You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hive.apache.org by ha...@apache.org on 2015/01/19 18:39:57 UTC

svn commit: r1653068 [1/2] - in /hive/trunk/ql/src: java/org/apache/hadoop/hive/ql/parse/ test/results/clientpositive/ test/results/clientpositive/spark/ test/results/clientpositive/tez/

Author: hashutosh
Date: Mon Jan 19 17:39:56 2015
New Revision: 1653068

URL: http://svn.apache.org/r1653068
Log:
HIVE-4809 : ReduceSinkOperator of PTFOperator can have redundant key columns (Navis via Ashutosh Chauhan)

Modified:
    hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/parse/PTFTranslator.java
    hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/parse/SemanticAnalyzer.java
    hive/trunk/ql/src/test/results/clientpositive/correlationoptimizer12.q.out
    hive/trunk/ql/src/test/results/clientpositive/ctas_colname.q.out
    hive/trunk/ql/src/test/results/clientpositive/groupby_resolution.q.out
    hive/trunk/ql/src/test/results/clientpositive/quotedid_basic.q.out
    hive/trunk/ql/src/test/results/clientpositive/spark/subquery_in.q.out
    hive/trunk/ql/src/test/results/clientpositive/spark/vectorized_ptf.q.out
    hive/trunk/ql/src/test/results/clientpositive/subquery_in.q.out
    hive/trunk/ql/src/test/results/clientpositive/subquery_in_having.q.out
    hive/trunk/ql/src/test/results/clientpositive/subquery_notin.q.out
    hive/trunk/ql/src/test/results/clientpositive/subquery_unqualcolumnrefs.q.out
    hive/trunk/ql/src/test/results/clientpositive/tez/subquery_in.q.out
    hive/trunk/ql/src/test/results/clientpositive/tez/vectorized_ptf.q.out
    hive/trunk/ql/src/test/results/clientpositive/union_remove_6_subq.q.out
    hive/trunk/ql/src/test/results/clientpositive/vectorized_ptf.q.out
    hive/trunk/ql/src/test/results/clientpositive/windowing_streaming.q.out

Modified: hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/parse/PTFTranslator.java
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/parse/PTFTranslator.java?rev=1653068&r1=1653067&r2=1653068&view=diff
==============================================================================
--- hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/parse/PTFTranslator.java (original)
+++ hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/parse/PTFTranslator.java Mon Jan 19 17:39:56 2015
@@ -926,18 +926,17 @@ public class PTFTranslator {
      */
     for (ColumnInfo inpCInfo : inputRR.getColumnInfos()) {
       ColumnInfo cInfo = new ColumnInfo(inpCInfo);
-      String colAlias = cInfo.getAlias();
 
-      String[] tabColAlias = inputRR.reverseLookup(inpCInfo.getInternalName());
-      if (tabColAlias != null) {
-        colAlias = tabColAlias[1];
-      }
-      ASTNode inExpr = null;
-      inExpr = PTFTranslator.getASTNode(inpCInfo, inpRR);
+      ASTNode inExpr = PTFTranslator.getASTNode(inpCInfo, inpRR);
       if (inExpr != null) {
         rr.putExpression(inExpr, cInfo);
       } else {
-        rr.put(cInfo.getTabAlias(), colAlias, cInfo);
+        String[] tabColAlias = inputRR.reverseLookup(inpCInfo.getInternalName());
+        if (tabColAlias != null) {
+          rr.put(tabColAlias[0], tabColAlias[1], cInfo);
+        } else {
+          rr.put(inpCInfo.getTabAlias(), inpCInfo.getAlias(), cInfo);
+        }
       }
       
       String[] altMapping = inputRR.getAlternateMappings(inpCInfo.getInternalName());

Modified: hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/parse/SemanticAnalyzer.java
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/parse/SemanticAnalyzer.java?rev=1653068&r1=1653067&r2=1653068&view=diff
==============================================================================
--- hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/parse/SemanticAnalyzer.java (original)
+++ hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/parse/SemanticAnalyzer.java Mon Jan 19 17:39:56 2015
@@ -7037,12 +7037,11 @@ public class SemanticAnalyzer extends Ba
 
   }
 
-  @SuppressWarnings("nls")
   private Operator genReduceSinkPlan(String dest, QB qb, Operator<?> input,
       int numReducers) throws SemanticException {
-
+    
     RowResolver inputRR = opParseCtx.get(input).getRowResolver();
-
+    
     // First generate the expression for the partition and sort keys
     // The cluster by clause / distribute by clause has the aliases for
     // partition function
@@ -7050,15 +7049,14 @@ public class SemanticAnalyzer extends Ba
     if (partitionExprs == null) {
       partitionExprs = qb.getParseInfo().getDistributeByForClause(dest);
     }
-    ArrayList<ExprNodeDesc> partitionCols = new ArrayList<ExprNodeDesc>();
+    ArrayList<ExprNodeDesc> partCols = new ArrayList<ExprNodeDesc>();
     if (partitionExprs != null) {
       int ccount = partitionExprs.getChildCount();
       for (int i = 0; i < ccount; ++i) {
         ASTNode cl = (ASTNode) partitionExprs.getChild(i);
-        partitionCols.add(genExprNodeDesc(cl, inputRR));
+        partCols.add(genExprNodeDesc(cl, inputRR));
       }
     }
-
     ASTNode sortExprs = qb.getParseInfo().getClusterByForClause(dest);
     if (sortExprs == null) {
       sortExprs = qb.getParseInfo().getSortByForClause(dest);
@@ -7078,11 +7076,7 @@ public class SemanticAnalyzer extends Ba
         }
       }
     }
-    Operator dummy = Operator.createDummy();
-    dummy.setParentOperators(Arrays.asList(input));
-
     ArrayList<ExprNodeDesc> sortCols = new ArrayList<ExprNodeDesc>();
-    ArrayList<ExprNodeDesc> sortColsBack = new ArrayList<ExprNodeDesc>();
     StringBuilder order = new StringBuilder();
     if (sortExprs != null) {
       int ccount = sortExprs.getChildCount();
@@ -7103,9 +7097,25 @@ public class SemanticAnalyzer extends Ba
         }
         ExprNodeDesc exprNode = genExprNodeDesc(cl, inputRR);
         sortCols.add(exprNode);
-        sortColsBack.add(ExprNodeDescUtils.backtrack(exprNode, dummy, input));
       }
     }
+    return genReduceSinkPlan(input, partCols, sortCols, order.toString(), numReducers);
+  }
+  
+  @SuppressWarnings("nls")
+  private Operator genReduceSinkPlan(Operator<?> input,
+      ArrayList<ExprNodeDesc> partitionCols, ArrayList<ExprNodeDesc> sortCols, 
+      String sortOrder, int numReducers) throws SemanticException {
+
+    RowResolver inputRR = opParseCtx.get(input).getRowResolver();
+    
+    Operator dummy = Operator.createDummy();
+    dummy.setParentOperators(Arrays.asList(input));
+
+    ArrayList<ExprNodeDesc> sortColsBack = new ArrayList<ExprNodeDesc>();
+    for (ExprNodeDesc sortCol : sortCols) {
+      sortColsBack.add(ExprNodeDescUtils.backtrack(sortCol, dummy, input));
+    }
     // For the generation of the values expression just get the inputs
     // signature and generate field expressions for those
     RowResolver rsRR = new RowResolver();
@@ -7163,7 +7173,7 @@ public class SemanticAnalyzer extends Ba
 
     // TODO Not 100% sure NOT_ACID is always right here.
     ReduceSinkDesc rsdesc = PlanUtils.getReduceSinkDesc(sortCols, valueCols, outputColumns,
-        false, -1, partitionCols, order.toString(), numReducers, AcidUtils.Operation.NOT_ACID);
+        false, -1, partitionCols, sortOrder, numReducers, AcidUtils.Operation.NOT_ACID);
     Operator interim = putOpInsertMap(OperatorFactory.getAndMakeChild(rsdesc,
         new RowSchema(rsRR.getColumnInfos()), input), rsRR);
 
@@ -12137,130 +12147,34 @@ public class SemanticAnalyzer extends Ba
   }
 
   private Operator genReduceSinkPlanForWindowing(WindowingSpec spec,
-      RowResolver inputRR,
-      Operator input) throws SemanticException{
+      RowResolver inputRR, Operator input) throws SemanticException{
+    
     ArrayList<ExprNodeDesc> partCols = new ArrayList<ExprNodeDesc>();
-    ArrayList<ExprNodeDesc> valueCols = new ArrayList<ExprNodeDesc>();
     ArrayList<ExprNodeDesc> orderCols = new ArrayList<ExprNodeDesc>();
-    Map<String, ExprNodeDesc> colExprMap = new HashMap<String, ExprNodeDesc>();
-    List<String> outputColumnNames = new ArrayList<String>();
-    StringBuilder orderString = new StringBuilder();
+    StringBuilder order = new StringBuilder();
 
-    ArrayList<PartitionExpression> partColList = spec.getQueryPartitionSpec().getExpressions();
-    for (PartitionExpression partCol : partColList) {
+    for (PartitionExpression partCol : spec.getQueryPartitionSpec().getExpressions()) {
       ExprNodeDesc partExpr = genExprNodeDesc(partCol.getExpression(), inputRR);
       partCols.add(partExpr);
       orderCols.add(partExpr);
-      orderString.append('+');
-    }
-
-    ArrayList<OrderExpression> orderColList = spec.getQueryOrderSpec() == null ?
-        new ArrayList<PTFInvocationSpec.OrderExpression>() :
-          spec.getQueryOrderSpec().getExpressions();
-    for (int i = 0; i < orderColList.size(); i++) {
-      OrderExpression orderCol = orderColList.get(i);
-      org.apache.hadoop.hive.ql.parse.PTFInvocationSpec.Order order = orderCol.getOrder();
-      if (order.name().equals("ASC")) {
-        orderString.append('+');
-      } else {
-        orderString.append('-');
-      }
-      ExprNodeDesc orderExpr = genExprNodeDesc(orderCol.getExpression(), inputRR);
-      orderCols.add(orderExpr);
-    }
-
-    ArrayList<ColumnInfo> colInfoList = inputRR.getColumnInfos();
-    RowResolver rsNewRR = new RowResolver();
-    int pos = 0;
-    for (ColumnInfo colInfo : colInfoList) {
-        ExprNodeDesc valueColExpr = new ExprNodeColumnDesc(colInfo);
-        valueCols.add(valueColExpr);
-        String internalName = SemanticAnalyzer.getColumnInternalName(pos++);
-        outputColumnNames.add(internalName);
-        colExprMap.put(internalName, valueColExpr);
-
-        String[] alias = inputRR.reverseLookup(colInfo.getInternalName());
-        ColumnInfo newColInfo = new ColumnInfo(
-            internalName, colInfo.getType(), alias[0],
-            colInfo.getIsVirtualCol(), colInfo.isHiddenVirtualCol());
-        rsNewRR.put(alias[0], alias[1], newColInfo);
-        String[] altMapping = inputRR.getAlternateMappings(colInfo.getInternalName());
-        if ( altMapping != null ) {
-          rsNewRR.put(altMapping[0], altMapping[1], newColInfo);
-        }
+      order.append('+');
     }
 
-    input = putOpInsertMap(OperatorFactory.getAndMakeChild(PlanUtils
-        .getReduceSinkDesc(orderCols,
-            valueCols, outputColumnNames, false,
-            -1, partCols, orderString.toString(), -1, AcidUtils.Operation.NOT_ACID),
-        new RowSchema(rsNewRR.getColumnInfos()), input), rsNewRR);
-    input.setColumnExprMap(colExprMap);
-
-
- // Construct the RR for extract operator
-    RowResolver extractRR = new RowResolver();
-    LinkedHashMap<String[], ColumnInfo> colsAddedByHaving =
-        new LinkedHashMap<String[], ColumnInfo>();
-    pos = 0;
-
-    for (ColumnInfo colInfo : colInfoList) {
-      String[] alias = inputRR.reverseLookup(colInfo.getInternalName());
-      /*
-       * if we have already encountered this colInfo internalName.
-       * We encounter it again because it must be put for the Having clause.
-       * We will add these entries in the end; in a loop on colsAddedByHaving. See below.
-       */
-      if ( colsAddedByHaving.containsKey(alias)) {
-        continue;
-      }
-      ASTNode astNode = PTFTranslator.getASTNode(colInfo, inputRR);
-      ColumnInfo eColInfo = new ColumnInfo(
-          SemanticAnalyzer.getColumnInternalName(pos++), colInfo.getType(), alias[0],
-          colInfo.getIsVirtualCol(), colInfo.isHiddenVirtualCol());
-
-      if ( astNode == null ) {
-        extractRR.put(alias[0], alias[1], eColInfo);
-      }
-      else {
-        /*
-         * in case having clause refers to this column may have been added twice;
-         * once with the ASTNode.toStringTree as the alias
-         * and then with the real alias.
-         */
-        extractRR.putExpression(astNode, eColInfo);
-        if ( !astNode.toStringTree().toLowerCase().equals(alias[1]) ) {
-          colsAddedByHaving.put(alias, eColInfo);
+    if (spec.getQueryOrderSpec() != null) {
+      for (OrderExpression orderCol : spec.getQueryOrderSpec().getExpressions()) {
+        String orderString = orderCol.getOrder().name();
+        if (orderString.equals("ASC")) {
+          order.append('+');
+        } else {
+          order.append('-');
         }
-      }
-      String[] altMapping = inputRR.getAlternateMappings(colInfo.getInternalName());
-      if ( altMapping != null ) {
-        extractRR.put(altMapping[0], altMapping[1], eColInfo);
+        orderCols.add(genExprNodeDesc(orderCol.getExpression(), inputRR));
       }
     }
 
-    for(Map.Entry<String[], ColumnInfo> columnAddedByHaving : colsAddedByHaving.entrySet() ) {
-      String[] alias = columnAddedByHaving.getKey();
-      ColumnInfo eColInfo = columnAddedByHaving.getValue();
-      extractRR.put(alias[0], alias[1], eColInfo);
-    }
-
-    /*
-     * b. Construct Extract Operator.
-     */
-    input = putOpInsertMap(OperatorFactory.getAndMakeChild(
-        new ExtractDesc(
-            new ExprNodeColumnDesc(TypeInfoFactory.stringTypeInfo,
-                Utilities.ReduceField.VALUE
-                .toString(), "", false)),
-        new RowSchema(inputRR.getColumnInfos()),
-        input), extractRR);
-
-
-    return input;
+    return genReduceSinkPlan(input, partCols, orderCols, order.toString(), -1);
   }
 
-
   public static ArrayList<WindowExpressionSpec> parseSelect(String selectExprStr)
       throws SemanticException
   {

Modified: hive/trunk/ql/src/test/results/clientpositive/correlationoptimizer12.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/correlationoptimizer12.q.out?rev=1653068&r1=1653067&r2=1653068&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/correlationoptimizer12.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/correlationoptimizer12.q.out Mon Jan 19 17:39:56 2015
@@ -36,9 +36,11 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col0 (type: string)
                 Statistics: Num rows: 500 Data size: 5312 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: string)
+                value expressions: _col1 (type: string)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: KEY.reducesinkkey0 (type: string), VALUE._col0 (type: string)
+          outputColumnNames: _col0, _col1
           Statistics: Num rows: 500 Data size: 5312 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 500 Data size: 5312 Basic stats: COMPLETE Column stats: NONE
@@ -109,9 +111,11 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col0 (type: string)
                 Statistics: Num rows: 25 Data size: 191 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: string)
+                value expressions: _col1 (type: string)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: KEY.reducesinkkey0 (type: string), VALUE._col0 (type: string)
+          outputColumnNames: _col0, _col1
           Statistics: Num rows: 25 Data size: 191 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 25 Data size: 191 Basic stats: COMPLETE Column stats: NONE

Modified: hive/trunk/ql/src/test/results/clientpositive/ctas_colname.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/ctas_colname.q.out?rev=1653068&r1=1653067&r2=1653068&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/ctas_colname.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/ctas_colname.q.out Mon Jan 19 17:39:56 2015
@@ -183,9 +183,10 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col0 (type: string)
                 Statistics: Num rows: 25 Data size: 191 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: string)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: string)
+          outputColumnNames: _col0, _col1
           Statistics: Num rows: 25 Data size: 191 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 25 Data size: 191 Basic stats: COMPLETE Column stats: NONE
@@ -329,9 +330,10 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col0 (type: string)
                 Statistics: Num rows: 500 Data size: 5312 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: string)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: string)
+          outputColumnNames: _col0, _col1
           Statistics: Num rows: 500 Data size: 5312 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 500 Data size: 5312 Basic stats: COMPLETE Column stats: NONE

Modified: hive/trunk/ql/src/test/results/clientpositive/groupby_resolution.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/groupby_resolution.q.out?rev=1653068&r1=1653067&r2=1653068&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/groupby_resolution.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/groupby_resolution.q.out Mon Jan 19 17:39:56 2015
@@ -704,7 +704,9 @@ STAGE PLANS:
               Statistics: Num rows: 83 Data size: 881 Basic stats: COMPLETE Column stats: NONE
               value expressions: _col0 (type: string), _col1 (type: bigint)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: VALUE._col0 (type: string), VALUE._col1 (type: bigint)
+          outputColumnNames: _col0, _col1
           Statistics: Num rows: 83 Data size: 881 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 83 Data size: 881 Basic stats: COMPLETE Column stats: NONE

Modified: hive/trunk/ql/src/test/results/clientpositive/quotedid_basic.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/quotedid_basic.q.out?rev=1653068&r1=1653067&r2=1653068&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/quotedid_basic.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/quotedid_basic.q.out Mon Jan 19 17:39:56 2015
@@ -195,9 +195,11 @@ STAGE PLANS:
               sort order: ++
               Map-reduce partition columns: _col2 (type: string)
               Statistics: Num rows: 0 Data size: 0 Basic stats: NONE Column stats: NONE
-              value expressions: _col0 (type: string), _col1 (type: string), _col2 (type: string)
+              value expressions: _col0 (type: string)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: VALUE._col0 (type: string), KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string)
+          outputColumnNames: _col0, _col1, _col2
           Statistics: Num rows: 0 Data size: 0 Basic stats: NONE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 0 Data size: 0 Basic stats: NONE Column stats: NONE
@@ -278,9 +280,11 @@ STAGE PLANS:
               sort order: ++
               Map-reduce partition columns: _col2 (type: string)
               Statistics: Num rows: 0 Data size: 0 Basic stats: NONE Column stats: NONE
-              value expressions: _col0 (type: string), _col1 (type: string), _col2 (type: string)
+              value expressions: _col0 (type: string)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: VALUE._col0 (type: string), KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string)
+          outputColumnNames: _col0, _col1, _col2
           Statistics: Num rows: 0 Data size: 0 Basic stats: NONE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 0 Data size: 0 Basic stats: NONE Column stats: NONE

Modified: hive/trunk/ql/src/test/results/clientpositive/spark/subquery_in.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/spark/subquery_in.q.out?rev=1653068&r1=1653067&r2=1653068&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/spark/subquery_in.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/spark/subquery_in.q.out Mon Jan 19 17:39:56 2015
@@ -140,7 +140,7 @@ STAGE PLANS:
   Stage: Stage-1
     Spark
       Edges:
-        Reducer 2 <- Map 1 (PARTITION-LEVEL SORT, 3), Map 3 (PARTITION-LEVEL SORT, 3)
+        Reducer 2 <- Map 1 (PARTITION-LEVEL SORT, 2), Map 3 (PARTITION-LEVEL SORT, 2)
 #### A masked pattern was here ####
       Vertices:
         Map 1 
@@ -265,9 +265,9 @@ STAGE PLANS:
   Stage: Stage-1
     Spark
       Edges:
-        Reducer 4 <- Map 3 (PARTITION-LEVEL SORT, 3)
+        Reducer 4 <- Map 3 (PARTITION-LEVEL SORT, 2)
         Reducer 5 <- Reducer 4 (GROUP, 1)
-        Reducer 2 <- Map 1 (PARTITION-LEVEL SORT, 3), Reducer 5 (PARTITION-LEVEL SORT, 3)
+        Reducer 2 <- Map 1 (PARTITION-LEVEL SORT, 2), Reducer 5 (PARTITION-LEVEL SORT, 2)
 #### A masked pattern was here ####
       Vertices:
         Map 1 
@@ -294,7 +294,6 @@ STAGE PLANS:
                     sort order: ++
                     Map-reduce partition columns: p_mfgr (type: string)
                     Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
-                    value expressions: p_mfgr (type: string), p_size (type: int)
         Reducer 2 
             Reduce Operator Tree:
               Join Operator
@@ -318,7 +317,9 @@ STAGE PLANS:
                         serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
         Reducer 4 
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+                outputColumnNames: _col2, _col5
                 Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
@@ -413,9 +414,9 @@ STAGE PLANS:
   Stage: Stage-1
     Spark
       Edges:
-        Reducer 4 <- Map 3 (PARTITION-LEVEL SORT, 3)
-        Reducer 5 <- Reducer 4 (GROUP, 3)
-        Reducer 2 <- Map 1 (PARTITION-LEVEL SORT, 3), Reducer 5 (PARTITION-LEVEL SORT, 3)
+        Reducer 4 <- Map 3 (PARTITION-LEVEL SORT, 2)
+        Reducer 5 <- Reducer 4 (GROUP, 2)
+        Reducer 2 <- Map 1 (PARTITION-LEVEL SORT, 2), Reducer 5 (PARTITION-LEVEL SORT, 2)
 #### A masked pattern was here ####
       Vertices:
         Map 1 
@@ -442,7 +443,6 @@ STAGE PLANS:
                     sort order: ++
                     Map-reduce partition columns: p_mfgr (type: string)
                     Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
-                    value expressions: p_mfgr (type: string), p_size (type: int)
         Reducer 2 
             Reduce Operator Tree:
               Join Operator
@@ -466,7 +466,9 @@ STAGE PLANS:
                         serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
         Reducer 4 
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+                outputColumnNames: _col2, _col5
                 Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
@@ -573,8 +575,8 @@ STAGE PLANS:
   Stage: Stage-1
     Spark
       Edges:
-        Reducer 4 <- Map 3 (GROUP, 3)
-        Reducer 2 <- Map 1 (PARTITION-LEVEL SORT, 3), Reducer 4 (PARTITION-LEVEL SORT, 3)
+        Reducer 4 <- Map 3 (GROUP, 2)
+        Reducer 2 <- Map 1 (PARTITION-LEVEL SORT, 2), Reducer 4 (PARTITION-LEVEL SORT, 2)
 #### A masked pattern was here ####
       Vertices:
         Map 1 
@@ -728,9 +730,9 @@ STAGE PLANS:
   Stage: Stage-1
     Spark
       Edges:
-        Reducer 2 <- Map 1 (GROUP, 3)
-        Reducer 3 <- Map 5 (PARTITION-LEVEL SORT, 3), Reducer 2 (PARTITION-LEVEL SORT, 3)
-        Reducer 4 <- Map 6 (PARTITION-LEVEL SORT, 3), Reducer 3 (PARTITION-LEVEL SORT, 3)
+        Reducer 2 <- Map 1 (GROUP, 2)
+        Reducer 3 <- Map 5 (PARTITION-LEVEL SORT, 2), Reducer 2 (PARTITION-LEVEL SORT, 2)
+        Reducer 4 <- Map 6 (PARTITION-LEVEL SORT, 2), Reducer 3 (PARTITION-LEVEL SORT, 2)
 #### A masked pattern was here ####
       Vertices:
         Map 1 

Modified: hive/trunk/ql/src/test/results/clientpositive/spark/vectorized_ptf.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/spark/vectorized_ptf.q.out?rev=1653068&r1=1653067&r2=1653068&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/spark/vectorized_ptf.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/spark/vectorized_ptf.q.out Mon Jan 19 17:39:56 2015
@@ -312,12 +312,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                      value expressions: _col5 (type: int), _col7 (type: double)
                       auto parallelism: false
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -689,12 +691,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 14 Data size: 8823 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: false
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 14 Data size: 8823 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 14 Data size: 8823 Basic stats: COMPLETE Column stats: NONE
@@ -1171,12 +1175,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                      value expressions: _col5 (type: int), _col7 (type: double)
                       auto parallelism: false
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -1459,12 +1465,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: false
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -1777,13 +1785,15 @@ STAGE PLANS:
                   Map-reduce partition columns: _col0 (type: string)
                   Statistics: Num rows: 13 Data size: 8021 Basic stats: COMPLETE Column stats: NONE
                   tag: -1
-                  value expressions: _col0 (type: string), _col1 (type: string), _col2 (type: int)
+                  value expressions: _col2 (type: int)
                   auto parallelism: false
             Execution mode: vectorized
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: string), VALUE._col0 (type: int)
+                outputColumnNames: _col0, _col1, _col2
                 Statistics: Num rows: 13 Data size: 8021 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 13 Data size: 8021 Basic stats: COMPLETE Column stats: NONE
@@ -2658,12 +2668,13 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
                       auto parallelism: false
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), KEY.reducesinkkey2 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -2935,12 +2946,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                      value expressions: _col5 (type: int), _col7 (type: double)
                       auto parallelism: false
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -3214,12 +3227,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                      value expressions: _col5 (type: int), _col7 (type: double)
                       auto parallelism: false
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -3520,12 +3535,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                      value expressions: _col5 (type: int), _col7 (type: double)
                       auto parallelism: false
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -3829,12 +3846,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                      value expressions: _col5 (type: int), _col7 (type: double)
                       auto parallelism: false
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -4305,12 +4324,14 @@ STAGE PLANS:
                   Map-reduce partition columns: _col2 (type: string)
                   Statistics: Num rows: 14 Data size: 8823 Basic stats: COMPLETE Column stats: NONE
                   tag: -1
-                  value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                  value expressions: _col5 (type: int), _col7 (type: double)
                   auto parallelism: false
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 14 Data size: 8823 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 14 Data size: 8823 Basic stats: COMPLETE Column stats: NONE
@@ -4838,12 +4859,14 @@ STAGE PLANS:
                     Map-reduce partition columns: _col0 (type: string)
                     Statistics: Num rows: 13 Data size: 8021 Basic stats: COMPLETE Column stats: NONE
                     tag: -1
-                    value expressions: _col0 (type: string), _col1 (type: string), _col2 (type: double)
+                    value expressions: _col2 (type: double)
                     auto parallelism: false
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: string), VALUE._col0 (type: double)
+                outputColumnNames: _col0, _col1, _col2
                 Statistics: Num rows: 13 Data size: 8021 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 13 Data size: 8021 Basic stats: COMPLETE Column stats: NONE
@@ -5270,7 +5293,9 @@ STAGE PLANS:
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: VALUE._col1 (type: string), KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -5280,12 +5305,14 @@ STAGE PLANS:
                     Map-reduce partition columns: _col2 (type: string)
                     Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                     tag: -1
-                    value expressions: _wcol0 (type: bigint), _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                    value expressions: _wcol0 (type: bigint), _col5 (type: int)
                     auto parallelism: false
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: VALUE._col0 (type: bigint), KEY.reducesinkkey2 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col4 (type: int)
+                outputColumnNames: _col0, _col2, _col3, _col6
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -5322,7 +5349,9 @@ STAGE PLANS:
         Reducer 5 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -5369,7 +5398,7 @@ STAGE PLANS:
                     Map-reduce partition columns: _col2 (type: string)
                     Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                     tag: -1
-                    value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                    value expressions: _col1 (type: string)
                     auto parallelism: false
         Reducer 7 
             Needs Tagging: false
@@ -5388,7 +5417,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                      value expressions: _col5 (type: int), _col7 (type: double)
                       auto parallelism: false
 
   Stage: Stage-1
@@ -5813,12 +5842,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string), _col1 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: false
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -6183,12 +6214,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: false
         Reducer 5 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -6521,12 +6554,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: false
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -6888,12 +6923,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string), _col1 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: false
         Reducer 5 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -7271,12 +7308,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string), _col1 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: false
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -7620,12 +7659,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: false
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE

Modified: hive/trunk/ql/src/test/results/clientpositive/subquery_in.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/subquery_in.q.out?rev=1653068&r1=1653067&r2=1653068&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/subquery_in.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/subquery_in.q.out Mon Jan 19 17:39:56 2015
@@ -271,9 +271,10 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col0 (type: string)
                 Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: int)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+          outputColumnNames: _col0, _col1
           Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
@@ -439,9 +440,10 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col0 (type: string)
                 Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: int)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+          outputColumnNames: _col0, _col1
           Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE

Modified: hive/trunk/ql/src/test/results/clientpositive/subquery_in_having.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/subquery_in_having.q.out?rev=1653068&r1=1653067&r2=1653068&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/subquery_in_having.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/subquery_in_having.q.out Mon Jan 19 17:39:56 2015
@@ -1379,9 +1379,11 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col1 (type: string)
                 Statistics: Num rows: 15 Data size: 3173 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: string), _col2 (type: int)
+                value expressions: _col0 (type: string)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: VALUE._col0 (type: string), KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+          outputColumnNames: _col0, _col1, _col2
           Statistics: Num rows: 15 Data size: 3173 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 15 Data size: 3173 Basic stats: COMPLETE Column stats: NONE

Modified: hive/trunk/ql/src/test/results/clientpositive/subquery_notin.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/subquery_notin.q.out?rev=1653068&r1=1653067&r2=1653068&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/subquery_notin.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/subquery_notin.q.out Mon Jan 19 17:39:56 2015
@@ -330,9 +330,11 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col1 (type: string)
                 Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: string), _col2 (type: int)
+                value expressions: _col0 (type: string)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: VALUE._col0 (type: string), KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+          outputColumnNames: _col0, _col1, _col2
           Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
@@ -468,9 +470,11 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col1 (type: string)
                 Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: string), _col2 (type: int)
+                value expressions: _col0 (type: string)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: VALUE._col0 (type: string), KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+          outputColumnNames: _col0, _col1, _col2
           Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
@@ -579,9 +583,10 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col0 (type: string)
                 Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: int)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+          outputColumnNames: _col0, _col1
           Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
@@ -714,9 +719,10 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col0 (type: string)
                 Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: int)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+          outputColumnNames: _col0, _col1
           Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
@@ -873,9 +879,10 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col0 (type: string)
                 Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: int)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+          outputColumnNames: _col0, _col1
           Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
@@ -1048,9 +1055,10 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col0 (type: string)
                 Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: int)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+          outputColumnNames: _col0, _col1
           Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE

Modified: hive/trunk/ql/src/test/results/clientpositive/subquery_unqualcolumnrefs.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/subquery_unqualcolumnrefs.q.out?rev=1653068&r1=1653067&r2=1653068&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/subquery_unqualcolumnrefs.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/subquery_unqualcolumnrefs.q.out Mon Jan 19 17:39:56 2015
@@ -224,9 +224,10 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col0 (type: string)
                 Statistics: Num rows: 0 Data size: 0 Basic stats: NONE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: int)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+          outputColumnNames: _col0, _col1
           Statistics: Num rows: 0 Data size: 0 Basic stats: NONE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 0 Data size: 0 Basic stats: NONE Column stats: NONE
@@ -376,9 +377,10 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col0 (type: string)
                 Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: int)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+          outputColumnNames: _col0, _col1
           Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
@@ -810,9 +812,11 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col1 (type: string)
                 Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: string), _col2 (type: int)
+                value expressions: _col0 (type: string)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: VALUE._col0 (type: string), KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+          outputColumnNames: _col0, _col1, _col2
           Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
@@ -948,9 +952,11 @@ STAGE PLANS:
                 sort order: ++
                 Map-reduce partition columns: _col1 (type: string)
                 Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
-                value expressions: _col0 (type: string), _col1 (type: string), _col2 (type: int)
+                value expressions: _col0 (type: string)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: VALUE._col0 (type: string), KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+          outputColumnNames: _col0, _col1, _col2
           Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE

Modified: hive/trunk/ql/src/test/results/clientpositive/tez/subquery_in.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/tez/subquery_in.q.out?rev=1653068&r1=1653067&r2=1653068&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/tez/subquery_in.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/tez/subquery_in.q.out Mon Jan 19 17:39:56 2015
@@ -310,7 +310,6 @@ STAGE PLANS:
                       sort order: ++
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
-                      value expressions: _col0 (type: string), _col1 (type: int)
         Reducer 2 
             Reduce Operator Tree:
               Merge Join Operator
@@ -334,7 +333,9 @@ STAGE PLANS:
                         serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
         Reducer 4 
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+                outputColumnNames: _col0, _col1
                 Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
@@ -466,7 +467,6 @@ STAGE PLANS:
                       sort order: ++
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
-                      value expressions: _col0 (type: string), _col1 (type: int)
         Reducer 2 
             Reduce Operator Tree:
               Merge Join Operator
@@ -490,7 +490,9 @@ STAGE PLANS:
                         serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
         Reducer 4 
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+                outputColumnNames: _col0, _col1
                 Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 3147 Basic stats: COMPLETE Column stats: NONE

Modified: hive/trunk/ql/src/test/results/clientpositive/tez/vectorized_ptf.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/tez/vectorized_ptf.q.out?rev=1653068&r1=1653067&r2=1653068&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/tez/vectorized_ptf.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/tez/vectorized_ptf.q.out Mon Jan 19 17:39:56 2015
@@ -312,12 +312,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                      value expressions: _col5 (type: int), _col7 (type: double)
                       auto parallelism: true
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -690,12 +692,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 14 Data size: 8823 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: true
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 14 Data size: 8823 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 14 Data size: 8823 Basic stats: COMPLETE Column stats: NONE
@@ -1172,12 +1176,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                      value expressions: _col5 (type: int), _col7 (type: double)
                       auto parallelism: true
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -1460,12 +1466,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: true
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -1778,13 +1786,15 @@ STAGE PLANS:
                   Map-reduce partition columns: _col0 (type: string)
                   Statistics: Num rows: 13 Data size: 8021 Basic stats: COMPLETE Column stats: NONE
                   tag: -1
-                  value expressions: _col0 (type: string), _col1 (type: string), _col2 (type: int)
+                  value expressions: _col2 (type: int)
                   auto parallelism: true
             Execution mode: vectorized
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: string), VALUE._col0 (type: int)
+                outputColumnNames: _col0, _col1, _col2
                 Statistics: Num rows: 13 Data size: 8021 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 13 Data size: 8021 Basic stats: COMPLETE Column stats: NONE
@@ -2661,12 +2671,13 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
                       auto parallelism: true
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), KEY.reducesinkkey2 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -2938,12 +2949,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                      value expressions: _col5 (type: int), _col7 (type: double)
                       auto parallelism: true
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -3217,12 +3230,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                      value expressions: _col5 (type: int), _col7 (type: double)
                       auto parallelism: true
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -3523,12 +3538,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                      value expressions: _col5 (type: int), _col7 (type: double)
                       auto parallelism: true
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -3832,12 +3849,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                      value expressions: _col5 (type: int), _col7 (type: double)
                       auto parallelism: true
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -4309,12 +4328,14 @@ STAGE PLANS:
                   Map-reduce partition columns: _col2 (type: string)
                   Statistics: Num rows: 14 Data size: 8823 Basic stats: COMPLETE Column stats: NONE
                   tag: -1
-                  value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                  value expressions: _col5 (type: int), _col7 (type: double)
                   auto parallelism: true
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 14 Data size: 8823 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 14 Data size: 8823 Basic stats: COMPLETE Column stats: NONE
@@ -4842,12 +4863,14 @@ STAGE PLANS:
                     Map-reduce partition columns: _col0 (type: string)
                     Statistics: Num rows: 13 Data size: 8021 Basic stats: COMPLETE Column stats: NONE
                     tag: -1
-                    value expressions: _col0 (type: string), _col1 (type: string), _col2 (type: double)
+                    value expressions: _col2 (type: double)
                     auto parallelism: true
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: string), VALUE._col0 (type: double)
+                outputColumnNames: _col0, _col1, _col2
                 Statistics: Num rows: 13 Data size: 8021 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 13 Data size: 8021 Basic stats: COMPLETE Column stats: NONE
@@ -5284,7 +5307,7 @@ STAGE PLANS:
                     Map-reduce partition columns: _col2 (type: string)
                     Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                     tag: -1
-                    value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                    value expressions: _col1 (type: string)
                     auto parallelism: true
                   Select Operator
                     expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
@@ -5296,12 +5319,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int), _col7 (type: double)
+                      value expressions: _col5 (type: int), _col7 (type: double)
                       auto parallelism: true
         Reducer 3 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: VALUE._col1 (type: string), KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -5311,12 +5336,14 @@ STAGE PLANS:
                     Map-reduce partition columns: _col2 (type: string)
                     Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                     tag: -1
-                    value expressions: _wcol0 (type: bigint), _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                    value expressions: _wcol0 (type: bigint), _col5 (type: int)
                     auto parallelism: true
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: VALUE._col0 (type: bigint), KEY.reducesinkkey2 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col4 (type: int)
+                outputColumnNames: _col0, _col2, _col3, _col6
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -5353,7 +5380,9 @@ STAGE PLANS:
         Reducer 5 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int), VALUE._col5 (type: double)
+                outputColumnNames: _col1, _col2, _col5, _col7
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -5813,12 +5842,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string), _col1 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: true
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -6183,12 +6214,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: true
         Reducer 5 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -6521,12 +6554,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: true
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -6888,12 +6923,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string), _col1 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: true
         Reducer 5 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -7271,12 +7308,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string), _col1 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: true
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
@@ -7620,12 +7659,14 @@ STAGE PLANS:
                       Map-reduce partition columns: _col2 (type: string)
                       Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                       tag: -1
-                      value expressions: _col1 (type: string), _col2 (type: string), _col5 (type: int)
+                      value expressions: _col5 (type: int)
                       auto parallelism: true
         Reducer 4 
             Needs Tagging: false
             Reduce Operator Tree:
-              Extract
+              Select Operator
+                expressions: KEY.reducesinkkey1 (type: string), KEY.reducesinkkey0 (type: string), VALUE._col3 (type: int)
+                outputColumnNames: _col1, _col2, _col5
                 Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE
                 PTF Operator
                   Statistics: Num rows: 26 Data size: 16042 Basic stats: COMPLETE Column stats: NONE

Modified: hive/trunk/ql/src/test/results/clientpositive/union_remove_6_subq.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/union_remove_6_subq.q.out?rev=1653068&r1=1653067&r2=1653068&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/union_remove_6_subq.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/union_remove_6_subq.q.out Mon Jan 19 17:39:56 2015
@@ -544,9 +544,10 @@ STAGE PLANS:
               sort order: ++
               Map-reduce partition columns: _col0 (type: string)
               Statistics: Num rows: 250 Data size: 2656 Basic stats: COMPLETE Column stats: NONE
-              value expressions: _col0 (type: string), _col1 (type: bigint)
       Reduce Operator Tree:
-        Extract
+        Select Operator
+          expressions: KEY.reducesinkkey0 (type: string), KEY.reducesinkkey1 (type: bigint)
+          outputColumnNames: _col0, _col1
           Statistics: Num rows: 250 Data size: 2656 Basic stats: COMPLETE Column stats: NONE
           PTF Operator
             Statistics: Num rows: 250 Data size: 2656 Basic stats: COMPLETE Column stats: NONE