Skip to content
Open
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
42 changes: 28 additions & 14 deletions client-v2/src/main/java/com/clickhouse/client/api/Client.java
Original file line number Diff line number Diff line change
Expand Up @@ -186,7 +186,6 @@ private Client(Collection<Endpoint> endpoints, Map<String,String> configuration,
}

this.endpoints = tmpEndpoints.build();
this.httpClientHelper = new HttpAPIClientHelper(this.configuration, metricsRegistry, initSslContext);

String retry = configuration.get(ClientConfigProperties.RETRY_ON_FAILURE.getKey());
this.retries = retry == null ? 0 : Integer.parseInt(retry);
Expand All @@ -197,6 +196,7 @@ private Client(Collection<Endpoint> endpoints, Map<String,String> configuration,
this.lz4Factory = LZ4Factory.fastestJavaInstance();
}

this.httpClientHelper = new HttpAPIClientHelper(this.configuration, metricsRegistry, initSslContext, lz4Factory);
this.serverVersion = configuration.getOrDefault(ClientConfigProperties.SERVER_VERSION.getKey(), "unknown");
this.dbUser = configuration.getOrDefault(ClientConfigProperties.USER.getKey(), ClientConfigProperties.USER.getDefObjVal());
this.typeHintMapping = (Map<ClickHouseDataType, Class<?>>) this.configuration.get(ClientConfigProperties.TYPE_HINT_MAPPING.getKey());
Expand Down Expand Up @@ -1073,6 +1073,18 @@ public Builder sslSocketSNI(String sni) {
return this;
}

/**
* Make sending statement parameters as HTTP Form data (in the body of a request).
* Works only for query methods and with disabled client request compression.
*
* @param enable - if feature enabled
* @return this builder instance
*/
public Builder useHttpFormDataForQuery(boolean enable) {
this.configuration.put(ClientConfigProperties.USE_HTTP_FORM_REQUEST_FOR_QUERY.getKey(), String.valueOf(enable));
return this;
}

public Client build() {
// check if endpoint are empty. so can not initiate client
if (this.endpoints.isEmpty()) {
Expand Down Expand Up @@ -1279,7 +1291,7 @@ public CompletableFuture<InsertResponse> insert(String tableName, List<?> data,
for (int i = 0; i <= maxRetries; i++) {
// Execute request
try (ClassicHttpResponse httpResponse =
httpClientHelper.executeRequest(selectedEndpoint, requestSettings.getAllSettings(), lz4Factory,
httpClientHelper.executeRequest(selectedEndpoint, requestSettings.getAllSettings(),
out -> {
out.write("INSERT INTO ".getBytes());
out.write(tableName.getBytes());
Expand Down Expand Up @@ -1496,7 +1508,7 @@ public CompletableFuture<InsertResponse> insert(String tableName,
for (int i = 0; i <= retries; i++) {
// Execute request
try (ClassicHttpResponse httpResponse =
httpClientHelper.executeRequest(selectedEndpoint, requestSettings.getAllSettings(), lz4Factory,
httpClientHelper.executeRequest(selectedEndpoint, requestSettings.getAllSettings(),
out -> {
writer.onOutput(out);
out.close();
Expand Down Expand Up @@ -1607,25 +1619,27 @@ public CompletableFuture<QueryResponse> query(String sqlQuery, Map<String, Objec
ClientStatisticsHolder clientStats = new ClientStatisticsHolder();
clientStats.start(ClientMetrics.OP_DURATION);

Supplier<QueryResponse> responseSupplier;
if (queryParams != null) {
requestSettings.setOption(HttpAPIClientHelper.KEY_STATEMENT_PARAMS, queryParams);
}

if (queryParams != null) {
requestSettings.setOption(HttpAPIClientHelper.KEY_STATEMENT_PARAMS, queryParams);
}
responseSupplier = () -> {
Supplier<QueryResponse> responseSupplier = () -> {
long startTime = System.nanoTime();
// Selecting some node
Endpoint selectedEndpoint = getNextAliveNode();
RuntimeException lastException = null;
for (int i = 0; i <= retries; i++) {
ClassicHttpResponse httpResponse = null;
try {
httpResponse =
httpClientHelper.executeRequest(selectedEndpoint, requestSettings.getAllSettings(), lz4Factory, output -> {
output.write(sqlQuery.getBytes(StandardCharsets.UTF_8));
output.close();
});

boolean useMultipart = ClientConfigProperties.USE_HTTP_FORM_REQUEST_FOR_QUERY.getOrDefault(requestSettings.getAllSettings());
if (queryParams != null && useMultipart) {
httpResponse = httpClientHelper.executeMultiPartRequest(selectedEndpoint,
requestSettings.getAllSettings(), sqlQuery);
} else {
httpResponse = httpClientHelper.executeRequest(selectedEndpoint,
requestSettings.getAllSettings(),
sqlQuery);
}
// Check response
if (httpResponse.getCode() == HttpStatus.SC_SERVICE_UNAVAILABLE) {
LOG.warn("Failed to get response. Server returned {}. Retrying. (Duration: {})", httpResponse.getCode(), durationSince(startTime));
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -182,6 +182,13 @@ public Object parseValue(String value) {
* SNI SSL parameter that will be set for each outbound SSL socket.
*/
SSL_SOCKET_SNI("ssl_socket_sni", String.class,""),

/**
* If parameters should be sent in request body.
* Note: work only with HTTP Compression

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

does it works with both HTTP compression and block level compression?

*/
USE_HTTP_FORM_REQUEST_FOR_QUERY("client.http.use_form_request_for_query", Boolean.class, "false"),

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

suggestion: Can we name the config HTTP_SEND_PARAMS_IN_BODY instead? Not sure if there are any standard way to name these in java client. But just looking at it, made me confuse. Mainly FOR_QUERY is too generic and no where it says about parameters?.

or USE_HTTP_FORM_REQUEST_FOR_QUERY_PARAMS but I felt that is bit too mouthful. Given it's a boolean we can avoid USE_ prefix I think.


;

private static final Logger LOG = LoggerFactory.getLogger(ClientConfigProperties.class);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -51,8 +51,8 @@ public void writeTo(OutputStream outStream) throws IOException {
throw new UnsupportedOperationException("Unsupported: writing compressed response to elsewhere");
}

try {
httpEntity.writeTo(compressorStreamFactory.createCompressorOutputStream(compressionAlgo, outStream));
try (OutputStream compressingStream = compressorStreamFactory.createCompressorOutputStream(compressionAlgo, outStream)){
httpEntity.writeTo(compressingStream);
} catch (CompressorException e) {
throw new IOException("Failed to create compressing output stream", e);
}
Expand All @@ -75,7 +75,8 @@ public void close() throws IOException {

@Override
public long getContentLength() {
return httpEntity.getContentLength();
// compressed request length is unknown event if it is a byte[]

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

I think it should be even not event?

return isResponse ? httpEntity.getContentLength() : -1;
}

@Override
Expand Down
Loading
Loading