You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hive.apache.org by se...@apache.org on 2016/03/24 00:16:57 UTC

[3/3] hive git commit: HIVE-13283 : LLAP: make sure IO elevator is enabled by default in the daemons (Sergey Shelukhin, reviewed by Vikram Dixit K)

HIVE-13283 : LLAP: make sure IO elevator is enabled by default in the daemons (Sergey Shelukhin, reviewed by Vikram Dixit K)


Project: http://git-wip-us.apache.org/repos/asf/hive/repo
Commit: http://git-wip-us.apache.org/repos/asf/hive/commit/6dc362f0
Tree: http://git-wip-us.apache.org/repos/asf/hive/tree/6dc362f0
Diff: http://git-wip-us.apache.org/repos/asf/hive/diff/6dc362f0

Branch: refs/heads/master
Commit: 6dc362f0f1989452be39cf710911749ea8d1d199
Parents: 39d029e
Author: Sergey Shelukhin <se...@apache.org>
Authored: Wed Mar 23 16:16:33 2016 -0700
Committer: Sergey Shelukhin <se...@apache.org>
Committed: Wed Mar 23 16:16:33 2016 -0700

----------------------------------------------------------------------
 .../org/apache/hadoop/hive/conf/HiveConf.java   |  2 +-
 .../hive/llap/cli/LlapOptionsProcessor.java     |  1 -
 .../hive/llap/daemon/impl/LlapDaemon.java       |  4 +-
 .../hadoop/hive/ql/exec/tez/TezJobMonitor.java  |  2 +-
 .../hadoop/hive/ql/io/HiveInputFormat.java      | 15 ++--
 .../org/apache/hadoop/hive/ql/plan/MapWork.java |  4 +-
 .../llap/bucket_map_join_tez1.q.out             | 35 ++++++++
 .../llap/bucket_map_join_tez2.q.out             | 15 ++++
 .../results/clientpositive/llap/cte_3.q.out     |  4 +
 .../results/clientpositive/llap/cte_5.q.out     |  2 +
 .../results/clientpositive/llap/cte_mat_1.q.out |  2 +
 .../results/clientpositive/llap/cte_mat_2.q.out |  2 +
 .../results/clientpositive/llap/cte_mat_3.q.out |  3 +
 .../results/clientpositive/llap/cte_mat_4.q.out |  6 ++
 .../results/clientpositive/llap/cte_mat_5.q.out |  3 +
 .../llap/dynamic_partition_pruning.q.out        | 95 ++++++++++++++++++++
 .../llap/dynamic_partition_pruning_2.q.out      | 16 ++++
 .../llap/hybridgrace_hashjoin_1.q.out           | 24 +++++
 .../llap/hybridgrace_hashjoin_2.q.out           | 36 ++++++++
 .../results/clientpositive/llap/llap_udf.q.out  |  3 +
 .../clientpositive/llap/llapdecider.q.out       | 12 +++
 .../clientpositive/llap/lvj_mapjoin.q.out       |  2 +
 .../clientpositive/llap/mapjoin_decimal.q.out   |  2 +
 .../test/results/clientpositive/llap/mrr.q.out  | 10 +++
 .../llap/tez_bmj_schema_evolution.q.out         |  2 +
 .../results/clientpositive/llap/tez_dml.q.out   |  3 +
 .../llap/tez_dynpart_hashjoin_1.q.out           | 12 +++
 .../llap/tez_dynpart_hashjoin_2.q.out           |  9 ++
 .../results/clientpositive/llap/tez_join.q.out  |  2 +
 .../clientpositive/llap/tez_join_hash.q.out     |  6 ++
 .../llap/tez_join_result_complex.q.out          |  4 +
 .../clientpositive/llap/tez_join_tests.q.out    |  3 +
 .../clientpositive/llap/tez_joins_explain.q.out |  3 +
 .../clientpositive/llap/tez_self_join.q.out     |  3 +
 .../results/clientpositive/llap/tez_smb_1.q.out |  6 ++
 .../clientpositive/llap/tez_smb_main.q.out      | 24 +++++
 .../results/clientpositive/llap/tez_union.q.out | 28 ++++++
 .../clientpositive/llap/tez_union2.q.out        |  3 +
 .../llap/tez_union_dynamic_partition.q.out      |  2 +
 .../llap/tez_union_group_by.q.out               |  5 ++
 .../llap/tez_union_multiinsert.q.out            | 13 +++
 .../llap/tez_vector_dynpart_hashjoin_1.q.out    | 12 +++
 .../llap/tez_vector_dynpart_hashjoin_2.q.out    |  9 ++
 .../llap/vector_join_part_col_char.q.out        |  2 +
 .../vectorized_dynamic_partition_pruning.q.out  | 89 ++++++++++++++++++
 .../clientpositive/tez/llapdecider.q.out        | 12 +++
 .../common/io/encoded/EncodedColumnBatch.java   |  2 +-
 47 files changed, 537 insertions(+), 17 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/common/src/java/org/apache/hadoop/hive/conf/HiveConf.java
----------------------------------------------------------------------
diff --git a/common/src/java/org/apache/hadoop/hive/conf/HiveConf.java b/common/src/java/org/apache/hadoop/hive/conf/HiveConf.java
index c14df20..b8b9dcf 100644
--- a/common/src/java/org/apache/hadoop/hive/conf/HiveConf.java
+++ b/common/src/java/org/apache/hadoop/hive/conf/HiveConf.java
@@ -2522,7 +2522,7 @@ public class HiveConf extends Configuration {
         "hive.tez.exec.inplace.progress",
         true,
         "Updates tez job execution progress in-place in the terminal."),
-    LLAP_IO_ENABLED("hive.llap.io.enabled", false, "Whether the LLAP IO layer is enabled."),
+    LLAP_IO_ENABLED("hive.llap.io.enabled", null, "Whether the LLAP IO layer is enabled."),
     LLAP_IO_MEMORY_MODE("hive.llap.io.memory.mode", "cache",
         new StringSet("cache", "allocator", "none"),
         "LLAP IO memory usage; 'cache' (the default) uses data and metadata cache with a\n" +

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/llap-server/src/java/org/apache/hadoop/hive/llap/cli/LlapOptionsProcessor.java
----------------------------------------------------------------------
diff --git a/llap-server/src/java/org/apache/hadoop/hive/llap/cli/LlapOptionsProcessor.java b/llap-server/src/java/org/apache/hadoop/hive/llap/cli/LlapOptionsProcessor.java
index c292b37..b7f019c 100644
--- a/llap-server/src/java/org/apache/hadoop/hive/llap/cli/LlapOptionsProcessor.java
+++ b/llap-server/src/java/org/apache/hadoop/hive/llap/cli/LlapOptionsProcessor.java
@@ -235,7 +235,6 @@ public class LlapOptionsProcessor {
     String name = commandLine.getOptionValue(OPTION_NAME, null);
 
     final int executors = Integer.parseInt(commandLine.getOptionValue(OPTION_EXECUTORS, "-1"));
-    // TODO# here
     final int ioThreads = Integer.parseInt(
         commandLine.getOptionValue(OPTION_IO_THREADS, Integer.toString(executors)));
     final long cache = parseSuffixed(commandLine.getOptionValue(OPTION_CACHE, "-1"));

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/llap-server/src/java/org/apache/hadoop/hive/llap/daemon/impl/LlapDaemon.java
----------------------------------------------------------------------
diff --git a/llap-server/src/java/org/apache/hadoop/hive/llap/daemon/impl/LlapDaemon.java b/llap-server/src/java/org/apache/hadoop/hive/llap/daemon/impl/LlapDaemon.java
index ff9ce59..22d7eec 100644
--- a/llap-server/src/java/org/apache/hadoop/hive/llap/daemon/impl/LlapDaemon.java
+++ b/llap-server/src/java/org/apache/hadoop/hive/llap/daemon/impl/LlapDaemon.java
@@ -348,8 +348,8 @@ public class LlapDaemon extends CompositeService implements ContainerRunner, Lla
 
       long ioMemoryBytes = HiveConf.getSizeVar(daemonConf, ConfVars.LLAP_IO_MEMORY_MAX_SIZE);
       boolean isDirectCache = HiveConf.getBoolVar(daemonConf, ConfVars.LLAP_ALLOCATOR_DIRECT);
-      boolean llapIoEnabled = HiveConf.getBoolVar(daemonConf, HiveConf.ConfVars.LLAP_IO_ENABLED);
-      llapDaemon = new LlapDaemon(daemonConf, numExecutors, executorMemoryBytes, llapIoEnabled,
+      boolean isLlapIo = HiveConf.getBoolVar(daemonConf, HiveConf.ConfVars.LLAP_IO_ENABLED, true);
+      llapDaemon = new LlapDaemon(daemonConf, numExecutors, executorMemoryBytes, isLlapIo,
               isDirectCache, ioMemoryBytes, localDirs, rpcPort, mngPort, shufflePort);
 
       LOG.info("Adding shutdown hook for LlapDaemon");

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/ql/src/java/org/apache/hadoop/hive/ql/exec/tez/TezJobMonitor.java
----------------------------------------------------------------------
diff --git a/ql/src/java/org/apache/hadoop/hive/ql/exec/tez/TezJobMonitor.java b/ql/src/java/org/apache/hadoop/hive/ql/exec/tez/TezJobMonitor.java
index 67f9da8..7a7e2ac 100644
--- a/ql/src/java/org/apache/hadoop/hive/ql/exec/tez/TezJobMonitor.java
+++ b/ql/src/java/org/apache/hadoop/hive/ql/exec/tez/TezJobMonitor.java
@@ -225,7 +225,7 @@ public class TezJobMonitor {
     long startTime = 0;
     boolean isProfileEnabled = HiveConf.getBoolVar(conf, HiveConf.ConfVars.TEZ_EXEC_SUMMARY) ||
         Utilities.isPerfOrAboveLogging(conf);
-    boolean llapIoEnabled = HiveConf.getBoolVar(conf, HiveConf.ConfVars.LLAP_IO_ENABLED);
+    boolean llapIoEnabled = HiveConf.getBoolVar(conf, HiveConf.ConfVars.LLAP_IO_ENABLED, true);
 
     boolean inPlaceEligible = InPlaceUpdates.inPlaceEligible(conf);
     synchronized(shutdownList) {

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/ql/src/java/org/apache/hadoop/hive/ql/io/HiveInputFormat.java
----------------------------------------------------------------------
diff --git a/ql/src/java/org/apache/hadoop/hive/ql/io/HiveInputFormat.java b/ql/src/java/org/apache/hadoop/hive/ql/io/HiveInputFormat.java
index 1ed9b9c..46a5413 100755
--- a/ql/src/java/org/apache/hadoop/hive/ql/io/HiveInputFormat.java
+++ b/ql/src/java/org/apache/hadoop/hive/ql/io/HiveInputFormat.java
@@ -202,13 +202,14 @@ public class HiveInputFormat<K extends WritableComparable, V extends Writable>
 
   public static InputFormat<WritableComparable, Writable> wrapForLlap(
       InputFormat<WritableComparable, Writable> inputFormat, Configuration conf) {
-    if (!HiveConf.getBoolVar(conf, ConfVars.LLAP_IO_ENABLED)) {
+    if (!HiveConf.getBoolVar(conf, ConfVars.LLAP_IO_ENABLED, LlapProxy.isDaemon())) {
       return inputFormat; // LLAP not enabled, no-op.
     }
     boolean isSupported = inputFormat instanceof LlapWrappableInputFormatInterface,
-        isVector = Utilities.isVectorMode(conf);
-    if (!isSupported || !isVector) {
-      LOG.info("Not using llap for " + inputFormat + ": " + isSupported + ", " + isVector);
+        isVectorized = Utilities.isVectorMode(conf);
+    if (!isSupported || !isVectorized) {
+      LOG.info("Not using llap for " + inputFormat + ": supported = " + isSupported
+          + ", vectorized = " + isVectorized);
       return inputFormat;
     }
     if (LOG.isDebugEnabled()) {
@@ -223,11 +224,6 @@ public class HiveInputFormat<K extends WritableComparable, V extends Writable>
     return castInputFormat(llapIo.getInputFormat(inputFormat));
   }
 
-  public static boolean isLlapEnabled(Configuration conf) {
-    // Don't check IO - it needn't be initialized on client.
-    return HiveConf.getBoolVar(conf, ConfVars.LLAP_IO_ENABLED);
-  }
-
   public static boolean canWrapAnyForLlap(Configuration conf, MapWork mapWork) {
     return Utilities.isVectorMode(conf, mapWork);
   }
@@ -271,7 +267,6 @@ public class HiveInputFormat<K extends WritableComparable, V extends Writable>
   public RecordReader getRecordReader(InputSplit split, JobConf job,
       Reporter reporter) throws IOException {
     HiveInputSplit hsplit = (HiveInputSplit) split;
-
     InputSplit inputSplit = hsplit.getInputSplit();
     String inputFormatClassName = null;
     Class inputFormatClass = null;

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/ql/src/java/org/apache/hadoop/hive/ql/plan/MapWork.java
----------------------------------------------------------------------
diff --git a/ql/src/java/org/apache/hadoop/hive/ql/plan/MapWork.java b/ql/src/java/org/apache/hadoop/hive/ql/plan/MapWork.java
index 2da881d..0851d9e 100644
--- a/ql/src/java/org/apache/hadoop/hive/ql/plan/MapWork.java
+++ b/ql/src/java/org/apache/hadoop/hive/ql/plan/MapWork.java
@@ -34,6 +34,7 @@ import java.util.Set;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.hive.conf.HiveConf;
+import org.apache.hadoop.hive.conf.HiveConf.ConfVars;
 import org.apache.hadoop.hive.ql.exec.FileSinkOperator;
 import org.apache.hadoop.hive.ql.exec.Operator;
 import org.apache.hadoop.hive.ql.exec.OperatorUtils;
@@ -207,7 +208,8 @@ public class MapWork extends BaseWork {
 
   public void deriveLlap(Configuration conf) {
     boolean hasLlap = false, hasNonLlap = false, hasAcid = false;
-    boolean isLlapOn = HiveInputFormat.isLlapEnabled(conf),
+    // Assume the IO is enabled on the daemon by default. We cannot reasonably check it here.
+    boolean isLlapOn = HiveConf.getBoolVar(conf, ConfVars.LLAP_IO_ENABLED, llapMode),
         canWrapAny = isLlapOn && HiveInputFormat.canWrapAnyForLlap(conf, this);
     boolean hasPathToPartInfo = (pathToPartitionInfo != null && !pathToPartitionInfo.isEmpty());
     if (canWrapAny && hasPathToPartInfo) {

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/ql/src/test/results/clientpositive/llap/bucket_map_join_tez1.q.out
----------------------------------------------------------------------
diff --git a/ql/src/test/results/clientpositive/llap/bucket_map_join_tez1.q.out b/ql/src/test/results/clientpositive/llap/bucket_map_join_tez1.q.out
index f5a036e..21cfa5c 100644
--- a/ql/src/test/results/clientpositive/llap/bucket_map_join_tez1.q.out
+++ b/ql/src/test/results/clientpositive/llap/bucket_map_join_tez1.q.out
@@ -147,6 +147,7 @@ STAGE PLANS:
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -182,6 +183,7 @@ STAGE PLANS:
                                 output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                                 serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-0
     Fetch Operator
@@ -231,6 +233,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: int), _col1 (type: string)
                         Statistics: Num rows: 500 Data size: 5312 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -249,6 +252,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: int)
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -373,6 +377,7 @@ STAGE PLANS:
                           Statistics: Num rows: 27 Data size: 2808 Basic stats: COMPLETE Column stats: NONE
                           value expressions: _col1 (type: double)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -408,6 +413,7 @@ STAGE PLANS:
                                 output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                                 serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -497,6 +503,7 @@ STAGE PLANS:
                               Statistics: Num rows: 550 Data size: 5843 Basic stats: COMPLETE Column stats: NONE
                               value expressions: _col1 (type: double)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -516,6 +523,7 @@ STAGE PLANS:
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -535,6 +543,7 @@ STAGE PLANS:
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -640,6 +649,7 @@ STAGE PLANS:
                               Statistics: Num rows: 266 Data size: 2822 Basic stats: COMPLETE Column stats: NONE
                               value expressions: _col1 (type: double)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -658,6 +668,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: int)
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -677,6 +688,7 @@ STAGE PLANS:
                         Statistics: Num rows: 500 Data size: 5312 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -779,6 +791,7 @@ STAGE PLANS:
                                 output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                                 serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -798,6 +811,7 @@ STAGE PLANS:
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -816,6 +830,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: int)
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-0
     Fetch Operator
@@ -889,6 +904,7 @@ STAGE PLANS:
                                   output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                                   serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -907,6 +923,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: int)
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -926,6 +943,7 @@ STAGE PLANS:
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-0
     Fetch Operator
@@ -983,6 +1001,7 @@ STAGE PLANS:
                           Statistics: Num rows: 27 Data size: 2808 Basic stats: COMPLETE Column stats: NONE
                           value expressions: _col1 (type: double)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -1018,6 +1037,7 @@ STAGE PLANS:
                                 output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                                 serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -1082,6 +1102,7 @@ STAGE PLANS:
                         Statistics: Num rows: 27 Data size: 2808 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -1117,6 +1138,7 @@ STAGE PLANS:
                                 output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                                 serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -1180,6 +1202,7 @@ STAGE PLANS:
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col0 (type: int)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -1211,6 +1234,7 @@ STAGE PLANS:
                               output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                               serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-0
     Fetch Operator
@@ -1279,6 +1303,7 @@ STAGE PLANS:
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -1314,6 +1339,7 @@ STAGE PLANS:
                                 output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                                 serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-0
     Fetch Operator
@@ -1383,6 +1409,7 @@ STAGE PLANS:
                                   output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                                   serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -1401,6 +1428,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: int)
                         Statistics: Num rows: 500 Data size: 5312 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -1420,6 +1448,7 @@ STAGE PLANS:
                         Statistics: Num rows: 500 Data size: 5312 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col0 (type: int)
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-0
     Fetch Operator
@@ -1481,6 +1510,7 @@ STAGE PLANS:
                             Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 2
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -1516,6 +1546,7 @@ STAGE PLANS:
                                 output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                                 serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-0
     Fetch Operator
@@ -1583,6 +1614,7 @@ STAGE PLANS:
                           sort order: 
                           Statistics: Num rows: 550 Data size: 5843 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -1601,6 +1633,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: int)
                         Statistics: Num rows: 1 Data size: 11 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -1700,6 +1733,7 @@ STAGE PLANS:
                           sort order: 
                           Statistics: Num rows: 550 Data size: 5843 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -1718,6 +1752,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: int)
                         Statistics: Num rows: 1 Data size: 11 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/ql/src/test/results/clientpositive/llap/bucket_map_join_tez2.q.out
----------------------------------------------------------------------
diff --git a/ql/src/test/results/clientpositive/llap/bucket_map_join_tez2.q.out b/ql/src/test/results/clientpositive/llap/bucket_map_join_tez2.q.out
index 2800ee5..86ffb45 100644
--- a/ql/src/test/results/clientpositive/llap/bucket_map_join_tez2.q.out
+++ b/ql/src/test/results/clientpositive/llap/bucket_map_join_tez2.q.out
@@ -170,6 +170,7 @@ STAGE PLANS:
                                   output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                                   serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -188,6 +189,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: int)
                         Statistics: Num rows: 500 Data size: 5312 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -207,6 +209,7 @@ STAGE PLANS:
                         Statistics: Num rows: 500 Data size: 5312 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col0 (type: int)
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-0
     Fetch Operator
@@ -275,6 +278,7 @@ STAGE PLANS:
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col0 (type: int), _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -310,6 +314,7 @@ STAGE PLANS:
                                 output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                                 serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-0
     Fetch Operator
@@ -366,6 +371,7 @@ STAGE PLANS:
                               output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                               serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -384,6 +390,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: int)
                         Statistics: Num rows: 55 Data size: 584 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-0
     Fetch Operator
@@ -440,6 +447,7 @@ STAGE PLANS:
                               output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                               serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -458,6 +466,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: int)
                         Statistics: Num rows: 55 Data size: 584 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-0
     Fetch Operator
@@ -501,6 +510,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: int)
                         Statistics: Num rows: 55 Data size: 584 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -532,6 +542,7 @@ STAGE PLANS:
                               output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                               serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-0
     Fetch Operator
@@ -574,6 +585,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: int)
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -592,6 +604,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: int)
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -660,6 +673,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -679,6 +693,7 @@ STAGE PLANS:
                         Statistics: Num rows: 242 Data size: 2566 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/ql/src/test/results/clientpositive/llap/cte_3.q.out
----------------------------------------------------------------------
diff --git a/ql/src/test/results/clientpositive/llap/cte_3.q.out b/ql/src/test/results/clientpositive/llap/cte_3.q.out
index 37796c0..82f6414 100644
--- a/ql/src/test/results/clientpositive/llap/cte_3.q.out
+++ b/ql/src/test/results/clientpositive/llap/cte_3.q.out
@@ -40,6 +40,7 @@ STAGE PLANS:
                             serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
                             name: default.q1
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-2
     Dependency Collection
@@ -124,6 +125,7 @@ STAGE PLANS:
                             serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
                             name: default.q1
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-2
     Dependency Collection
@@ -211,6 +213,7 @@ STAGE PLANS:
                             serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
                             name: default.q2
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-2
     Dependency Collection
@@ -240,6 +243,7 @@ STAGE PLANS:
                             serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
                             name: default.q1
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-5
     Dependency Collection

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/ql/src/test/results/clientpositive/llap/cte_5.q.out
----------------------------------------------------------------------
diff --git a/ql/src/test/results/clientpositive/llap/cte_5.q.out b/ql/src/test/results/clientpositive/llap/cte_5.q.out
index eb6d5da..9f9b718 100644
--- a/ql/src/test/results/clientpositive/llap/cte_5.q.out
+++ b/ql/src/test/results/clientpositive/llap/cte_5.q.out
@@ -106,6 +106,7 @@ STAGE PLANS:
                         Statistics: Num rows: 1 Data size: 3 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col0 (type: int)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -122,6 +123,7 @@ STAGE PLANS:
                         Map-reduce partition columns: 5.0 (type: double)
                         Statistics: Num rows: 250 Data size: 2656 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/ql/src/test/results/clientpositive/llap/cte_mat_1.q.out
----------------------------------------------------------------------
diff --git a/ql/src/test/results/clientpositive/llap/cte_mat_1.q.out b/ql/src/test/results/clientpositive/llap/cte_mat_1.q.out
index d8c74b5..9ba574d 100644
--- a/ql/src/test/results/clientpositive/llap/cte_mat_1.q.out
+++ b/ql/src/test/results/clientpositive/llap/cte_mat_1.q.out
@@ -38,6 +38,7 @@ STAGE PLANS:
                         Map-reduce partition columns: '5' (type: string)
                         Statistics: Num rows: 250 Data size: 2656 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -54,6 +55,7 @@ STAGE PLANS:
                         Map-reduce partition columns: '5' (type: string)
                         Statistics: Num rows: 250 Data size: 2656 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/ql/src/test/results/clientpositive/llap/cte_mat_2.q.out
----------------------------------------------------------------------
diff --git a/ql/src/test/results/clientpositive/llap/cte_mat_2.q.out b/ql/src/test/results/clientpositive/llap/cte_mat_2.q.out
index d8c74b5..9ba574d 100644
--- a/ql/src/test/results/clientpositive/llap/cte_mat_2.q.out
+++ b/ql/src/test/results/clientpositive/llap/cte_mat_2.q.out
@@ -38,6 +38,7 @@ STAGE PLANS:
                         Map-reduce partition columns: '5' (type: string)
                         Statistics: Num rows: 250 Data size: 2656 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -54,6 +55,7 @@ STAGE PLANS:
                         Map-reduce partition columns: '5' (type: string)
                         Statistics: Num rows: 250 Data size: 2656 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/ql/src/test/results/clientpositive/llap/cte_mat_3.q.out
----------------------------------------------------------------------
diff --git a/ql/src/test/results/clientpositive/llap/cte_mat_3.q.out b/ql/src/test/results/clientpositive/llap/cte_mat_3.q.out
index 3c38c21..ff188ad 100644
--- a/ql/src/test/results/clientpositive/llap/cte_mat_3.q.out
+++ b/ql/src/test/results/clientpositive/llap/cte_mat_3.q.out
@@ -43,6 +43,7 @@ STAGE PLANS:
                             serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
                             name: default.q1
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-2
     Dependency Collection
@@ -72,6 +73,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 1 Data size: 0 Basic stats: PARTIAL Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -90,6 +92,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 1 Data size: 0 Basic stats: PARTIAL Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 3 
             Execution mode: llap
             Reduce Operator Tree:

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/ql/src/test/results/clientpositive/llap/cte_mat_4.q.out
----------------------------------------------------------------------
diff --git a/ql/src/test/results/clientpositive/llap/cte_mat_4.q.out b/ql/src/test/results/clientpositive/llap/cte_mat_4.q.out
index 3245b72..2671d9f 100644
--- a/ql/src/test/results/clientpositive/llap/cte_mat_4.q.out
+++ b/ql/src/test/results/clientpositive/llap/cte_mat_4.q.out
@@ -84,6 +84,7 @@ STAGE PLANS:
                             serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
                             name: default.q1
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-2
     Dependency Collection
@@ -113,6 +114,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 1 Data size: 0 Basic stats: PARTIAL Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -131,6 +133,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 1 Data size: 0 Basic stats: PARTIAL Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 3 
             Execution mode: llap
             Reduce Operator Tree:
@@ -298,6 +301,7 @@ STAGE PLANS:
                             serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
                             name: default.q1
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-2
     Dependency Collection
@@ -327,6 +331,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 1 Data size: 0 Basic stats: PARTIAL Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -345,6 +350,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 1 Data size: 0 Basic stats: PARTIAL Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 3 
             Execution mode: llap
             Reduce Operator Tree:

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/ql/src/test/results/clientpositive/llap/cte_mat_5.q.out
----------------------------------------------------------------------
diff --git a/ql/src/test/results/clientpositive/llap/cte_mat_5.q.out b/ql/src/test/results/clientpositive/llap/cte_mat_5.q.out
index 44371c2..ee5bac1 100644
--- a/ql/src/test/results/clientpositive/llap/cte_mat_5.q.out
+++ b/ql/src/test/results/clientpositive/llap/cte_mat_5.q.out
@@ -108,6 +108,7 @@ STAGE PLANS:
                             serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
                             name: default.q1
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-2
     Dependency Collection
@@ -138,6 +139,7 @@ STAGE PLANS:
                         Statistics: Num rows: 1 Data size: 3 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col0 (type: int)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -156,6 +158,7 @@ STAGE PLANS:
                         Map-reduce partition columns: UDFToDouble(_col0) (type: double)
                         Statistics: Num rows: 1 Data size: 0 Basic stats: PARTIAL Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 3 
             Execution mode: llap
             Reduce Operator Tree:

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/ql/src/test/results/clientpositive/llap/dynamic_partition_pruning.q.out
----------------------------------------------------------------------
diff --git a/ql/src/test/results/clientpositive/llap/dynamic_partition_pruning.q.out b/ql/src/test/results/clientpositive/llap/dynamic_partition_pruning.q.out
index 05f05aa..c5a0aea 100644
--- a/ql/src/test/results/clientpositive/llap/dynamic_partition_pruning.q.out
+++ b/ql/src/test/results/clientpositive/llap/dynamic_partition_pruning.q.out
@@ -73,6 +73,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -232,6 +233,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -266,6 +268,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -359,6 +362,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -378,6 +382,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -486,6 +491,7 @@ STAGE PLANS:
                       Map-reduce partition columns: day(_col0) (type: int)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -520,6 +526,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -613,6 +620,7 @@ STAGE PLANS:
                       Map-reduce partition columns: day(_col0) (type: int)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -632,6 +640,7 @@ STAGE PLANS:
                         Map-reduce partition columns: day(_col0) (type: int)
                         Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -730,6 +739,7 @@ STAGE PLANS:
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
                       value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 5 
             Map Operator Tree:
                 TableScan
@@ -764,6 +774,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Map 6 
             Map Operator Tree:
                 TableScan
@@ -798,6 +809,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 5 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -915,6 +927,7 @@ STAGE PLANS:
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
                       value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 5 
             Map Operator Tree:
                 TableScan
@@ -934,6 +947,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 6 
             Map Operator Tree:
                 TableScan
@@ -953,6 +967,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 1 Data size: 5 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -1078,6 +1093,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string), _col1 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -1127,6 +1143,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 27 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -1220,6 +1237,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string), _col1 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -1239,6 +1257,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string), _col2 (type: string)
                         Statistics: Num rows: 1 Data size: 27 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -1345,6 +1364,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -1379,6 +1399,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -1472,6 +1493,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -1491,6 +1513,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -1595,6 +1618,7 @@ STAGE PLANS:
                       Map-reduce partition columns: UDFToDouble(_col0) (type: double)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -1629,6 +1653,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 7 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -1722,6 +1747,7 @@ STAGE PLANS:
                       Map-reduce partition columns: (UDFToDouble(_col0) * UDFToDouble(2)) (type: double)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -1756,6 +1782,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 7 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -1849,6 +1876,7 @@ STAGE PLANS:
                       Map-reduce partition columns: UDFToDouble(_col0) (type: double)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -1868,6 +1896,7 @@ STAGE PLANS:
                         Map-reduce partition columns: UDFToDouble(UDFToInteger((_col0 / UDFToDouble(2)))) (type: double)
                         Statistics: Num rows: 1 Data size: 7 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -1961,6 +1990,7 @@ STAGE PLANS:
                       Map-reduce partition columns: (UDFToDouble(_col0) * UDFToDouble(2)) (type: double)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -1980,6 +2010,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: double)
                         Statistics: Num rows: 1 Data size: 7 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -2086,6 +2117,7 @@ STAGE PLANS:
                       Map-reduce partition columns: UDFToString((UDFToDouble(_col0) * UDFToDouble(2))) (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -2120,6 +2152,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 7 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -2229,6 +2262,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -2246,6 +2280,7 @@ STAGE PLANS:
                       Map-reduce partition columns: '2008-04-08' (type: string)
                       Statistics: Num rows: 1000 Data size: 10624 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -2381,6 +2416,7 @@ STAGE PLANS:
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
                       value expressions: _col0 (type: string), _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -2399,6 +2435,7 @@ STAGE PLANS:
                         Statistics: Num rows: 1 Data size: 27 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col0 (type: string), _col2 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -2500,6 +2537,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string), _col1 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -2549,6 +2587,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 27 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -2644,6 +2683,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -2678,6 +2718,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -2770,6 +2811,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 4
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -2785,6 +2827,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -2860,6 +2903,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -2894,6 +2938,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -2973,6 +3018,7 @@ STAGE PLANS:
                       Statistics: Num rows: 1000 Data size: 10624 Basic stats: COMPLETE Column stats: NONE
                       value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 5 
             Map Operator Tree:
                 TableScan
@@ -3007,6 +3053,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Map 6 
             Map Operator Tree:
                 TableScan
@@ -3026,6 +3073,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 1 Data size: 5 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -3142,6 +3190,7 @@ STAGE PLANS:
                         Statistics: Num rows: 1 Data size: 0 Basic stats: PARTIAL Column stats: NONE
                         value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 5 
             Map Operator Tree:
                 TableScan
@@ -3161,6 +3210,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 6 
             Map Operator Tree:
                 TableScan
@@ -3180,6 +3230,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 1 Data size: 5 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -3289,6 +3340,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -3308,6 +3360,7 @@ STAGE PLANS:
                         Statistics: Num rows: 1 Data size: 84 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col0 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 7 
             Map Operator Tree:
                 TableScan
@@ -3327,6 +3380,7 @@ STAGE PLANS:
                         Statistics: Num rows: 1 Data size: 84 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col0 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -3494,6 +3548,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -3513,6 +3568,7 @@ STAGE PLANS:
                         Statistics: Num rows: 1 Data size: 84 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col0 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 7 
             Map Operator Tree:
                 TableScan
@@ -3532,6 +3588,7 @@ STAGE PLANS:
                         Statistics: Num rows: 1 Data size: 84 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col0 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -3704,6 +3761,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 10 
             Map Operator Tree:
                 TableScan
@@ -3723,6 +3781,7 @@ STAGE PLANS:
                         Statistics: Num rows: 1 Data size: 84 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col0 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 5 
             Map Operator Tree:
                 TableScan
@@ -3740,6 +3799,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 7 
             Map Operator Tree:
                 TableScan
@@ -3759,6 +3819,7 @@ STAGE PLANS:
                         Statistics: Num rows: 1 Data size: 84 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col0 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 11 
             Execution mode: llap
             Reduce Operator Tree:
@@ -3985,6 +4046,7 @@ STAGE PLANS:
                           Statistics: Num rows: 1 Data size: 8 Basic stats: COMPLETE Column stats: NONE
                           value expressions: _col0 (type: bigint)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -4019,6 +4081,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -4122,6 +4185,7 @@ STAGE PLANS:
                           Statistics: Num rows: 1 Data size: 8 Basic stats: COMPLETE Column stats: NONE
                           value expressions: _col0 (type: bigint)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -4156,6 +4220,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -4258,6 +4323,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 8 Basic stats: COMPLETE Column stats: NONE
                             value expressions: _col0 (type: bigint)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -4292,6 +4358,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -4326,6 +4393,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 5 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -4430,6 +4498,7 @@ STAGE PLANS:
                           Statistics: Num rows: 1 Data size: 8 Basic stats: COMPLETE Column stats: NONE
                           value expressions: _col0 (type: bigint)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -4479,6 +4548,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 27 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -4580,6 +4650,7 @@ STAGE PLANS:
                           Statistics: Num rows: 1 Data size: 8 Basic stats: COMPLETE Column stats: NONE
                           value expressions: _col0 (type: bigint)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -4614,6 +4685,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -4691,6 +4763,7 @@ STAGE PLANS:
                           Statistics: Num rows: 1 Data size: 8 Basic stats: COMPLETE Column stats: NONE
                           value expressions: _col0 (type: bigint)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -4725,6 +4798,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 7 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -4813,6 +4887,7 @@ STAGE PLANS:
                           Statistics: Num rows: 1 Data size: 8 Basic stats: COMPLETE Column stats: NONE
                           value expressions: _col0 (type: bigint)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -4847,6 +4922,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 7 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -4951,6 +5027,7 @@ STAGE PLANS:
                           Statistics: Num rows: 1 Data size: 8 Basic stats: COMPLETE Column stats: NONE
                           value expressions: _col0 (type: bigint)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -4968,6 +5045,7 @@ STAGE PLANS:
                       Map-reduce partition columns: '2008-04-08' (type: string)
                       Statistics: Num rows: 1000 Data size: 10624 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -5099,6 +5177,7 @@ STAGE PLANS:
                           Statistics: Num rows: 1 Data size: 8 Basic stats: COMPLETE Column stats: NONE
                           value expressions: _col0 (type: bigint)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -5133,6 +5212,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -5205,6 +5285,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 8 Basic stats: COMPLETE Column stats: NONE
                             value expressions: _col0 (type: bigint)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -5220,6 +5301,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -5276,6 +5358,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -5309,6 +5392,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 8 Basic stats: COMPLETE Column stats: NONE
                             value expressions: _col0 (type: bigint)
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 3 
             Execution mode: llap
             Reduce Operator Tree:
@@ -5392,6 +5476,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 8 Basic stats: COMPLETE Column stats: NONE
                             value expressions: _col0 (type: bigint)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -5426,6 +5511,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 21 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -5445,6 +5531,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 1 Data size: 5 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -5525,6 +5612,7 @@ STAGE PLANS:
                         Statistics: Num rows: 1 Data size: 0 Basic stats: PARTIAL Column stats: NONE
                         value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -5569,6 +5657,7 @@ STAGE PLANS:
                               Statistics: Num rows: 1 Data size: 8 Basic stats: COMPLETE Column stats: NONE
                               value expressions: _col0 (type: bigint)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -5588,6 +5677,7 @@ STAGE PLANS:
                         Map-reduce partition columns: _col0 (type: string)
                         Statistics: Num rows: 1 Data size: 5 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 3 
             Execution mode: llap
             Reduce Operator Tree:
@@ -5671,6 +5761,7 @@ STAGE PLANS:
                           Map-reduce partition columns: _col0 (type: string)
                           Statistics: Num rows: 2200 Data size: 23372 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -5690,6 +5781,7 @@ STAGE PLANS:
                         Statistics: Num rows: 1 Data size: 84 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col0 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 6 
             Map Operator Tree:
                 TableScan
@@ -5709,6 +5801,7 @@ STAGE PLANS:
                         Statistics: Num rows: 1 Data size: 84 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col0 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -5906,6 +5999,7 @@ STAGE PLANS:
                           Statistics: Num rows: 1 Data size: 8 Basic stats: COMPLETE Column stats: NONE
                           value expressions: _col0 (type: bigint)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -5955,6 +6049,7 @@ STAGE PLANS:
                             Statistics: Num rows: 2 Data size: 54 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:

http://git-wip-us.apache.org/repos/asf/hive/blob/6dc362f0/ql/src/test/results/clientpositive/llap/dynamic_partition_pruning_2.q.out
----------------------------------------------------------------------
diff --git a/ql/src/test/results/clientpositive/llap/dynamic_partition_pruning_2.q.out b/ql/src/test/results/clientpositive/llap/dynamic_partition_pruning_2.q.out
index 67c9aa5..6f93b6a 100644
--- a/ql/src/test/results/clientpositive/llap/dynamic_partition_pruning_2.q.out
+++ b/ql/src/test/results/clientpositive/llap/dynamic_partition_pruning_2.q.out
@@ -199,6 +199,7 @@ STAGE PLANS:
                             Statistics: Num rows: 9 Data size: 29 Basic stats: COMPLETE Column stats: NONE
                             value expressions: _col1 (type: bigint), _col2 (type: decimal(20,0))
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -234,6 +235,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 5 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -371,6 +373,7 @@ STAGE PLANS:
                             Statistics: Num rows: 9 Data size: 29 Basic stats: COMPLETE Column stats: NONE
                             value expressions: _col1 (type: bigint), _col2 (type: decimal(20,0))
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -391,6 +394,7 @@ STAGE PLANS:
                         Statistics: Num rows: 1 Data size: 5 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -513,6 +517,7 @@ STAGE PLANS:
                               output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                               serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -533,6 +538,7 @@ STAGE PLANS:
                         Statistics: Num rows: 3 Data size: 15 Basic stats: COMPLETE Column stats: NONE
                         value expressions: _col1 (type: string)
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-0
     Fetch Operator
@@ -624,6 +630,7 @@ STAGE PLANS:
                             output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                             serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
         Map 2 
             Map Operator Tree:
                 TableScan
@@ -639,6 +646,7 @@ STAGE PLANS:
                         sort order: 
                         Statistics: Num rows: 1 Data size: 5 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
 
   Stage: Stage-0
     Fetch Operator
@@ -740,6 +748,7 @@ STAGE PLANS:
                             Statistics: Num rows: 9 Data size: 29 Basic stats: COMPLETE Column stats: NONE
                             value expressions: _col1 (type: bigint), _col2 (type: decimal(20,0))
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -775,6 +784,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 5 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree:
@@ -894,6 +904,7 @@ STAGE PLANS:
                             output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                             serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -928,6 +939,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 5 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 1
             Execution mode: llap
+            LLAP IO: no inputs
         Map 4 
             Map Operator Tree:
                 TableScan
@@ -957,6 +969,7 @@ STAGE PLANS:
                             output format: org.apache.hadoop.hive.ql.io.HiveSequenceFileOutputFormat
                             serde: org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe
             Execution mode: llap
+            LLAP IO: no inputs
         Map 5 
             Map Operator Tree:
                 TableScan
@@ -991,6 +1004,7 @@ STAGE PLANS:
                             Statistics: Num rows: 1 Data size: 5 Basic stats: COMPLETE Column stats: NONE
                             Target Vertex: Map 4
             Execution mode: llap
+            LLAP IO: no inputs
         Union 2 
             Vertex: Union 2
 
@@ -1085,6 +1099,7 @@ STAGE PLANS:
                           Statistics: Num rows: 1 Data size: 8 Basic stats: COMPLETE Column stats: NONE
                           value expressions: _col0 (type: bigint)
             Execution mode: llap
+            LLAP IO: no inputs
         Map 3 
             Map Operator Tree:
                 TableScan
@@ -1101,6 +1116,7 @@ STAGE PLANS:
                       Map-reduce partition columns: _col0 (type: string)
                       Statistics: Num rows: 2000 Data size: 21248 Basic stats: COMPLETE Column stats: NONE
             Execution mode: llap
+            LLAP IO: no inputs
         Reducer 2 
             Execution mode: llap
             Reduce Operator Tree: