Skip to content

Commit 1baf186

Browse files
authored
Merge pull request #1909 from ClickHouse/v2_json_support
[client-v2] Added support of reading new JSON as string
2 parents 4689c2d + 70cdc62 commit 1baf186

File tree

12 files changed

+298
-31
lines changed

12 files changed

+298
-31
lines changed

client-v2/src/main/java/com/clickhouse/client/api/Client.java

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -1903,7 +1903,7 @@ public ClickHouseBinaryFormatReader newBinaryFormatReader(QueryResponse response
19031903
byteBufferPool);
19041904
break;
19051905
default:
1906-
throw new IllegalArgumentException("Unsupported format: " + response.getFormat());
1906+
throw new IllegalArgumentException("Binary readers doesn't support format: " + response.getFormat());
19071907
}
19081908
return reader;
19091909
}

client-v2/src/main/java/com/clickhouse/client/api/ClientSettings.java

Lines changed: 12 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -39,4 +39,16 @@ public static List<String> valuesFromCommaSeparated(String value) {
3939
public static final String SETTING_LOG_COMMENT = SERVER_SETTING_PREFIX + "log_comment";
4040

4141
public static final String HTTP_USE_BASIC_AUTH = "http_use_basic_auth";
42+
43+
// -- Experimental features --
44+
45+
/**
46+
* Server will expect a string in JSON format and parse it into a JSON object.
47+
*/
48+
public static final String INPUT_FORMAT_BINARY_READ_JSON_AS_STRING = "input_format_binary_read_json_as_string";
49+
50+
/**
51+
* Server will return a JSON object as a string.
52+
*/
53+
public static final String OUTPUT_FORMAT_BINARY_WRITE_JSON_AS_STRING = "output_format_binary_write_json_as_string";
4254
}

client-v2/src/main/java/com/clickhouse/client/api/data_formats/internal/AbstractBinaryFormatReader.java

Lines changed: 7 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -1,7 +1,9 @@
11
package com.clickhouse.client.api.data_formats.internal;
22

33
import com.clickhouse.client.api.ClientException;
4+
import com.clickhouse.client.api.ClientSettings;
45
import com.clickhouse.client.api.data_formats.ClickHouseBinaryFormatReader;
6+
import com.clickhouse.client.api.internal.MapUtils;
57
import com.clickhouse.client.api.metadata.TableSchema;
68
import com.clickhouse.client.api.query.NullValueException;
79
import com.clickhouse.client.api.query.POJOSetter;
@@ -32,8 +34,10 @@
3234
import java.time.temporal.ChronoUnit;
3335
import java.util.Collections;
3436
import java.util.HashMap;
37+
import java.util.HashSet;
3538
import java.util.List;
3639
import java.util.Map;
40+
import java.util.Set;
3741
import java.util.TimeZone;
3842
import java.util.UUID;
3943
import java.util.concurrent.ConcurrentHashMap;
@@ -71,7 +75,9 @@ protected AbstractBinaryFormatReader(InputStream inputStream, QuerySettings quer
7175
if (timeZone == null) {
7276
throw new ClientException("Time zone is not set. (useServerTimezone:" + useServerTimeZone + ")");
7377
}
74-
this.binaryStreamReader = new BinaryStreamReader(inputStream, timeZone, LOG, byteBufferAllocator);
78+
boolean jsonAsString = MapUtils.getFlag(this.settings,
79+
ClientSettings.SERVER_SETTING_PREFIX + ClientSettings.OUTPUT_FORMAT_BINARY_WRITE_JSON_AS_STRING, false);
80+
this.binaryStreamReader = new BinaryStreamReader(inputStream, timeZone, LOG, byteBufferAllocator, jsonAsString);
7581
if (schema != null) {
7682
setSchema(schema);
7783
}

client-v2/src/main/java/com/clickhouse/client/api/data_formats/internal/BinaryStreamReader.java

Lines changed: 10 additions & 13 deletions
Original file line numberDiff line numberDiff line change
@@ -45,16 +45,7 @@ public class BinaryStreamReader {
4545

4646
private final ByteBufferAllocator bufferAllocator;
4747

48-
/**
49-
* Creates a BinaryStreamReader instance that will use {@link DefaultByteBufferAllocator} to allocate buffers.
50-
*
51-
* @param input - source of raw data in a suitable format
52-
* @param timeZone - timezone to use for date and datetime values
53-
* @param log - logger
54-
*/
55-
BinaryStreamReader(InputStream input, TimeZone timeZone, Logger log) {
56-
this(input, timeZone, log, new DefaultByteBufferAllocator());
57-
}
48+
private final boolean jsonAsString;
5849

5950
/**
6051
* Createa a BinaryStreamReader instance that will use the provided buffer allocator.
@@ -64,11 +55,12 @@ public class BinaryStreamReader {
6455
* @param log - logger
6556
* @param bufferAllocator - byte buffer allocator
6657
*/
67-
BinaryStreamReader(InputStream input, TimeZone timeZone, Logger log, ByteBufferAllocator bufferAllocator) {
58+
BinaryStreamReader(InputStream input, TimeZone timeZone, Logger log, ByteBufferAllocator bufferAllocator, boolean jsonAsString) {
6859
this.log = log == null ? NOPLogger.NOP_LOGGER : log;
6960
this.timeZone = timeZone;
7061
this.input = input;
7162
this.bufferAllocator = bufferAllocator;
63+
this.jsonAsString = jsonAsString;
7264
}
7365

7466
/**
@@ -203,8 +195,13 @@ public <T> T readValue(ClickHouseColumn column, Class<?> typeHint) throws IOExce
203195
case Ring:
204196
return (T) readGeoRing();
205197

206-
// case JSON: // obsolete https://clickhouse.com/docs/en/sql-reference/data-types/json#displaying-json-column
207-
// case Object:
198+
case JSON: // experimental https://clickhouse.com/docs/en/sql-reference/data-types/newjson
199+
if (jsonAsString) {
200+
return (T) readString(input);
201+
} else {
202+
throw new RuntimeException("Reading JSON from binary is not implemented yet");
203+
}
204+
// case Object: // deprecated https://clickhouse.com/docs/en/sql-reference/data-types/object-data-type
208205
case Array:
209206
return convertArray(readArray(column), typeHint);
210207
case Map:

client-v2/src/main/java/com/clickhouse/client/api/data_formats/internal/SerializerUtils.java

Lines changed: 11 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -205,11 +205,22 @@ private static void serializePrimitiveData(OutputStream stream, Object value, Cl
205205
case IPv6:
206206
BinaryStreamUtils.writeInet6Address(stream, (Inet6Address) value);
207207
break;
208+
case JSON:
209+
serializeJSON(stream, value);
210+
break;
208211
default:
209212
throw new UnsupportedOperationException("Unsupported data type: " + column.getDataType());
210213
}
211214
}
212215

216+
private static void serializeJSON(OutputStream stream, Object value) throws IOException {
217+
if (value instanceof String) {
218+
BinaryStreamUtils.writeString(stream, (String)value);
219+
} else {
220+
throw new UnsupportedOperationException("Serialization of Java object to JSON is not supported yet.");
221+
}
222+
}
223+
213224
private static void serializeAggregateFunction(OutputStream stream, Object value, ClickHouseColumn column) throws IOException {
214225
if (column.getAggregateFunction() == ClickHouseAggregateFunction.groupBitmap) {
215226
if (value == null) {

client-v2/src/main/java/com/clickhouse/client/api/internal/MapUtils.java

Lines changed: 11 additions & 8 deletions
Original file line numberDiff line numberDiff line change
@@ -68,21 +68,24 @@ public static boolean getFlag(Map<String, String> map, String key) {
6868
throw new IllegalArgumentException("Invalid non-boolean value for the key '" + key + "': '" + val + "'");
6969
}
7070

71-
public static boolean getFlag(Map<String, String> map, String key, boolean defaultValue) {
72-
String val = map.get(key);
71+
public static boolean getFlag(Map<String, ?> map, String key, boolean defaultValue) {
72+
Object val = map.get(key);
7373
if (val == null) {
7474
return defaultValue;
7575
}
76-
if (val.equalsIgnoreCase("true")) {
77-
return true;
78-
} else if (val.equalsIgnoreCase("false")) {
79-
return false;
76+
if (val instanceof Boolean) {
77+
return (Boolean) val;
78+
} else if (val instanceof String) {
79+
String str = (String) val;
80+
if (str.equalsIgnoreCase("true") || str.equalsIgnoreCase("1")) {
81+
return true;
82+
} else if (str.equalsIgnoreCase("false") || str.equalsIgnoreCase("0")) {
83+
return false;
84+
}
8085
}
81-
8286
throw new IllegalArgumentException("Invalid non-boolean value for the key '" + key + "': '" + val + "'");
8387
}
8488

85-
8689
public static boolean getFlag(Map<String, ?> p1, Map<String, ?> p2, String key) {
8790
Object val = p1.get(key);
8891
if (val == null) {

client-v2/src/test/java/com/clickhouse/client/insert/InsertTests.java

Lines changed: 46 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -9,7 +9,9 @@
99
import com.clickhouse.client.ClickHouseProtocol;
1010
import com.clickhouse.client.api.Client;
1111
import com.clickhouse.client.api.ClientException;
12+
import com.clickhouse.client.api.ClientSettings;
1213
import com.clickhouse.client.api.command.CommandResponse;
14+
import com.clickhouse.client.api.command.CommandSettings;
1315
import com.clickhouse.client.api.data_formats.ClickHouseBinaryFormatReader;
1416
import com.clickhouse.client.api.enums.Protocol;
1517
import com.clickhouse.client.api.insert.InsertResponse;
@@ -19,19 +21,25 @@
1921
import com.clickhouse.client.api.metrics.ServerMetrics;
2022
import com.clickhouse.client.api.query.GenericRecord;
2123
import com.clickhouse.client.api.query.QueryResponse;
24+
import com.clickhouse.client.api.query.QuerySettings;
2225
import com.clickhouse.data.ClickHouseFormat;
26+
import com.clickhouse.data.ClickHouseVersion;
27+
import org.apache.commons.lang3.StringEscapeUtils;
2328
import org.testcontainers.shaded.org.apache.commons.lang3.RandomStringUtils;
2429
import org.testng.Assert;
2530
import org.testng.annotations.AfterMethod;
2631
import org.testng.annotations.BeforeMethod;
2732
import org.testng.annotations.DataProvider;
2833
import org.testng.annotations.Test;
2934

35+
import java.io.BufferedReader;
3036
import java.io.ByteArrayInputStream;
3137
import java.io.ByteArrayOutputStream;
3238
import java.io.IOException;
39+
import java.io.InputStreamReader;
3340
import java.io.PrintWriter;
3441
import java.util.ArrayList;
42+
import java.util.Arrays;
3543
import java.util.Collections;
3644
import java.util.List;
3745
import java.util.UUID;
@@ -121,6 +129,44 @@ public void insertSimplePOJOs() throws Exception {
121129
assertEquals(response.getQueryId(), uuid);
122130
}
123131

132+
133+
@Test(groups = { "integration" }, enabled = true)
134+
public void insertPOJOWithJSON() throws Exception {
135+
List<GenericRecord> serverVersion = client.queryAll("SELECT version()");
136+
if (ClickHouseVersion.of(serverVersion.get(0).getString(1)).check("(,24.8]")) {
137+
System.out.println("Test is skipped: feature is supported since 24.8");
138+
return;
139+
}
140+
141+
final String tableName = "pojo_with_json_table";
142+
final String createSQL = PojoWithJSON.createTable(tableName);
143+
final String originalJsonStr = "{\"a\":{\"b\":\"42\"},\"c\":[\"1\",\"2\",\"3\"]}";
144+
145+
146+
CommandSettings commandSettings = new CommandSettings();
147+
commandSettings.serverSetting("allow_experimental_json_type", "1");
148+
client.execute("DROP TABLE IF EXISTS " + tableName, commandSettings).get(1, TimeUnit.SECONDS);
149+
client.execute(createSQL, commandSettings).get(1, TimeUnit.SECONDS);
150+
151+
client.register(PojoWithJSON.class, client.getTableSchema(tableName, "default"));
152+
PojoWithJSON pojo = new PojoWithJSON();
153+
pojo.setEventPayload(originalJsonStr);
154+
List<Object> data = Arrays.asList(pojo);
155+
156+
InsertSettings insertSettings = new InsertSettings()
157+
.serverSetting(ClientSettings.INPUT_FORMAT_BINARY_READ_JSON_AS_STRING, "1");
158+
InsertResponse response = client.insert(tableName, data, insertSettings).get(30, TimeUnit.SECONDS);
159+
assertEquals(response.getWrittenRows(), 1);
160+
161+
QuerySettings settings = new QuerySettings()
162+
.setFormat(ClickHouseFormat.CSV);
163+
try (QueryResponse resp = client.query("SELECT * FROM " + tableName, settings).get(1, TimeUnit.SECONDS)) {
164+
BufferedReader reader = new BufferedReader(new InputStreamReader(resp.getInputStream()));
165+
String jsonStr = StringEscapeUtils.unescapeCsv(reader.lines().findFirst().get());
166+
Assert.assertEquals(jsonStr, originalJsonStr);
167+
}
168+
}
169+
124170
@Test(groups = { "integration" }, enabled = true)
125171
public void insertPOJOAndReadBack() throws Exception {
126172
final String tableName = "single_pojo_table";
Lines changed: 41 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,41 @@
1+
package com.clickhouse.client.insert;
2+
3+
import java.util.Objects;
4+
5+
public class PojoWithJSON {
6+
7+
// This field is a string representation of a JSON object
8+
private String eventPayload;
9+
10+
public String getEventPayload() {
11+
return eventPayload;
12+
}
13+
14+
public void setEventPayload(String eventPayload) {
15+
this.eventPayload = eventPayload;
16+
}
17+
18+
@Override
19+
public boolean equals(Object o) {
20+
if (this == o) return true;
21+
if (o == null || getClass() != o.getClass()) return false;
22+
PojoWithJSON that = (PojoWithJSON) o;
23+
return Objects.equals(eventPayload, that.eventPayload);
24+
}
25+
26+
@Override
27+
public int hashCode() {
28+
return Objects.hash(eventPayload);
29+
}
30+
31+
@Override
32+
public String toString() {
33+
return "PojoWithJSON{" +
34+
"eventPayload='" + eventPayload + '\'' +
35+
'}';
36+
}
37+
38+
public static String createTable(String tableName) {
39+
return "CREATE TABLE " + tableName + " (eventPayload JSON) ENGINE = MergeTree() ORDER BY tuple()";
40+
}
41+
}

client-v2/src/test/java/com/clickhouse/client/query/QueryTests.java

Lines changed: 32 additions & 8 deletions
Original file line numberDiff line numberDiff line change
@@ -16,8 +16,8 @@
1616
import com.clickhouse.client.api.DataTypeUtils;
1717
import com.clickhouse.client.api.ServerException;
1818
import com.clickhouse.client.api.command.CommandResponse;
19+
import com.clickhouse.client.api.command.CommandSettings;
1920
import com.clickhouse.client.api.data_formats.ClickHouseBinaryFormatReader;
20-
import com.clickhouse.client.api.data_formats.internal.BinaryStreamReader;
2121
import com.clickhouse.client.api.enums.Protocol;
2222
import com.clickhouse.client.api.insert.InsertResponse;
2323
import com.clickhouse.client.api.insert.InsertSettings;
@@ -30,14 +30,13 @@
3030
import com.clickhouse.client.api.query.QueryResponse;
3131
import com.clickhouse.client.api.query.QuerySettings;
3232
import com.clickhouse.client.api.query.Records;
33-
import com.clickhouse.client.http.config.HttpConnectionProvider;
34-
import com.clickhouse.client.insert.SamplePOJO;
3533
import com.clickhouse.data.ClickHouseDataType;
3634
import com.clickhouse.data.ClickHouseFormat;
3735
import com.clickhouse.data.ClickHouseVersion;
3836
import com.fasterxml.jackson.databind.JsonNode;
3937
import com.fasterxml.jackson.databind.MappingIterator;
4038
import com.fasterxml.jackson.databind.ObjectMapper;
39+
import org.apache.commons.lang3.StringEscapeUtils;
4140
import org.testng.Assert;
4241
import org.testng.annotations.AfterMethod;
4342
import org.testng.annotations.BeforeMethod;
@@ -57,15 +56,13 @@
5756
import java.net.Inet4Address;
5857
import java.net.Inet6Address;
5958
import java.net.InetAddress;
60-
import java.nio.file.Files;
6159
import java.time.LocalDate;
6260
import java.time.LocalDateTime;
6361
import java.time.ZoneId;
6462
import java.time.ZonedDateTime;
6563
import java.util.ArrayList;
6664
import java.util.Arrays;
6765
import java.util.Collections;
68-
import java.util.Comparator;
6966
import java.util.HashMap;
7067
import java.util.HashSet;
7168
import java.util.Iterator;
@@ -74,7 +71,6 @@
7471
import java.util.Random;
7572
import java.util.Set;
7673
import java.util.UUID;
77-
import java.util.concurrent.CompletableFuture;
7874
import java.util.concurrent.CountDownLatch;
7975
import java.util.concurrent.ExecutionException;
8076
import java.util.concurrent.ExecutorService;
@@ -123,7 +119,6 @@ public void setUp() {
123119
.compressClientRequest(false)
124120
.compressServerResponse(useServerCompression)
125121
.useHttpCompression(useHttpCompression)
126-
.useNewImplementation(true)
127122
.build();
128123

129124
delayForProfiler(0);
@@ -340,7 +335,7 @@ public void testQueryAllTableNames() {
340335
}
341336

342337
@Test(groups = {"integration"})
343-
public void testQueryJSON() throws ExecutionException, InterruptedException {
338+
public void testQueryJSONEachRow() throws ExecutionException, InterruptedException {
344339
Map<String, Object> datasetRecord = prepareSimpleDataSet();
345340
QuerySettings settings = new QuerySettings().setFormat(ClickHouseFormat.JSONEachRow);
346341
Future<QueryResponse> response = client.query("SELECT * FROM " + DATASET_TABLE, settings);
@@ -1793,6 +1788,35 @@ public void testReadingBitmap() throws Exception {
17931788
}
17941789
}
17951790

1791+
@Test(groups = {"integration"})
1792+
public void testReadingJSONValues() throws Exception {
1793+
List<GenericRecord> serverVersion = client.queryAll("SELECT version()");
1794+
if (ClickHouseVersion.of(serverVersion.get(0).getString(1)).check("(,24.8]")) {
1795+
System.out.println("Test is skipped: feature is supported since 24.8");
1796+
return;
1797+
}
1798+
CommandSettings commandSettings = new CommandSettings();
1799+
commandSettings.serverSetting("allow_experimental_json_type", "1");
1800+
client.execute("DROP TABLE IF EXISTS test_json_values", commandSettings).get(1, TimeUnit.SECONDS);
1801+
client.execute("CREATE TABLE test_json_values (json JSON) ENGINE = MergeTree ORDER BY ()", commandSettings).get(1, TimeUnit.SECONDS);
1802+
client.execute("INSERT INTO test_json_values VALUES ('{\"a\" : {\"b\" : 42}, \"c\" : [1, 2, 3]}')", commandSettings).get(1, TimeUnit.SECONDS);
1803+
1804+
1805+
QuerySettings settings = new QuerySettings().setFormat(ClickHouseFormat.CSV);
1806+
try (QueryResponse resp = client.query("SELECT json FROM test_json_values", settings).get(1, TimeUnit.SECONDS)) {
1807+
BufferedReader reader = new BufferedReader(new InputStreamReader(resp.getInputStream()));
1808+
Assert.assertEquals(StringEscapeUtils.unescapeCsv(reader.lines().findFirst().get()), "{\"a\":{\"b\":\"42\"},\"c\":[\"1\",\"2\",\"3\"]}");
1809+
}
1810+
1811+
settings = new QuerySettings()
1812+
.serverSetting(ClientSettings.OUTPUT_FORMAT_BINARY_WRITE_JSON_AS_STRING, "1");
1813+
try (QueryResponse resp = client.query("SELECT json FROM test_json_values", settings).get(1, TimeUnit.SECONDS)) {
1814+
ClickHouseBinaryFormatReader reader = client.newBinaryFormatReader(resp);
1815+
Assert.assertNotNull(reader.next());
1816+
Assert.assertEquals(reader.getString(1), "{\"a\":{\"b\":\"42\"},\"c\":[\"1\",\"2\",\"3\"]}");
1817+
}
1818+
}
1819+
17961820
protected Client.Builder newClient() {
17971821
ClickHouseNode node = getServer(ClickHouseProtocol.HTTP);
17981822
return new Client.Builder()

0 commit comments

Comments
 (0)