You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hive.apache.org by jc...@apache.org on 2020/05/11 14:10:09 UTC

[hive] branch master updated: HIVE-23389: FilterMergeRule can lead to AssertionError (Jesus Camacho Rodriguez, reviewed by Zoltan Haindrich)

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

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


The following commit(s) were added to refs/heads/master by this push:
     new 6be3278  HIVE-23389: FilterMergeRule can lead to AssertionError (Jesus Camacho Rodriguez, reviewed by Zoltan Haindrich)
6be3278 is described below

commit 6be3278629945e34213f3f47d29eff7850c4b40f
Author: Jesus Camacho Rodriguez <jc...@apache.org>
AuthorDate: Wed May 6 16:08:33 2020 -0700

    HIVE-23389: FilterMergeRule can lead to AssertionError (Jesus Camacho Rodriguez, reviewed by Zoltan Haindrich)
    
    Close apache/hive#1010
---
 .../hadoop/hive/ql/optimizer/calcite/Bug.java      | 39 ++++++++++++++
 .../calcite/rules/HiveFilterMergeRule.java         | 59 ++++++++++++++++++++++
 .../hadoop/hive/ql/parse/CalcitePlanner.java       |  3 +-
 3 files changed, 100 insertions(+), 1 deletion(-)

diff --git a/ql/src/java/org/apache/hadoop/hive/ql/optimizer/calcite/Bug.java b/ql/src/java/org/apache/hadoop/hive/ql/optimizer/calcite/Bug.java
new file mode 100644
index 0000000..349f1a9
--- /dev/null
+++ b/ql/src/java/org/apache/hadoop/hive/ql/optimizer/calcite/Bug.java
@@ -0,0 +1,39 @@
+/*
+ * 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.hadoop.hive.ql.optimizer.calcite;
+
+/**
+ * Holder for a list of constants describing which bugs have not been
+ * fixed.
+ *
+ * <p>The usage of the constant is a convenient way to identify the impact of
+ * the bug. When someone fixes the bug, they will remove the constant and all
+ * usages of it. Also, the constant helps track the propagation of the fix: as
+ * the fix is integrated into other branches, the constant will be removed from
+ * those branches.</p>
+ *
+ */
+public final class Bug {
+
+  /**
+   * Whether <a href="https://issues.apache.org/jira/browse/CALCITE-3982">issue
+   * CALCITE-3982</a> is fixed.
+   */
+  public static final boolean CALCITE_3982_FIXED = false;
+
+}
diff --git a/ql/src/java/org/apache/hadoop/hive/ql/optimizer/calcite/rules/HiveFilterMergeRule.java b/ql/src/java/org/apache/hadoop/hive/ql/optimizer/calcite/rules/HiveFilterMergeRule.java
new file mode 100644
index 0000000..4f820af
--- /dev/null
+++ b/ql/src/java/org/apache/hadoop/hive/ql/optimizer/calcite/rules/HiveFilterMergeRule.java
@@ -0,0 +1,59 @@
+/*
+ * 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.hadoop.hive.ql.optimizer.calcite.rules;
+
+import org.apache.calcite.plan.RelOptRule;
+import org.apache.calcite.plan.RelOptRuleCall;
+import org.apache.calcite.tools.RelBuilder;
+import org.apache.hadoop.hive.ql.optimizer.calcite.Bug;
+import org.apache.hadoop.hive.ql.optimizer.calcite.HiveRelFactories;
+import org.apache.hadoop.hive.ql.optimizer.calcite.reloperators.HiveFilter;
+
+/**
+ * Mostly a copy of {@link org.apache.calcite.rel.rules.FilterMergeRule}.
+ * However, it relies in relBuilder to create the new condition and thus
+ * simplifies/flattens the predicate before creating the new filter.
+ */
+public class HiveFilterMergeRule extends RelOptRule {
+
+  public static final HiveFilterMergeRule INSTANCE =
+      new HiveFilterMergeRule();
+
+  /** Private constructor. */
+  private HiveFilterMergeRule() {
+    super(operand(HiveFilter.class,
+        operand(HiveFilter.class, any())),
+        HiveRelFactories.HIVE_BUILDER, null);
+    if (Bug.CALCITE_3982_FIXED) {
+      throw new AssertionError("Remove logic in HiveFilterMergeRule when [CALCITE-3982] "
+          + "has been fixed and use directly Calcite's FilterMergeRule instead.");
+    }
+  }
+
+  //~ Methods ----------------------------------------------------------------
+
+  public void onMatch(RelOptRuleCall call) {
+    final HiveFilter topFilter = call.rel(0);
+    final HiveFilter bottomFilter = call.rel(1);
+
+    final RelBuilder relBuilder = call.builder();
+    relBuilder.push(bottomFilter.getInput())
+        .filter(bottomFilter.getCondition(), topFilter.getCondition());
+
+    call.transformTo(relBuilder.build());
+  }
+}
diff --git a/ql/src/java/org/apache/hadoop/hive/ql/parse/CalcitePlanner.java b/ql/src/java/org/apache/hadoop/hive/ql/parse/CalcitePlanner.java
index 085de48..32ad4c1 100644
--- a/ql/src/java/org/apache/hadoop/hive/ql/parse/CalcitePlanner.java
+++ b/ql/src/java/org/apache/hadoop/hive/ql/parse/CalcitePlanner.java
@@ -207,6 +207,7 @@ import org.apache.hadoop.hive.ql.optimizer.calcite.rules.HiveExpandDistinctAggre
 import org.apache.hadoop.hive.ql.optimizer.calcite.rules.HiveFieldTrimmerRule;
 import org.apache.hadoop.hive.ql.optimizer.calcite.rules.HiveFilterAggregateTransposeRule;
 import org.apache.hadoop.hive.ql.optimizer.calcite.rules.HiveFilterJoinRule;
+import org.apache.hadoop.hive.ql.optimizer.calcite.rules.HiveFilterMergeRule;
 import org.apache.hadoop.hive.ql.optimizer.calcite.rules.HiveFilterProjectTSTransposeRule;
 import org.apache.hadoop.hive.ql.optimizer.calcite.rules.HiveFilterProjectTransposeRule;
 import org.apache.hadoop.hive.ql.optimizer.calcite.rules.HiveFilterSetOpTransposeRule;
@@ -2011,7 +2012,7 @@ public class CalcitePlanner extends SemanticAnalyzer {
       rules.add(HiveFilterJoinRule.FILTER_ON_JOIN);
       rules.add(new HiveFilterAggregateTransposeRule(Filter.class, HiveRelFactories.HIVE_BUILDER,
           Aggregate.class));
-      rules.add(new FilterMergeRule(HiveRelFactories.HIVE_BUILDER));
+      rules.add(HiveFilterMergeRule.INSTANCE);
       if (conf.getBoolVar(HiveConf.ConfVars.HIVE_OPTIMIZE_REDUCE_WITH_STATS)) {
         rules.add(HiveReduceExpressionsWithStatsRule.INSTANCE);
       }