1010import org .apache .lucene .util .SetOnce ;
1111import org .elasticsearch .action .ActionListener ;
1212import org .elasticsearch .test .ESTestCase ;
13+ import org .elasticsearch .threadpool .ThreadPool ;
1314import org .elasticsearch .xpack .esql .EsqlTestUtils ;
1415import org .elasticsearch .xpack .esql .core .expression .Alias ;
1516import org .elasticsearch .xpack .esql .core .expression .Expression ;
1617import org .elasticsearch .xpack .esql .core .expression .FoldContext ;
1718import org .elasticsearch .xpack .esql .core .tree .Source ;
1819import org .elasticsearch .xpack .esql .expression .function .scalar .string .Concat ;
1920import org .elasticsearch .xpack .esql .expression .predicate .operator .arithmetic .Add ;
21+ import org .elasticsearch .xpack .esql .optimizer .rules .logical .preoptimizer .PreOptimizerRule ;
2022import org .elasticsearch .xpack .esql .plan .logical .Eval ;
2123import org .elasticsearch .xpack .esql .plan .logical .Filter ;
2224import org .elasticsearch .xpack .esql .plan .logical .Limit ;
2325import org .elasticsearch .xpack .esql .plan .logical .LogicalPlan ;
2426import org .elasticsearch .xpack .esql .plan .logical .Project ;
27+ import org .junit .After ;
28+ import org .junit .Before ;
2529
2630import java .util .List ;
31+ import java .util .concurrent .TimeUnit ;
32+ import java .util .concurrent .atomic .AtomicInteger ;
2733
2834import static org .elasticsearch .xpack .esql .EsqlTestUtils .as ;
2935import static org .elasticsearch .xpack .esql .EsqlTestUtils .fieldAttribute ;
3339
3440public class LogicalPlanPreOptimizerTests extends ESTestCase {
3541
42+ private ThreadPool threadPool ;
43+
44+ @ Before
45+ public void setUpThreadPool () {
46+ threadPool = createThreadPool ();
47+ }
48+
49+ @ After
50+ public void tearDownThreadPool () {
51+ terminate (threadPool );
52+ }
53+
3654 public void testPlanIsMarkedAsPreOptimized () throws Exception {
3755 for (int round = 0 ; round < 100 ; round ++) {
38- // We want to make sure that the pre-optimizer woks for a wide range of plans
56+ // We want to make sure that the pre-optimizer works for a wide range of plans
3957 preOptimizedPlan (randomPlan ());
4058 }
4159 }
@@ -52,6 +70,75 @@ public void testPreOptimizeFailsIfPlanIsNotAnalyzed() throws Exception {
5270 });
5371 }
5472
73+ public void testPreOptimizerRulesAreAppliedInOrder () throws Exception {
74+ LogicalPlan plan = EsqlTestUtils .relation ();
75+ plan .setPreOptimized ();
76+
77+ StringBuilder executionOrder = new StringBuilder ();
78+
79+ // Create mock rules that track execution order
80+ PreOptimizerRule rule1 = createOrderTrackingRule ("A" , executionOrder );
81+ PreOptimizerRule rule2 = createOrderTrackingRule ("B" , executionOrder );
82+ PreOptimizerRule rule3 = createOrderTrackingRule ("C" , executionOrder );
83+
84+ LogicalPlanPreOptimizer preOptimizer = new LogicalPlanPreOptimizer (List .of (rule1 , rule2 , rule3 ));
85+
86+ SetOnce <LogicalPlan > resultHolder = new SetOnce <>();
87+
88+ preOptimizer .preOptimize (plan , ActionListener .wrap (resultHolder ::set , ESTestCase ::fail ));
89+
90+ assertBusy (() -> {
91+ assertThat (resultHolder .get (), notNullValue ());
92+ // Rules should be applied in the order they were provided
93+ assertThat (executionOrder .toString (), equalTo ("ABC" ));
94+ assertThat (resultHolder .get ().preOptimized (), equalTo (true ));
95+ });
96+ }
97+
98+ public void testPreOptimizerWithEmptyRulesList () throws Exception {
99+ LogicalPlan plan = EsqlTestUtils .relation ();
100+ plan .setPreOptimized ();
101+
102+ LogicalPlanPreOptimizer preOptimizer = new LogicalPlanPreOptimizer (List .of ());
103+
104+ SetOnce <LogicalPlan > resultHolder = new SetOnce <>();
105+
106+ preOptimizer .preOptimize (plan , ActionListener .wrap (resultHolder ::set , ESTestCase ::fail ));
107+
108+ assertBusy (() -> {
109+ assertThat (resultHolder .get (), notNullValue ());
110+ assertThat (resultHolder .get ().preOptimized (), equalTo (true ));
111+ // The plan should be the same as the original (no modifications)
112+ assertThat (resultHolder .get (), equalTo (plan ));
113+ });
114+ }
115+
116+ public void testPreOptimizerRuleFailurePropagatesError () throws Exception {
117+ LogicalPlan plan = EsqlTestUtils .relation ();
118+ plan .setPreOptimized ();
119+
120+ RuntimeException expectedError = new RuntimeException ("Mock rule failure" );
121+
122+ AtomicInteger ruleACounter = new AtomicInteger ();
123+ PreOptimizerRule ruleA = createMockRule (ruleACounter );
124+ PreOptimizerRule ruleB = createFailingRule (expectedError );
125+ AtomicInteger ruleCCounter = new AtomicInteger ();
126+ PreOptimizerRule ruleC = createMockRule (ruleCCounter );
127+
128+ LogicalPlanPreOptimizer preOptimizer = new LogicalPlanPreOptimizer (List .of (ruleA , ruleB , ruleC ));
129+
130+ SetOnce <Exception > exceptionHolder = new SetOnce <>();
131+
132+ preOptimizer .preOptimize (plan , ActionListener .wrap (r -> fail ("Should have failed" ), exceptionHolder ::set ));
133+
134+ assertBusy (() -> {
135+ assertThat (exceptionHolder .get (), notNullValue ());
136+ assertThat (exceptionHolder .get (), equalTo (expectedError ));
137+ assertThat (ruleACounter .get (), equalTo (1 ));
138+ assertThat (ruleCCounter .get (), equalTo (0 ));
139+ });
140+ }
141+
55142 public LogicalPlan preOptimizedPlan (LogicalPlan plan ) throws Exception {
56143 // set plan as analyzed
57144 plan .setPreOptimized ();
@@ -107,4 +194,28 @@ private Expression randomCondition() {
107194
108195 return EsqlTestUtils .greaterThanOf (randomExpression (), randomExpression ());
109196 }
197+
198+ // Helper methods for creating mock rules
199+
200+ private PreOptimizerRule createMockRule (AtomicInteger executionCounter ) {
201+ return (plan , listener ) -> {
202+ threadPool .schedule (() -> {
203+ executionCounter .incrementAndGet ();
204+ listener .onResponse (plan ); // Return the plan unchanged
205+ }, randomTimeValue (1 , 100 , TimeUnit .MILLISECONDS ), threadPool .executor (ThreadPool .Names .GENERIC ));
206+ };
207+ }
208+
209+ private PreOptimizerRule createOrderTrackingRule (String ruleId , StringBuilder executionOrder ) {
210+ return (plan , listener ) -> {
211+ threadPool .schedule (() -> {
212+ executionOrder .append (ruleId );
213+ listener .onResponse (plan ); // Return the plan unchanged
214+ }, randomTimeValue (1 , 100 , TimeUnit .MILLISECONDS ), threadPool .executor (ThreadPool .Names .GENERIC ));
215+ };
216+ }
217+
218+ private PreOptimizerRule createFailingRule (Exception error ) {
219+ return (plan , listener ) -> listener .onFailure (error );
220+ }
110221}
0 commit comments