Skip to content

Commit 07cb14e

Browse files
authored
Expose more detailed profiling information (#126525)
1 parent 9e3476e commit 07cb14e

File tree

8 files changed

+139
-51
lines changed

8 files changed

+139
-51
lines changed

server/src/main/java/org/elasticsearch/TransportVersions.java

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -219,6 +219,7 @@ static TransportVersion def(int id) {
219219
public static final TransportVersion REPO_ANALYSIS_COPY_BLOB = def(9_048_00_0);
220220
public static final TransportVersion AMAZON_BEDROCK_TASK_SETTINGS = def(9_049_00_0);
221221
public static final TransportVersion ESQL_REPORT_SHARD_PARTITIONING = def(9_050_00_0);
222+
public static final TransportVersion ESQL_QUERY_PLANNING_DURATION = def(9_051_00_0);
222223

223224
/*
224225
* STOP! READ THIS FIRST! No, really,

test/framework/src/main/java/org/elasticsearch/test/rest/ESRestTestCase.java

Lines changed: 7 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -151,6 +151,7 @@
151151
import static org.hamcrest.Matchers.everyItem;
152152
import static org.hamcrest.Matchers.greaterThanOrEqualTo;
153153
import static org.hamcrest.Matchers.in;
154+
import static org.hamcrest.Matchers.instanceOf;
154155
import static org.hamcrest.Matchers.notNullValue;
155156

156157
/**
@@ -2665,6 +2666,12 @@ public static Request newXContentRequest(HttpMethod method, String endpoint, ToX
26652666
return request;
26662667
}
26672668

2669+
protected static MapMatcher getProfileMatcher() {
2670+
return matchesMap().entry("query", instanceOf(Map.class))
2671+
.entry("planning", instanceOf(Map.class))
2672+
.entry("drivers", instanceOf(List.class));
2673+
}
2674+
26682675
protected static MapMatcher getResultMatcher(boolean includeMetadata, boolean includePartial) {
26692676
MapMatcher mapMatcher = matchesMap();
26702677
if (includeMetadata) {

x-pack/plugin/esql/qa/server/single-node/src/javaRestTest/java/org/elasticsearch/xpack/esql/qa/single_node/RestEsqlIT.java

Lines changed: 3 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -299,7 +299,7 @@ public void testProfile() throws IOException {
299299
Map<String, Object> result = runEsql(builder);
300300
assertResultMap(
301301
result,
302-
getResultMatcher(result).entry("profile", matchesMap().entry("drivers", instanceOf(List.class))),
302+
getResultMatcher(result).entry("profile", getProfileMatcher()),
303303
matchesList().item(matchesMap().entry("name", "AVG(value)").entry("type", "double")),
304304
equalTo(List.of(List.of(499.5d)))
305305
);
@@ -502,7 +502,7 @@ public void testInlineStatsProfile() throws IOException {
502502
}
503503
assertResultMap(
504504
result,
505-
getResultMatcher(result).entry("profile", matchesMap().entry("drivers", instanceOf(List.class))),
505+
getResultMatcher(result).entry("profile", getProfileMatcher()),
506506
matchesList().item(matchesMap().entry("name", "@timestamp").entry("type", "date"))
507507
.item(matchesMap().entry("name", "test").entry("type", "text"))
508508
.item(matchesMap().entry("name", "test.keyword").entry("type", "keyword"))
@@ -605,7 +605,7 @@ public void testForceSleepsProfile() throws IOException {
605605
}
606606
assertResultMap(
607607
result,
608-
getResultMatcher(result).entry("profile", matchesMap().entry("drivers", instanceOf(List.class))),
608+
getResultMatcher(result).entry("profile", getProfileMatcher()),
609609
matchesList().item(matchesMap().entry("name", "AVG(value)").entry("type", "double"))
610610
.item(matchesMap().entry("name", "MAX(value)").entry("type", "long"))
611611
.item(matchesMap().entry("name", "MIN(value)").entry("type", "long"))

x-pack/plugin/esql/src/main/java/org/elasticsearch/xpack/esql/action/EsqlExecutionInfo.java

Lines changed: 33 additions & 23 deletions
Original file line numberDiff line numberDiff line change
@@ -37,7 +37,6 @@
3737
import java.util.Objects;
3838
import java.util.Set;
3939
import java.util.concurrent.ConcurrentMap;
40-
import java.util.concurrent.TimeUnit;
4140
import java.util.function.BiFunction;
4241
import java.util.function.Predicate;
4342
import java.util.stream.Stream;
@@ -67,17 +66,20 @@ public class EsqlExecutionInfo implements ChunkedToXContentObject, Writeable {
6766
// Updates to the Cluster occur with the updateCluster method that given the key to map transforms an
6867
// old Cluster Object to a new Cluster Object with the remapping function.
6968
public final Map<String, Cluster> clusterInfo;
70-
private TimeValue overallTook;
7169
// whether the user has asked for CCS metadata to be in the JSON response (the overall took will always be present)
7270
private final boolean includeCCSMetadata;
7371

7472
// fields that are not Writeable since they are only needed on the primary CCS coordinator
7573
private final transient Predicate<String> skipUnavailablePredicate;
76-
private final transient Long relativeStartNanos; // start time for an ESQL query for calculating took times
77-
private transient TimeValue planningTookTime; // time elapsed since start of query to calling ComputeService.execute
7874
private volatile boolean isPartial; // Does this request have partial results?
7975
private transient volatile boolean isStopped; // Have we received stop command?
8076

77+
// start time for the ESQL query for calculating time spans relative to the beginning of the query
78+
private final transient TimeSpan.Builder relativeStart;
79+
private transient TimeSpan overallTimeSpan;
80+
private transient TimeSpan planningTimeSpan; // time elapsed since start of query to calling ComputeService.execute
81+
private TimeValue overallTook;
82+
8183
public EsqlExecutionInfo(boolean includeCCSMetadata) {
8284
this(Predicates.always(), includeCCSMetadata); // default all clusters to skip_unavailable=true
8385
}
@@ -90,18 +92,17 @@ public EsqlExecutionInfo(Predicate<String> skipUnavailablePredicate, boolean inc
9092
this.clusterInfo = ConcurrentCollections.newConcurrentMap();
9193
this.skipUnavailablePredicate = skipUnavailablePredicate;
9294
this.includeCCSMetadata = includeCCSMetadata;
93-
this.relativeStartNanos = System.nanoTime();
95+
this.relativeStart = TimeSpan.start();
9496
}
9597

9698
/**
9799
* For testing use with fromXContent parsing only
98-
* @param clusterInfo
99100
*/
100101
EsqlExecutionInfo(ConcurrentMap<String, Cluster> clusterInfo, boolean includeCCSMetadata) {
101102
this.clusterInfo = clusterInfo;
102103
this.includeCCSMetadata = includeCCSMetadata;
103104
this.skipUnavailablePredicate = Predicates.always();
104-
this.relativeStartNanos = null;
105+
this.relativeStart = null;
105106
}
106107

107108
public EsqlExecutionInfo(StreamInput in) throws IOException {
@@ -127,7 +128,11 @@ public EsqlExecutionInfo(StreamInput in) throws IOException {
127128
}
128129

129130
this.skipUnavailablePredicate = Predicates.always();
130-
this.relativeStartNanos = null;
131+
this.relativeStart = null;
132+
if (in.getTransportVersion().onOrAfter(TransportVersions.ESQL_QUERY_PLANNING_DURATION)) {
133+
this.overallTimeSpan = in.readOptional(TimeSpan::readFrom);
134+
this.planningTimeSpan = in.readOptional(TimeSpan::readFrom);
135+
}
131136
}
132137

133138
@Override
@@ -144,37 +149,38 @@ public void writeTo(StreamOutput out) throws IOException {
144149
if (out.getTransportVersion().onOrAfter(TransportVersions.ESQL_RESPONSE_PARTIAL)) {
145150
out.writeBoolean(isPartial);
146151
}
152+
if (out.getTransportVersion().onOrAfter(TransportVersions.ESQL_QUERY_PLANNING_DURATION)) {
153+
out.writeOptionalWriteable(overallTimeSpan);
154+
out.writeOptionalWriteable(planningTimeSpan);
155+
}
147156
}
148157

149158
public boolean includeCCSMetadata() {
150159
return includeCCSMetadata;
151160
}
152161

153-
public Long getRelativeStartNanos() {
154-
return relativeStartNanos;
155-
}
156-
157162
/**
158163
* Call when ES|QL "planning" phase is complete and query execution (in ComputeService) is about to start.
159164
* Note this is currently only built for a single phase planning/execution model. When INLINESTATS
160165
* moves towards GA we may need to revisit this model. Currently, it should never be called more than once.
161166
*/
162167
public void markEndPlanning() {
163-
assert planningTookTime == null : "markEndPlanning should only be called once";
164-
assert relativeStartNanos != null : "Relative start time must be set when markEndPlanning is called";
165-
planningTookTime = new TimeValue(System.nanoTime() - relativeStartNanos, TimeUnit.NANOSECONDS);
168+
assert planningTimeSpan == null : "markEndPlanning should only be called once";
169+
assert relativeStart != null : "Relative start time must be set when markEndPlanning is called";
170+
planningTimeSpan = relativeStart.stop();
166171
}
167172

168173
public TimeValue planningTookTime() {
169-
return planningTookTime;
174+
return planningTimeSpan != null ? planningTimeSpan.toTimeValue() : null;
170175
}
171176

172177
/**
173178
* Call when ES|QL execution is complete in order to set the overall took time for an ES|QL query.
174179
*/
175180
public void markEndQuery() {
176-
assert relativeStartNanos != null : "Relative start time must be set when markEndQuery is called";
177-
overallTook = new TimeValue(System.nanoTime() - relativeStartNanos, TimeUnit.NANOSECONDS);
181+
assert relativeStart != null : "Relative start time must be set when markEndQuery is called";
182+
overallTimeSpan = relativeStart.stop();
183+
overallTook = overallTimeSpan.toTimeValue();
178184
}
179185

180186
// for testing only - use markEndQuery in production code
@@ -190,11 +196,15 @@ public TimeValue overallTook() {
190196
* How much time the query took since starting.
191197
*/
192198
public TimeValue tookSoFar() {
193-
if (relativeStartNanos == null) {
194-
return new TimeValue(0);
195-
} else {
196-
return new TimeValue(System.nanoTime() - relativeStartNanos, TimeUnit.NANOSECONDS);
197-
}
199+
return relativeStart != null ? relativeStart.stop().toTimeValue() : TimeValue.ZERO;
200+
}
201+
202+
public TimeSpan overallTimeSpan() {
203+
return overallTimeSpan;
204+
}
205+
206+
public TimeSpan planningTimeSpan() {
207+
return planningTimeSpan;
198208
}
199209

200210
public Set<String> clusterAliases() {

x-pack/plugin/esql/src/main/java/org/elasticsearch/xpack/esql/action/EsqlQueryResponse.java

Lines changed: 15 additions & 19 deletions
Original file line numberDiff line numberDiff line change
@@ -225,9 +225,6 @@ public Iterator<? extends ToXContent> toXContentChunked(ToXContent.Params params
225225
)
226226
: ResponseXContentUtils.allColumns(columns, "columns");
227227
Iterator<? extends ToXContent> valuesIt = ResponseXContentUtils.columnValues(this.columns, this.pages, columnar, nullColumns);
228-
Iterator<ToXContent> profileRender = profile != null
229-
? ChunkedToXContentHelper.field("profile", profile, params)
230-
: Collections.emptyIterator();
231228
Iterator<ToXContent> executionInfoRender = executionInfo != null && executionInfo.hasMetadataToReport()
232229
? ChunkedToXContentHelper.field("_clusters", executionInfo, params)
233230
: Collections.emptyIterator();
@@ -238,11 +235,24 @@ public Iterator<? extends ToXContent> toXContentChunked(ToXContent.Params params
238235
columnHeadings,
239236
ChunkedToXContentHelper.array("values", valuesIt),
240237
executionInfoRender,
241-
profileRender,
238+
profileRenderer(params),
242239
ChunkedToXContentHelper.endObject()
243240
);
244241
}
245242

243+
private Iterator<ToXContent> profileRenderer(ToXContent.Params params) {
244+
if (profile == null) {
245+
return Collections.emptyIterator();
246+
}
247+
return Iterators.concat(ChunkedToXContentHelper.startObject("profile"), ChunkedToXContentHelper.chunk((b, p) -> {
248+
if (executionInfo != null) {
249+
b.field("query", executionInfo.overallTimeSpan());
250+
b.field("planning", executionInfo.planningTimeSpan());
251+
}
252+
return b;
253+
}), ChunkedToXContentHelper.array("drivers", profile.drivers.iterator(), params), ChunkedToXContentHelper.endObject());
254+
}
255+
246256
public boolean[] nullColumns() {
247257
boolean[] nullColumns = new boolean[columns.size()];
248258
for (int c = 0; c < nullColumns.length; c++) {
@@ -260,11 +270,6 @@ private boolean allColumnsAreNull(int c) {
260270
return true;
261271
}
262272

263-
@Override
264-
public boolean isFragment() {
265-
return false;
266-
}
267-
268273
@Override
269274
public boolean equals(Object o) {
270275
if (this == o) return true;
@@ -344,7 +349,7 @@ public EsqlResponse responseInternal() {
344349
return esqlResponse;
345350
}
346351

347-
public static class Profile implements Writeable, ChunkedToXContentObject {
352+
public static class Profile implements Writeable {
348353
private final List<DriverProfile> drivers;
349354

350355
public Profile(List<DriverProfile> drivers) {
@@ -377,15 +382,6 @@ public int hashCode() {
377382
return Objects.hash(drivers);
378383
}
379384

380-
@Override
381-
public Iterator<? extends ToXContent> toXContentChunked(ToXContent.Params params) {
382-
return Iterators.concat(
383-
ChunkedToXContentHelper.startObject(),
384-
ChunkedToXContentHelper.array("drivers", drivers.iterator(), params),
385-
ChunkedToXContentHelper.endObject()
386-
);
387-
}
388-
389385
List<DriverProfile> drivers() {
390386
return drivers;
391387
}
Lines changed: 77 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,77 @@
1+
/*
2+
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
3+
* or more contributor license agreements. Licensed under the Elastic License
4+
* 2.0; you may not use this file except in compliance with the Elastic License
5+
* 2.0.
6+
*/
7+
8+
package org.elasticsearch.xpack.esql.action;
9+
10+
import org.elasticsearch.common.io.stream.StreamInput;
11+
import org.elasticsearch.common.io.stream.StreamOutput;
12+
import org.elasticsearch.common.io.stream.Writeable;
13+
import org.elasticsearch.core.TimeValue;
14+
import org.elasticsearch.xcontent.ToXContentObject;
15+
import org.elasticsearch.xcontent.XContentBuilder;
16+
17+
import java.io.IOException;
18+
19+
import static org.elasticsearch.core.TimeValue.timeValueNanos;
20+
21+
/**
22+
* THis class is used to capture a duration of some process, including start and stop point int time.
23+
*/
24+
public record TimeSpan(long startMillis, long startNanos, long stopMillis, long stopNanos) implements Writeable, ToXContentObject {
25+
26+
public static TimeSpan readFrom(StreamInput in) throws IOException {
27+
return new TimeSpan(in.readVLong(), in.readVLong(), in.readVLong(), in.readVLong());
28+
}
29+
30+
@Override
31+
public void writeTo(StreamOutput out) throws IOException {
32+
out.writeVLong(startMillis);
33+
out.writeVLong(startNanos);
34+
out.writeVLong(stopMillis);
35+
out.writeVLong(stopNanos);
36+
}
37+
38+
@Override
39+
public XContentBuilder toXContent(XContentBuilder builder, Params params) throws IOException {
40+
builder.startObject();
41+
builder.timestampFieldsFromUnixEpochMillis("start_millis", "start", startMillis);
42+
builder.timestampFieldsFromUnixEpochMillis("stop_millis", "stop", stopMillis);
43+
if (builder.humanReadable()) {
44+
builder.field("took_time", toTimeValue());
45+
}
46+
builder.field("took_millis", durationInMillis());
47+
builder.field("took_nanos", durationInNanos());
48+
builder.endObject();
49+
return builder;
50+
}
51+
52+
public TimeValue toTimeValue() {
53+
return timeValueNanos(stopNanos - startNanos);
54+
}
55+
56+
public long durationInMillis() {
57+
return stopMillis - startMillis;
58+
}
59+
60+
public long durationInNanos() {
61+
return stopNanos - startNanos;
62+
}
63+
64+
public static Builder start() {
65+
return new Builder();
66+
}
67+
68+
public static class Builder {
69+
70+
private final long startMillis = System.currentTimeMillis();
71+
private final long startNanos = System.nanoTime();
72+
73+
public TimeSpan stop() {
74+
return new TimeSpan(startMillis, startNanos, System.currentTimeMillis(), System.nanoTime());
75+
}
76+
}
77+
}

x-pack/plugin/esql/src/main/java/org/elasticsearch/xpack/esql/plugin/ComputeService.java

Lines changed: 1 addition & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -26,7 +26,6 @@
2626
import org.elasticsearch.compute.operator.exchange.ExchangeSourceHandler;
2727
import org.elasticsearch.core.Releasable;
2828
import org.elasticsearch.core.Releasables;
29-
import org.elasticsearch.core.TimeValue;
3029
import org.elasticsearch.core.Tuple;
3130
import org.elasticsearch.index.query.SearchExecutionContext;
3231
import org.elasticsearch.logging.LogManager;
@@ -261,7 +260,7 @@ public void execute(
261260
computeListener.acquireCompute().delegateFailure((l, profiles) -> {
262261
if (execInfo.clusterInfo.containsKey(LOCAL_CLUSTER)) {
263262
execInfo.swapCluster(LOCAL_CLUSTER, (k, v) -> {
264-
var tookTime = TimeValue.timeValueNanos(System.nanoTime() - execInfo.getRelativeStartNanos());
263+
var tookTime = execInfo.tookSoFar();
265264
var builder = new EsqlExecutionInfo.Cluster.Builder(v).setTook(tookTime);
266265
if (v.getStatus() == EsqlExecutionInfo.Cluster.Status.RUNNING) {
267266
final Integer failedShards = execInfo.getCluster(LOCAL_CLUSTER).getFailedShards();

x-pack/plugin/esql/src/test/java/org/elasticsearch/xpack/esql/session/EsqlCCSUtilsTests.java

Lines changed: 2 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -502,12 +502,10 @@ public void testUpdateExecutionInfoAtEndOfPlanning() {
502502
(k, v) -> new EsqlExecutionInfo.Cluster(REMOTE1_ALIAS, "*", true, EsqlExecutionInfo.Cluster.Status.SKIPPED)
503503
);
504504
executionInfo.swapCluster(REMOTE2_ALIAS, (k, v) -> new EsqlExecutionInfo.Cluster(REMOTE2_ALIAS, "mylogs1,mylogs2,logs*", false));
505-
506505
assertNull(executionInfo.planningTookTime());
507506
assertNull(executionInfo.overallTook());
508-
try {
509-
Thread.sleep(1);
510-
} catch (InterruptedException e) {}
507+
508+
safeSleep(1);
511509

512510
EsqlCCSUtils.updateExecutionInfoAtEndOfPlanning(executionInfo);
513511

0 commit comments

Comments
 (0)