You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@kylin.apache.org by sh...@apache.org on 2018/10/24 06:30:37 UTC

[kylin] branch master updated: KYLIN-2200 Verify CompileException on UNION ALL query when result only contains one column and add IT.

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

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


The following commit(s) were added to refs/heads/master by this push:
     new 494933e  KYLIN-2200 Verify CompileException on UNION ALL query when result only contains one column and add IT.
494933e is described below

commit 494933e08307d741a99417c6c43a9d1b833d77f4
Author: gaodayue <ga...@meituan.com>
AuthorDate: Thu Nov 17 15:52:29 2016 +0800

    KYLIN-2200 Verify CompileException on UNION ALL query when result only contains one column and add IT.
---
 .../org/apache/kylin/query/ITKylinQueryTest.java   |  5 ++
 .../test/resources/query/sql_unionall/query01.sql  |  7 ++
 .../test/resources/query/sql_unionall/query02.sql  |  8 ++
 .../test/resources/query/sql_unionall/query03.sql  | 10 +++
 .../kylin/query/relnode/KylinEnumerableUnion.java  | 95 ++++++++++++++++++++++
 .../apache/kylin/query/relnode/OLAPUnionRel.java   |  5 +-
 6 files changed, 126 insertions(+), 4 deletions(-)

diff --git a/kylin-it/src/test/java/org/apache/kylin/query/ITKylinQueryTest.java b/kylin-it/src/test/java/org/apache/kylin/query/ITKylinQueryTest.java
index 59cb4d5..c6d1f62 100644
--- a/kylin-it/src/test/java/org/apache/kylin/query/ITKylinQueryTest.java
+++ b/kylin-it/src/test/java/org/apache/kylin/query/ITKylinQueryTest.java
@@ -245,6 +245,11 @@ public class ITKylinQueryTest extends KylinTestBase {
     }
 
     @Test
+    public void testUnionallQuery() throws Exception {
+        execAndCompQuery(getQueryFolderPrefix() + "src/test/resources/query/sql_unionall", null, true);
+    }
+
+    @Test
     public void testTimeStampAdd() throws Exception {
         execAndCompQuery(getQueryFolderPrefix() + "src/test/resources/query/sql_timestamp", null, true);
     }
diff --git a/kylin-it/src/test/resources/query/sql_unionall/query01.sql b/kylin-it/src/test/resources/query/sql_unionall/query01.sql
new file mode 100644
index 0000000..4289811
--- /dev/null
+++ b/kylin-it/src/test/resources/query/sql_unionall/query01.sql
@@ -0,0 +1,7 @@
+-- unionall
+select count(ORDER_ID) as ORDER_ID
+from TEST_KYLIN_FACT as TEST_A
+where ORDER_ID <> 1
+union all
+select count(ORDER_ID) as ORDER_ID
+from TEST_KYLIN_FACT as TEST_B
diff --git a/kylin-it/src/test/resources/query/sql_unionall/query02.sql b/kylin-it/src/test/resources/query/sql_unionall/query02.sql
new file mode 100644
index 0000000..8828285
--- /dev/null
+++ b/kylin-it/src/test/resources/query/sql_unionall/query02.sql
@@ -0,0 +1,8 @@
+-- unionall
+select count(*) as count_a
+from TEST_KYLIN_FACT as TEST_A
+where lstg_format_name='FP-GTC'
+union all
+select count(*) as count_a
+from TEST_KYLIN_FACT as TEST_A
+where lstg_format_name='FP-GTC'
\ No newline at end of file
diff --git a/kylin-it/src/test/resources/query/sql_unionall/query03.sql b/kylin-it/src/test/resources/query/sql_unionall/query03.sql
new file mode 100644
index 0000000..b1366f1
--- /dev/null
+++ b/kylin-it/src/test/resources/query/sql_unionall/query03.sql
@@ -0,0 +1,10 @@
+-- unionall subquery under join
+select count(*) as cnt
+FROM TEST_KYLIN_FACT as TEST_A
+join (
+    select * from TEST_KYLIN_FACT where CAL_DT < DATE '2012-02-01'
+    union all
+    select * from TEST_KYLIN_FACT where CAL_DT > DATE '2013-12-31'
+) TEST_B
+on TEST_A.ORDER_ID = TEST_B.ORDER_ID
+group by TEST_A.SELLER_ID
diff --git a/query/src/main/java/org/apache/kylin/query/relnode/KylinEnumerableUnion.java b/query/src/main/java/org/apache/kylin/query/relnode/KylinEnumerableUnion.java
new file mode 100644
index 0000000..67ed0ee
--- /dev/null
+++ b/query/src/main/java/org/apache/kylin/query/relnode/KylinEnumerableUnion.java
@@ -0,0 +1,95 @@
+/*
+ * 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.kylin.query.relnode;
+
+import org.apache.calcite.adapter.enumerable.EnumerableRel;
+import org.apache.calcite.adapter.enumerable.EnumerableRelImplementor;
+import org.apache.calcite.adapter.enumerable.EnumerableUnion;
+import org.apache.calcite.adapter.enumerable.JavaRowFormat;
+import org.apache.calcite.adapter.enumerable.PhysType;
+import org.apache.calcite.adapter.enumerable.PhysTypeImpl;
+import org.apache.calcite.linq4j.Enumerable;
+import org.apache.calcite.linq4j.ExtendedEnumerable;
+import org.apache.calcite.linq4j.Ord;
+import org.apache.calcite.linq4j.function.EqualityComparer;
+import org.apache.calcite.linq4j.function.Functions;
+import org.apache.calcite.linq4j.tree.BlockBuilder;
+import org.apache.calcite.linq4j.tree.Expression;
+import org.apache.calcite.linq4j.tree.Expressions;
+import org.apache.calcite.linq4j.tree.Types;
+import org.apache.calcite.plan.RelOptCluster;
+import org.apache.calcite.plan.RelTraitSet;
+import org.apache.calcite.rel.RelNode;
+import org.apache.calcite.util.BuiltInMethod;
+
+import java.lang.reflect.Method;
+import java.util.List;
+
+/**
+ * KYLIN-2200
+ */
+public class KylinEnumerableUnion extends EnumerableUnion {
+    private Method unionArray;
+    private Method arrayComparer;
+
+    public KylinEnumerableUnion(RelOptCluster cluster, RelTraitSet traitSet, List<RelNode> inputs, boolean all) {
+        super(cluster, traitSet, inputs, all);
+
+        unionArray = Types.lookupMethod(ExtendedEnumerable.class, "union", Enumerable.class, EqualityComparer.class);
+        arrayComparer = Types.lookupMethod(Functions.class, "arrayComparer");
+    }
+
+    private Expression createUnionExpression(Expression left, Expression right, boolean arrayInput) {
+        if (all) {
+            return Expressions.call(left, BuiltInMethod.CONCAT.method, right);
+        }
+
+        return arrayInput
+                ? Expressions.call(left, unionArray, right, Expressions.call(arrayComparer))
+                : Expressions.call(left, BuiltInMethod.UNION.method, right);
+    }
+
+    @Override
+    public Result implement(EnumerableRelImplementor implementor, Prefer pref) {
+        final BlockBuilder builder = new BlockBuilder();
+        Expression unionExp = null;
+        for (Ord<RelNode> ord : Ord.zip(inputs)) {
+            EnumerableRel input = (EnumerableRel) ord.e;
+            final Result result = implementor.visitChild(this, ord.i, input, pref);
+            Expression childExp =
+                    builder.append(
+                            "child" + ord.i,
+                            result.block);
+
+            if (unionExp == null) {
+                unionExp = childExp;
+            } else {
+                unionExp = createUnionExpression(unionExp, childExp, result.format == JavaRowFormat.ARRAY);
+            }
+        }
+
+        builder.add(unionExp);
+        final PhysType physType =
+                PhysTypeImpl.of(
+                        implementor.getTypeFactory(),
+                        getRowType(),
+                        pref.prefer(JavaRowFormat.CUSTOM));
+        return implementor.result(physType, builder.toBlock());
+    }
+}
diff --git a/query/src/main/java/org/apache/kylin/query/relnode/OLAPUnionRel.java b/query/src/main/java/org/apache/kylin/query/relnode/OLAPUnionRel.java
index 11f0a92..7366eeb 100644
--- a/query/src/main/java/org/apache/kylin/query/relnode/OLAPUnionRel.java
+++ b/query/src/main/java/org/apache/kylin/query/relnode/OLAPUnionRel.java
@@ -23,7 +23,6 @@ import java.util.List;
 
 import org.apache.calcite.adapter.enumerable.EnumerableConvention;
 import org.apache.calcite.adapter.enumerable.EnumerableRel;
-import org.apache.calcite.adapter.enumerable.EnumerableUnion;
 import org.apache.calcite.plan.RelOptCluster;
 import org.apache.calcite.plan.RelOptCost;
 import org.apache.calcite.plan.RelOptPlanner;
@@ -46,7 +45,6 @@ import com.google.common.collect.Lists;
  */
 public class OLAPUnionRel extends Union implements OLAPRel {
 
-    final boolean localAll ; // avoid same name in parent class
     ColumnRowType columnRowType;
     OLAPContext context;
 
@@ -56,7 +54,6 @@ public class OLAPUnionRel extends Union implements OLAPRel {
         for (RelNode child : inputs) {
             Preconditions.checkArgument(getConvention() == child.getConvention());
         }
-        this.localAll = all;
     }
 
     @Override
@@ -138,7 +135,7 @@ public class OLAPUnionRel extends Union implements OLAPRel {
             }
             relInputs.add(input);
         }
-        return new EnumerableUnion(getCluster(), traitSet.replace(EnumerableConvention.INSTANCE), relInputs, localAll);
+        return new KylinEnumerableUnion(getCluster(), traitSet.replace(EnumerableConvention.INSTANCE), relInputs, all);
     }
 
     @Override