Skip to content

Commit e13173c

Browse files
authored
[ES|QL] COMPLETION command logical plan optimizer (#126763)
1 parent 8c9a091 commit e13173c

File tree

7 files changed

+259
-1
lines changed

7 files changed

+259
-1
lines changed

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

Lines changed: 2 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -38,6 +38,7 @@
3838
import org.elasticsearch.xpack.esql.optimizer.rules.logical.PushDownAndCombineFilters;
3939
import org.elasticsearch.xpack.esql.optimizer.rules.logical.PushDownAndCombineLimits;
4040
import org.elasticsearch.xpack.esql.optimizer.rules.logical.PushDownAndCombineOrderBy;
41+
import org.elasticsearch.xpack.esql.optimizer.rules.logical.PushDownCompletion;
4142
import org.elasticsearch.xpack.esql.optimizer.rules.logical.PushDownEnrich;
4243
import org.elasticsearch.xpack.esql.optimizer.rules.logical.PushDownEval;
4344
import org.elasticsearch.xpack.esql.optimizer.rules.logical.PushDownRegexExtract;
@@ -190,6 +191,7 @@ protected static Batch<LogicalPlan> operators() {
190191
new PruneLiteralsInOrderBy(),
191192
new PushDownAndCombineLimits(),
192193
new PushDownAndCombineFilters(),
194+
new PushDownCompletion(),
193195
new PushDownEval(),
194196
new PushDownRegexExtract(),
195197
new PushDownEnrich(),

x-pack/plugin/esql/src/main/java/org/elasticsearch/xpack/esql/optimizer/rules/logical/PushDownAndCombineFilters.java

Lines changed: 5 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -24,6 +24,7 @@
2424
import org.elasticsearch.xpack.esql.plan.logical.Project;
2525
import org.elasticsearch.xpack.esql.plan.logical.RegexExtract;
2626
import org.elasticsearch.xpack.esql.plan.logical.UnaryPlan;
27+
import org.elasticsearch.xpack.esql.plan.logical.inference.Completion;
2728
import org.elasticsearch.xpack.esql.plan.logical.join.Join;
2829
import org.elasticsearch.xpack.esql.plan.logical.join.JoinTypes;
2930

@@ -70,6 +71,10 @@ protected LogicalPlan rule(Filter filter) {
7071
// Push down filters that do not rely on attributes created by RegexExtract
7172
var attributes = AttributeSet.of(Expressions.asAttributes(re.extractedFields()));
7273
plan = maybePushDownPastUnary(filter, re, attributes::contains, NO_OP);
74+
} else if (child instanceof Completion completion) {
75+
// Push down filters that do not rely on attributes created by Cpmpletion
76+
var attributes = AttributeSet.of(completion.generatedAttributes());
77+
plan = maybePushDownPastUnary(filter, completion, attributes::contains, NO_OP);
7378
} else if (child instanceof Enrich enrich) {
7479
// Push down filters that do not rely on attributes created by Enrich
7580
var attributes = AttributeSet.of(Expressions.asAttributes(enrich.enrichFields()));

x-pack/plugin/esql/src/main/java/org/elasticsearch/xpack/esql/optimizer/rules/logical/PushDownAndCombineLimits.java

Lines changed: 6 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -17,6 +17,7 @@
1717
import org.elasticsearch.xpack.esql.plan.logical.Project;
1818
import org.elasticsearch.xpack.esql.plan.logical.RegexExtract;
1919
import org.elasticsearch.xpack.esql.plan.logical.UnaryPlan;
20+
import org.elasticsearch.xpack.esql.plan.logical.inference.Completion;
2021
import org.elasticsearch.xpack.esql.plan.logical.join.Join;
2122
import org.elasticsearch.xpack.esql.plan.logical.join.JoinTypes;
2223

@@ -38,7 +39,11 @@ public LogicalPlan rule(Limit limit, LogicalOptimizerContext ctx) {
3839
// We want to preserve the duplicated() value of the smaller limit, so we'll use replaceChild.
3940
return parentLimitValue < childLimitValue ? limit.replaceChild(childLimit.child()) : childLimit;
4041
} else if (limit.child() instanceof UnaryPlan unary) {
41-
if (unary instanceof Eval || unary instanceof Project || unary instanceof RegexExtract || unary instanceof Enrich) {
42+
if (unary instanceof Eval
43+
|| unary instanceof Project
44+
|| unary instanceof RegexExtract
45+
|| unary instanceof Enrich
46+
|| unary instanceof Completion) {
4247
return unary.replaceChild(limit.replaceChild(unary.child()));
4348
} else if (unary instanceof MvExpand) {
4449
// MV_EXPAND can increase the number of rows, so we cannot just push the limit down
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,18 @@
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.optimizer.rules.logical;
9+
10+
import org.elasticsearch.xpack.esql.plan.logical.LogicalPlan;
11+
import org.elasticsearch.xpack.esql.plan.logical.inference.Completion;
12+
13+
public final class PushDownCompletion extends OptimizerRules.OptimizerRule<Completion> {
14+
@Override
15+
protected LogicalPlan rule(Completion p) {
16+
return PushDownUtils.pushGeneratingPlanPastProjectAndOrderBy(p);
17+
}
18+
}

x-pack/plugin/esql/src/test/java/org/elasticsearch/xpack/esql/optimizer/LogicalPlanOptimizerTests.java

Lines changed: 15 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -100,6 +100,7 @@
100100
import org.elasticsearch.xpack.esql.optimizer.rules.logical.OptimizerRules;
101101
import org.elasticsearch.xpack.esql.optimizer.rules.logical.PruneRedundantOrderBy;
102102
import org.elasticsearch.xpack.esql.optimizer.rules.logical.PushDownAndCombineLimits;
103+
import org.elasticsearch.xpack.esql.optimizer.rules.logical.PushDownCompletion;
103104
import org.elasticsearch.xpack.esql.optimizer.rules.logical.PushDownEnrich;
104105
import org.elasticsearch.xpack.esql.optimizer.rules.logical.PushDownEval;
105106
import org.elasticsearch.xpack.esql.optimizer.rules.logical.PushDownRegexExtract;
@@ -123,6 +124,7 @@
123124
import org.elasticsearch.xpack.esql.plan.logical.TimeSeriesAggregate;
124125
import org.elasticsearch.xpack.esql.plan.logical.TopN;
125126
import org.elasticsearch.xpack.esql.plan.logical.UnaryPlan;
127+
import org.elasticsearch.xpack.esql.plan.logical.inference.Completion;
126128
import org.elasticsearch.xpack.esql.plan.logical.join.InlineJoin;
127129
import org.elasticsearch.xpack.esql.plan.logical.join.Join;
128130
import org.elasticsearch.xpack.esql.plan.logical.join.JoinConfig;
@@ -162,6 +164,7 @@
162164
import static org.elasticsearch.xpack.esql.EsqlTestUtils.getFieldAttribute;
163165
import static org.elasticsearch.xpack.esql.EsqlTestUtils.loadMapping;
164166
import static org.elasticsearch.xpack.esql.EsqlTestUtils.localSource;
167+
import static org.elasticsearch.xpack.esql.EsqlTestUtils.randomLiteral;
165168
import static org.elasticsearch.xpack.esql.EsqlTestUtils.referenceAttribute;
166169
import static org.elasticsearch.xpack.esql.EsqlTestUtils.singleValue;
167170
import static org.elasticsearch.xpack.esql.EsqlTestUtils.unboundLogicalOptimizerContext;
@@ -176,6 +179,7 @@
176179
import static org.elasticsearch.xpack.esql.core.type.DataType.INTEGER;
177180
import static org.elasticsearch.xpack.esql.core.type.DataType.KEYWORD;
178181
import static org.elasticsearch.xpack.esql.core.type.DataType.LONG;
182+
import static org.elasticsearch.xpack.esql.core.type.DataType.TEXT;
179183
import static org.elasticsearch.xpack.esql.expression.predicate.operator.comparison.EsqlBinaryComparison.BinaryComparisonOperation.EQ;
180184
import static org.elasticsearch.xpack.esql.expression.predicate.operator.comparison.EsqlBinaryComparison.BinaryComparisonOperation.GT;
181185
import static org.elasticsearch.xpack.esql.expression.predicate.operator.comparison.EsqlBinaryComparison.BinaryComparisonOperation.GTE;
@@ -5556,6 +5560,17 @@ record PushdownShadowingGeneratingPlanTestCase(
55565560
)
55575561
),
55585562
new PushDownEnrich()
5563+
),
5564+
// | COMPLETION CONCAT(some text, x) WITH inferenceID AS y
5565+
new PushdownShadowingGeneratingPlanTestCase(
5566+
(plan, attr) -> new Completion(
5567+
EMPTY,
5568+
plan,
5569+
randomLiteral(TEXT),
5570+
new Concat(EMPTY, randomLiteral(TEXT), List.of(attr)),
5571+
new ReferenceAttribute(EMPTY, "y", KEYWORD)
5572+
),
5573+
new PushDownCompletion()
55595574
) };
55605575

55615576
/**

x-pack/plugin/esql/src/test/java/org/elasticsearch/xpack/esql/optimizer/rules/logical/PushDownAndCombineFiltersTests.java

Lines changed: 54 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -8,12 +8,15 @@
88
package org.elasticsearch.xpack.esql.optimizer.rules.logical;
99

1010
import org.elasticsearch.index.IndexMode;
11+
import org.elasticsearch.index.query.QueryBuilder;
1112
import org.elasticsearch.test.ESTestCase;
1213
import org.elasticsearch.xpack.esql.core.expression.Alias;
1314
import org.elasticsearch.xpack.esql.core.expression.Attribute;
1415
import org.elasticsearch.xpack.esql.core.expression.Expression;
1516
import org.elasticsearch.xpack.esql.core.expression.FieldAttribute;
17+
import org.elasticsearch.xpack.esql.core.type.DataType;
1618
import org.elasticsearch.xpack.esql.expression.function.aggregate.Count;
19+
import org.elasticsearch.xpack.esql.expression.function.fulltext.Match;
1720
import org.elasticsearch.xpack.esql.expression.function.scalar.math.Pow;
1821
import org.elasticsearch.xpack.esql.expression.function.scalar.string.RLike;
1922
import org.elasticsearch.xpack.esql.expression.function.scalar.string.WildcardLike;
@@ -28,6 +31,7 @@
2831
import org.elasticsearch.xpack.esql.plan.logical.Filter;
2932
import org.elasticsearch.xpack.esql.plan.logical.LogicalPlan;
3033
import org.elasticsearch.xpack.esql.plan.logical.Project;
34+
import org.elasticsearch.xpack.esql.plan.logical.inference.Completion;
3135
import org.elasticsearch.xpack.esql.plan.logical.local.EsqlProject;
3236

3337
import java.util.ArrayList;
@@ -45,9 +49,12 @@
4549
import static org.elasticsearch.xpack.esql.EsqlTestUtils.greaterThanOf;
4650
import static org.elasticsearch.xpack.esql.EsqlTestUtils.greaterThanOrEqualOf;
4751
import static org.elasticsearch.xpack.esql.EsqlTestUtils.lessThanOf;
52+
import static org.elasticsearch.xpack.esql.EsqlTestUtils.randomLiteral;
53+
import static org.elasticsearch.xpack.esql.EsqlTestUtils.referenceAttribute;
4854
import static org.elasticsearch.xpack.esql.EsqlTestUtils.rlike;
4955
import static org.elasticsearch.xpack.esql.EsqlTestUtils.wildcardLike;
5056
import static org.elasticsearch.xpack.esql.core.tree.Source.EMPTY;
57+
import static org.mockito.Mockito.mock;
5158

5259
public class PushDownAndCombineFiltersTests extends ESTestCase {
5360

@@ -238,6 +245,53 @@ public void testSelectivelyPushDownFilterPastFunctionAgg() {
238245
assertEquals(expected, new PushDownAndCombineFilters().apply(fb));
239246
}
240247

248+
// from ... | where a > 1 | COMPLETION "some prompt" WITH reranker AS completion | where b < 2 and match(completion, some text)
249+
// => ... | where a > 1 AND b < 2| COMPLETION "some prompt" WITH reranker AS completion | match(completion, some text)
250+
public void testPushDownFilterPastCompletion() {
251+
FieldAttribute a = getFieldAttribute("a");
252+
FieldAttribute b = getFieldAttribute("b");
253+
EsRelation relation = relation(List.of(a, b));
254+
255+
GreaterThan conditionA = greaterThanOf(getFieldAttribute("a"), ONE);
256+
Filter filterA = new Filter(EMPTY, relation, conditionA);
257+
258+
Completion completion = completion(filterA);
259+
260+
LessThan conditionB = lessThanOf(getFieldAttribute("b"), TWO);
261+
Match conditionCompletion = new Match(
262+
EMPTY,
263+
completion.targetField(),
264+
randomLiteral(DataType.TEXT),
265+
mock(Expression.class),
266+
mock(QueryBuilder.class)
267+
);
268+
Filter filterB = new Filter(EMPTY, completion, new And(EMPTY, conditionB, conditionCompletion));
269+
270+
LogicalPlan expectedOptimizedPlan = new Filter(
271+
EMPTY,
272+
new Completion(
273+
EMPTY,
274+
new Filter(EMPTY, relation, new And(EMPTY, conditionA, conditionB)),
275+
completion.inferenceId(),
276+
completion.prompt(),
277+
completion.targetField()
278+
),
279+
conditionCompletion
280+
);
281+
282+
assertEquals(expectedOptimizedPlan, new PushDownAndCombineFilters().apply(filterB));
283+
}
284+
285+
private static Completion completion(LogicalPlan child) {
286+
return new Completion(
287+
EMPTY,
288+
child,
289+
randomLiteral(DataType.TEXT),
290+
randomLiteral(DataType.TEXT),
291+
referenceAttribute(randomIdentifier(), DataType.TEXT)
292+
);
293+
}
294+
241295
private static EsRelation relation() {
242296
return relation(List.of());
243297
}
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,159 @@
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.optimizer.rules.logical;
9+
10+
import org.elasticsearch.test.ESTestCase;
11+
import org.elasticsearch.xpack.esql.core.expression.Alias;
12+
import org.elasticsearch.xpack.esql.core.expression.Attribute;
13+
import org.elasticsearch.xpack.esql.core.expression.FieldAttribute;
14+
import org.elasticsearch.xpack.esql.core.expression.Literal;
15+
import org.elasticsearch.xpack.esql.expression.Order;
16+
import org.elasticsearch.xpack.esql.expression.function.scalar.convert.ToInteger;
17+
import org.elasticsearch.xpack.esql.expression.predicate.operator.comparison.Equals;
18+
import org.elasticsearch.xpack.esql.plan.logical.EsRelation;
19+
import org.elasticsearch.xpack.esql.plan.logical.Eval;
20+
import org.elasticsearch.xpack.esql.plan.logical.Filter;
21+
import org.elasticsearch.xpack.esql.plan.logical.Limit;
22+
import org.elasticsearch.xpack.esql.plan.logical.LogicalPlan;
23+
import org.elasticsearch.xpack.esql.plan.logical.OrderBy;
24+
import org.elasticsearch.xpack.esql.plan.logical.UnaryPlan;
25+
import org.elasticsearch.xpack.esql.plan.logical.inference.Completion;
26+
27+
import java.util.List;
28+
import java.util.function.BiConsumer;
29+
import java.util.function.BiFunction;
30+
31+
import static org.elasticsearch.xpack.esql.EsqlTestUtils.as;
32+
import static org.elasticsearch.xpack.esql.EsqlTestUtils.getFieldAttribute;
33+
import static org.elasticsearch.xpack.esql.EsqlTestUtils.randomLiteral;
34+
import static org.elasticsearch.xpack.esql.EsqlTestUtils.unboundLogicalOptimizerContext;
35+
import static org.elasticsearch.xpack.esql.core.tree.Source.EMPTY;
36+
import static org.elasticsearch.xpack.esql.core.type.DataType.INTEGER;
37+
import static org.elasticsearch.xpack.esql.core.type.DataType.TEXT;
38+
import static org.elasticsearch.xpack.esql.optimizer.LocalLogicalPlanOptimizerTests.relation;
39+
40+
public class PushDownAndCombineLimitsTests extends ESTestCase {
41+
42+
private static class PushDownLimitTestCase<PlanType extends UnaryPlan> {
43+
private final Class<PlanType> clazz;
44+
private final BiFunction<LogicalPlan, Attribute, PlanType> planBuilder;
45+
private final BiConsumer<PlanType, PlanType> planChecker;
46+
47+
PushDownLimitTestCase(
48+
Class<PlanType> clazz,
49+
BiFunction<LogicalPlan, Attribute, PlanType> planBuilder,
50+
BiConsumer<PlanType, PlanType> planChecker
51+
) {
52+
this.clazz = clazz;
53+
this.planBuilder = planBuilder;
54+
this.planChecker = planChecker;
55+
}
56+
57+
public PlanType buildPlan(LogicalPlan child, Attribute attr) {
58+
return planBuilder.apply(child, attr);
59+
}
60+
61+
public void checkOptimizedPlan(LogicalPlan basePlan, LogicalPlan optimizedPlan) {
62+
planChecker.accept(as(basePlan, clazz), as(optimizedPlan, clazz));
63+
}
64+
}
65+
66+
private static final List<PushDownLimitTestCase<? extends UnaryPlan>> PUSHABLE_LIMIT_TEST_CASES = List.of(
67+
new PushDownLimitTestCase<>(
68+
Eval.class,
69+
(plan, attr) -> new Eval(EMPTY, plan, List.of(new Alias(EMPTY, "y", new ToInteger(EMPTY, attr)))),
70+
(basePlan, optimizedPlan) -> {
71+
assertEquals(basePlan.source(), optimizedPlan.source());
72+
assertEquals(basePlan.fields(), optimizedPlan.fields());
73+
}
74+
),
75+
new PushDownLimitTestCase<>(
76+
Completion.class,
77+
(plan, attr) -> new Completion(EMPTY, plan, randomLiteral(TEXT), randomLiteral(TEXT), attr),
78+
(basePlan, optimizedPlan) -> {
79+
assertEquals(basePlan.source(), optimizedPlan.source());
80+
assertEquals(basePlan.inferenceId(), optimizedPlan.inferenceId());
81+
assertEquals(basePlan.prompt(), optimizedPlan.prompt());
82+
assertEquals(basePlan.targetField(), optimizedPlan.targetField());
83+
}
84+
)
85+
);
86+
87+
private static final List<PushDownLimitTestCase<? extends UnaryPlan>> NON_PUSHABLE_LIMIT_TEST_CASES = List.of(
88+
new PushDownLimitTestCase<>(
89+
Filter.class,
90+
(plan, attr) -> new Filter(EMPTY, plan, new Equals(EMPTY, attr, new Literal(EMPTY, "right", TEXT))),
91+
(basePlan, optimizedPlan) -> {
92+
assertEquals(basePlan.source(), optimizedPlan.source());
93+
assertEquals(basePlan.condition(), optimizedPlan.condition());
94+
}
95+
),
96+
new PushDownLimitTestCase<>(
97+
OrderBy.class,
98+
(plan, attr) -> new OrderBy(EMPTY, plan, List.of(new Order(EMPTY, attr, Order.OrderDirection.DESC, null))),
99+
(basePlan, optimizedPlan) -> {
100+
assertEquals(basePlan.source(), optimizedPlan.source());
101+
assertEquals(basePlan.order(), optimizedPlan.order());
102+
}
103+
)
104+
);
105+
106+
public void testPushableLimit() {
107+
FieldAttribute a = getFieldAttribute("a");
108+
FieldAttribute b = getFieldAttribute("b");
109+
EsRelation relation = relation().withAttributes(List.of(a, b));
110+
111+
for (PushDownLimitTestCase<? extends UnaryPlan> pushableLimitTestCase : PUSHABLE_LIMIT_TEST_CASES) {
112+
int precedingLimitValue = randomIntBetween(1, 10_000);
113+
Limit precedingLimit = new Limit(EMPTY, new Literal(EMPTY, precedingLimitValue, INTEGER), relation);
114+
115+
LogicalPlan pushableLimitTestPlan = pushableLimitTestCase.buildPlan(precedingLimit, a);
116+
117+
int pushableLimitValue = randomIntBetween(1, 10_000);
118+
Limit pushableLimit = new Limit(EMPTY, new Literal(EMPTY, pushableLimitValue, INTEGER), pushableLimitTestPlan);
119+
120+
LogicalPlan optimizedPlan = optimizePlan(pushableLimit);
121+
122+
pushableLimitTestCase.checkOptimizedPlan(pushableLimitTestPlan, optimizedPlan);
123+
124+
assertEquals(
125+
as(optimizedPlan, UnaryPlan.class).child(),
126+
new Limit(EMPTY, new Literal(EMPTY, Math.min(pushableLimitValue, precedingLimitValue), INTEGER), relation)
127+
);
128+
}
129+
}
130+
131+
public void testNonPushableLimit() {
132+
FieldAttribute a = getFieldAttribute("a");
133+
FieldAttribute b = getFieldAttribute("b");
134+
EsRelation relation = relation().withAttributes(List.of(a, b));
135+
136+
for (PushDownLimitTestCase<? extends UnaryPlan> nonPushableLimitTestCase : NON_PUSHABLE_LIMIT_TEST_CASES) {
137+
int precedingLimitValue = randomIntBetween(1, 10_000);
138+
Limit precedingLimit = new Limit(EMPTY, new Literal(EMPTY, precedingLimitValue, INTEGER), relation);
139+
UnaryPlan nonPushableLimitTestPlan = nonPushableLimitTestCase.buildPlan(precedingLimit, a);
140+
int nonPushableLimitValue = randomIntBetween(1, 10_000);
141+
Limit nonPushableLimit = new Limit(EMPTY, new Literal(EMPTY, nonPushableLimitValue, INTEGER), nonPushableLimitTestPlan);
142+
Limit optimizedPlan = as(optimizePlan(nonPushableLimit), Limit.class);
143+
nonPushableLimitTestCase.checkOptimizedPlan(nonPushableLimitTestPlan, optimizedPlan.child());
144+
assertEquals(
145+
optimizedPlan,
146+
new Limit(
147+
EMPTY,
148+
new Literal(EMPTY, Math.min(nonPushableLimitValue, precedingLimitValue), INTEGER),
149+
nonPushableLimitTestPlan
150+
)
151+
);
152+
assertEquals(as(optimizedPlan.child(), UnaryPlan.class).child(), nonPushableLimitTestPlan.child());
153+
}
154+
}
155+
156+
private LogicalPlan optimizePlan(LogicalPlan plan) {
157+
return new PushDownAndCombineLimits().apply(plan, unboundLogicalOptimizerContext());
158+
}
159+
}

0 commit comments

Comments
 (0)