Skip to content

Commit 311e8b0

Browse files
committed
Enable partial results by default in ES|QL
1 parent 18df4d0 commit 311e8b0

File tree

18 files changed

+77
-21
lines changed

18 files changed

+77
-21
lines changed

test/external-modules/esql-heap-attack/src/javaRestTest/java/org/elasticsearch/xpack/esql/heap_attack/HeapAttackIT.java

+2
Original file line numberDiff line numberDiff line change
@@ -117,6 +117,7 @@ public void testSortByManyLongsTooMuchMemoryAsync() throws IOException {
117117
initManyLongs();
118118
Request request = new Request("POST", "/_query/async");
119119
request.addParameter("error_trace", "");
120+
request.addParameter("allow_partial_results", Boolean.toString(false));
120121
request.setJsonEntity(makeSortByManyLongs(5000).toString().replace("\n", "\\n"));
121122
request.setOptions(
122123
RequestOptions.DEFAULT.toBuilder()
@@ -517,6 +518,7 @@ private Map<String, Object> manyEval(int evalLines) throws IOException {
517518
private Response query(String query, String filterPath) throws IOException {
518519
Request request = new Request("POST", "/_query");
519520
request.addParameter("error_trace", "");
521+
request.addParameter("allow_partial_results", Boolean.toString(false));
520522
if (filterPath != null) {
521523
request.addParameter("filter_path", filterPath);
522524
}

x-pack/plugin/build.gradle

+1
Original file line numberDiff line numberDiff line change
@@ -102,6 +102,7 @@ tasks.named("yamlRestCompatTestTransform").configure({ task ->
102102
task.skipTest("esql/190_lookup_join/alias-pattern-multiple", "LOOKUP JOIN does not support index aliases for now")
103103
task.skipTest("esql/190_lookup_join/alias-pattern-single", "LOOKUP JOIN does not support index aliases for now")
104104
task.skipTest("esql/180_match_operator/match with disjunctions", "Disjunctions in full text functions work now")
105+
task.skipTest("esql/63_enrich_int_range/Invalid age as double", "ES|QL now can return partial results instead of failing the query")
105106
// Expected deprecation warning to compat yaml tests:
106107
task.addAllowedWarningRegex(".*rollup functionality will be removed in Elasticsearch.*")
107108
task.skipTest("esql/40_tsdb/from doc with aggregate_metric_double", "TODO: support for subset of metric fields")

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

+2
Original file line numberDiff line numberDiff line change
@@ -39,4 +39,6 @@ public final ActionType<Response> action() {
3939

4040
public abstract EsqlQueryRequestBuilder<Request, Response> filter(QueryBuilder filter);
4141

42+
public abstract EsqlQueryRequestBuilder<Request, Response> allowPartialResults(boolean allowPartialResults);
43+
4244
}

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

+2
Original file line numberDiff line numberDiff line change
@@ -97,6 +97,7 @@ public void testInvalidPragma() throws IOException {
9797
assertOK(client().performRequest(request));
9898
}
9999
RequestObjectBuilder builder = requestObjectBuilder().query("from test-index | limit 1 | keep f");
100+
builder.allPartialResults(false);
100101
builder.pragmas(Settings.builder().put("data_partitioning", "invalid-option").build());
101102
ResponseException re = expectThrows(ResponseException.class, () -> runEsqlSync(builder));
102103
assertThat(EntityUtils.toString(re.getResponse().getEntity()), containsString("No enum constant"));
@@ -107,6 +108,7 @@ public void testInvalidPragma() throws IOException {
107108
public void testPragmaNotAllowed() throws IOException {
108109
assumeFalse("pragma only disabled on release builds", Build.current().isSnapshot());
109110
RequestObjectBuilder builder = requestObjectBuilder().query("row a = 1, b = 2");
111+
builder.allPartialResults(false);
110112
builder.pragmas(Settings.builder().put("data_partitioning", "shard").build());
111113
ResponseException re = expectThrows(ResponseException.class, () -> runEsqlSync(builder));
112114
assertThat(EntityUtils.toString(re.getResponse().getEntity()), containsString("[pragma] only allowed in snapshot builds"));

x-pack/plugin/esql/qa/server/src/main/java/org/elasticsearch/xpack/esql/qa/rest/RestEsqlTestCase.java

+9
Original file line numberDiff line numberDiff line change
@@ -127,6 +127,7 @@ public static class RequestObjectBuilder {
127127
private Boolean includeCCSMetadata = null;
128128

129129
private CheckedConsumer<XContentBuilder, IOException> filter;
130+
private Boolean allPartialResults = null;
130131

131132
public RequestObjectBuilder() throws IOException {
132133
this(randomFrom(XContentType.values()));
@@ -204,6 +205,11 @@ public RequestObjectBuilder filter(CheckedConsumer<XContentBuilder, IOException>
204205
return this;
205206
}
206207

208+
public RequestObjectBuilder allPartialResults(boolean allPartialResults) {
209+
this.allPartialResults = allPartialResults;
210+
return this;
211+
}
212+
207213
public RequestObjectBuilder build() throws IOException {
208214
if (isBuilt == false) {
209215
if (tables != null) {
@@ -1150,6 +1156,9 @@ static Request prepareRequestWithOptions(RequestObjectBuilder requestObject, Mod
11501156
requestObject.build();
11511157
Request request = prepareRequest(mode);
11521158
String mediaType = attachBody(requestObject, request);
1159+
if (requestObject.allPartialResults != null) {
1160+
request.addParameter("allow_partial_results", String.valueOf(requestObject.allPartialResults));
1161+
}
11531162

11541163
RequestOptions.Builder options = request.getOptions().toBuilder();
11551164
options.setWarningsHandler(WarningsHandler.PERMISSIVE); // We assert the warnings ourselves

x-pack/plugin/esql/src/internalClusterTest/java/org/elasticsearch/xpack/esql/action/AbstractCrossClusterTestCase.java

+10
Original file line numberDiff line numberDiff line change
@@ -25,6 +25,7 @@
2525
import org.elasticsearch.transport.RemoteClusterAware;
2626
import org.elasticsearch.xcontent.XContentBuilder;
2727
import org.elasticsearch.xcontent.json.JsonXContent;
28+
import org.elasticsearch.xpack.esql.plugin.EsqlPlugin;
2829
import org.junit.After;
2930
import org.junit.Before;
3031

@@ -117,6 +118,12 @@ public void releaseLatches() {
117118
CrossClusterAsyncQueryIT.CountingPauseFieldPlugin.release();
118119
}
119120

121+
@Override
122+
protected Settings nodeSettings() {
123+
// TODO: remove this override
124+
return Settings.builder().put(super.nodeSettings()).put(EsqlPlugin.QUERY_ALLOW_PARTIAL_RESULTS.getKey(), false).build();
125+
}
126+
120127
protected void assertClusterInfoSuccess(EsqlExecutionInfo.Cluster cluster, int numShards) {
121128
assertThat(cluster.getTook().millis(), greaterThanOrEqualTo(0L));
122129
assertThat(cluster.getStatus(), equalTo(EsqlExecutionInfo.Cluster.Status.SUCCESSFUL));
@@ -261,6 +268,9 @@ protected void clearSkipUnavailable() {
261268
}
262269

263270
protected EsqlQueryResponse runQuery(EsqlQueryRequest request) {
271+
if (request.allowPartialResults() == null) {
272+
request.allowPartialResults(randomBoolean());
273+
}
264274
return client(LOCAL_CLUSTER).execute(EsqlQueryAction.INSTANCE, request).actionGet(30, TimeUnit.SECONDS);
265275
}
266276

x-pack/plugin/esql/src/internalClusterTest/java/org/elasticsearch/xpack/esql/action/AbstractEsqlIntegTestCase.java

+8
Original file line numberDiff line numberDiff line change
@@ -121,6 +121,14 @@ protected Collection<Class<? extends Plugin>> nodePlugins() {
121121
return CollectionUtils.appendToCopy(super.nodePlugins(), EsqlPlugin.class);
122122
}
123123

124+
@Override
125+
protected Settings nodeSettings(int nodeOrdinal, Settings otherSettings) {
126+
return Settings.builder()
127+
.put(super.nodeSettings(nodeOrdinal, otherSettings))
128+
.put(EsqlPlugin.QUERY_ALLOW_PARTIAL_RESULTS.getKey(), false)
129+
.build();
130+
}
131+
124132
protected void setRequestCircuitBreakerLimit(ByteSizeValue limit) {
125133
if (limit != null) {
126134
assertAcked(

x-pack/plugin/esql/src/internalClusterTest/java/org/elasticsearch/xpack/esql/action/EnrichIT.java

+10-1
Original file line numberDiff line numberDiff line change
@@ -139,15 +139,24 @@ protected EsqlQueryResponse run(EsqlQueryRequest request) {
139139
}
140140
if (randomBoolean()) {
141141
setRequestCircuitBreakerLimit(ByteSizeValue.ofBytes(between(256, 4096)));
142+
EsqlQueryResponse resp = null;
142143
try {
143-
return client.execute(EsqlQueryAction.INSTANCE, request).actionGet(2, TimeUnit.MINUTES);
144+
resp = client.execute(EsqlQueryAction.INSTANCE, request).actionGet(2, TimeUnit.MINUTES);
144145
} catch (Exception e) {
145146
logger.info("request failed", e);
146147
EsqlTestUtils.assertEsqlFailure(e);
147148
ensureBlocksReleased();
148149
} finally {
149150
setRequestCircuitBreakerLimit(null);
150151
}
152+
if (resp != null) {
153+
if (resp.isPartial()) {
154+
resp.close();
155+
assertTrue(request.allowPartialResults());
156+
} else {
157+
return resp;
158+
}
159+
}
151160
}
152161
return client.execute(EsqlQueryAction.INSTANCE, request).actionGet(30, TimeUnit.SECONDS);
153162
}

x-pack/plugin/esql/src/internalClusterTest/java/org/elasticsearch/xpack/esql/action/EsqlActionBreakerIT.java

+1-3
Original file line numberDiff line numberDiff line change
@@ -95,9 +95,7 @@ private EsqlQueryResponse runWithBreaking(EsqlQueryRequest request) throws Circu
9595

9696
@Override
9797
protected EsqlQueryResponse run(EsqlQueryRequest request) {
98-
if (randomBoolean()) {
99-
request.allowPartialResults(randomBoolean());
100-
}
98+
request.allowPartialResults(randomBoolean());
10199
Exception failure = null;
102100
try {
103101
final EsqlQueryResponse resp = runWithBreaking(request);

x-pack/plugin/esql/src/internalClusterTest/java/org/elasticsearch/xpack/esql/action/EsqlDisruptionIT.java

+1-3
Original file line numberDiff line numberDiff line change
@@ -83,9 +83,7 @@ private EsqlQueryResponse runQueryWithDisruption(EsqlQueryRequest request) {
8383
logger.info("--> start disruption scheme [{}]", disruptionScheme);
8484
disruptionScheme.startDisrupting();
8585
logger.info("--> executing esql query with disruption {} ", request.query());
86-
if (randomBoolean()) {
87-
request.allowPartialResults(randomBoolean());
88-
}
86+
request.allowPartialResults(randomBoolean());
8987
ActionFuture<EsqlQueryResponse> future = client().execute(EsqlQueryAction.INSTANCE, request);
9088
EsqlQueryResponse resp = null;
9189
try {

x-pack/plugin/esql/src/internalClusterTest/java/org/elasticsearch/xpack/esql/action/ManyShardsIT.java

+2
Original file line numberDiff line numberDiff line change
@@ -69,6 +69,7 @@ protected Collection<Class<? extends Plugin>> nodePlugins() {
6969
@Override
7070
protected Settings nodeSettings(int nodeOrdinal, Settings otherSettings) {
7171
return Settings.builder()
72+
.put(super.nodeSettings(nodeOrdinal, otherSettings))
7273
.put(ExchangeService.INACTIVE_SINKS_INTERVAL_SETTING, TimeValue.timeValueMillis(between(3000, 5000)))
7374
.build();
7475
}
@@ -190,6 +191,7 @@ public void sendResponse(Exception exception) {
190191
request.query("from single-node-index | stats count(user) by tags");
191192
request.acceptedPragmaRisks(true);
192193
request.pragmas(randomPragmas());
194+
request.allowPartialResults(false);
193195
CountDownLatch queryLatch = new CountDownLatch(1);
194196
client().execute(EsqlQueryAction.INSTANCE, request, ActionListener.runAfter(ActionListener.wrap(r -> {
195197
r.close();

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

+4-3
Original file line numberDiff line numberDiff line change
@@ -52,7 +52,7 @@ public class EsqlQueryRequest extends org.elasticsearch.xpack.core.esql.action.E
5252
private boolean keepOnCompletion;
5353
private boolean onSnapshotBuild = Build.current().isSnapshot();
5454
private boolean acceptedPragmaRisks = false;
55-
private boolean allowPartialResults = false;
55+
private Boolean allowPartialResults = null;
5656

5757
/**
5858
* "Tables" provided in the request for use with things like {@code LOOKUP}.
@@ -232,12 +232,13 @@ public Map<String, Map<String, Column>> tables() {
232232
return tables;
233233
}
234234

235-
public boolean allowPartialResults() {
235+
public Boolean allowPartialResults() {
236236
return allowPartialResults;
237237
}
238238

239-
public void allowPartialResults(boolean allowPartialResults) {
239+
public EsqlQueryRequest allowPartialResults(boolean allowPartialResults) {
240240
this.allowPartialResults = allowPartialResults;
241+
return this;
241242
}
242243

243244
@Override

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

+6
Original file line numberDiff line numberDiff line change
@@ -66,6 +66,12 @@ public EsqlQueryRequestBuilder keepOnCompletion(boolean keepOnCompletion) {
6666
return this;
6767
}
6868

69+
@Override
70+
public EsqlQueryRequestBuilder allowPartialResults(boolean allowPartialResults) {
71+
request.allowPartialResults(allowPartialResults);
72+
return this;
73+
}
74+
6975
static { // plumb access from x-pack core
7076
SharedSecrets.setEsqlQueryRequestBuilderAccess(EsqlQueryRequestBuilder::newSyncEsqlQueryRequestBuilder);
7177
}

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

-2
Original file line numberDiff line numberDiff line change
@@ -85,7 +85,6 @@ String fields() {
8585
static final ParseField WAIT_FOR_COMPLETION_TIMEOUT = new ParseField("wait_for_completion_timeout");
8686
static final ParseField KEEP_ALIVE = new ParseField("keep_alive");
8787
static final ParseField KEEP_ON_COMPLETION = new ParseField("keep_on_completion");
88-
static final ParseField ALLOW_PARTIAL_RESULTS = new ParseField("allow_partial_results");
8988

9089
private static final ObjectParser<EsqlQueryRequest, Void> SYNC_PARSER = objectParserSync(EsqlQueryRequest::syncEsqlQueryRequest);
9190
private static final ObjectParser<EsqlQueryRequest, Void> ASYNC_PARSER = objectParserAsync(EsqlQueryRequest::asyncEsqlQueryRequest);
@@ -115,7 +114,6 @@ private static void objectParserCommon(ObjectParser<EsqlQueryRequest, ?> parser)
115114
parser.declareString((request, localeTag) -> request.locale(Locale.forLanguageTag(localeTag)), LOCALE_FIELD);
116115
parser.declareBoolean(EsqlQueryRequest::profile, PROFILE_FIELD);
117116
parser.declareField((p, r, c) -> new ParseTables(r, p).parseTables(), TABLES_FIELD, ObjectParser.ValueType.OBJECT);
118-
parser.declareBoolean(EsqlQueryRequest::allowPartialResults, ALLOW_PARTIAL_RESULTS);
119117
}
120118

121119
private static ObjectParser<EsqlQueryRequest, Void> objectParserSync(Supplier<EsqlQueryRequest> supplier) {

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

+4
Original file line numberDiff line numberDiff line change
@@ -51,6 +51,10 @@ protected RestChannelConsumer prepareRequest(RestRequest request, NodeClient cli
5151
}
5252

5353
protected static RestChannelConsumer restChannelConsumer(EsqlQueryRequest esqlRequest, RestRequest request, NodeClient client) {
54+
final Boolean partialResults = request.paramAsBoolean("allow_partial_results", null);
55+
if (partialResults != null) {
56+
esqlRequest.allowPartialResults(partialResults);
57+
}
5458
LOGGER.debug("Beginning execution of ESQL query.\nQuery string: [{}]", esqlRequest.query());
5559

5660
return channel -> {

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

+8-1
Original file line numberDiff line numberDiff line change
@@ -101,6 +101,13 @@ public class EsqlPlugin extends Plugin implements ActionPlugin {
101101
Setting.Property.Dynamic
102102
);
103103

104+
public static final Setting<Boolean> QUERY_ALLOW_PARTIAL_RESULTS = Setting.boolSetting(
105+
"esql.query.allow_partial_results",
106+
true,
107+
Setting.Property.NodeScope,
108+
Setting.Property.Dynamic
109+
);
110+
104111
@Override
105112
public Collection<?> createComponents(PluginServices services) {
106113
CircuitBreaker circuitBreaker = services.indicesService().getBigArrays().breakerService().getBreaker("request");
@@ -141,7 +148,7 @@ protected XPackLicenseState getLicenseState() {
141148
*/
142149
@Override
143150
public List<Setting<?>> getSettings() {
144-
return List.of(QUERY_RESULT_TRUNCATION_DEFAULT_SIZE, QUERY_RESULT_TRUNCATION_MAX_SIZE);
151+
return List.of(QUERY_RESULT_TRUNCATION_DEFAULT_SIZE, QUERY_RESULT_TRUNCATION_MAX_SIZE, QUERY_ALLOW_PARTIAL_RESULTS);
145152
}
146153

147154
@Override

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

+7
Original file line numberDiff line numberDiff line change
@@ -80,6 +80,7 @@ public class TransportEsqlQueryAction extends HandledTransportAction<EsqlQueryRe
8080
private final RemoteClusterService remoteClusterService;
8181
private final UsageService usageService;
8282
private final TransportActionServices services;
83+
private volatile boolean defaultAllowPartialResults;
8384

8485
@Inject
8586
@SuppressWarnings("this-escape")
@@ -158,6 +159,9 @@ public TransportEsqlQueryAction(
158159
indexNameExpressionResolver,
159160
usageService
160161
);
162+
defaultAllowPartialResults = EsqlPlugin.QUERY_ALLOW_PARTIAL_RESULTS.get(clusterService.getSettings());
163+
clusterService.getClusterSettings()
164+
.addSettingsUpdateConsumer(EsqlPlugin.QUERY_ALLOW_PARTIAL_RESULTS, v -> defaultAllowPartialResults = v);
161165
}
162166

163167
@Override
@@ -194,6 +198,9 @@ public void execute(EsqlQueryRequest request, EsqlQueryTask task, ActionListener
194198
}
195199

196200
private void innerExecute(Task task, EsqlQueryRequest request, ActionListener<EsqlQueryResponse> listener) {
201+
if (request.allowPartialResults() == null) {
202+
request.allowPartialResults(defaultAllowPartialResults);
203+
}
197204
Configuration configuration = new Configuration(
198205
ZoneOffset.UTC,
199206
request.locale() != null ? request.locale() : Locale.US,

x-pack/plugin/src/yamlRestTest/resources/rest-api-spec/test/esql/63_enrich_int_range.yml

-8
Original file line numberDiff line numberDiff line change
@@ -189,11 +189,3 @@ teardown:
189189

190190
- length: { values: 1 }
191191
- match: { values.0: [ 5, null ] }
192-
193-
---
194-
"Invalid age as double":
195-
- do:
196-
catch: /ENRICH range and input types are incompatible. range\[INTEGER\], input\[DOUBLE\]/
197-
esql.query:
198-
body:
199-
query: 'FROM employees | ENRICH ages-policy ON salary | STATS count=COUNT(*) BY description | SORT count DESC, description ASC'

0 commit comments

Comments
 (0)