[ 
https://issues.apache.org/jira/browse/DRILL-786?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16688807#comment-16688807
 ] 

ASF GitHub Bot commented on DRILL-786:
--------------------------------------

asfgit closed pull request #1488: DRILL-786: Allow CROSS JOIN syntax
URL: https://github.com/apache/drill/pull/1488
 
 
   

This is a PR merged from a forked repository.
As GitHub hides the original diff on merge, it is displayed below for
the sake of provenance:

As this is a foreign pull request (from a fork), the diff is supplied
below (as it won't show otherwise due to GitHub magic):

diff --git 
a/exec/java-exec/src/main/java/org/apache/drill/exec/physical/impl/join/JoinUtils.java
 
b/exec/java-exec/src/main/java/org/apache/drill/exec/physical/impl/join/JoinUtils.java
index 52871e2b82b..90e85581cd3 100644
--- 
a/exec/java-exec/src/main/java/org/apache/drill/exec/physical/impl/join/JoinUtils.java
+++ 
b/exec/java-exec/src/main/java/org/apache/drill/exec/physical/impl/join/JoinUtils.java
@@ -51,11 +51,14 @@
 import org.apache.drill.exec.planner.logical.DrillLimitRel;
 import org.apache.drill.exec.record.VectorAccessible;
 import org.apache.drill.exec.resolver.TypeCastRules;
+import org.apache.drill.exec.work.foreman.UnsupportedRelOperatorException;
 
 import java.util.ArrayList;
 import java.util.LinkedList;
 import java.util.List;
 
+import static 
org.apache.drill.exec.planner.physical.PlannerSettings.NLJOIN_FOR_SCALAR;
+
 public class JoinUtils {
 
   public enum JoinCategory {
@@ -65,6 +68,11 @@
   }
   private static final org.slf4j.Logger logger = 
org.slf4j.LoggerFactory.getLogger(JoinUtils.class);
 
+  public static final String FAILED_TO_PLAN_CARTESIAN_JOIN = String.format(
+      "This query cannot be planned possibly due to either a cartesian join or 
an inequality join. %n" +
+          "If a cartesian or inequality join is used intentionally, set the 
option '%s' to false and try again.",
+      NLJOIN_FOR_SCALAR.getOptionName());
+
   // Check the comparator is supported in join condition. Note that a similar 
check is also
   // done in JoinPrel; however we have to repeat it here because a physical 
plan
   // may be submitted directly to Drill.
@@ -127,6 +135,18 @@ public static boolean checkCartesianJoin(RelNode relNode, 
List<Integer> leftKeys
     return false;
   }
 
+  /**
+   * Check if the given RelNode contains any Cartesian join.
+   * Return true if find one. Otherwise, return false.
+   *
+   * @param relNode     {@link RelNode} instance to be inspected
+   * @return            Return true if the given relNode contains Cartesian 
join.
+   *                    Otherwise, return false
+   */
+  public static boolean checkCartesianJoin(RelNode relNode) {
+    return checkCartesianJoin(relNode, new LinkedList<>(), new LinkedList<>(), 
new LinkedList<>());
+  }
+
   /**
    * Checks if implicit cast is allowed between the two input types of the 
join condition. Currently we allow
    * implicit casts in join condition only between numeric types and 
varchar/varbinary types.
@@ -299,6 +319,16 @@ public static boolean hasScalarSubqueryInput(RelNode left, 
RelNode right) {
     return isScalarSubquery(left) || isScalarSubquery(right);
   }
 
+  /**
+   * Creates new exception for queries that cannot be planned due
+   * to presence of cartesian or inequality join.
+   *
+   * @return new {@link UnsupportedRelOperatorException} instance
+   */
+  public static UnsupportedRelOperatorException 
cartesianJoinPlanningException() {
+    return new UnsupportedRelOperatorException(FAILED_TO_PLAN_CARTESIAN_JOIN);
+  }
+
   /**
    * Collects expressions list from the input project.
    * For the case when input rel node has single input, its input is taken.
diff --git 
a/exec/java-exec/src/main/java/org/apache/drill/exec/planner/sql/handlers/DefaultSqlHandler.java
 
b/exec/java-exec/src/main/java/org/apache/drill/exec/planner/sql/handlers/DefaultSqlHandler.java
index c75311f4ff1..f7d11f8a9bc 100644
--- 
a/exec/java-exec/src/main/java/org/apache/drill/exec/planner/sql/handlers/DefaultSqlHandler.java
+++ 
b/exec/java-exec/src/main/java/org/apache/drill/exec/planner/sql/handlers/DefaultSqlHandler.java
@@ -18,7 +18,6 @@
 package org.apache.drill.exec.planner.sql.handlers;
 
 import java.io.IOException;
-import java.util.ArrayList;
 import java.util.Collection;
 import java.util.List;
 import java.util.concurrent.TimeUnit;
@@ -106,7 +105,6 @@
 import org.apache.drill.exec.util.Pointer;
 import org.apache.drill.exec.work.foreman.ForemanSetupException;
 import org.apache.drill.exec.work.foreman.SqlUnsupportedException;
-import org.apache.drill.exec.work.foreman.UnsupportedRelOperatorException;
 import org.slf4j.Logger;
 
 import org.apache.drill.shaded.guava.com.google.common.base.Preconditions;
@@ -294,8 +292,8 @@ protected DrillRel convertToRawDrel(final RelNode relNode) 
throws SqlUnsupported
     } catch (RelOptPlanner.CannotPlanException ex) {
       logger.error(ex.getMessage());
 
-      if (JoinUtils.checkCartesianJoin(relNode, new ArrayList<>(), new 
ArrayList<>(), new ArrayList<>())) {
-        throw new UnsupportedRelOperatorException("This query cannot be 
planned possibly due to either a cartesian join or an inequality join");
+      if (JoinUtils.checkCartesianJoin(relNode)) {
+        throw JoinUtils.cartesianJoinPlanningException();
       } else {
         throw ex;
       }
@@ -459,8 +457,8 @@ protected Prel convertToPrel(RelNode drel, RelDataType 
validatedRowType) throws
     } catch (RelOptPlanner.CannotPlanException ex) {
       logger.error(ex.getMessage());
 
-      if (JoinUtils.checkCartesianJoin(drel, new ArrayList<>(), new 
ArrayList<>(), new ArrayList<>())) {
-        throw new UnsupportedRelOperatorException("This query cannot be 
planned possibly due to either a cartesian join or an inequality join");
+      if (JoinUtils.checkCartesianJoin(drel)) {
+        throw JoinUtils.cartesianJoinPlanningException();
       } else {
         throw ex;
       }
@@ -482,8 +480,8 @@ protected Prel convertToPrel(RelNode drel, RelDataType 
validatedRowType) throws
       } catch (RelOptPlanner.CannotPlanException ex) {
         logger.error(ex.getMessage());
 
-        if (JoinUtils.checkCartesianJoin(drel, new ArrayList<>(), new 
ArrayList<>(), new ArrayList<>())) {
-          throw new UnsupportedRelOperatorException("This query cannot be 
planned possibly due to either a cartesian join or an inequality join");
+        if (JoinUtils.checkCartesianJoin(drel)) {
+          throw JoinUtils.cartesianJoinPlanningException();
         } else {
           throw ex;
         }
diff --git 
a/exec/java-exec/src/main/java/org/apache/drill/exec/planner/sql/parser/UnsupportedOperatorsVisitor.java
 
b/exec/java-exec/src/main/java/org/apache/drill/exec/planner/sql/parser/UnsupportedOperatorsVisitor.java
index 8505a68a108..b9728410a97 100644
--- 
a/exec/java-exec/src/main/java/org/apache/drill/exec/planner/sql/parser/UnsupportedOperatorsVisitor.java
+++ 
b/exec/java-exec/src/main/java/org/apache/drill/exec/planner/sql/parser/UnsupportedOperatorsVisitor.java
@@ -36,7 +36,6 @@
 import org.apache.calcite.sql.SqlCall;
 import org.apache.calcite.sql.SqlKind;
 import org.apache.calcite.sql.SqlJoin;
-import org.apache.calcite.sql.JoinType;
 import org.apache.calcite.sql.SqlNode;
 import org.apache.calcite.sql.type.SqlTypeName;
 import org.apache.calcite.sql.util.SqlShuttle;
@@ -256,14 +255,6 @@ public SqlNode visit(SqlCall sqlCall) {
             "See Apache Drill JIRA: DRILL-1986");
         throw new UnsupportedOperationException();
       }
-
-      // Block Cross Join
-      if(join.getJoinType() == JoinType.CROSS) {
-        
unsupportedOperatorCollector.setException(SqlUnsupportedException.ExceptionType.RELATIONAL,
-            "CROSS JOIN is not supported\n" +
-            "See Apache Drill JIRA: DRILL-1921");
-        throw new UnsupportedOperationException();
-      }
     }
 
     //Disable UNNEST if the configuration disable it
diff --git 
a/exec/java-exec/src/test/java/org/apache/drill/TestDisabledFunctionality.java 
b/exec/java-exec/src/test/java/org/apache/drill/TestDisabledFunctionality.java
index 679d01eeb05..1dcd6919956 100644
--- 
a/exec/java-exec/src/test/java/org/apache/drill/TestDisabledFunctionality.java
+++ 
b/exec/java-exec/src/test/java/org/apache/drill/TestDisabledFunctionality.java
@@ -16,6 +16,7 @@
  * limitations under the License.
  */
 package org.apache.drill;
+
 import org.apache.drill.categories.UnlikelyTest;
 import org.apache.drill.common.exceptions.UserException;
 import org.apache.drill.exec.planner.physical.PlannerSettings;
@@ -92,15 +93,6 @@ public void testDisabledNaturalJoin() throws Exception {
     }
   }
 
-  @Test(expected = UnsupportedRelOperatorException.class) // see DRILL-1921
-  public void testDisabledCrossJoin() throws Exception {
-    try {
-      test("select * from cp.`tpch/nation.parquet` CROSS JOIN 
cp.`tpch/region.parquet`");
-    } catch(UserException ex) {
-      throwAsUnsupportedException(ex);
-    }
-  }
-
   @Test(expected = UnsupportedDataTypeException.class) // see DRILL-1959
   public void testDisabledCastTINYINT() throws Exception {
     try {
diff --git 
a/exec/java-exec/src/test/java/org/apache/drill/exec/planner/sql/CrossJoinTest.java
 
b/exec/java-exec/src/test/java/org/apache/drill/exec/planner/sql/CrossJoinTest.java
new file mode 100644
index 00000000000..348df328c92
--- /dev/null
+++ 
b/exec/java-exec/src/test/java/org/apache/drill/exec/planner/sql/CrossJoinTest.java
@@ -0,0 +1,201 @@
+/*
+ * 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.drill.exec.planner.sql;
+
+import org.apache.drill.categories.SqlTest;
+import org.apache.drill.common.exceptions.UserRemoteException;
+import org.apache.drill.exec.planner.physical.PlannerSettings;
+import org.apache.drill.test.ClusterFixture;
+import org.apache.drill.test.ClusterTest;
+import org.junit.After;
+import org.junit.BeforeClass;
+import org.junit.Test;
+import org.junit.experimental.categories.Category;
+
+import static 
org.apache.drill.exec.physical.impl.join.JoinUtils.FAILED_TO_PLAN_CARTESIAN_JOIN;
+
+@Category(SqlTest.class)
+public class CrossJoinTest extends ClusterTest {
+
+  private static int NATION_TABLE_RECORDS_COUNT = 25;
+
+  private static int EXPECTED_COUNT = NATION_TABLE_RECORDS_COUNT * 
NATION_TABLE_RECORDS_COUNT;
+
+  @BeforeClass
+  public static void setUp() throws Exception {
+    startCluster(ClusterFixture.builder(dirTestWatcher));
+  }
+
+  @After
+  public void tearDown() {
+    client.resetSession(PlannerSettings.NLJOIN_FOR_SCALAR.getOptionName());
+  }
+
+  @Test
+  public void testCrossJoinFailsForEnabledOption() throws Exception {
+    enableNlJoinForScalarOnly();
+
+    thrownException.expect(UserRemoteException.class);
+    thrownException.expectMessage(FAILED_TO_PLAN_CARTESIAN_JOIN);
+
+    queryBuilder().sql(
+        "SELECT l.n_name, r.n_name " +
+            "FROM cp.`tpch/nation.parquet` l " +
+            "CROSS JOIN cp.`tpch/nation.parquet` r")
+        .run();
+  }
+
+  @Test
+  public void testCrossJoinSucceedsForDisabledOption() throws Exception {
+    disableNlJoinForScalarOnly();
+    client.testBuilder().sqlQuery(
+        "SELECT l.n_name,r.n_name " +
+            "FROM cp.`tpch/nation.parquet` l " +
+            "CROSS JOIN cp.`tpch/nation.parquet` r")
+        .expectsNumRecords(EXPECTED_COUNT)
+        .go();
+  }
+
+  @Test
+  public void testCommaJoinFailsForEnabledOption() throws Exception {
+    enableNlJoinForScalarOnly();
+
+    thrownException.expect(UserRemoteException.class);
+    thrownException.expectMessage(FAILED_TO_PLAN_CARTESIAN_JOIN);
+
+    queryBuilder().sql(
+        "SELECT l.n_name,r.n_name " +
+            "FROM cp.`tpch/nation.parquet` l, cp.`tpch/nation.parquet` r")
+        .run();
+  }
+
+  @Test
+  public void testCommaJoinSucceedsForDisabledOption() throws Exception {
+    disableNlJoinForScalarOnly();
+    client.testBuilder().sqlQuery(
+        "SELECT l.n_name,r.n_name " +
+            "FROM cp.`tpch/nation.parquet` l, cp.`tpch/nation.parquet` r")
+        .expectsNumRecords(EXPECTED_COUNT)
+        .go();
+  }
+
+  @Test
+  public void testSubSelectCrossJoinFailsForEnabledOption() throws Exception {
+    enableNlJoinForScalarOnly();
+
+    thrownException.expect(UserRemoteException.class);
+    thrownException.expectMessage(FAILED_TO_PLAN_CARTESIAN_JOIN);
+
+    queryBuilder().sql(
+        "SELECT COUNT(*) c " +
+            "FROM (" +
+            "SELECT l.n_name,r.n_name " +
+            "FROM cp.`tpch/nation.parquet` l " +
+            "CROSS JOIN cp.`tpch/nation.parquet` r" +
+            ")")
+        .run();
+  }
+
+  @Test
+  public void testSubSelectCrossJoinSucceedsForDisabledOption() throws 
Exception {
+    disableNlJoinForScalarOnly();
+
+    client.testBuilder()
+        .sqlQuery(
+            "SELECT COUNT(*) c " +
+                "FROM (SELECT l.n_name,r.n_name " +
+                "FROM cp.`tpch/nation.parquet` l " +
+                "CROSS JOIN cp.`tpch/nation.parquet` r)")
+        .unOrdered()
+        .baselineColumns("c")
+        .baselineValues((long) EXPECTED_COUNT)
+        .go();
+  }
+
+  @Test
+  public void textCrossAndCommaJoinFailsForEnabledOption() throws Exception {
+    enableNlJoinForScalarOnly();
+
+    thrownException.expect(UserRemoteException.class);
+    thrownException.expectMessage(FAILED_TO_PLAN_CARTESIAN_JOIN);
+
+    queryBuilder().sql(
+        "SELECT * " +
+            "FROM cp.`tpch/nation.parquet` a, cp.`tpch/nation.parquet` b " +
+            "CROSS JOIN cp.`tpch/nation.parquet` c")
+        .run();
+  }
+
+  @Test
+  public void textCrossAndCommaJoinSucceedsForDisabledOption() throws 
Exception {
+    disableNlJoinForScalarOnly();
+
+    client.testBuilder().sqlQuery(
+        "SELECT * " +
+            "FROM cp.`tpch/nation.parquet` a, cp.`tpch/nation.parquet` b " +
+            "CROSS JOIN cp.`tpch/nation.parquet` c")
+        .expectsNumRecords(NATION_TABLE_RECORDS_COUNT * EXPECTED_COUNT)
+        .go();
+  }
+
+  @Test
+  public void testCrossApplyFailsForEnabledOption() throws Exception {
+    enableNlJoinForScalarOnly();
+
+    thrownException.expect(UserRemoteException.class);
+    thrownException.expectMessage(FAILED_TO_PLAN_CARTESIAN_JOIN);
+
+    queryBuilder().sql(
+        "SELECT * " +
+            "FROM cp.`tpch/nation.parquet` l " +
+            "CROSS APPLY cp.`tpch/nation.parquet` r")
+        .run();
+  }
+
+  @Test
+  public void testCrossApplySucceedsForDisabledOption() throws Exception {
+    disableNlJoinForScalarOnly();
+
+    client.testBuilder().sqlQuery(
+        "SELECT * " +
+            "FROM cp.`tpch/nation.parquet` l " +
+            "CROSS APPLY cp.`tpch/nation.parquet` r")
+        .expectsNumRecords(EXPECTED_COUNT)
+        .go();
+  }
+
+  @Test
+  public void testCrossJoinSucceedsForEnabledOptionAndScalarInput() throws 
Exception {
+    enableNlJoinForScalarOnly();
+
+    client.testBuilder().sqlQuery(
+        "SELECT * " +
+            "FROM cp.`tpch/nation.parquet` l " +
+            "CROSS JOIN (SELECT * FROM cp.`tpch/nation.parquet` r LIMIT 1)")
+        .expectsNumRecords(NATION_TABLE_RECORDS_COUNT)
+        .go();
+  }
+
+  private static void disableNlJoinForScalarOnly() {
+    client.alterSession(PlannerSettings.NLJOIN_FOR_SCALAR.getOptionName(), 
false);
+  }
+
+  private static void enableNlJoinForScalarOnly() {
+    client.alterSession(PlannerSettings.NLJOIN_FOR_SCALAR.getOptionName(), 
true);
+  }
+}


 

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
us...@infra.apache.org


> Implement CROSS JOIN
> --------------------
>
>                 Key: DRILL-786
>                 URL: https://issues.apache.org/jira/browse/DRILL-786
>             Project: Apache Drill
>          Issue Type: New Feature
>          Components: Query Planning &amp; Optimization
>    Affects Versions: 1.14.0
>            Reporter: Krystal
>            Assignee: Igor Guzenko
>            Priority: Major
>              Labels: doc-impacting, ready-to-commit
>             Fix For: 1.15.0
>
>
> *For documentation:*
> Due to it's nature cross joins can produce extremely large results, and we 
> don't recommend to use the feature if you don't know that results won't cause 
> out of memory errors. That's why cross joins are disabled by default, to 
> allow explicit cross join syntax you'll have to enable it by setting  
> planner.enable_nljoin_for_scalar_only option to false. There is also another 
> limitation related to usage of aggregation function over cross join relation. 
> When input row count for aggregate function is bigger than value of 
> planner.slice_target option then query can't be planned (because 2 phase 
> aggregation can't be created in such case), as a workaround you should set 
> planner.enable_multiphase_agg to false. This limitation will be active until 
> fix of https://issues.apache.org/jira/browse/DRILL-6839. 
> ---------------------------------------------------------------------------------------------------------------------
> git.commit.id.abbrev=5d7e3d3
> 0: jdbc:drill:schema=dfs> select student.name, student.age, 
> student.studentnum from student cross join voter where student.age = 20 and 
> voter.age = 20;
> Query failed: org.apache.drill.exec.rpc.RpcException: Remote failure while 
> running query.[error_id: "af90e65a-c4d7-4635-a436-bbc1444c8db2"
> Root: rel#318:Subset#28.PHYSICAL.SINGLETON([]).[]
> Original rel:
> AbstractConverter(subset=[rel#318:Subset#28.PHYSICAL.SINGLETON([]).[]], 
> convention=[PHYSICAL], DrillDistributionTraitDef=[SINGLETON([])], sort=[[]]): 
> rowcount = 22500.0, cumulative cost = {inf}, id = 320
>   DrillScreenRel(subset=[rel#317:Subset#28.LOGICAL.ANY([]).[]]): rowcount = 
> 22500.0, cumulative cost = {2250.0 rows, 2250.0 cpu, 0.0 io, 0.0 network}, id 
> = 316
>     DrillProjectRel(subset=[rel#315:Subset#27.LOGICAL.ANY([]).[]], name=[$2], 
> age=[$1], studentnum=[$3]): rowcount = 22500.0, cumulative cost = {22500.0 
> rows, 12.0 cpu, 0.0 io, 0.0 network}, id = 314
>       DrillJoinRel(subset=[rel#313:Subset#26.LOGICAL.ANY([]).[]], 
> condition=[true], joinType=[inner]): rowcount = 22500.0, cumulative cost = 
> {22500.0 rows, 0.0 cpu, 0.0 io, 0.0 network}, id = 312
>         DrillFilterRel(subset=[rel#308:Subset#23.LOGICAL.ANY([]).[]], 
> condition=[=(CAST($1):INTEGER, 20)]): rowcount = 150.0, cumulative cost = 
> {1000.0 rows, 4000.0 cpu, 0.0 io, 0.0 network}, id = 307
>           DrillScanRel(subset=[rel#306:Subset#22.LOGICAL.ANY([]).[]], 
> table=[[dfs, student]]): rowcount = 1000.0, cumulative cost = {1000.0 rows, 
> 4000.0 cpu, 0.0 io, 0.0 network}, id = 129
>         DrillFilterRel(subset=[rel#311:Subset#25.LOGICAL.ANY([]).[]], 
> condition=[=(CAST($1):INTEGER, 20)]): rowcount = 150.0, cumulative cost = 
> {1000.0 rows, 4000.0 cpu, 0.0 io, 0.0 network}, id = 310
>           DrillScanRel(subset=[rel#309:Subset#24.LOGICAL.ANY([]).[]], 
> table=[[dfs, voter]]): rowcount = 1000.0, cumulative cost = {1000.0 rows, 
> 2000.0 cpu, 0.0 io, 0.0 network}, id = 140
> Stack trace:
> org.eigenbase.relopt.RelOptPlanner$CannotPlanException: Node 
> [rel#318:Subset#28.PHYSICAL.SINGLETON([]).[]] could not be implemented; 
> planner state:
> Root: rel#318:Subset#28.PHYSICAL.SINGLETON([]).[]
> Original rel:
> AbstractConverter(subset=[rel#318:Subset#28.PHYSICAL.SINGLETON([]).[]], 
> convention=[PHYSICAL], DrillDistributionTraitDef=[SINGLETON([])], sort=[[]]): 
> rowcount = 22500.0, cumulative cost = {inf}, id = 320
>   DrillScreenRel(subset=[rel#317:Subset#28.LOGICAL.ANY([]).[]]): rowcount = 
> 22500.0, cumulative cost = {2250.0 rows, 2250.0 cpu, 0.0 io, 0.0 network}, id 
> = 316
>     DrillProjectRel(subset=[rel#315:Subset#27.LOGICAL.ANY([]).[]], name=[$2], 
> age=[$1], studentnum=[$3]): rowcount = 22500.0, cumulative cost = {22500.0 
> rows, 12.0 cpu, 0.0 io, 0.0 network}, id = 314
>       DrillJoinRel(subset=[rel#313:Subset#26.LOGICAL.ANY([]).[]], 
> condition=[true], joinType=[inner]): rowcount = 22500.0, cumulative cost = 
> {22500.0 rows, 0.0 cpu, 0.0 io, 0.0 network}, id = 312
>         DrillFilterRel(subset=[rel#308:Subset#23.LOGICAL.ANY([]).[]], 
> condition=[=(CAST($1):INTEGER, 20)]): rowcount = 150.0, cumulative cost = 
> {1000.0 rows, 4000.0 cpu, 0.0 io, 0.0 network}, id = 307
>           DrillScanRel(subset=[rel#306:Subset#22.LOGICAL.ANY([]).[]], 
> table=[[dfs, student]]): rowcount = 1000.0, cumulative cost = {1000.0 rows, 
> 4000.0 cpu, 0.0 io, 0.0 network}, id = 129
>         DrillFilterRel(subset=[rel#311:Subset#25.LOGICAL.ANY([]).[]], 
> condition=[=(CAST($1):INTEGER, 20)]): rowcount = 150.0, cumulative cost = 
> {1000.0 rows, 4000.0 cpu, 0.0 io, 0.0 network}, id = 310
>           DrillScanRel(subset=[rel#309:Subset#24.LOGICAL.ANY([]).[]], 
> table=[[dfs, voter]]): rowcount = 1000.0, cumulative cost = {1000.0 rows, 
> 2000.0 cpu, 0.0 io, 0.0 network}, id = 140
> Sets:
> Set#22, type: (DrillRecordRow[*, age, name, studentnum])
> rel#306:Subset#22.LOGICAL.ANY([]).[], best=rel#129, 
> importance=0.5904900000000001
> rel#129:DrillScanRel.LOGICAL.ANY([]).[](table=[dfs, student]), 
> rowcount=1000.0, cumulative cost={1000.0 rows, 4000.0 cpu, 0.0 io, 0.0 
> network}
> rel#333:AbstractConverter.LOGICAL.ANY([]).[](child=rel#332:Subset#22.PHYSICAL.ANY([]).[],convention=LOGICAL,DrillDistributionTraitDef=ANY([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#337:AbstractConverter.LOGICAL.ANY([]).[](child=rel#336:Subset#22.PHYSICAL.SINGLETON([]).[],convention=LOGICAL,DrillDistributionTraitDef=ANY([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#332:Subset#22.PHYSICAL.ANY([]).[], best=rel#335, importance=0.531441
> rel#334:AbstractConverter.PHYSICAL.ANY([]).[](child=rel#306:Subset#22.LOGICAL.ANY([]).[],convention=PHYSICAL,DrillDistributionTraitDef=ANY([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#338:AbstractConverter.PHYSICAL.ANY([]).[](child=rel#336:Subset#22.PHYSICAL.SINGLETON([]).[],convention=PHYSICAL,DrillDistributionTraitDef=ANY([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#339:AbstractConverter.PHYSICAL.SINGLETON([]).[](child=rel#306:Subset#22.LOGICAL.ANY([]).[],convention=PHYSICAL,DrillDistributionTraitDef=SINGLETON([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#340:AbstractConverter.PHYSICAL.SINGLETON([]).[](child=rel#332:Subset#22.PHYSICAL.ANY([]).[],convention=PHYSICAL,DrillDistributionTraitDef=SINGLETON([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#335:ScanPrel.PHYSICAL.SINGLETON([]).[](groupscan=ParquetGroupScan 
> [entries=[ReadEntryWithPath [path=maprfs:/drill/testdata/p1tests/student]], 
> selectionRoot=/drill/testdata/p1tests/student, columns=[SchemaPath [`age`], 
> SchemaPath [`name`], SchemaPath [`studentnum`]]]), rowcount=1000.0, 
> cumulative cost={1000.0 rows, 4000.0 cpu, 0.0 io, 0.0 network}
> rel#336:Subset#22.PHYSICAL.SINGLETON([]).[], best=rel#335, 
> importance=0.4782969000000001
> rel#339:AbstractConverter.PHYSICAL.SINGLETON([]).[](child=rel#306:Subset#22.LOGICAL.ANY([]).[],convention=PHYSICAL,DrillDistributionTraitDef=SINGLETON([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#340:AbstractConverter.PHYSICAL.SINGLETON([]).[](child=rel#332:Subset#22.PHYSICAL.ANY([]).[],convention=PHYSICAL,DrillDistributionTraitDef=SINGLETON([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#335:ScanPrel.PHYSICAL.SINGLETON([]).[](groupscan=ParquetGroupScan 
> [entries=[ReadEntryWithPath [path=maprfs:/drill/testdata/p1tests/student]], 
> selectionRoot=/drill/testdata/p1tests/student, columns=[SchemaPath [`age`], 
> SchemaPath [`name`], SchemaPath [`studentnum`]]]), rowcount=1000.0, 
> cumulative cost={1000.0 rows, 4000.0 cpu, 0.0 io, 0.0 network}
> Set#23, type: (DrillRecordRow[*, age, name, studentnum])
> rel#308:Subset#23.LOGICAL.ANY([]).[], best=rel#307, importance=0.6561
> rel#307:DrillFilterRel.LOGICAL.ANY([]).[](child=rel#306:Subset#22.LOGICAL.ANY([]).[],condition==(CAST($1):INTEGER,
>  20)), rowcount=150.0, cumulative cost={2000.0 rows, 8000.0 cpu, 0.0 io, 0.0 
> network}
> rel#343:AbstractConverter.LOGICAL.ANY([]).[](child=rel#342:Subset#23.PHYSICAL.SINGLETON([]).[],convention=LOGICAL,DrillDistributionTraitDef=ANY([]),sort=[]),
>  rowcount=150.0, cumulative cost={inf}
> rel#342:Subset#23.PHYSICAL.SINGLETON([]).[], best=rel#341, 
> importance=0.5904900000000001
> rel#344:AbstractConverter.PHYSICAL.SINGLETON([]).[](child=rel#308:Subset#23.LOGICAL.ANY([]).[],convention=PHYSICAL,DrillDistributionTraitDef=SINGLETON([]),sort=[]),
>  rowcount=150.0, cumulative cost={inf}
> rel#341:FilterPrel.PHYSICAL.SINGLETON([]).[](child=rel#332:Subset#22.PHYSICAL.ANY([]).[],condition==(CAST($1):INTEGER,
>  20)), rowcount=150.0, cumulative cost={2000.0 rows, 8000.0 cpu, 0.0 io, 0.0 
> network}
> Set#24, type: (DrillRecordRow[*, age])
> rel#309:Subset#24.LOGICAL.ANY([]).[], best=rel#140, 
> importance=0.5904900000000001
> rel#140:DrillScanRel.LOGICAL.ANY([]).[](table=[dfs, voter]), rowcount=1000.0, 
> cumulative cost={1000.0 rows, 2000.0 cpu, 0.0 io, 0.0 network}
> rel#330:AbstractConverter.LOGICAL.ANY([]).[](child=rel#329:Subset#24.PHYSICAL.ANY([]).[],convention=LOGICAL,DrillDistributionTraitDef=ANY([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#349:AbstractConverter.LOGICAL.ANY([]).[](child=rel#348:Subset#24.PHYSICAL.SINGLETON([]).[],convention=LOGICAL,DrillDistributionTraitDef=ANY([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#329:Subset#24.PHYSICAL.ANY([]).[], best=rel#347, importance=0.531441
> rel#331:AbstractConverter.PHYSICAL.ANY([]).[](child=rel#309:Subset#24.LOGICAL.ANY([]).[],convention=PHYSICAL,DrillDistributionTraitDef=ANY([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#350:AbstractConverter.PHYSICAL.ANY([]).[](child=rel#348:Subset#24.PHYSICAL.SINGLETON([]).[],convention=PHYSICAL,DrillDistributionTraitDef=ANY([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#351:AbstractConverter.PHYSICAL.SINGLETON([]).[](child=rel#309:Subset#24.LOGICAL.ANY([]).[],convention=PHYSICAL,DrillDistributionTraitDef=SINGLETON([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#352:AbstractConverter.PHYSICAL.SINGLETON([]).[](child=rel#329:Subset#24.PHYSICAL.ANY([]).[],convention=PHYSICAL,DrillDistributionTraitDef=SINGLETON([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#347:ScanPrel.PHYSICAL.SINGLETON([]).[](groupscan=ParquetGroupScan 
> [entries=[ReadEntryWithPath [path=maprfs:/drill/testdata/p1tests/voter]], 
> selectionRoot=/drill/testdata/p1tests/voter, columns=[SchemaPath [`age`]]]), 
> rowcount=1000.0, cumulative cost={1000.0 rows, 2000.0 cpu, 0.0 io, 0.0 
> network}
> rel#348:Subset#24.PHYSICAL.SINGLETON([]).[], best=rel#347, 
> importance=0.4782969000000001
> rel#351:AbstractConverter.PHYSICAL.SINGLETON([]).[](child=rel#309:Subset#24.LOGICAL.ANY([]).[],convention=PHYSICAL,DrillDistributionTraitDef=SINGLETON([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#352:AbstractConverter.PHYSICAL.SINGLETON([]).[](child=rel#329:Subset#24.PHYSICAL.ANY([]).[],convention=PHYSICAL,DrillDistributionTraitDef=SINGLETON([]),sort=[]),
>  rowcount=1000.0, cumulative cost={inf}
> rel#347:ScanPrel.PHYSICAL.SINGLETON([]).[](groupscan=ParquetGroupScan 
> [entries=[ReadEntryWithPath [path=maprfs:/drill/testdata/p1tests/voter]], 
> selectionRoot=/drill/testdata/p1tests/voter, columns=[SchemaPath [`age`]]]), 
> rowcount=1000.0, cumulative cost={1000.0 rows, 2000.0 cpu, 0.0 io, 0.0 
> network}
> Set#25, type: (DrillRecordRow[*, age])
> rel#311:Subset#25.LOGICAL.ANY([]).[], best=rel#310, importance=0.6561
> rel#310:DrillFilterRel.LOGICAL.ANY([]).[](child=rel#309:Subset#24.LOGICAL.ANY([]).[],condition==(CAST($1):INTEGER,
>  20)), rowcount=150.0, cumulative cost={2000.0 rows, 6000.0 cpu, 0.0 io, 0.0 
> network}
> rel#355:AbstractConverter.LOGICAL.ANY([]).[](child=rel#354:Subset#25.PHYSICAL.SINGLETON([]).[],convention=LOGICAL,DrillDistributionTraitDef=ANY([]),sort=[]),
>  rowcount=150.0, cumulative cost={inf}
> rel#354:Subset#25.PHYSICAL.SINGLETON([]).[], best=rel#353, 
> importance=0.5904900000000001
> rel#356:AbstractConverter.PHYSICAL.SINGLETON([]).[](child=rel#311:Subset#25.LOGICAL.ANY([]).[],convention=PHYSICAL,DrillDistributionTraitDef=SINGLETON([]),sort=[]),
>  rowcount=150.0, cumulative cost={inf}
> rel#353:FilterPrel.PHYSICAL.SINGLETON([]).[](child=rel#329:Subset#24.PHYSICAL.ANY([]).[],condition==(CAST($1):INTEGER,
>  20)), rowcount=150.0, cumulative cost={2000.0 rows, 6000.0 cpu, 0.0 io, 0.0 
> network}
> Set#26, type: RecordType(ANY *, ANY age, ANY name, ANY studentnum, ANY *0, 
> ANY age0)
> rel#313:Subset#26.LOGICAL.ANY([]).[], best=rel#312, 
> importance=0.7290000000000001
> rel#312:DrillJoinRel.LOGICAL.ANY([]).[](left=rel#308:Subset#23.LOGICAL.ANY([]).[],right=rel#311:Subset#25.LOGICAL.ANY([]).[],condition=true,joinType=inner),
>  rowcount=22500.0, cumulative cost={4001.0 rows, 14001.0 cpu, 0.0 io, 0.0 
> network}
> rel#327:AbstractConverter.LOGICAL.ANY([]).[](child=rel#326:Subset#26.PHYSICAL.ANY([]).[],convention=LOGICAL,DrillDistributionTraitDef=ANY([]),sort=[]),
>  rowcount=1.7976931348623157E308, cumulative cost={inf}
> rel#326:Subset#26.PHYSICAL.ANY([]).[], best=null, importance=0.6561
> rel#328:AbstractConverter.PHYSICAL.ANY([]).[](child=rel#313:Subset#26.LOGICAL.ANY([]).[],convention=PHYSICAL,DrillDistributionTraitDef=ANY([]),sort=[]),
>  rowcount=22500.0, cumulative cost={inf}
> Set#27, type: RecordType(ANY name, ANY age, ANY studentnum)
> rel#315:Subset#27.LOGICAL.ANY([]).[], best=rel#314, importance=0.81
> rel#314:DrillProjectRel.LOGICAL.ANY([]).[](child=rel#313:Subset#26.LOGICAL.ANY([]).[],name=$2,age=$1,studentnum=$3),
>  rowcount=22500.0, cumulative cost={26501.0 rows, 14013.0 cpu, 0.0 io, 0.0 
> network}
> rel#322:AbstractConverter.LOGICAL.ANY([]).[](child=rel#321:Subset#27.PHYSICAL.SINGLETON([]).[],convention=LOGICAL,DrillDistributionTraitDef=ANY([]),sort=[]),
>  rowcount=1.7976931348623157E308, cumulative cost={inf}
> rel#321:Subset#27.PHYSICAL.SINGLETON([]).[], best=null, 
> importance=0.7290000000000001
> rel#323:AbstractConverter.PHYSICAL.SINGLETON([]).[](child=rel#315:Subset#27.LOGICAL.ANY([]).[],convention=PHYSICAL,DrillDistributionTraitDef=SINGLETON([]),sort=[]),
>  rowcount=22500.0, cumulative cost={inf}
> Set#28, type: RecordType(ANY name, ANY age, ANY studentnum)
> rel#317:Subset#28.LOGICAL.ANY([]).[], best=rel#316, importance=0.9
> rel#316:DrillScreenRel.LOGICAL.ANY([]).[](child=rel#315:Subset#27.LOGICAL.ANY([]).[]),
>  rowcount=22500.0, cumulative cost={28751.0 rows, 16263.0 cpu, 0.0 io, 0.0 
> network}
> rel#319:AbstractConverter.LOGICAL.ANY([]).[](child=rel#318:Subset#28.PHYSICAL.SINGLETON([]).[],convention=LOGICAL,DrillDistributionTraitDef=ANY([]),sort=[]),
>  rowcount=1.7976931348623157E308, cumulative cost={inf}
> rel#318:Subset#28.PHYSICAL.SINGLETON([]).[], best=null, importance=1.0
> rel#320:AbstractConverter.PHYSICAL.SINGLETON([]).[](child=rel#317:Subset#28.LOGICAL.ANY([]).[],convention=PHYSICAL,DrillDistributionTraitDef=SINGLETON([]),sort=[]),
>  rowcount=22500.0, cumulative cost={inf}
> rel#324:ScreenPrel.PHYSICAL.SINGLETON([]).[](child=rel#321:Subset#27.PHYSICAL.SINGLETON([]).[]),
>  rowcount=1.7976931348623157E308, cumulative cost={inf}
> org.eigenbase.relopt.volcano.RelSubset$CheapestPlanReplacer.visit(RelSubset.java:445)
>  ~[optiq-core-0.7-20140513.013236-5.jar:na]
> org.eigenbase.relopt.volcano.RelSubset.buildCheapestPlan(RelSubset.java:287) 
> ~[optiq-core-0.7-20140513.013236-5.jar:na]
> org.eigenbase.relopt.volcano.VolcanoPlanner.findBestExp(VolcanoPlanner.java:669)
>  ~[optiq-core-0.7-20140513.013236-5.jar:na]
> net.hydromatic.optiq.prepare.PlannerImpl.transform(PlannerImpl.java:271) 
> ~[optiq-core-0.7-20140513.013236-5.jar:na]
> org.apache.drill.exec.planner.sql.handlers.DefaultSqlHandler.convertToPrel(DefaultSqlHandler.java:119)
>  
> ~[drill-java-exec-1.0.0-m2-incubating-SNAPSHOT-rebuffed.jar:1.0.0-m2-incubating-SNAPSHOT]
> org.apache.drill.exec.planner.sql.handlers.DefaultSqlHandler.getPlan(DefaultSqlHandler.java:89)
>  
> ~[drill-java-exec-1.0.0-m2-incubating-SNAPSHOT-rebuffed.jar:1.0.0-m2-incubating-SNAPSHOT]
> org.apache.drill.exec.planner.sql.DrillSqlWorker.getPlan(DrillSqlWorker.java:134)
>  
> ~[drill-java-exec-1.0.0-m2-incubating-SNAPSHOT-rebuffed.jar:1.0.0-m2-incubating-SNAPSHOT]
> org.apache.drill.exec.work.foreman.Foreman.runSQL(Foreman.java:338) 
> [drill-java-exec-1.0.0-m2-incubating-SNAPSHOT-rebuffed.jar:1.0.0-m2-incubating-SNAPSHOT]
> org.apache.drill.exec.work.foreman.Foreman.run(Foreman.java:186) 
> [drill-java-exec-1.0.0-m2-incubating-SNAPSHOT-rebuffed.jar:1.0.0-m2-incubating-SNAPSHOT]
> java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1145)
>  [na:1.7.0_45]
> java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:615)
>  [na:1.7.0_45]
> java.lang.Thread.run(Thread.java:744) [na:1.7.0_45]



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)

Reply via email to