Skip to content

Commit 04d3443

Browse files
authored
Merge pull request #288 from ClickHouse/handle-null-arrays
Send an empty array when field is null
2 parents ec796ae + 30fb894 commit 04d3443

File tree

2 files changed

+43
-20
lines changed

2 files changed

+43
-20
lines changed

src/main/java/com/clickhouse/kafka/connect/sink/db/ClickHouseWriter.java

Lines changed: 21 additions & 18 deletions
Original file line numberDiff line numberDiff line change
@@ -23,6 +23,7 @@
2323
import com.google.gson.reflect.TypeToken;
2424

2525
import java.math.BigDecimal;
26+
import java.util.*;
2627
import java.util.concurrent.TimeUnit;
2728

2829
import org.apache.kafka.connect.data.Field;
@@ -35,11 +36,6 @@
3536
import java.io.IOException;
3637
import java.nio.charset.StandardCharsets;
3738
import java.util.concurrent.atomic.AtomicBoolean;
38-
import java.util.Date;
39-
import java.util.HashMap;
40-
import java.util.List;
41-
import java.util.Map;
42-
import java.util.UUID;
4339
import java.util.concurrent.CompletableFuture;
4440
import java.util.concurrent.ExecutionException;
4541
import java.util.stream.Collectors;
@@ -354,8 +350,10 @@ private void doWriteCol(Record record, Column col, ClickHousePipedOutputStream s
354350
BinaryStreamUtils.writeNonNull(stream);
355351
}
356352
if (!col.isNullable() && value.getObject() == null) {
357-
// this the situation when the col is not isNullable, but the data is null here we need to drop the records
358-
throw new RuntimeException(String.format("An attempt to write null into not nullable column '%s'", col.getName()));
353+
if (colType == Type.ARRAY)
354+
BinaryStreamUtils.writeNonNull(stream);
355+
else
356+
throw new RuntimeException(String.format("An attempt to write null into not nullable column '%s'", col.getName()));
359357
}
360358
switch (colType) {
361359
case INT8:
@@ -403,18 +401,23 @@ private void doWriteCol(Record record, Column col, ClickHousePipedOutputStream s
403401
break;
404402
case ARRAY:
405403
List<?> arrObject = (List<?>) value.getObject();
406-
int sizeArrObject = arrObject.size();
407-
BinaryStreamUtils.writeVarInt(stream, sizeArrObject);
408-
arrObject.forEach(v -> {
409-
try {
410-
if (col.getSubType().isNullable() && v != null) {
411-
BinaryStreamUtils.writeNonNull(stream);
404+
405+
if (arrObject == null) {
406+
doWritePrimitive(colType, value.getFieldType(), stream, new ArrayList<>());
407+
} else {
408+
int sizeArrObject = arrObject.size();
409+
BinaryStreamUtils.writeVarInt(stream, sizeArrObject);
410+
arrObject.forEach(v -> {
411+
try {
412+
if (col.getSubType().isNullable() && v != null) {
413+
BinaryStreamUtils.writeNonNull(stream);
414+
}
415+
doWritePrimitive(col.getSubType().getType(), value.getFieldType(), stream, v);
416+
} catch (IOException e) {
417+
throw new RuntimeException(e);
412418
}
413-
doWritePrimitive(col.getSubType().getType(), value.getFieldType(), stream, v);
414-
} catch (IOException e) {
415-
throw new RuntimeException(e);
416-
}
417-
});
419+
});
420+
}
418421
break;
419422
}
420423
} else {

src/test/java/com/clickhouse/kafka/connect/sink/ClickHouseSinkTaskWithSchemaTest.java

Lines changed: 22 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -120,7 +120,7 @@ public void mapTypesTest() {
120120
}
121121

122122
@Test
123-
public void nullableArrayTypeTest() {
123+
public void nullArrayTypeTest() {
124124
Map<String, String> props = getTestProperties();
125125
ClickHouseHelperClient chc = createClient(props);
126126

@@ -131,8 +131,28 @@ public void nullableArrayTypeTest() {
131131

132132
ClickHouseSinkTask chst = new ClickHouseSinkTask();
133133
chst.start(props);
134-
assertThrowsExactly(RuntimeException.class, () -> chst.put(sr), "An attempt to write null into not nullable column 'arr'");
134+
chst.put(sr);
135135
chst.stop();
136+
137+
assertEquals(sr.size(), ClickHouseTestHelpers.countRows(chc, topic));
138+
}
139+
140+
@Test
141+
public void nullableArrayTypeTest() {
142+
Map<String, String> props = getTestProperties();
143+
ClickHouseHelperClient chc = createClient(props);
144+
145+
String topic = "nullable_array_string_table_test";
146+
ClickHouseTestHelpers.dropTable(chc, topic);
147+
ClickHouseTestHelpers.createTable(chc, topic, "CREATE TABLE %s ( `off16` Int16, `arr` Array(Nullable(String)) ) Engine = MergeTree ORDER BY off16");
148+
Collection<SinkRecord> sr = SchemaTestData.createNullableArrayType(topic, 1);
149+
150+
ClickHouseSinkTask chst = new ClickHouseSinkTask();
151+
chst.start(props);
152+
chst.put(sr);
153+
chst.stop();
154+
155+
assertEquals(sr.size(), ClickHouseTestHelpers.countRows(chc, topic));
136156
}
137157

138158
@Test

0 commit comments

Comments
 (0)