You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@calcite.apache.org by se...@apache.org on 2019/03/29 21:57:26 UTC

[calcite] branch master updated: [CALCITE-2941] EnumerableLimitRule on Sort with no collation creates EnumerableLimit with wrong traitSet and cluster (Ruben Quesada Lopez)

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

sereda pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/calcite.git


The following commit(s) were added to refs/heads/master by this push:
     new 382f62c  [CALCITE-2941] EnumerableLimitRule on Sort with no collation creates EnumerableLimit with wrong traitSet and cluster (Ruben Quesada Lopez)
382f62c is described below

commit 382f62c84ac1c976c6569fde8be225b32d865730
Author: rubenada <ru...@gmail.com>
AuthorDate: Thu Mar 21 11:45:05 2019 +0100

    [CALCITE-2941] EnumerableLimitRule on Sort with no collation creates EnumerableLimit with wrong traitSet and cluster (Ruben Quesada Lopez)
---
 .../adapter/enumerable/EnumerableLimitRule.java    |  12 +--
 .../calcite/rel/rules/EnumerableLimitRuleTest.java | 100 +++++++++++++++++++++
 2 files changed, 102 insertions(+), 10 deletions(-)

diff --git a/core/src/main/java/org/apache/calcite/adapter/enumerable/EnumerableLimitRule.java b/core/src/main/java/org/apache/calcite/adapter/enumerable/EnumerableLimitRule.java
index 4f8ab10..a5b9623 100644
--- a/core/src/main/java/org/apache/calcite/adapter/enumerable/EnumerableLimitRule.java
+++ b/core/src/main/java/org/apache/calcite/adapter/enumerable/EnumerableLimitRule.java
@@ -18,7 +18,6 @@ package org.apache.calcite.adapter.enumerable;
 
 import org.apache.calcite.plan.RelOptRule;
 import org.apache.calcite.plan.RelOptRuleCall;
-import org.apache.calcite.plan.RelTraitSet;
 import org.apache.calcite.rel.RelNode;
 import org.apache.calcite.rel.core.Sort;
 
@@ -40,8 +39,6 @@ class EnumerableLimitRule extends RelOptRule {
     if (sort.offset == null && sort.fetch == null) {
       return;
     }
-    final RelTraitSet traitSet =
-        sort.getTraitSet().replace(EnumerableConvention.INSTANCE);
     RelNode input = sort.getInput();
     if (!sort.getCollation().getFieldCollations().isEmpty()) {
       // Create a sort with the same sort key, but no offset or fetch.
@@ -52,14 +49,9 @@ class EnumerableLimitRule extends RelOptRule {
           null,
           null);
     }
-    RelNode x = convert(
-        input,
-        input.getTraitSet().replace(EnumerableConvention.INSTANCE));
     call.transformTo(
-        new EnumerableLimit(
-            sort.getCluster(),
-            traitSet,
-            x,
+        EnumerableLimit.create(
+            convert(input, input.getTraitSet().replace(EnumerableConvention.INSTANCE)),
             sort.offset,
             sort.fetch));
   }
diff --git a/core/src/test/java/org/apache/calcite/rel/rules/EnumerableLimitRuleTest.java b/core/src/test/java/org/apache/calcite/rel/rules/EnumerableLimitRuleTest.java
new file mode 100644
index 0000000..6c597ad
--- /dev/null
+++ b/core/src/test/java/org/apache/calcite/rel/rules/EnumerableLimitRuleTest.java
@@ -0,0 +1,100 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to you under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.calcite.rel.rules;
+
+import org.apache.calcite.adapter.enumerable.EnumerableConvention;
+import org.apache.calcite.adapter.enumerable.EnumerableRules;
+import org.apache.calcite.plan.ConventionTraitDef;
+import org.apache.calcite.plan.RelTraitSet;
+import org.apache.calcite.rel.RelCollation;
+import org.apache.calcite.rel.RelCollationTraitDef;
+import org.apache.calcite.rel.RelFieldCollation;
+import org.apache.calcite.rel.RelNode;
+import org.apache.calcite.schema.SchemaPlus;
+import org.apache.calcite.schemas.HrClusteredSchema;
+import org.apache.calcite.sql.parser.SqlParser;
+import org.apache.calcite.tools.FrameworkConfig;
+import org.apache.calcite.tools.Frameworks;
+import org.apache.calcite.tools.Program;
+import org.apache.calcite.tools.Programs;
+import org.apache.calcite.tools.RelBuilder;
+import org.apache.calcite.tools.RuleSet;
+import org.apache.calcite.tools.RuleSets;
+
+import com.google.common.collect.ImmutableList;
+
+import org.junit.Test;
+
+import java.util.List;
+
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertNotNull;
+
+/**
+ * Tests the application of the {@link org.apache.calcite.adapter.enumerable.EnumerableLimitRule}.
+ */
+public final class EnumerableLimitRuleTest {
+
+  /** Test case for
+   * <a href="https://issues.apache.org/jira/browse/CALCITE-2941">[CALCITE-2941]
+   * EnumerableLimitRule on Sort with no collation creates EnumerableLimit with
+   * wrong traitSet and cluster</a>.
+   */
+  @Test public void enumerableLimitOnEmptySort() throws Exception {
+    RuleSet prepareRules =
+        RuleSets.ofList(
+            EnumerableRules.ENUMERABLE_FILTER_RULE,
+            EnumerableRules.ENUMERABLE_SORT_RULE,
+            EnumerableRules.ENUMERABLE_LIMIT_RULE,
+            EnumerableRules.ENUMERABLE_TABLE_SCAN_RULE);
+    SchemaPlus rootSchema = Frameworks.createRootSchema(true);
+    SchemaPlus defSchema = rootSchema.add("hr", new HrClusteredSchema());
+    FrameworkConfig config = Frameworks.newConfigBuilder()
+        .parserConfig(SqlParser.Config.DEFAULT)
+        .defaultSchema(defSchema)
+        .traitDefs(ConventionTraitDef.INSTANCE, RelCollationTraitDef.INSTANCE)
+        .programs(Programs.of(prepareRules))
+        .build();
+
+    RelBuilder builder = RelBuilder.create(config);
+    RelNode planBefore = builder
+        .scan("hr", "emps")
+        .sort(builder.field(0)) // will produce collation [0] in the plan
+        .filter(
+            builder.notEquals(
+                builder.field(0),
+                builder.literal(100)))
+        .limit(1, 5) // force a limit inside an "empty" Sort (with no collation)
+        .build();
+
+    RelTraitSet desiredTraits = planBefore.getTraitSet()
+        .replace(EnumerableConvention.INSTANCE);
+    Program program = Programs.of(prepareRules);
+    RelNode planAfter = program.run(planBefore.getCluster().getPlanner(), planBefore,
+        desiredTraits, ImmutableList.of(), ImmutableList.of());
+
+    // verify that the collation [0] is not lost in the final plan
+    RelCollation collation = planAfter.getTraitSet().getTrait(RelCollationTraitDef.INSTANCE);
+    assertNotNull(collation);
+    List<RelFieldCollation> fieldCollationList = collation.getFieldCollations();
+    assertNotNull(fieldCollationList);
+    assertEquals(1, fieldCollationList.size());
+    assertEquals(0, fieldCollationList.get(0).getFieldIndex());
+  }
+}
+
+// End EnumerableLimitRuleTest.java