Skip to content

Commit 21af845

Browse files
authored
Merge pull request #1675 from ClickHouse/feat_generic_metrics
Refactored metrics to make them more generic
2 parents 7103661 + 9685e5f commit 21af845

File tree

15 files changed

+382
-215
lines changed

15 files changed

+382
-215
lines changed

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

Lines changed: 11 additions & 10 deletions
Original file line numberDiff line numberDiff line change
@@ -21,6 +21,8 @@
2121
import com.clickhouse.client.api.internal.TableSchemaParser;
2222
import com.clickhouse.client.api.internal.ValidationUtils;
2323
import com.clickhouse.client.api.metadata.TableSchema;
24+
import com.clickhouse.client.api.internal.ClientStatisticsHolder;
25+
import com.clickhouse.client.api.metrics.ClientMetrics;
2426
import com.clickhouse.client.api.query.GenericRecord;
2527
import com.clickhouse.client.api.query.QueryResponse;
2628
import com.clickhouse.client.api.query.QuerySettings;
@@ -103,7 +105,7 @@ public class Client {
103105
private static final Logger LOG = LoggerFactory.getLogger(Client.class);
104106
private ExecutorService queryExecutor;
105107

106-
private Map<String, OperationStatistics.ClientStatistics> globalClientStats = new ConcurrentHashMap<>();
108+
private Map<String, ClientStatisticsHolder> globalClientStats = new ConcurrentHashMap<>();
107109

108110
private Client(Set<String> endpoints, Map<String,String> configuration) {
109111
this.endpoints = endpoints;
@@ -526,7 +528,7 @@ public CompletableFuture<InsertResponse> insert(String tableName, List<?> data,
526528

527529
String operationId = startOperation();
528530
settings.setOperationId(operationId);
529-
globalClientStats.get(operationId).start("serialization");
531+
globalClientStats.get(operationId).start(ClientMetrics.OP_SERIALIZATION);
530532

531533
if (data == null || data.isEmpty()) {
532534
throw new IllegalArgumentException("Data cannot be empty");
@@ -560,7 +562,7 @@ public CompletableFuture<InsertResponse> insert(String tableName, List<?> data,
560562
}
561563
}
562564

563-
globalClientStats.get(operationId).stop("serialization");
565+
globalClientStats.get(operationId).stop(ClientMetrics.OP_SERIALIZATION);
564566
LOG.debug("Total serialization time: {}", globalClientStats.get(operationId).getElapsedTime("serialization"));
565567
return insert(tableName, new ByteArrayInputStream(stream.toByteArray()), format, settings);
566568
}
@@ -598,8 +600,8 @@ public CompletableFuture<InsertResponse> insert(String tableName,
598600
operationId = startOperation();
599601
settings.setOperationId(operationId);
600602
}
601-
OperationStatistics.ClientStatistics clientStats = globalClientStats.remove(operationId);
602-
clientStats.start("insert");
603+
ClientStatisticsHolder clientStats = globalClientStats.remove(operationId);
604+
clientStats.start(ClientMetrics.OP_DURATION);
603605

604606
CompletableFuture<InsertResponse> responseFuture = new CompletableFuture<>();
605607

@@ -627,7 +629,6 @@ public CompletableFuture<InsertResponse> insert(String tableName,
627629
responseFuture.completeExceptionally(new ClientException("Operation has likely timed out.", e));
628630
}
629631
}
630-
clientStats.stop("insert");
631632
LOG.debug("Total insert (InputStream) time: {}", clientStats.getElapsedTime("insert"));
632633
}
633634

@@ -695,8 +696,8 @@ public CompletableFuture<QueryResponse> query(String sqlQuery, Map<String, Objec
695696
if (settings.getFormat() == null) {
696697
settings.setFormat(ClickHouseFormat.RowBinaryWithNamesAndTypes);
697698
}
698-
OperationStatistics.ClientStatistics clientStats = new OperationStatistics.ClientStatistics();
699-
clientStats.start("query");
699+
ClientStatisticsHolder clientStats = new ClientStatisticsHolder();
700+
clientStats.start(ClientMetrics.OP_DURATION);
700701
ClickHouseClient client = createClient();
701702
ClickHouseRequest<?> request = client.read(getServerNode());
702703

@@ -751,7 +752,7 @@ public CompletableFuture<Records> queryRecords(String sqlQuery, QuerySettings se
751752
settings = new QuerySettings();
752753
}
753754
settings.setFormat(ClickHouseFormat.RowBinaryWithNamesAndTypes);
754-
OperationStatistics.ClientStatistics clientStats = new OperationStatistics.ClientStatistics();
755+
ClientStatisticsHolder clientStats = new ClientStatisticsHolder();
755756
clientStats.start("query");
756757
ClickHouseClient client = createClient();
757758
ClickHouseRequest<?> request = client.read(getServerNode());
@@ -865,7 +866,7 @@ private ClickHouseRequest.Mutation createMutationRequest(ClickHouseRequest.Mutat
865866

866867
private String startOperation() {
867868
String operationId = UUID.randomUUID().toString();
868-
globalClientStats.put(operationId, new OperationStatistics.ClientStatistics());
869+
globalClientStats.put(operationId, new ClientStatisticsHolder());
869870
return operationId;
870871
}
871872
}

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

Lines changed: 0 additions & 156 deletions
This file was deleted.

client-v2/src/main/java/com/clickhouse/client/api/insert/InsertResponse.java

Lines changed: 62 additions & 12 deletions
Original file line numberDiff line numberDiff line change
@@ -2,25 +2,22 @@
22

33
import com.clickhouse.client.ClickHouseClient;
44
import com.clickhouse.client.ClickHouseResponse;
5-
import com.clickhouse.client.ClickHouseResponseSummary;
6-
import com.clickhouse.client.api.OperationStatistics;
7-
8-
import java.util.concurrent.ExecutionException;
9-
import java.util.concurrent.TimeUnit;
10-
import java.util.concurrent.TimeoutException;
5+
import com.clickhouse.client.api.internal.ClientStatisticsHolder;
6+
import com.clickhouse.client.api.metrics.OperationMetrics;
7+
import com.clickhouse.client.api.metrics.ServerMetrics;
118

129
public class InsertResponse implements AutoCloseable {
1310
private final ClickHouseResponse responseRef;
1411
private final ClickHouseClient client;
1512

16-
private OperationStatistics operationStatistics;
13+
private OperationMetrics operationMetrics;
1714

1815
public InsertResponse(ClickHouseClient client, ClickHouseResponse responseRef,
19-
OperationStatistics.ClientStatistics clientStatistics) {
16+
ClientStatisticsHolder clientStatisticsHolder) {
2017
this.responseRef = responseRef;
2118
this.client = client;
22-
this.operationStatistics = new OperationStatistics(clientStatistics);
23-
this.operationStatistics.updateServerStats(responseRef.getSummary());
19+
this.operationMetrics = new OperationMetrics(clientStatisticsHolder);
20+
this.operationMetrics.operationComplete(responseRef.getSummary());
2421
}
2522

2623
@Override
@@ -32,7 +29,60 @@ public void close() {
3229
}
3330
}
3431

35-
public OperationStatistics getOperationStatistics() {
36-
return operationStatistics;
32+
/**
33+
* Returns the metrics of this operation.
34+
*
35+
* @return metrics of this operation
36+
*/
37+
public OperationMetrics getMetrics() {
38+
return operationMetrics;
39+
}
40+
41+
/**
42+
* Alias for {@link ServerMetrics#NUM_ROWS_READ}
43+
* @return number of rows read by server from the storage
44+
*/
45+
public long getReadRows() {
46+
return operationMetrics.getMetric(ServerMetrics.NUM_ROWS_READ).getLong();
47+
}
48+
49+
/**
50+
* Alias for {@link ServerMetrics#NUM_BYTES_READ}
51+
* @return number of bytes read by server from the storage
52+
*/
53+
public long getReadBytes() {
54+
return operationMetrics.getMetric(ServerMetrics.NUM_BYTES_READ).getLong();
55+
}
56+
57+
/**
58+
* Alias for {@link ServerMetrics#NUM_ROWS_WRITTEN}
59+
* @return number of rows written by server to the storage
60+
*/
61+
public long getWrittenRows() {
62+
return operationMetrics.getMetric(ServerMetrics.NUM_ROWS_WRITTEN).getLong();
63+
}
64+
65+
/**
66+
* Alias for {@link ServerMetrics#NUM_BYTES_WRITTEN}
67+
* @return number of bytes written by server to the storage
68+
*/
69+
public long getWrittenBytes() {
70+
return operationMetrics.getMetric(ServerMetrics.NUM_BYTES_WRITTEN).getLong();
71+
}
72+
73+
/**
74+
* Alias for {@link ServerMetrics#ELAPSED_TIME}
75+
* @return elapsed time in nanoseconds
76+
*/
77+
public long getServerTime() {
78+
return operationMetrics.getMetric(ServerMetrics.ELAPSED_TIME).getLong();
79+
}
80+
81+
/**
82+
* Alias for {@link ServerMetrics#RESULT_ROWS}
83+
* @return number of returned rows
84+
*/
85+
public long getResultRows() {
86+
return operationMetrics.getMetric(ServerMetrics.RESULT_ROWS).getLong();
3787
}
3888
}
Lines changed: 45 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,45 @@
1+
package com.clickhouse.client.api.internal;
2+
3+
import com.clickhouse.client.api.metrics.ClientMetrics;
4+
5+
import java.util.HashMap;
6+
import java.util.Map;
7+
8+
public class ClientStatisticsHolder {
9+
10+
private final Map<String, StopWatch> stopWatches = new HashMap<>();
11+
12+
public void start(ClientMetrics metric) {
13+
start(metric.getKey());
14+
}
15+
16+
public void start(String spanName) {
17+
stopWatches.computeIfAbsent(spanName, k -> new StopWatch()).start();
18+
}
19+
20+
public StopWatch stop(ClientMetrics metric) {
21+
return stop(metric.getKey());
22+
}
23+
24+
public StopWatch stop(String spanName) {
25+
StopWatch timer = stopWatches.computeIfAbsent(spanName, k -> new StopWatch());
26+
timer.stop();
27+
return timer;
28+
}
29+
30+
public long getElapsedTime(String spanName) {
31+
StopWatch sw = stopWatches.get(spanName);
32+
return sw == null ? -1 : sw.getElapsedTime();
33+
}
34+
35+
public Map<String, StopWatch> getStopWatches() {
36+
return stopWatches;
37+
}
38+
39+
@Override
40+
public String toString() {
41+
return "ClientStatistics{" +
42+
"\"spans\"=" + stopWatches +
43+
'}';
44+
}
45+
}

0 commit comments

Comments
 (0)