Uploaded image for project: 'Calcite'
  1. Calcite
  2. CALCITE-3618

WindowedAggRelSplitter.isDependent is incorrect

    XMLWordPrintableJSON

Details

    Description

      WindowedAggRelSplitter isDependent is incorrect

            while (!dfs.isEmpty()) {
              int source = dfs.pop();
              if (visited.contains(source)) {
                continue;
              }
      
              if (source == ordinal1) {
                return true;
              }
      
              visited.add(source);
              for (DefaultEdge e : graph.getOutwardEdges(source)) {
                int target = (int) e.target;
                if (rank.get(target) < rank.get(ordinal1)) {
                  dfs.push(target);
                }
              }
            }
      

      It only pushes target into dfs queue if target rank is smaller than ordinal1's rank, which makes it impossible to find the dependency.

      For SQL like

      select 
       lag(a2, 1, 0) over (partition by "deptno" order by a1) as lagx 
      from 
       (
        select 
         "deptno", 
         "salary" / "commission" as a1, 
         sum("commission") over ( partition by "deptno" order by "salary" / "commission") / sum("commission") over (partition by "deptno") as a2 
        from 
         "hr"."emps"
       )
      

      It generates levels of exprs like the attached picture below which is incorrect.

      To reproduce the error, you can execute the below SQL in the CsvTest

      "SELECT 
           LAG(A2, 1, 0) OVER (PARTITION BY "GENDER" ORDER BY A1) AS LAGX 
          FROM 
           (
            SELECT 
             "GENDER", 
             "EMPNO" / "DEPTNO" AS A1, 
             SUM("DEPTNO") OVER ( PARTITION BY "GENDER" ORDER BY "EMPNO" / "DEPTNO") / SUM("DEPTNO") OVER (PARTITION BY "GENDER") AS A2 
            FROM 
             "EMPS"
           ) 

      And Calcite will throw the below error

      Error while applying rule ProjectToWindowRule:project, args [rel#12:LogicalProject.NONE.[](input=RelSubset#11,LAGX=LAG(/(CASE(>(COUNT($2) OVER (PARTITION BY $3 ORDER BY /($0, $2) RANGE BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW), 0), $SUM0($2) OVER (PARTITION BY $3 ORDER BY /($0, $2) RANGE BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW), null:INTEGER), CASE(>(COUNT($2) OVER (PARTITION BY $3 RANGE BETWEEN UNBOUNDED PRECEDING AND UNBOUNDED FOLLOWING), 0), $SUM0($2) OVER (PARTITION BY $3 RANGE BETWEEN UNBOUNDED PRECEDING AND UNBOUNDED FOLLOWING), null:INTEGER)), 1, 0) OVER (PARTITION BY $3 ORDER BY /($0, $2) RANGE BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW))]
              at org.apache.calcite.avatica.Helper.createException(Helper.java:56)
              at org.apache.calcite.avatica.Helper.createException(Helper.java:41)
              at org.apache.calcite.avatica.AvaticaStatement.executeInternal(AvaticaStatement.java:163)
              at org.apache.calcite.avatica.AvaticaStatement.executeQuery(AvaticaStatement.java:227)
              at org.apache.calcite.test.CsvTest.checkSql(CsvTest.java:408)
              at org.apache.calcite.test.CsvTest.access$300(CsvTest.java:71)
              at org.apache.calcite.test.CsvTest$Fluent.ok(CsvTest.java:1066)
              at org.apache.calcite.test.CsvTest.testSelect(CsvTest.java:192)
              Caused by:
              java.lang.RuntimeException: Error while applying rule ProjectToWindowRule:project, args [rel#12:LogicalProject.NONE.[](input=RelSubset#11,LAGX=LAG(/(CASE(>(COUNT($2) OVER (PARTITION BY $3 ORDER BY /($0, $2) RANGE BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW), 0), $SUM0($2) OVER (PARTITION BY $3 ORDER BY /($0, $2) RANGE BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW), null:INTEGER), CASE(>(COUNT($2) OVER (PARTITION BY $3 RANGE BETWEEN UNBOUNDED PRECEDING AND UNBOUNDED FOLLOWING), 0), $SUM0($2) OVER (PARTITION BY $3 RANGE BETWEEN UNBOUNDED PRECEDING AND UNBOUNDED FOLLOWING), null:INTEGER)), 1, 0) OVER (PARTITION BY $3 ORDER BY /($0, $2) RANGE BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW))]
                  at org.apache.calcite.plan.volcano.VolcanoRuleCall.onMatch(VolcanoRuleCall.java:238)
                  at org.apache.calcite.plan.volcano.VolcanoPlanner.findBestExp(VolcanoPlanner.java:632)
                  at org.apache.calcite.tools.Programs.lambda$standard$3(Programs.java:287)
                  at org.apache.calcite.tools.Programs$SequenceProgram.run(Programs.java:347)
                  at org.apache.calcite.prepare.Prepare.optimize(Prepare.java:189)
                  at org.apache.calcite.prepare.Prepare.prepareSql(Prepare.java:320)
                  at org.apache.calcite.prepare.Prepare.prepareSql(Prepare.java:231)
                  at org.apache.calcite.prepare.CalcitePrepareImpl.prepare2_(CalcitePrepareImpl.java:638)
                  at org.apache.calcite.prepare.CalcitePrepareImpl.prepare_(CalcitePrepareImpl.java:502)
                  at org.apache.calcite.prepare.CalcitePrepareImpl.prepareSql(CalcitePrepareImpl.java:472)
                  at org.apache.calcite.jdbc.CalciteConnectionImpl.parseQuery(CalciteConnectionImpl.java:231)
                  at org.apache.calcite.jdbc.CalciteMetaImpl.prepareAndExecute(CalciteMetaImpl.java:550)
                  at org.apache.calcite.avatica.AvaticaConnection.prepareAndExecuteInternal(AvaticaConnection.java:675)
                  at org.apache.calcite.avatica.AvaticaStatement.executeInternal(AvaticaStatement.java:156)
                  ... 5 more
                  Caused by:
                  java.lang.IllegalArgumentException
                      at org.apache.calcite.rex.RexSlot$SelfPopulatingList.get(RexSlot.java:90)
                      at org.apache.calcite.rex.RexSlot$SelfPopulatingList.get(RexSlot.java:60)
                      at org.apache.calcite.rex.RexLocalRef.createName(RexLocalRef.java:83)
                      at org.apache.calcite.rex.RexLocalRef.<init>(RexLocalRef.java:52)
                      at org.apache.calcite.rel.rules.CalcRelSplitter$InputToCommonExprConverter.visitLocalRef(CalcRelSplitter.java:921)
                      at org.apache.calcite.rel.rules.CalcRelSplitter$InputToCommonExprConverter.visitLocalRef(CalcRelSplitter.java:872)
                      at org.apache.calcite.rex.RexLocalRef.accept(RexLocalRef.java:75)
                      at org.apache.calcite.rex.RexShuttle.visitList(RexShuttle.java:149)
                      at org.apache.calcite.rex.RexShuttle.visitCall(RexShuttle.java:101)
                      at org.apache.calcite.rex.RexShuttle.visitCall(RexShuttle.java:34)
                      at org.apache.calcite.rex.RexCall.accept(RexCall.java:191)
                      at org.apache.calcite.rel.rules.CalcRelSplitter.createProgramForLevel(CalcRelSplitter.java:561)
                      at org.apache.calcite.rel.rules.CalcRelSplitter.execute(CalcRelSplitter.java:218)
                      at org.apache.calcite.rel.rules.ProjectToWindowRule$ProjectToLogicalProjectAndWindowRule.onMatch(ProjectToWindowRule.java:187)
                      at org.apache.calcite.plan.volcano.VolcanoRuleCall.onMatch(VolcanoRuleCall.java:211)
                      ... 18 more
      

       

      I  have submitted a PR to fix this https://github.com/apache/calcite/pull/1678

      Attachments

        Issue Links

          Activity

            People

              Unassigned Unassigned
              lxian2 Li Xian
              Votes:
              0 Vote for this issue
              Watchers:
              3 Start watching this issue

              Dates

                Created:
                Updated:
                Resolved:

                Time Tracking

                  Estimated:
                  Original Estimate - Not Specified
                  Not Specified
                  Remaining:
                  Remaining Estimate - 0h
                  0h
                  Logged:
                  Time Spent - 50m
                  50m