You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hive.apache.org by rh...@apache.org on 2014/09/08 20:43:31 UTC

svn commit: r1623457 [1/2] - in /hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql: optimizer/ optimizer/optiq/ optimizer/optiq/cost/ optimizer/optiq/reloperators/ optimizer/optiq/rules/ optimizer/optiq/stats/ optimizer/optiq/translator/ parse/ u...

Author: rhbutani
Date: Mon Sep  8 18:43:30 2014
New Revision: 1623457

URL: http://svn.apache.org/r1623457
Log:
HIVE-8016 CBO: PPD to honor hive Join Cond, Casting fixes, Add annotations for IF, Code cleanup (Jojn Pullokkaran via Harish Butani)

Removed:
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/PreCBOOptimizer.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/JoinUtil.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/Pair.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/rules/HivePullUpProjectsAboveJoinRule.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/rules/HivePushJoinThroughJoinRule.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/rules/HiveSwapJoinRule.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/CBOTableStatsValidator.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/RelNodeConverter.java
Modified:
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/HiveDefaultRelMetadataProvider.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/HiveOptiqUtil.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/RelOptHiveTable.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/TraitsUtil.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/cost/HiveCost.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/cost/HiveCostUtil.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveAggregateRel.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveFilterRel.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveJoinRel.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveLimitRel.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveProjectRel.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveRel.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveSortRel.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveTableScanRel.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveUnionRel.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/rules/HiveMergeProjectRule.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/rules/HivePushFilterPastJoinRule.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/FilterSelectivityEstimator.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/HiveRelMdDistinctRowCount.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/HiveRelMdSelectivity.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/ASTBuilder.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/ASTConverter.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/DerivedTableInjector.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/ExprNodeConverter.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/RexNodeConverter.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/SqlFunctionConverter.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/TypeConverter.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/parse/SemanticAnalyzer.java
    hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/udf/generic/GenericUDFIf.java

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/HiveDefaultRelMetadataProvider.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/HiveDefaultRelMetadataProvider.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/HiveDefaultRelMetadataProvider.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/HiveDefaultRelMetadataProvider.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq;
 
 import com.google.common.collect.ImmutableList;
@@ -10,14 +27,6 @@ import org.eigenbase.rel.metadata.Chaine
 import org.eigenbase.rel.metadata.DefaultRelMetadataProvider;
 import org.eigenbase.rel.metadata.RelMetadataProvider;
 
-/**
- * Distinct row count and Selectivity is overridden for Hive.<br>
- * <p>
- * Distinct Row Count is overridden for:<br>
- * 1) Join 2) TableScan.<br>
- * Selectivity is overridden for:<br>
- * 1) Join 2) TableScan & Filter.
- */
 public class HiveDefaultRelMetadataProvider {
   private HiveDefaultRelMetadataProvider() {
   }

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/HiveOptiqUtil.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/HiveOptiqUtil.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/HiveOptiqUtil.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/HiveOptiqUtil.java Mon Sep  8 18:43:30 2014
@@ -1,20 +1,49 @@
+/**
+ * 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.optiq;
 
 import java.util.ArrayList;
 import java.util.BitSet;
+import java.util.HashMap;
+import java.util.HashSet;
 import java.util.List;
+import java.util.Map;
+import java.util.Set;
+import java.util.Map.Entry;
 
+import org.apache.hadoop.hive.ql.optimizer.optiq.reloperators.HiveJoinRel;
 import org.eigenbase.rel.RelFactories.ProjectFactory;
 import org.eigenbase.rel.RelNode;
+import org.eigenbase.relopt.RelOptUtil;
+import org.eigenbase.relopt.RelOptUtil.InputReferencedVisitor;
 import org.eigenbase.reltype.RelDataTypeField;
 import org.eigenbase.rex.RexBuilder;
 import org.eigenbase.rex.RexInputRef;
 import org.eigenbase.rex.RexNode;
+import org.eigenbase.sql.SqlKind;
 import org.eigenbase.sql.fun.SqlStdOperatorTable;
 import org.eigenbase.sql.validate.SqlValidatorUtil;
 import org.eigenbase.util.Pair;
 
 import com.google.common.base.Function;
+import com.google.common.collect.ImmutableList;
+import com.google.common.collect.ImmutableMap;
+import com.google.common.collect.ImmutableSet;
 import com.google.common.collect.Lists;
 
 /**
@@ -26,7 +55,7 @@ public class HiveOptiqUtil {
   /**
    * Get list of virtual columns from the given list of projections.
    * <p>
-   *
+   * 
    * @param exps
    *          list of rex nodes representing projections
    * @return List of Virtual Columns, will not be null.
@@ -66,14 +95,10 @@ public class HiveOptiqUtil {
     return projIndxLst;
   }
 
-  @Deprecated
-  public static void todo(String s) {
-  }
-
   /**
    * Push any equi join conditions that are not column references as Projections
    * on top of the children.
-   *
+   * 
    * @param factory
    *          Project factory to use.
    * @param inputRels
@@ -85,16 +110,17 @@ public class HiveOptiqUtil {
    * @param systemColCount
    *          number of system columns, usually zero. These columns are
    *          projected at the leading edge of the output row.
-   * @param leftKeys       on return this contains the join key positions from
-   *                       the new project rel on the LHS.
-   * @param rightKeys      on return this contains the join key positions from
-   *                       the new project rel on the RHS.
+   * @param leftKeys
+   *          on return this contains the join key positions from the new
+   *          project rel on the LHS.
+   * @param rightKeys
+   *          on return this contains the join key positions from the new
+   *          project rel on the RHS.
    * @return the join condition after the equi expressions pushed down.
    */
-  public static RexNode projectNonColumnEquiConditions(ProjectFactory factory,
-      RelNode[] inputRels, List<RexNode> leftJoinKeys,
-      List<RexNode> rightJoinKeys, int systemColCount, List<Integer> leftKeys,
-      List<Integer> rightKeys) {
+  public static RexNode projectNonColumnEquiConditions(ProjectFactory factory, RelNode[] inputRels,
+      List<RexNode> leftJoinKeys, List<RexNode> rightJoinKeys, int systemColCount,
+      List<Integer> leftKeys, List<Integer> rightKeys) {
     RelNode leftRel = inputRels[0];
     RelNode rightRel = inputRels[1];
     RexBuilder rexBuilder = leftRel.getCluster().getRexBuilder();
@@ -118,8 +144,7 @@ public class HiveOptiqUtil {
     }
 
     for (i = 0; i < origRightInputSize; i++) {
-      final RelDataTypeField field = rightRel.getRowType().getFieldList()
-          .get(i);
+      final RelDataTypeField field = rightRel.getRowType().getFieldList().get(i);
       newRightFields.add(rexBuilder.makeInputRef(field.getType(), i));
       newRightFieldNames.add(field.getName());
     }
@@ -151,13 +176,12 @@ public class HiveOptiqUtil {
       RexNode cond = rexBuilder.makeCall(
           SqlStdOperatorTable.EQUALS,
           rexBuilder.makeInputRef(leftKey.getType(), systemColCount + p.left),
-          rexBuilder.makeInputRef(rightKey.getType(), systemColCount
-              + origLeftInputSize + newKeyCount + p.right));
+          rexBuilder.makeInputRef(rightKey.getType(), systemColCount + origLeftInputSize
+              + newKeyCount + p.right));
       if (outJoinCond == null) {
         outJoinCond = cond;
       } else {
-        outJoinCond = rexBuilder.makeCall(SqlStdOperatorTable.AND, outJoinCond,
-            cond);
+        outJoinCond = rexBuilder.makeCall(SqlStdOperatorTable.AND, outJoinCond, cond);
       }
     }
 
@@ -166,20 +190,17 @@ public class HiveOptiqUtil {
     }
 
     int newLeftOffset = systemColCount + origLeftInputSize;
-    int newRightOffset = systemColCount + origLeftInputSize
-        + origRightInputSize + newKeyCount;
+    int newRightOffset = systemColCount + origLeftInputSize + origRightInputSize + newKeyCount;
     for (i = 0; i < newKeyCount; i++) {
       leftKeys.add(origLeftInputSize + i);
       rightKeys.add(origRightInputSize + i);
-      RexNode cond = rexBuilder.makeCall(SqlStdOperatorTable.EQUALS, rexBuilder
-          .makeInputRef(newLeftFields.get(i).getType(), newLeftOffset + i),
-          rexBuilder.makeInputRef(newLeftFields.get(i).getType(),
-              newRightOffset + i));
+      RexNode cond = rexBuilder.makeCall(SqlStdOperatorTable.EQUALS,
+          rexBuilder.makeInputRef(newLeftFields.get(i).getType(), newLeftOffset + i),
+          rexBuilder.makeInputRef(newLeftFields.get(i).getType(), newRightOffset + i));
       if (outJoinCond == null) {
         outJoinCond = cond;
       } else {
-        outJoinCond = rexBuilder.makeCall(SqlStdOperatorTable.AND, outJoinCond,
-            cond);
+        outJoinCond = rexBuilder.makeCall(SqlStdOperatorTable.AND, outJoinCond, cond);
       }
     }
 
@@ -197,4 +218,260 @@ public class HiveOptiqUtil {
 
     return outJoinCond;
   }
+
+  /**
+   * JoinPredicateInfo represents Join condition; JoinPredicate Info uses
+   * JoinLeafPredicateInfo to represent individual conjunctive elements in the
+   * predicate.<br>
+   * JoinPredicateInfo = JoinLeafPredicateInfo1 and JoinLeafPredicateInfo2...<br>
+   * <p>
+   * JoinPredicateInfo:<br>
+   * 1. preserves the order of conjuctive elements for
+   * equi-join(m_equiJoinPredicateElements)<br>
+   * 2. Stores set of projection indexes from left and right child which is part
+   * of equi join keys; the indexes are both in child and Join node schema.<br>
+   * 3. Keeps a map of projection indexes that are part of join keys to list of
+   * conjuctive elements(JoinLeafPredicateInfo) that uses them.
+   * 
+   */
+  public static class JoinPredicateInfo {
+    private final ImmutableList<JoinLeafPredicateInfo>                        nonEquiJoinPredicateElements;
+    private final ImmutableList<JoinLeafPredicateInfo>                        equiJoinPredicateElements;
+    private final ImmutableSet<Integer>                                       projsFromLeftPartOfJoinKeysInChildSchema;
+    private final ImmutableSet<Integer>                                       projsFromRightPartOfJoinKeysInChildSchema;
+    private final ImmutableSet<Integer>                                       projsFromRightPartOfJoinKeysInJoinSchema;
+    private final ImmutableMap<Integer, ImmutableList<JoinLeafPredicateInfo>> mapOfProjIndxInJoinSchemaToLeafPInfo;
+
+    public JoinPredicateInfo(List<JoinLeafPredicateInfo> nonEquiJoinPredicateElements,
+        List<JoinLeafPredicateInfo> equiJoinPredicateElements,
+        Set<Integer> projsFromLeftPartOfJoinKeysInChildSchema,
+        Set<Integer> projsFromRightPartOfJoinKeysInChildSchema,
+        Set<Integer> projsFromRightPartOfJoinKeysInJoinSchema,
+        Map<Integer, ImmutableList<JoinLeafPredicateInfo>> mapOfProjIndxInJoinSchemaToLeafPInfo) {
+      this.nonEquiJoinPredicateElements = ImmutableList.copyOf(nonEquiJoinPredicateElements);
+      this.equiJoinPredicateElements = ImmutableList.copyOf(equiJoinPredicateElements);
+      this.projsFromLeftPartOfJoinKeysInChildSchema = ImmutableSet
+          .copyOf(projsFromLeftPartOfJoinKeysInChildSchema);
+      this.projsFromRightPartOfJoinKeysInChildSchema = ImmutableSet
+          .copyOf(projsFromRightPartOfJoinKeysInChildSchema);
+      this.projsFromRightPartOfJoinKeysInJoinSchema = ImmutableSet
+          .copyOf(projsFromRightPartOfJoinKeysInJoinSchema);
+      this.mapOfProjIndxInJoinSchemaToLeafPInfo = ImmutableMap
+          .copyOf(mapOfProjIndxInJoinSchemaToLeafPInfo);
+    }
+
+    public List<JoinLeafPredicateInfo> getNonEquiJoinPredicateElements() {
+      return nonEquiJoinPredicateElements;
+    }
+
+    public List<JoinLeafPredicateInfo> getEquiJoinPredicateElements() {
+      return equiJoinPredicateElements;
+    }
+
+    public Set<Integer> getProjsFromLeftPartOfJoinKeysInChildSchema() {
+      return projsFromLeftPartOfJoinKeysInChildSchema;
+    }
+
+    public Set<Integer> getProjsFromRightPartOfJoinKeysInChildSchema() {
+      return projsFromRightPartOfJoinKeysInChildSchema;
+    }
+
+    /**
+     * NOTE: Join Schema = left Schema + (right Schema offset by
+     * left.fieldcount). Hence its ok to return projections from left in child
+     * schema.
+     */
+    public Set<Integer> getProjsFromLeftPartOfJoinKeysInJoinSchema() {
+      return projsFromLeftPartOfJoinKeysInChildSchema;
+    }
+
+    public Set<Integer> getProjsFromRightPartOfJoinKeysInJoinSchema() {
+      return projsFromRightPartOfJoinKeysInJoinSchema;
+    }
+
+    public Map<Integer, ImmutableList<JoinLeafPredicateInfo>> getMapOfProjIndxToLeafPInfo() {
+      return mapOfProjIndxInJoinSchemaToLeafPInfo;
+    }
+
+    public static JoinPredicateInfo constructJoinPredicateInfo(HiveJoinRel j) {
+      return constructJoinPredicateInfo(j, j.getCondition());
+    }
+
+    public static JoinPredicateInfo constructJoinPredicateInfo(HiveJoinRel j, RexNode predicate) {
+      JoinPredicateInfo jpi = null;
+      JoinLeafPredicateInfo jlpi = null;
+      List<JoinLeafPredicateInfo> equiLPIList = new ArrayList<JoinLeafPredicateInfo>();
+      List<JoinLeafPredicateInfo> nonEquiLPIList = new ArrayList<JoinLeafPredicateInfo>();
+      Set<Integer> projsFromLeftPartOfJoinKeys = new HashSet<Integer>();
+      Set<Integer> projsFromRightPartOfJoinKeys = new HashSet<Integer>();
+      Set<Integer> projsFromRightPartOfJoinKeysInJoinSchema = new HashSet<Integer>();
+      Map<Integer, List<JoinLeafPredicateInfo>> tmpMapOfProjIndxInJoinSchemaToLeafPInfo = new HashMap<Integer, List<JoinLeafPredicateInfo>>();
+      Map<Integer, ImmutableList<JoinLeafPredicateInfo>> mapOfProjIndxInJoinSchemaToLeafPInfo = new HashMap<Integer, ImmutableList<JoinLeafPredicateInfo>>();
+      List<JoinLeafPredicateInfo> tmpJLPILst = null;
+      int rightOffSet = j.getLeft().getRowType().getFieldCount();
+      int projIndxInJoin;
+      List<RexNode> conjuctiveElements;
+
+      // 1. Decompose Join condition to a number of leaf predicates
+      // (conjuctive elements)
+      conjuctiveElements = RelOptUtil.conjunctions(predicate);
+
+      // 2. Walk through leaf predicates building up JoinLeafPredicateInfo
+      for (RexNode ce : conjuctiveElements) {
+        // 2.1 Construct JoinLeafPredicateInfo
+        jlpi = JoinLeafPredicateInfo.constructJoinLeafPredicateInfo(j, ce);
+
+        // 2.2 Classify leaf predicate as Equi vs Non Equi
+        if (jlpi.m_comparisonType.equals(SqlKind.EQUALS)) {
+          equiLPIList.add(jlpi);
+        } else {
+          nonEquiLPIList.add(jlpi);
+        }
+
+        // 2.3 Maintain join keys coming from left vs right (in child &
+        // Join Schema)
+        projsFromLeftPartOfJoinKeys.addAll(jlpi.getProjsFromLeftPartOfJoinKeysInChildSchema());
+        projsFromRightPartOfJoinKeys.addAll(jlpi.getProjsFromRightPartOfJoinKeysInChildSchema());
+        projsFromRightPartOfJoinKeysInJoinSchema.addAll(jlpi
+            .getProjsFromRightPartOfJoinKeysInJoinSchema());
+
+        // 2.4 Update Join Key to JoinLeafPredicateInfo map with keys
+        // from left
+        for (Integer projIndx : jlpi.getProjsFromLeftPartOfJoinKeysInChildSchema()) {
+          tmpJLPILst = tmpMapOfProjIndxInJoinSchemaToLeafPInfo.get(projIndx);
+          if (tmpJLPILst == null)
+            tmpJLPILst = new ArrayList<JoinLeafPredicateInfo>();
+          tmpJLPILst.add(jlpi);
+          tmpMapOfProjIndxInJoinSchemaToLeafPInfo.put(projIndx, tmpJLPILst);
+        }
+
+        // 2.5 Update Join Key to JoinLeafPredicateInfo map with keys
+        // from right
+        for (Integer projIndx : jlpi.getProjsFromRightPartOfJoinKeysInChildSchema()) {
+          projIndxInJoin = projIndx + rightOffSet;
+          tmpJLPILst = tmpMapOfProjIndxInJoinSchemaToLeafPInfo.get(projIndxInJoin);
+          if (tmpJLPILst == null)
+            tmpJLPILst = new ArrayList<JoinLeafPredicateInfo>();
+          tmpJLPILst.add(jlpi);
+          tmpMapOfProjIndxInJoinSchemaToLeafPInfo.put(projIndxInJoin, tmpJLPILst);
+        }
+
+      }
+
+      // 3. Update Update Join Key to List<JoinLeafPredicateInfo> to use
+      // ImmutableList
+      for (Entry<Integer, List<JoinLeafPredicateInfo>> e : tmpMapOfProjIndxInJoinSchemaToLeafPInfo
+          .entrySet()) {
+        mapOfProjIndxInJoinSchemaToLeafPInfo.put(e.getKey(), ImmutableList.copyOf(e.getValue()));
+      }
+
+      // 4. Construct JoinPredicateInfo
+      jpi = new JoinPredicateInfo(nonEquiLPIList, equiLPIList, projsFromLeftPartOfJoinKeys,
+          projsFromRightPartOfJoinKeys, projsFromRightPartOfJoinKeysInJoinSchema,
+          mapOfProjIndxInJoinSchemaToLeafPInfo);
+      return jpi;
+    }
+  }
+
+  /**
+   * JoinLeafPredicateInfo represents leaf predicate in Join condition
+   * (conjuctive lement).<br>
+   * <p>
+   * JoinLeafPredicateInfo:<br>
+   * 1. Stores list of expressions from left and right child which is part of
+   * equi join keys.<br>
+   * 2. Stores set of projection indexes from left and right child which is part
+   * of equi join keys; the indexes are both in child and Join node schema.<br>
+   */
+  public static class JoinLeafPredicateInfo {
+    private final SqlKind                m_comparisonType;
+    private final ImmutableList<RexNode> m_joinKeyExprsFromLeft;
+    private final ImmutableList<RexNode> m_joinKeyExprsFromRight;
+    private final ImmutableSet<Integer>  m_projsFromLeftPartOfJoinKeysInChildSchema;
+    private final ImmutableSet<Integer>  m_projsFromRightPartOfJoinKeysInChildSchema;
+    private final ImmutableSet<Integer>  m_projsFromRightPartOfJoinKeysInJoinSchema;
+
+    public JoinLeafPredicateInfo(SqlKind comparisonType, List<RexNode> joinKeyExprsFromLeft,
+        List<RexNode> joinKeyExprsFromRight, Set<Integer> projsFromLeftPartOfJoinKeysInChildSchema,
+        Set<Integer> projsFromRightPartOfJoinKeysInChildSchema,
+        Set<Integer> projsFromRightPartOfJoinKeysInJoinSchema) {
+      m_comparisonType = comparisonType;
+      m_joinKeyExprsFromLeft = ImmutableList.copyOf(joinKeyExprsFromLeft);
+      m_joinKeyExprsFromRight = ImmutableList.copyOf(joinKeyExprsFromRight);
+      m_projsFromLeftPartOfJoinKeysInChildSchema = ImmutableSet
+          .copyOf(projsFromLeftPartOfJoinKeysInChildSchema);
+      m_projsFromRightPartOfJoinKeysInChildSchema = ImmutableSet
+          .copyOf(projsFromRightPartOfJoinKeysInChildSchema);
+      m_projsFromRightPartOfJoinKeysInJoinSchema = ImmutableSet
+          .copyOf(projsFromRightPartOfJoinKeysInJoinSchema);
+    }
+
+    public List<RexNode> getJoinKeyExprsFromLeft() {
+      return m_joinKeyExprsFromLeft;
+    }
+
+    public List<RexNode> getJoinKeyExprsFromRight() {
+      return m_joinKeyExprsFromRight;
+    }
+
+    public Set<Integer> getProjsFromLeftPartOfJoinKeysInChildSchema() {
+      return m_projsFromLeftPartOfJoinKeysInChildSchema;
+    }
+
+    /**
+     * NOTE: Join Schema = left Schema + (right Schema offset by
+     * left.fieldcount). Hence its ok to return projections from left in child
+     * schema.
+     */
+    public Set<Integer> getProjsFromLeftPartOfJoinKeysInJoinSchema() {
+      return m_projsFromLeftPartOfJoinKeysInChildSchema;
+    }
+
+    public Set<Integer> getProjsFromRightPartOfJoinKeysInChildSchema() {
+      return m_projsFromRightPartOfJoinKeysInChildSchema;
+    }
+
+    public Set<Integer> getProjsFromRightPartOfJoinKeysInJoinSchema() {
+      return m_projsFromRightPartOfJoinKeysInJoinSchema;
+    }
+
+    private static JoinLeafPredicateInfo constructJoinLeafPredicateInfo(HiveJoinRel j, RexNode pe) {
+      JoinLeafPredicateInfo jlpi = null;
+      List<Integer> filterNulls = new ArrayList<Integer>();
+      List<RexNode> joinKeyExprsFromLeft = new ArrayList<RexNode>();
+      List<RexNode> joinKeyExprsFromRight = new ArrayList<RexNode>();
+      Set<Integer> projsFromLeftPartOfJoinKeysInChildSchema = new HashSet<Integer>();
+      Set<Integer> projsFromRightPartOfJoinKeysInChildSchema = new HashSet<Integer>();
+      Set<Integer> projsFromRightPartOfJoinKeysInJoinSchema = new HashSet<Integer>();
+      int rightOffSet = j.getLeft().getRowType().getFieldCount();
+
+      // 1. Split leaf join predicate to expressions from left, right
+      @SuppressWarnings("unused")
+      RexNode nonEquiPredicate = RelOptUtil.splitJoinCondition(j.getSystemFieldList(), j.getLeft(),
+          j.getRight(), pe, joinKeyExprsFromLeft, joinKeyExprsFromRight, filterNulls, null);
+
+      // 2. For left expressions, collect child projection indexes used
+      InputReferencedVisitor irvLeft = new InputReferencedVisitor();
+      irvLeft.apply(joinKeyExprsFromLeft);
+      projsFromLeftPartOfJoinKeysInChildSchema.addAll(irvLeft.inputPosReferenced);
+
+      // 3. For right expressions, collect child projection indexes used
+      InputReferencedVisitor irvRight = new InputReferencedVisitor();
+      irvRight.apply(joinKeyExprsFromRight);
+      projsFromRightPartOfJoinKeysInChildSchema.addAll(irvRight.inputPosReferenced);
+
+      // 3. Translate projection indexes from right to join schema, by adding
+      // offset.
+      for (Integer indx : projsFromRightPartOfJoinKeysInChildSchema) {
+        projsFromRightPartOfJoinKeysInJoinSchema.add(indx + rightOffSet);
+      }
+
+      // 4. Construct JoinLeafPredicateInfo
+      jlpi = new JoinLeafPredicateInfo(pe.getKind(), joinKeyExprsFromLeft, joinKeyExprsFromRight,
+          projsFromLeftPartOfJoinKeysInChildSchema, projsFromRightPartOfJoinKeysInChildSchema,
+          projsFromRightPartOfJoinKeysInJoinSchema);
+
+      return jlpi;
+    }
+  }
 }

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/RelOptHiveTable.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/RelOptHiveTable.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/RelOptHiveTable.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/RelOptHiveTable.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq;
 
 import java.util.ArrayList;
@@ -7,7 +24,6 @@ import java.util.HashSet;
 import java.util.List;
 import java.util.Map;
 import java.util.Set;
-import java.util.concurrent.atomic.AtomicBoolean;
 import java.util.concurrent.atomic.AtomicInteger;
 
 import org.apache.commons.logging.Log;
@@ -38,16 +54,15 @@ import com.google.common.collect.Immutab
 import com.google.common.collect.ImmutableMap.Builder;
 
 public class RelOptHiveTable extends RelOptAbstractTable {
-  private final Table                             m_hiveTblMetadata;
-  private final ImmutableList<ColumnInfo>         m_hiveNonPartitionCols;
-  private final ImmutableMap<Integer, ColumnInfo> m_hiveNonPartitionColsMap;
-  private final ImmutableMap<Integer, ColumnInfo> m_hivePartitionColsMap;
-  private final int                               m_noOfProjs;
-  final HiveConf                                  m_hiveConf;
-
-  private double                                  m_rowCount        = -1;
-  Map<Integer, ColStatistics>                     m_hiveColStatsMap = new HashMap<Integer, ColStatistics>();
-  private Integer                                 m_numPartitions;
+  private final Table                             hiveTblMetadata;
+  private final ImmutableList<ColumnInfo>         hiveNonPartitionCols;
+  private final ImmutableMap<Integer, ColumnInfo> hiveNonPartitionColsMap;
+  private final ImmutableMap<Integer, ColumnInfo> hivePartitionColsMap;
+  private final int                               noOfProjs;
+  final HiveConf                                  hiveConf;
+
+  private double                                  rowCount        = -1;
+  Map<Integer, ColStatistics>                     hiveColStatsMap = new HashMap<Integer, ColStatistics>();
   PrunedPartitionList                             partitionList;
   Map<String, PrunedPartitionList>                partitionCache;
   AtomicInteger                                   noColsMissingStats;
@@ -60,12 +75,12 @@ public class RelOptHiveTable extends Rel
       Table hiveTblMetadata, List<ColumnInfo> hiveNonPartitionCols,
       List<ColumnInfo> hivePartitionCols, HiveConf hconf, Map<String, PrunedPartitionList> partitionCache, AtomicInteger noColsMissingStats) {
     super(optiqSchema, name, rowType);
-    m_hiveTblMetadata = hiveTblMetadata;
-    m_hiveNonPartitionCols = ImmutableList.copyOf(hiveNonPartitionCols);
-    m_hiveNonPartitionColsMap = getColInfoMap(hiveNonPartitionCols, 0);
-    m_hivePartitionColsMap = getColInfoMap(hivePartitionCols, m_hiveNonPartitionColsMap.size());
-    m_noOfProjs = hiveNonPartitionCols.size() + hivePartitionCols.size();
-    m_hiveConf = hconf;
+    this.hiveTblMetadata = hiveTblMetadata;
+    this.hiveNonPartitionCols = ImmutableList.copyOf(hiveNonPartitionCols);
+    this.hiveNonPartitionColsMap = getColInfoMap(hiveNonPartitionCols, 0);
+    this.hivePartitionColsMap = getColInfoMap(hivePartitionCols, hiveNonPartitionColsMap.size());
+    this.noOfProjs = hiveNonPartitionCols.size() + hivePartitionCols.size();
+    this.hiveConf = hconf;
     this.partitionCache = partitionCache;
     this.noColsMissingStats = noColsMissingStats;
   }
@@ -100,27 +115,27 @@ public class RelOptHiveTable extends Rel
 
   @Override
   public double getRowCount() {
-    if (m_rowCount == -1) {
+    if (rowCount == -1) {
       if (null == partitionList) {
         // we are here either unpartitioned table or partitioned table with no predicates
-        computePartitionList(m_hiveConf, null);
+        computePartitionList(hiveConf, null);
       }
-      if (m_hiveTblMetadata.isPartitioned()) {
+      if (hiveTblMetadata.isPartitioned()) {
         List<Long> rowCounts = StatsUtils.getBasicStatForPartitions(
-            m_hiveTblMetadata, partitionList.getNotDeniedPartns(),
+            hiveTblMetadata, partitionList.getNotDeniedPartns(),
             StatsSetupConst.ROW_COUNT);
-        m_rowCount = StatsUtils.getSumIgnoreNegatives(rowCounts);
+        rowCount = StatsUtils.getSumIgnoreNegatives(rowCounts);
 
       } else {
-        m_rowCount = StatsUtils.getNumRows(m_hiveTblMetadata);
+        rowCount = StatsUtils.getNumRows(hiveTblMetadata);
       }
     }
 
-    return m_rowCount;
+    return rowCount;
   }
 
   public Table getHiveTableMD() {
-    return m_hiveTblMetadata;
+    return hiveTblMetadata;
   }
 
   private String getColNamesForLogging(Set<String> colLst) {
@@ -140,16 +155,16 @@ public class RelOptHiveTable extends Rel
   public void computePartitionList(HiveConf conf, RexNode pruneNode) {
 
     try {
-      if (!m_hiveTblMetadata.isPartitioned() || pruneNode == null || InputFinder.bits(pruneNode).length() == 0 ) {
+      if (!hiveTblMetadata.isPartitioned() || pruneNode == null || InputFinder.bits(pruneNode).length() == 0 ) {
         // there is no predicate on partitioning column, we need all partitions in this case.
-        partitionList = PartitionPruner.prune(m_hiveTblMetadata, null, conf, getName(), partitionCache);
+        partitionList = PartitionPruner.prune(hiveTblMetadata, null, conf, getName(), partitionCache);
         return;
       }
 
       // We have valid pruning expressions, only retrieve qualifying partitions
       ExprNodeDesc pruneExpr = pruneNode.accept(new ExprNodeConverter(getName(), getRowType(), true));
 
-      partitionList = PartitionPruner.prune(m_hiveTblMetadata, pruneExpr, conf, getName(), partitionCache);
+      partitionList = PartitionPruner.prune(hiveTblMetadata, pruneExpr, conf, getName(), partitionCache);
     } catch (HiveException he) {
       throw new RuntimeException(he);
     }
@@ -165,16 +180,16 @@ public class RelOptHiveTable extends Rel
     // 1. Separate required columns to Non Partition and Partition Cols
     ColumnInfo tmp;
     for (Integer pi : projIndxLst) {
-      if (m_hiveColStatsMap.get(pi) == null) {
-        if ((tmp = m_hiveNonPartitionColsMap.get(pi)) != null) {
+      if (hiveColStatsMap.get(pi) == null) {
+        if ((tmp = hiveNonPartitionColsMap.get(pi)) != null) {
           nonPartColNamesThatRqrStats.add(tmp.getInternalName());
           nonPartColIndxsThatRqrStats.add(pi);
-        } else if ((tmp = m_hivePartitionColsMap.get(pi)) != null) {
+        } else if ((tmp = hivePartitionColsMap.get(pi)) != null) {
           partColNamesThatRqrStats.add(tmp.getInternalName());
           partColIndxsThatRqrStats.add(pi);
         } else {
           String logMsg = "Unable to find Column Index: " + pi + ", in "
-              + m_hiveTblMetadata.getCompleteName();
+              + hiveTblMetadata.getCompleteName();
           LOG.error(logMsg);
           throw new RuntimeException(logMsg);
         }
@@ -184,16 +199,16 @@ public class RelOptHiveTable extends Rel
     if (null == partitionList) {
       // We could be here either because its an unpartitioned table or because
       // there are no pruning predicates on a partitioned table.
-      computePartitionList(m_hiveConf, null);
+      computePartitionList(hiveConf, null);
     }
 
     // 2. Obtain Col Stats for Non Partition Cols
     if (nonPartColNamesThatRqrStats.size() > 0) {
       List<ColStatistics> hiveColStats;
 
-      if (!m_hiveTblMetadata.isPartitioned()) {
+      if (!hiveTblMetadata.isPartitioned()) {
         // 2.1 Handle the case for unpartitioned table.
-        hiveColStats = StatsUtils.getTableColumnStats(m_hiveTblMetadata, m_hiveNonPartitionCols,
+        hiveColStats = StatsUtils.getTableColumnStats(hiveTblMetadata, hiveNonPartitionCols,
             nonPartColNamesThatRqrStats);
 
         // 2.1.1 Record Column Names that we needed stats for but couldn't
@@ -215,17 +230,17 @@ public class RelOptHiveTable extends Rel
         try {
           if (partitionList.getNotDeniedPartns().isEmpty()) {
             // no need to make a metastore call
-            m_rowCount = 0;
+            rowCount = 0;
             hiveColStats = new ArrayList<ColStatistics>();
             for (String c : nonPartColNamesThatRqrStats) {
               // add empty stats object for each column
-              hiveColStats.add(new ColStatistics(m_hiveTblMetadata.getTableName(), c, null));
+              hiveColStats.add(new ColStatistics(hiveTblMetadata.getTableName(), c, null));
             }
             colNamesFailedStats.clear();
           } else {
-          Statistics stats = StatsUtils.collectStatistics(m_hiveConf, partitionList,
-              m_hiveTblMetadata, m_hiveNonPartitionCols, nonPartColNamesThatRqrStats, true, true);
-          m_rowCount = stats.getNumRows();
+          Statistics stats = StatsUtils.collectStatistics(hiveConf, partitionList,
+              hiveTblMetadata, hiveNonPartitionCols, nonPartColNamesThatRqrStats, true, true);
+          rowCount = stats.getNumRows();
           hiveColStats = new ArrayList<ColStatistics>();
           for (String c : nonPartColNamesThatRqrStats) {
             ColStatistics cs = stats.getColumnStatisticsFromColName(c);
@@ -245,27 +260,26 @@ public class RelOptHiveTable extends Rel
 
       if (hiveColStats != null && hiveColStats.size() == nonPartColNamesThatRqrStats.size()) {
         for (int i = 0; i < hiveColStats.size(); i++) {
-          m_hiveColStatsMap.put(nonPartColIndxsThatRqrStats.get(i), hiveColStats.get(i));
+          hiveColStatsMap.put(nonPartColIndxsThatRqrStats.get(i), hiveColStats.get(i));
         }
       }
     }
 
     // 3. Obtain Stats for Partition Cols
     if (colNamesFailedStats.isEmpty() && !partColNamesThatRqrStats.isEmpty()) {
-       m_numPartitions = partitionList.getPartitions().size();
       ColStatistics cStats = null;
       for (int i = 0; i < partColNamesThatRqrStats.size(); i++) {
-        cStats = new ColStatistics(m_hiveTblMetadata.getTableName(),
-            partColNamesThatRqrStats.get(i), m_hivePartitionColsMap.get(
+        cStats = new ColStatistics(hiveTblMetadata.getTableName(),
+            partColNamesThatRqrStats.get(i), hivePartitionColsMap.get(
                 partColIndxsThatRqrStats.get(i)).getTypeName());
         cStats.setCountDistint(getDistinctCount(partitionList.getPartitions(),partColNamesThatRqrStats.get(i)));
-        m_hiveColStatsMap.put(partColIndxsThatRqrStats.get(i), cStats);
+        hiveColStatsMap.put(partColIndxsThatRqrStats.get(i), cStats);
       }
     }
 
     // 4. Warn user if we could get stats for required columns
     if (!colNamesFailedStats.isEmpty()) {
-      String logMsg = "No Stats for " + m_hiveTblMetadata.getCompleteName() + ", Columns: "
+      String logMsg = "No Stats for " + hiveTblMetadata.getCompleteName() + ", Columns: "
           + getColNamesForLogging(colNamesFailedStats);
       LOG.error(logMsg);
       noColsMissingStats.getAndAdd(colNamesFailedStats.size());
@@ -287,16 +301,16 @@ public class RelOptHiveTable extends Rel
     if (projIndxLst != null) {
       updateColStats(new HashSet<Integer>(projIndxLst));
       for (Integer i : projIndxLst) {
-        colStatsBldr.add(m_hiveColStatsMap.get(i));
+        colStatsBldr.add(hiveColStatsMap.get(i));
       }
     } else {
       List<Integer> pILst = new ArrayList<Integer>();
-      for (Integer i = 0; i < m_noOfProjs; i++) {
+      for (Integer i = 0; i < noOfProjs; i++) {
         pILst.add(i);
       }
       updateColStats(new HashSet<Integer>(pILst));
       for (Integer pi : pILst) {
-        colStatsBldr.add(m_hiveColStatsMap.get(pi));
+        colStatsBldr.add(hiveColStatsMap.get(pi));
       }
     }
 
@@ -312,7 +326,7 @@ public class RelOptHiveTable extends Rel
   public boolean containsPartitionColumnsOnly(BitSet cols) {
 
     for (int i = cols.nextSetBit(0); i >= 0; i++, i = cols.nextSetBit(i + 1)) {
-      if (!m_hivePartitionColsMap.containsKey(i)) {
+      if (!hivePartitionColsMap.containsKey(i)) {
         return false;
       }
     }

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/TraitsUtil.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/TraitsUtil.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/TraitsUtil.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/TraitsUtil.java Mon Sep  8 18:43:30 2014
@@ -1,52 +1,19 @@
 package org.apache.hadoop.hive.ql.optimizer.optiq;
 
-import java.util.List;
 
 import org.apache.hadoop.hive.ql.optimizer.optiq.reloperators.HiveRel;
-import org.eigenbase.rel.AggregateCall;
 import org.eigenbase.rel.RelCollation;
 import org.eigenbase.rel.RelCollationImpl;
-import org.eigenbase.rel.RelNode;
 import org.eigenbase.relopt.RelOptCluster;
 import org.eigenbase.relopt.RelTraitSet;
-import org.eigenbase.reltype.RelDataType;
 
 public class TraitsUtil {
-
-  public static RelTraitSet getSelectTraitSet(RelOptCluster cluster, RelNode child) {
-    return cluster.traitSetOf(HiveRel.CONVENTION, RelCollationImpl.EMPTY);
-  }
-
   public static RelTraitSet getSortTraitSet(RelOptCluster cluster, RelTraitSet traitSet,
       RelCollation collation) {
     return traitSet.plus(collation);
   }
 
-  public static RelTraitSet getFilterTraitSet(RelOptCluster cluster, RelTraitSet traitSet,
-      RelNode child) {
-    return cluster.traitSetOf(HiveRel.CONVENTION, RelCollationImpl.EMPTY);
-  }
-
-  public static RelTraitSet getLimitTraitSet(RelOptCluster cluster, RelTraitSet traitSet,
-      RelNode child) {
-    return cluster.traitSetOf(HiveRel.CONVENTION, RelCollationImpl.EMPTY);
-  }
-
-  public static RelTraitSet getAggregateTraitSet(RelOptCluster cluster, RelTraitSet traitSet,
-      List<Integer> gbCols, List<AggregateCall> aggCalls, RelNode child) {
-    return cluster.traitSetOf(HiveRel.CONVENTION, RelCollationImpl.EMPTY);
-  }
-
-  public static RelTraitSet getTableScanTraitSet(RelOptCluster cluster, RelTraitSet traitSet,
-      RelOptHiveTable table, RelDataType rowtype) {
-    return cluster.traitSetOf(HiveRel.CONVENTION, RelCollationImpl.EMPTY);
-  }
-
-  public static RelTraitSet getJoinTraitSet(RelOptCluster cluster, RelTraitSet traitSet) {
-    return cluster.traitSetOf(HiveRel.CONVENTION, RelCollationImpl.EMPTY);
-  }
-
-  public static RelTraitSet getUnionTraitSet(RelOptCluster cluster, RelTraitSet traitSet) {
+  public static RelTraitSet getDefaultTraitSet(RelOptCluster cluster) {
     return cluster.traitSetOf(HiveRel.CONVENTION, RelCollationImpl.EMPTY);
   }
 }

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/cost/HiveCost.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/cost/HiveCost.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/cost/HiveCost.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/cost/HiveCost.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.cost;
 
 import org.eigenbase.relopt.RelOptCost;

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/cost/HiveCostUtil.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/cost/HiveCostUtil.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/cost/HiveCostUtil.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/cost/HiveCostUtil.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.cost;
 
 import org.apache.hadoop.hive.ql.optimizer.optiq.reloperators.HiveRel;
@@ -10,6 +27,7 @@ public class HiveCostUtil {
   private static final double localFSWriteCostInNanoSec = 4 * netCostInNanoSec;
   private static final double localFSReadCostInNanoSec  = 4 * netCostInNanoSec;
   private static final double hDFSWriteCostInNanoSec    = 10 * localFSWriteCostInNanoSec;
+  @SuppressWarnings("unused")
   private static final double hDFSReadCostInNanoSec     = 1.5 * localFSReadCostInNanoSec;
 
   public static RelOptCost computCardinalityBasedCost(HiveRel hr) {

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveAggregateRel.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveAggregateRel.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveAggregateRel.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveAggregateRel.java Mon Sep  8 18:43:30 2014
@@ -1,10 +1,25 @@
+/**
+ * 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.optiq.reloperators;
 
 import java.util.BitSet;
 import java.util.List;
 
-import net.hydromatic.optiq.util.BitSets;
-
 import org.apache.hadoop.hive.ql.optimizer.optiq.TraitsUtil;
 import org.apache.hadoop.hive.ql.optimizer.optiq.cost.HiveCost;
 import org.eigenbase.rel.AggregateCall;
@@ -24,8 +39,7 @@ public class HiveAggregateRel extends Ag
 
   public HiveAggregateRel(RelOptCluster cluster, RelTraitSet traitSet, RelNode child,
       BitSet groupSet, List<AggregateCall> aggCalls) throws InvalidRelException {
-    super(cluster, TraitsUtil.getAggregateTraitSet(cluster, traitSet, BitSets.toList(groupSet),
-        aggCalls, child), child, groupSet, aggCalls);
+    super(cluster, TraitsUtil.getDefaultTraitSet(cluster), child, groupSet, aggCalls);
   }
 
   @Override

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveFilterRel.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveFilterRel.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveFilterRel.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveFilterRel.java Mon Sep  8 18:43:30 2014
@@ -1,7 +1,22 @@
+/**
+ * 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.optiq.reloperators;
 
-import java.util.List;
-
 import org.apache.hadoop.hive.ql.optimizer.optiq.TraitsUtil;
 import org.apache.hadoop.hive.ql.optimizer.optiq.cost.HiveCost;
 import org.eigenbase.rel.FilterRelBase;
@@ -18,7 +33,7 @@ public class HiveFilterRel extends Filte
   public static final FilterFactory DEFAULT_FILTER_FACTORY = new HiveFilterFactoryImpl();
 
   public HiveFilterRel(RelOptCluster cluster, RelTraitSet traits, RelNode child, RexNode condition) {
-    super(cluster, TraitsUtil.getFilterTraitSet(cluster, traits, child), child, condition);
+    super(cluster, TraitsUtil.getDefaultTraitSet(cluster), child, condition);
   }
 
   @Override
@@ -45,8 +60,7 @@ public class HiveFilterRel extends Filte
     @Override
     public RelNode createFilter(RelNode child, RexNode condition) {
       RelOptCluster cluster = child.getCluster();
-      HiveFilterRel filter = new HiveFilterRel(cluster, TraitsUtil.getFilterTraitSet(cluster, null,
-          child), child, condition);
+      HiveFilterRel filter = new HiveFilterRel(cluster, TraitsUtil.getDefaultTraitSet(cluster), child, condition);
       return filter;
     }
   }

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveJoinRel.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveJoinRel.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveJoinRel.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveJoinRel.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.reloperators;
 
 import java.util.Collections;
@@ -36,9 +53,10 @@ public class HiveJoinRel extends JoinRel
 
   public static final JoinFactory HIVE_JOIN_FACTORY = new HiveJoinFactoryImpl();
 
-  private final boolean m_leftSemiJoin;
-  private final JoinAlgorithm      m_joinAlgorithm;
-  private final MapJoinStreamingRelation m_mapJoinStreamingSide = MapJoinStreamingRelation.NONE;
+  private final boolean leftSemiJoin;
+  private final JoinAlgorithm      joinAlgorithm;
+  @SuppressWarnings("unused")
+  private final MapJoinStreamingRelation mapJoinStreamingSide = MapJoinStreamingRelation.NONE;
 
   public static HiveJoinRel getJoin(RelOptCluster cluster, RelNode left, RelNode right,
       RexNode condition, JoinRelType joinType, boolean leftSemiJoin) {
@@ -55,10 +73,10 @@ public class HiveJoinRel extends JoinRel
       RexNode condition, JoinRelType joinType, Set<String> variablesStopped,
       JoinAlgorithm joinAlgo, MapJoinStreamingRelation streamingSideForMapJoin, boolean leftSemiJoin)
       throws InvalidRelException {
-    super(cluster, TraitsUtil.getJoinTraitSet(cluster, traits), left, right, condition, joinType,
+    super(cluster, TraitsUtil.getDefaultTraitSet(cluster), left, right, condition, joinType,
         variablesStopped);
-    this.m_joinAlgorithm = joinAlgo;
-    m_leftSemiJoin = leftSemiJoin;
+    this.joinAlgorithm = joinAlgo;
+    this.leftSemiJoin = leftSemiJoin;
   }
 
   @Override
@@ -71,7 +89,7 @@ public class HiveJoinRel extends JoinRel
     try {
       Set<String> variablesStopped = Collections.emptySet();
       return new HiveJoinRel(getCluster(), traitSet, left, right, conditionExpr, joinType,
-          variablesStopped, JoinAlgorithm.NONE, null, m_leftSemiJoin);
+          variablesStopped, JoinAlgorithm.NONE, null, leftSemiJoin);
     } catch (InvalidRelException e) {
       // Semantic error not possible. Must be a bug. Convert to
       // internal error.
@@ -80,11 +98,11 @@ public class HiveJoinRel extends JoinRel
   }
 
   public JoinAlgorithm getJoinAlgorithm() {
-    return m_joinAlgorithm;
+    return joinAlgorithm;
   }
 
   public boolean isLeftSemiJoin() {
-    return m_leftSemiJoin;
+    return leftSemiJoin;
   }
 
   /**
@@ -102,7 +120,7 @@ public class HiveJoinRel extends JoinRel
    */
   @Override
   public RelDataType deriveRowType() {
-    if (m_leftSemiJoin) {
+    if (leftSemiJoin) {
       return deriveJoinRowType(left.getRowType(), null, JoinRelType.INNER,
           getCluster().getTypeFactory(), null,
           Collections.<RelDataTypeField> emptyList());

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveLimitRel.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveLimitRel.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveLimitRel.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveLimitRel.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.reloperators;
 
 import java.util.List;
@@ -18,7 +35,7 @@ public class HiveLimitRel extends Single
 
   HiveLimitRel(RelOptCluster cluster, RelTraitSet traitSet, RelNode child, RexNode offset,
       RexNode fetch) {
-    super(cluster, TraitsUtil.getLimitTraitSet(cluster, traitSet, child), child);
+    super(cluster, TraitsUtil.getDefaultTraitSet(cluster), child);
     this.offset = offset;
     this.fetch = fetch;
     assert getConvention() instanceof HiveRel;

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveProjectRel.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveProjectRel.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveProjectRel.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveProjectRel.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.reloperators;
 
 import java.util.ArrayList;
@@ -30,7 +47,7 @@ public class HiveProjectRel extends Proj
 
   public static final ProjectFactory DEFAULT_PROJECT_FACTORY = new HiveProjectFactoryImpl();
 
-  private final List<Integer>        m_virtualCols;
+  private final List<Integer>        virtualCols;
 
   /**
    * Creates a HiveProjectRel.
@@ -49,7 +66,7 @@ public class HiveProjectRel extends Proj
   public HiveProjectRel(RelOptCluster cluster, RelTraitSet traitSet, RelNode child,
       List<? extends RexNode> exps, RelDataType rowType, int flags) {
     super(cluster, traitSet, child, exps, rowType, flags);
-    m_virtualCols = ImmutableList.copyOf(HiveOptiqUtil.getVirtualCols(exps));
+    virtualCols = ImmutableList.copyOf(HiveOptiqUtil.getVirtualCols(exps));
   }
 
   /**
@@ -73,7 +90,7 @@ public class HiveProjectRel extends Proj
    */
   public static HiveProjectRel create(RelOptCluster cluster, RelNode child, List<? extends RexNode> exps,
       RelDataType rowType, final List<RelCollation> collationList) {
-    RelTraitSet traitSet = TraitsUtil.getSelectTraitSet(cluster, child);
+    RelTraitSet traitSet = TraitsUtil.getDefaultTraitSet(cluster);
     return new HiveProjectRel(cluster, traitSet, child, exps, rowType, Flags.BOXED);
   }
 
@@ -145,7 +162,7 @@ public class HiveProjectRel extends Proj
   }
 
   public List<Integer> getVirtualCols() {
-    return m_virtualCols;
+    return virtualCols;
   }
 
   /**

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveRel.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveRel.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveRel.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveRel.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.reloperators;
 
 import org.eigenbase.rel.RelNode;

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveSortRel.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveSortRel.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveSortRel.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveSortRel.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.reloperators;
 
 import org.apache.hadoop.hive.ql.optimizer.optiq.TraitsUtil;

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveTableScanRel.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveTableScanRel.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveTableScanRel.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveTableScanRel.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.reloperators;
 
 import java.util.List;
@@ -24,7 +41,6 @@ import org.eigenbase.reltype.RelDataType
  * </p>
  */
 public class HiveTableScanRel extends TableAccessRelBase implements HiveRel {
-  private List<ColStatistics> m_hiveColStat;
 
   /**
    * Creates a HiveTableScan.
@@ -40,7 +56,7 @@ public class HiveTableScanRel extends Ta
    */
   public HiveTableScanRel(RelOptCluster cluster, RelTraitSet traitSet, RelOptHiveTable table,
       RelDataType rowtype) {
-    super(cluster, TraitsUtil.getTableScanTraitSet(cluster, traitSet, table, rowtype), table);
+    super(cluster, TraitsUtil.getDefaultTraitSet(cluster), table);
     assert getConvention() == HiveRel.CONVENTION;
   }
 

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveUnionRel.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveUnionRel.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveUnionRel.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/reloperators/HiveUnionRel.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.reloperators;
 
 import java.util.List;

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/rules/HiveMergeProjectRule.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/rules/HiveMergeProjectRule.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/rules/HiveMergeProjectRule.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/rules/HiveMergeProjectRule.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.rules;
 
 import org.apache.hadoop.hive.ql.optimizer.optiq.reloperators.HiveProjectRel;

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/rules/HivePushFilterPastJoinRule.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/rules/HivePushFilterPastJoinRule.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/rules/HivePushFilterPastJoinRule.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/rules/HivePushFilterPastJoinRule.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.rules;
 
 import java.util.ArrayList;
@@ -24,8 +41,6 @@ import org.eigenbase.rex.RexUtil;
 import org.eigenbase.sql.SqlKind;
 import org.eigenbase.util.Holder;
 
-import com.google.common.collect.ImmutableList;
-
 public abstract class HivePushFilterPastJoinRule extends RelOptRule {
 
   public static final HivePushFilterPastJoinRule FILTER_ON_JOIN = new HivePushFilterPastJoinRule(
@@ -96,7 +111,7 @@ public abstract class HivePushFilterPast
     }
 
     final List<RexNode> aboveFilters = filter != null ? RelOptUtil
-        .conjunctions(filter.getCondition()) : ImmutableList.<RexNode> of();
+        .conjunctions(filter.getCondition()) : new ArrayList<RexNode>();
 
     List<RexNode> leftFilters = new ArrayList<RexNode>();
     List<RexNode> rightFilters = new ArrayList<RexNode>();

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/FilterSelectivityEstimator.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/FilterSelectivityEstimator.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/FilterSelectivityEstimator.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/FilterSelectivityEstimator.java Mon Sep  8 18:43:30 2014
@@ -5,7 +5,6 @@ import java.util.BitSet;
 import org.apache.hadoop.hive.ql.optimizer.optiq.RelOptHiveTable;
 import org.apache.hadoop.hive.ql.optimizer.optiq.reloperators.HiveTableScanRel;
 import org.eigenbase.rel.FilterRelBase;
-import org.eigenbase.rel.ProjectRel;
 import org.eigenbase.rel.ProjectRelBase;
 import org.eigenbase.rel.RelNode;
 import org.eigenbase.rel.metadata.RelMetadataQuery;
@@ -14,7 +13,6 @@ import org.eigenbase.relopt.RelOptUtil.I
 import org.eigenbase.rex.RexCall;
 import org.eigenbase.rex.RexInputRef;
 import org.eigenbase.rex.RexNode;
-import org.eigenbase.rex.RexUtil;
 import org.eigenbase.rex.RexVisitorImpl;
 import org.eigenbase.sql.SqlKind;
 

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/HiveRelMdDistinctRowCount.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/HiveRelMdDistinctRowCount.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/HiveRelMdDistinctRowCount.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/HiveRelMdDistinctRowCount.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.stats;
 
 import java.util.BitSet;

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/HiveRelMdSelectivity.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/HiveRelMdSelectivity.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/HiveRelMdSelectivity.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/stats/HiveRelMdSelectivity.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.stats;
 
 import java.util.ArrayList;
@@ -8,8 +25,8 @@ import java.util.Set;
 
 import net.hydromatic.optiq.BuiltinMethod;
 
-import org.apache.hadoop.hive.ql.optimizer.optiq.JoinUtil.JoinLeafPredicateInfo;
-import org.apache.hadoop.hive.ql.optimizer.optiq.JoinUtil.JoinPredicateInfo;
+import org.apache.hadoop.hive.ql.optimizer.optiq.HiveOptiqUtil.JoinLeafPredicateInfo;
+import org.apache.hadoop.hive.ql.optimizer.optiq.HiveOptiqUtil.JoinPredicateInfo;
 import org.apache.hadoop.hive.ql.optimizer.optiq.reloperators.HiveJoinRel;
 import org.apache.hadoop.hive.ql.optimizer.optiq.reloperators.HiveTableScanRel;
 import org.eigenbase.rel.JoinRelType;

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/ASTBuilder.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/ASTBuilder.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/ASTBuilder.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/ASTBuilder.java Mon Sep  8 18:43:30 2014
@@ -1,10 +1,28 @@
+/**
+ * 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.optiq.translator;
 
-import java.sql.Date;
 import java.text.DateFormat;
 import java.text.SimpleDateFormat;
 import java.util.Calendar;
 
+import net.hydromatic.avatica.ByteString;
+
 import org.apache.hadoop.hive.ql.optimizer.optiq.RelOptHiveTable;
 import org.apache.hadoop.hive.ql.parse.ASTNode;
 import org.apache.hadoop.hive.ql.parse.HiveParser;
@@ -117,6 +135,11 @@ class ASTBuilder {
     SqlTypeName sqlType = literal.getType().getSqlTypeName();
 
     switch (sqlType) {
+    case BINARY:
+      ByteString bs = (ByteString) literal.getValue();
+      val = bs.byteAt(0);
+      type = HiveParser.BigintLiteral;
+      break;
     case TINYINT:
       val = literal.getValue3();
       type = HiveParser.TinyintLiteral;

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/ASTConverter.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/ASTConverter.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/ASTConverter.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/ASTConverter.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.translator;
 
 import java.util.ArrayList;

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/DerivedTableInjector.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/DerivedTableInjector.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/DerivedTableInjector.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/DerivedTableInjector.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.translator;
 
 import java.util.ArrayList;
@@ -24,13 +41,9 @@ import org.eigenbase.rel.ValuesRelBase;
 import org.eigenbase.rel.rules.MultiJoinRel;
 import org.eigenbase.relopt.hep.HepRelVertex;
 import org.eigenbase.relopt.volcano.RelSubset;
-import org.eigenbase.reltype.RelDataTypeField;
 import org.eigenbase.rex.RexInputRef;
 import org.eigenbase.rex.RexNode;
 
-import com.google.common.base.Function;
-import com.google.common.collect.Lists;
-
 public class DerivedTableInjector {
 
   public static RelNode convertOpTree(RelNode rel, List<FieldSchema> resultSchema) {

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/ExprNodeConverter.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/ExprNodeConverter.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/ExprNodeConverter.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/ExprNodeConverter.java Mon Sep  8 18:43:30 2014
@@ -27,7 +27,6 @@ import org.apache.hadoop.hive.ql.plan.Ex
 import org.apache.hadoop.hive.ql.plan.ExprNodeConstantDesc;
 import org.apache.hadoop.hive.ql.plan.ExprNodeDesc;
 import org.apache.hadoop.hive.ql.plan.ExprNodeGenericFuncDesc;
-import org.apache.hadoop.hive.serde2.typeinfo.TypeInfo;
 import org.apache.hadoop.hive.serde2.typeinfo.TypeInfoFactory;
 import org.eigenbase.reltype.RelDataType;
 import org.eigenbase.reltype.RelDataTypeField;

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/RexNodeConverter.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/RexNodeConverter.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/RexNodeConverter.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/RexNodeConverter.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.translator;
 
 import java.math.BigDecimal;
@@ -10,6 +27,8 @@ import java.util.LinkedList;
 import java.util.List;
 import java.util.Map;
 
+import net.hydromatic.avatica.ByteString;
+
 import org.apache.hadoop.hive.common.type.Decimal128;
 import org.apache.hadoop.hive.common.type.HiveChar;
 import org.apache.hadoop.hive.common.type.HiveDecimal;
@@ -28,12 +47,10 @@ import org.apache.hadoop.hive.ql.udf.gen
 import org.apache.hadoop.hive.ql.udf.generic.GenericUDFBaseCompare;
 import org.apache.hadoop.hive.ql.udf.generic.GenericUDFBaseNumeric;
 import org.apache.hadoop.hive.ql.udf.generic.GenericUDFBridge;
-import org.apache.hadoop.hive.ql.udf.generic.GenericUDFTimestamp;
 import org.apache.hadoop.hive.ql.udf.generic.GenericUDFToBinary;
 import org.apache.hadoop.hive.ql.udf.generic.GenericUDFToChar;
 import org.apache.hadoop.hive.ql.udf.generic.GenericUDFToDate;
 import org.apache.hadoop.hive.ql.udf.generic.GenericUDFToDecimal;
-import org.apache.hadoop.hive.ql.udf.generic.GenericUDFToUnixTimeStamp;
 import org.apache.hadoop.hive.ql.udf.generic.GenericUDFToVarchar;
 import org.apache.hadoop.hive.serde2.objectinspector.ConstantObjectInspector;
 import org.apache.hadoop.hive.serde2.objectinspector.ObjectInspectorUtils;
@@ -204,7 +221,7 @@ public class RexNodeConverter {
           || (udf instanceof GenericUDFToBinary) || castExprUsingUDFBridge(udf)) {
         // || (udf instanceof GenericUDFToUnixTimeStamp) || (udf instanceof
         // GenericUDFTimestamp) || castExprUsingUDFBridge(udf)) {
-        castExpr = m_cluster.getRexBuilder().makeCast(
+        castExpr = m_cluster.getRexBuilder().makeAbstractCast(
             TypeConverter.convert(func.getTypeInfo(), m_cluster.getTypeFactory()),
             childRexNodeLst.get(0));
       }
@@ -264,7 +281,9 @@ public class RexNodeConverter {
       optiqLiteral = rexBuilder.makeLiteral(((Boolean) value).booleanValue());
       break;
     case BYTE:
-      optiqLiteral = rexBuilder.makeExactLiteral(new BigDecimal((Short) value));
+      byte[] byteArray = new byte[] { (Byte) value};
+      ByteString bs = new ByteString(byteArray);
+      optiqLiteral = rexBuilder.makeBinaryLiteral(bs);
       break;
     case SHORT:
       optiqLiteral = rexBuilder.makeExactLiteral(new BigDecimal((Short) value));

Modified: hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/SqlFunctionConverter.java
URL: http://svn.apache.org/viewvc/hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/SqlFunctionConverter.java?rev=1623457&r1=1623456&r2=1623457&view=diff
==============================================================================
--- hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/SqlFunctionConverter.java (original)
+++ hive/branches/cbo/ql/src/java/org/apache/hadoop/hive/ql/optimizer/optiq/translator/SqlFunctionConverter.java Mon Sep  8 18:43:30 2014
@@ -1,3 +1,20 @@
+/**
+ * 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.optiq.translator;
 
 import java.lang.annotation.Annotation;