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

AggregateJoinTransposeRule fails when process aggregateCall above SqlSumEmptyIsZeroAggFunction without groupKeys

VotersWatch issueWatchersLinkCloneUpdate Comment AuthorReplace String in CommentUpdate Comment VisibilityDelete Comments
    XMLWordPrintableJSON

Details

    • Bug
    • Status: Closed
    • Major
    • Resolution: Fixed
    • None
    • 1.16.0
    • None
    • None

    Description

         final HepProgram preProgram = new HepProgramBuilder()
                  .addRuleInstance(AggregateProjectMergeRule.INSTANCE)
                  .build();
          final HepProgram program = new HepProgramBuilder()
                  .addRuleInstance(AggregateReduceFunctionsRule.INSTANCE)
                  .addRuleInstance(AggregateJoinTransposeRule.EXTENDED)
                  .build();
          final String sql = "select sum(sal)\n"
                  + "from (select * from sales.emp where empno = 10) as e\n"
                  + "join sales.dept as d on e.job = d.name";
      

      AggregateJoinTransposeRule fails when run the above sql, the exception is as following.

      java.lang.AssertionError: type mismatch:
      aggCall type:
      INTEGER NOT NULL
      inferred type:
      INTEGER
      
      	at org.apache.calcite.util.Litmus$1.fail(Litmus.java:31)
      	at org.apache.calcite.plan.RelOptUtil.eq(RelOptUtil.java:1838)
      	at org.apache.calcite.rel.core.Aggregate.typeMatchesInferred(Aggregate.java:428)
      	at org.apache.calcite.rel.core.Aggregate.<init>(Aggregate.java:161)
      	at org.apache.calcite.rel.logical.LogicalAggregate.<init>(LogicalAggregate.java:65)
      	at org.apache.calcite.rel.logical.LogicalAggregate.create_(LogicalAggregate.java:110)
      	at org.apache.calcite.rel.logical.LogicalAggregate.create(LogicalAggregate.java:100)
      	at org.apache.calcite.rel.core.RelFactories$AggregateFactoryImpl.createAggregate(RelFactories.java:213)
      	at org.apache.calcite.tools.RelBuilder.aggregate(RelBuilder.java:1267)
      	at org.apache.calcite.tools.RelBuilder.aggregate(RelBuilder.java:1825)
      	at org.apache.calcite.rel.rules.AggregateJoinTransposeRule.onMatch(AggregateJoinTransposeRule.java:337)
      

      Notes: This is different with the issue referred in https://issues.apache.org/jira/browse/CALCITE-2105. This question caused by the SqlSplittableAggFunction of SqlSumEmptyIsZeroAggFunction is SumSplitter. If groupByKeys is empty, the aggregateCalls returnTypes of Sum is nullable, while the returnTypes of Sum0 is not nullable.

      Attachments

        Issue Links

        Activity

          This comment will be Viewable by All Users Viewable by All Users
          Cancel

          People

            julianhyde Julian Hyde
            jingzhang Jing Zhang
            Votes:
            0 Vote for this issue
            Watchers:
            2 Start watching this issue

            Dates

              Created:
              Updated:
              Resolved:

              Slack

                Issue deployment