Skip to content

Commit f9767d8

Browse files
committed
fixed issue
1 parent df5b1e1 commit f9767d8

File tree

3 files changed

+14
-28
lines changed

3 files changed

+14
-28
lines changed

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

Lines changed: 11 additions & 12 deletions
Original file line numberDiff line numberDiff line change
@@ -143,6 +143,7 @@ public class Client implements AutoCloseable {
143143
// Server context
144144
private String serverVersion;
145145
private Object metricsRegistry;
146+
private int retries;
146147

147148
private Client(Set<String> endpoints, Map<String,String> configuration, boolean useNewImplementation,
148149
ExecutorService sharedOperationExecutor, ColumnToMethodMatchingStrategy columnToMethodMatchingStrategy) {
@@ -172,6 +173,9 @@ private Client(Set<String> endpoints, Map<String,String> configuration, boolean
172173
boolean initSslContext = getEndpoints().stream().anyMatch(s -> s.toLowerCase().contains("https://"));
173174
this.httpClientHelper = new HttpAPIClientHelper(configuration, metricsRegistry, initSslContext);
174175
this.columnToMethodMatchingStrategy = columnToMethodMatchingStrategy;
176+
177+
String retry = configuration.get(ClientConfigProperties.RETRY_ON_FAILURE.getKey());
178+
this.retries = retry == null ? 0 : Integer.parseInt(retry);
175179
}
176180

177181
/**
@@ -1471,8 +1475,6 @@ public CompletableFuture<InsertResponse> insert(String tableName,
14711475
Supplier<InsertResponse> responseSupplier;
14721476

14731477

1474-
String retry = configuration.get(ClientConfigProperties.RETRY_ON_FAILURE.getKey());
1475-
final int maxRetries = retry == null ? 0 : Integer.parseInt(retry);
14761478
final int writeBufferSize = settings.getInputStreamCopyBufferSize() <= 0 ?
14771479
Integer.parseInt(configuration.getOrDefault(ClientConfigProperties.CLIENT_NETWORK_BUFFER_SIZE.getKey(), "8192")) :
14781480
settings.getInputStreamCopyBufferSize();
@@ -1491,7 +1493,7 @@ public CompletableFuture<InsertResponse> insert(String tableName,
14911493
ClickHouseNode selectedNode = getNextAliveNode();
14921494

14931495
RuntimeException lastException = null;
1494-
for (int i = 0; i <= maxRetries; i++) {
1496+
for (int i = 0; i <= retries; i++) {
14951497
// Execute request
14961498
try (ClassicHttpResponse httpResponse =
14971499
httpClientHelper.executeRequest(selectedNode, finalSettings.getAllSettings(),
@@ -1517,7 +1519,7 @@ public CompletableFuture<InsertResponse> insert(String tableName,
15171519
return new InsertResponse(metrics);
15181520
} catch (Exception e) {
15191521
lastException = httpClientHelper.wrapException(String.format("Insert failed (Attempt: %s/%s - Duration: %s)",
1520-
(i + 1), (maxRetries + 1), System.nanoTime() - startTime), e);
1522+
(i + 1), (retries + 1), System.nanoTime() - startTime), e);
15211523
if (httpClientHelper.shouldRetry(e, finalSettings.getAllSettings())) {
15221524
LOG.warn("Retrying.", e);
15231525
selectedNode = getNextAliveNode();
@@ -1526,15 +1528,15 @@ public CompletableFuture<InsertResponse> insert(String tableName,
15261528
}
15271529
}
15281530

1529-
if (i < maxRetries) {
1531+
if (i < retries) {
15301532
try {
15311533
writer.onRetry();
15321534
} catch (IOException ioe) {
15331535
throw new ClientException("Failed to reset stream before next attempt", ioe);
15341536
}
15351537
}
15361538
}
1537-
throw new ClientException("Insert request failed after attempts: " + (maxRetries + 1) + " - Duration: " + (System.nanoTime() - startTime), lastException);
1539+
throw new ClientException("Insert request failed after attempts: " + (retries + 1) + " - Duration: " + (System.nanoTime() - startTime), lastException);
15381540
};
15391541

15401542
return runAsyncOperation(responseSupplier, settings.getAllSettings());
@@ -1605,9 +1607,6 @@ public CompletableFuture<QueryResponse> query(String sqlQuery, Map<String, Objec
16051607

16061608
Supplier<QueryResponse> responseSupplier;
16071609

1608-
String retry = configuration.get(ClientConfigProperties.RETRY_ON_FAILURE.getKey());
1609-
final int maxRetries = retry == null ? 0 : Integer.parseInt(retry);
1610-
16111610
if (queryParams != null) {
16121611
settings.setOption("statement_params", queryParams);
16131612
}
@@ -1617,7 +1616,7 @@ public CompletableFuture<QueryResponse> query(String sqlQuery, Map<String, Objec
16171616
// Selecting some node
16181617
ClickHouseNode selectedNode = getNextAliveNode();
16191618
RuntimeException lastException = null;
1620-
for (int i = 0; i <= maxRetries; i++) {
1619+
for (int i = 0; i <= retries; i++) {
16211620
try {
16221621
ClassicHttpResponse httpResponse =
16231622
httpClientHelper.executeRequest(selectedNode, finalSettings.getAllSettings(), output -> {
@@ -1650,7 +1649,7 @@ public CompletableFuture<QueryResponse> query(String sqlQuery, Map<String, Objec
16501649

16511650
} catch (Exception e) {
16521651
lastException = httpClientHelper.wrapException(String.format("Query request failed (Attempt: %s/%s - Duration: %s)",
1653-
(i + 1), (maxRetries + 1), System.nanoTime() - startTime), e);
1652+
(i + 1), (retries + 1), System.nanoTime() - startTime), e);
16541653
if (httpClientHelper.shouldRetry(e, finalSettings.getAllSettings())) {
16551654
LOG.warn("Retrying.", e);
16561655
selectedNode = getNextAliveNode();
@@ -1660,7 +1659,7 @@ public CompletableFuture<QueryResponse> query(String sqlQuery, Map<String, Objec
16601659
}
16611660
}
16621661

1663-
throw new ClientException("Query request failed after attempts: " + (maxRetries + 1) + " - Duration: " + (System.nanoTime() - startTime), lastException);
1662+
throw new ClientException("Query request failed after attempts: " + (retries + 1) + " - Duration: " + (System.nanoTime() - startTime), lastException);
16641663
};
16651664

16661665
return runAsyncOperation(responseSupplier, settings.getAllSettings());

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

Lines changed: 2 additions & 16 deletions
Original file line numberDiff line numberDiff line change
@@ -13,20 +13,11 @@
1313
import com.clickhouse.client.api.data_formats.internal.SerializerUtils;
1414
import com.clickhouse.client.api.enums.ProxyType;
1515
import com.clickhouse.client.api.http.ClickHouseHttpProto;
16-
import io.micrometer.core.annotation.Timed;
17-
import org.apache.hc.client5.http.AuthenticationStrategy;
1816
import org.apache.hc.client5.http.ConnectTimeoutException;
19-
import org.apache.hc.client5.http.classic.ExecChain;
20-
import org.apache.hc.client5.http.classic.ExecChainHandler;
2117
import org.apache.hc.client5.http.classic.methods.HttpPost;
2218
import org.apache.hc.client5.http.config.ConnectionConfig;
2319
import org.apache.hc.client5.http.config.RequestConfig;
24-
import org.apache.hc.client5.http.impl.ChainElement;
25-
import org.apache.hc.client5.http.impl.DefaultAuthenticationStrategy;
26-
import org.apache.hc.client5.http.impl.DefaultClientConnectionReuseStrategy;
27-
import org.apache.hc.client5.http.impl.DefaultSchemePortResolver;
2820
import org.apache.hc.client5.http.impl.classic.CloseableHttpClient;
29-
import org.apache.hc.client5.http.impl.classic.ConnectExec;
3021
import org.apache.hc.client5.http.impl.classic.HttpClientBuilder;
3122
import org.apache.hc.client5.http.impl.io.BasicHttpClientConnectionManager;
3223
import org.apache.hc.client5.http.impl.io.ManagedHttpClientConnectionFactory;
@@ -39,13 +30,11 @@
3930
import org.apache.hc.client5.http.socket.LayeredConnectionSocketFactory;
4031
import org.apache.hc.client5.http.socket.PlainConnectionSocketFactory;
4132
import org.apache.hc.client5.http.ssl.SSLConnectionSocketFactory;
42-
import org.apache.hc.core5.http.ClassicHttpRequest;
4333
import org.apache.hc.core5.http.ClassicHttpResponse;
4434
import org.apache.hc.core5.http.ConnectionRequestTimeoutException;
4535
import org.apache.hc.core5.http.ContentType;
4636
import org.apache.hc.core5.http.Header;
4737
import org.apache.hc.core5.http.HttpEntity;
48-
import org.apache.hc.core5.http.HttpException;
4938
import org.apache.hc.core5.http.HttpHeaders;
5039
import org.apache.hc.core5.http.HttpHost;
5140
import org.apache.hc.core5.http.HttpRequest;
@@ -57,10 +46,7 @@
5746
import org.apache.hc.core5.http.impl.io.DefaultHttpResponseParserFactory;
5847
import org.apache.hc.core5.http.io.SocketConfig;
5948
import org.apache.hc.core5.http.io.entity.EntityTemplate;
60-
import org.apache.hc.core5.http.protocol.DefaultHttpProcessor;
6149
import org.apache.hc.core5.http.protocol.HttpContext;
62-
import org.apache.hc.core5.http.protocol.RequestTargetHost;
63-
import org.apache.hc.core5.http.protocol.RequestUserAgent;
6450
import org.apache.hc.core5.io.CloseMode;
6551
import org.apache.hc.core5.io.IOCallback;
6652
import org.apache.hc.core5.net.URIBuilder;
@@ -533,8 +519,8 @@ private void addHeaders(HttpPost req, Map<String, String> chConfig, Map<String,
533519

534520
private void addQueryParams(URIBuilder req, Map<String, String> chConfig, Map<String, Object> requestConfig) {
535521
for (String key : chConfig.keySet()) {
536-
if (key.startsWith(ClientConfigProperties.HTTP_HEADER_PREFIX)) {
537-
req.addParameter(key.substring(ClientConfigProperties.HTTP_HEADER_PREFIX.length()), chConfig.get(key));
522+
if (key.startsWith(ClientConfigProperties.SERVER_SETTING_PREFIX)) {
523+
req.addParameter(key.substring(ClientConfigProperties.SERVER_SETTING_PREFIX.length()), chConfig.get(key));
538524
}
539525
}
540526

performance/src/test/com/clickhouse/benchmark/clients/BenchmarkBase.java

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -222,6 +222,7 @@ protected static Client getClientV2(boolean includeDb) {
222222
.setUsername(getUsername())
223223
.setPassword(getPassword())
224224
.setMaxRetries(0)
225+
.setClientNetworkBufferSize(1024)
225226
.setDefaultDatabase(includeDb ? DB_NAME : "default")
226227
.build();
227228
}

0 commit comments

Comments
 (0)