You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@calcite.apache.org by dm...@apache.org on 2022/11/17 12:48:11 UTC

[calcite] 01/01: [CALCITE-5209] ArrayIndexOutOfBoundsException during SqlToRelConverter for group-by on `case` having `in` expression predicates exceeding SqlRelConverter.Config InSubQueryThreshold

This is an automated email from the ASF dual-hosted git repository.

dmsysolyatin pushed a commit to branch CALCITE-5209
in repository https://gitbox.apache.org/repos/asf/calcite.git

commit dffd907222c1afca9a971ea5b712b6d0d3c3f30a
Author: dssysolyatin <dm...@gmail.com>
AuthorDate: Tue Nov 15 16:41:38 2022 +0200

    [CALCITE-5209] ArrayIndexOutOfBoundsException during SqlToRelConverter for group-by on `case` having `in` expression predicates exceeding SqlRelConverter.Config InSubQueryThreshold
---
 .../apache/calcite/sql2rel/SqlToRelConverter.java  | 64 ++++++++++++++--------
 core/src/test/resources/sql/agg.iq                 | 34 ++++++++++++
 2 files changed, 75 insertions(+), 23 deletions(-)

diff --git a/core/src/main/java/org/apache/calcite/sql2rel/SqlToRelConverter.java b/core/src/main/java/org/apache/calcite/sql2rel/SqlToRelConverter.java
index 1e5a9ab8aa..948d2bbd50 100644
--- a/core/src/main/java/org/apache/calcite/sql2rel/SqlToRelConverter.java
+++ b/core/src/main/java/org/apache/calcite/sql2rel/SqlToRelConverter.java
@@ -71,6 +71,7 @@ import org.apache.calcite.rel.logical.LogicalUnion;
 import org.apache.calcite.rel.logical.LogicalValues;
 import org.apache.calcite.rel.metadata.RelColumnMapping;
 import org.apache.calcite.rel.metadata.RelMetadataQuery;
+import org.apache.calcite.rel.rel2sql.SqlImplementor;
 import org.apache.calcite.rel.stream.Delta;
 import org.apache.calcite.rel.stream.LogicalDelta;
 import org.apache.calcite.rel.type.RelDataType;
@@ -1136,7 +1137,15 @@ public class SqlToRelConverter {
       final Blackboard bb,
       final SqlNode expr,
       RelOptUtil.Logic logic) {
-    findSubQueries(bb, expr, logic, false);
+    replaceSubQueries(bb, expr, logic, null);
+  }
+
+  private void replaceSubQueries(
+      final Blackboard bb,
+      final SqlNode expr,
+      RelOptUtil.Logic logic,
+      final SqlImplementor.@Nullable Clause parentClause) {
+    findSubQueries(bb, expr, logic, false, parentClause);
     for (SubQuery node : bb.subQueryList) {
       substituteSubQuery(bb, node);
     }
@@ -2018,12 +2027,14 @@ public class SqlToRelConverter {
    *                                     corresponds to a variation of a select
    *                                     node, only register it if it's a scalar
    *                                     sub-query
+   * @param parentClause A clause inside which sub-query is searched
    */
   private void findSubQueries(
       Blackboard bb,
       SqlNode node,
       RelOptUtil.Logic logic,
-      boolean registerOnlyScalarSubQueries) {
+      boolean registerOnlyScalarSubQueries,
+      SqlImplementor.@Nullable Clause parentClause) {
     final SqlKind kind = node.getKind();
     switch (kind) {
     case EXISTS:
@@ -2038,7 +2049,7 @@ public class SqlToRelConverter {
     case SCALAR_QUERY:
       if (!registerOnlyScalarSubQueries
           || (kind == SqlKind.SCALAR_QUERY)) {
-        bb.registerSubQuery(node, RelOptUtil.Logic.TRUE_FALSE);
+        bb.registerSubQuery(node, RelOptUtil.Logic.TRUE_FALSE, parentClause);
       }
       return;
     case IN:
@@ -2070,7 +2081,7 @@ public class SqlToRelConverter {
           findSubQueries(bb, operand, logic,
               kind == SqlKind.IN || kind == SqlKind.NOT_IN
                   || kind == SqlKind.SOME || kind == SqlKind.ALL
-                  || registerOnlyScalarSubQueries);
+                  || registerOnlyScalarSubQueries, parentClause);
         }
       }
     } else if (node instanceof SqlNodeList) {
@@ -2078,7 +2089,7 @@ public class SqlToRelConverter {
         findSubQueries(bb, child, logic,
             kind == SqlKind.IN || kind == SqlKind.NOT_IN
                 || kind == SqlKind.SOME || kind == SqlKind.ALL
-                || registerOnlyScalarSubQueries);
+                || registerOnlyScalarSubQueries, parentClause);
       }
     }
 
@@ -2107,7 +2118,7 @@ public class SqlToRelConverter {
       default:
         break;
       }
-      bb.registerSubQuery(node, logic);
+      bb.registerSubQuery(node, logic, parentClause);
       break;
     default:
       break;
@@ -3376,13 +3387,13 @@ public class SqlToRelConverter {
     // also replace sub-queries inside ordering spec in the aggregates
     replaceSubQueries(bb, aggregateFinder.orderList,
         RelOptUtil.Logic.TRUE_FALSE_UNKNOWN);
-
     // If group-by clause is missing, pretend that it has zero elements.
     if (groupList == null) {
       groupList = SqlNodeList.EMPTY;
     }
 
-    replaceSubQueries(bb, groupList, RelOptUtil.Logic.TRUE_FALSE_UNKNOWN);
+    replaceSubQueries(bb, groupList, RelOptUtil.Logic.TRUE_FALSE_UNKNOWN, 
+        SqlImplementor.Clause.GROUP_BY);
 
     // register the group exprs
 
@@ -3474,7 +3485,8 @@ public class SqlToRelConverter {
       // This needs to be done separately from the sub-query inside
       // any aggregate in the select list, and after the aggregate rel
       // is allocated.
-      replaceSubQueries(bb, selectList, RelOptUtil.Logic.TRUE_FALSE_UNKNOWN);
+      replaceSubQueries(bb, selectList, RelOptUtil.Logic.TRUE_FALSE_UNKNOWN, 
+          SqlImplementor.Clause.SELECT);
 
       // Now sub-queries in the entire select list have been converted.
       // Convert the select expressions to get the final list to be
@@ -5140,26 +5152,30 @@ public class SqlToRelConverter {
       }
     }
 
-    void registerSubQuery(SqlNode node, RelOptUtil.Logic logic) {
-      for (SubQuery subQuery : subQueryList) {
-        // Compare the reference to make sure the matched node has
-        // exact scope where it belongs.
-        if (node == subQuery.node) {
-          return;
-        }
+    void registerSubQuery(SqlNode node, RelOptUtil.Logic logic, 
+        SqlImplementor.@Nullable Clause parentClause) {
+      if (getSubQuery(node, parentClause) == null) {
+        subQueryList.add(new SubQuery(node, logic, parentClause));
       }
-      subQueryList.add(new SubQuery(node, logic));
     }
 
-    @Nullable SubQuery getSubQuery(SqlNode expr) {
+    @Nullable SubQuery getSubQuery(SqlNode expr, SqlImplementor.@Nullable Clause exprParentClause) {
       for (SubQuery subQuery : subQueryList) {
         // Compare the reference to make sure the matched node has
         // exact scope where it belongs.
         if (expr == subQuery.node) {
           return subQuery;
         }
-      }
 
+        // Reference comparing does not work in case when select list has column which refers
+        // to the column inside `GROUP BY` clause.
+        // For example: SELECT deptno IN (1,2) FROM emp.deptno GROUP BY deptno IN (1,2);
+        if (exprParentClause == SqlImplementor.Clause.SELECT &&
+            subQuery.parentClause == SqlImplementor.Clause.GROUP_BY &&
+            expr.equalsDeep(subQuery.node, Litmus.IGNORE)) {
+          return subQuery;
+        }
+      }
       return null;
     }
 
@@ -5307,7 +5323,7 @@ public class SqlToRelConverter {
       case CURSOR:
       case IN:
       case NOT_IN:
-        subQuery = requireNonNull(getSubQuery(expr));
+        subQuery = requireNonNull(getSubQuery(expr, null));
         rex = requireNonNull(subQuery.expr);
         return StandardConvertletTable.castToValidatedType(expr, rex,
             validator(), rexBuilder);
@@ -5318,7 +5334,7 @@ public class SqlToRelConverter {
       case ARRAY_QUERY_CONSTRUCTOR:
       case MAP_QUERY_CONSTRUCTOR:
       case MULTISET_QUERY_CONSTRUCTOR:
-        subQuery = getSubQuery(expr);
+        subQuery = getSubQuery(expr, null);
         assert subQuery != null;
         rex = subQuery.expr;
         assert rex != null : "rex != null";
@@ -5503,7 +5519,7 @@ public class SqlToRelConverter {
     }
 
     @Override public RexRangeRef getSubQueryExpr(SqlCall call) {
-      final SubQuery subQuery = getSubQuery(call);
+      final SubQuery subQuery = getSubQuery(call, null);
       assert subQuery != null;
       return (RexRangeRef) requireNonNull(subQuery.expr, () -> "subQuery.expr for " + call);
     }
@@ -6374,10 +6390,12 @@ public class SqlToRelConverter {
     final SqlNode node;
     final RelOptUtil.Logic logic;
     @Nullable RexNode expr;
+    final SqlImplementor.@Nullable Clause parentClause;
 
-    private SubQuery(SqlNode node, RelOptUtil.Logic logic) {
+    private SubQuery(SqlNode node, RelOptUtil.Logic logic, SqlImplementor.Clause parentClause) {
       this.node = node;
       this.logic = logic;
+      this.parentClause = parentClause;
     }
   }
 
diff --git a/core/src/test/resources/sql/agg.iq b/core/src/test/resources/sql/agg.iq
index 8718313948..59c8badaf2 100644
--- a/core/src/test/resources/sql/agg.iq
+++ b/core/src/test/resources/sql/agg.iq
@@ -3467,4 +3467,38 @@ order by ename, deptno;
 
 !ok
 
+# Test cases for [CALCITE-5209] ArrayIndexOutOfBoundsException during SqlToRelConverter for group-by on `case`
+# having `in` expression predicates exceeding SqlRelConverter.Config InSubQueryThreshold
+!use scott
+select
+    case when deptno in (1, 2, 3, 4, 5) THEN 1 else 0 end
+from emp
+group by
+    case when deptno in (1, 2, 3, 4, 5) THEN 1 else 0 end;
+
++--------+
+| EXPR$0 |
++--------+
+|      0 |
++--------+
+(1 row)
+
+!ok
+
+!set insubquerythreshold 5
+select
+    case when deptno in (1, 2, 3, 4, 5) THEN 1 else 0 end
+from emp
+group by
+    case when deptno in (1, 2, 3, 4, 5) THEN 1 else 0 end;
+
++--------+
+| EXPR$0 |
++--------+
+|      0 |
++--------+
+(1 row)
+
+!ok
+
 # End agg.iq