merge from zheilbron/hyracks_msr
diff --git a/hivesterix/hivesterix-dist/src/main/assembly/binary-assembly.xml b/hivesterix/hivesterix-dist/src/main/assembly/binary-assembly.xml
index aeb3fb4..e2da26a 100755
--- a/hivesterix/hivesterix-dist/src/main/assembly/binary-assembly.xml
+++ b/hivesterix/hivesterix-dist/src/main/assembly/binary-assembly.xml
@@ -1,17 +1,12 @@
-<!--
- ! Copyright 2009-2013 by The Regents of the University of California
- ! Licensed under the Apache License, Version 2.0 (the "License");
- ! you may not use this file except in compliance with the License.
- ! you may obtain a copy of the License from
- ! 
- !     http://www.apache.org/licenses/LICENSE-2.0
- ! 
- ! Unless required by applicable law or agreed to in writing, software
- ! distributed under the License is distributed on an "AS IS" BASIS,
- ! WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- ! See the License for the specific language governing permissions and
- ! limitations under the License.
- !-->
+<!-- ! Copyright 2009-2013 by The Regents of the University of California 
+	! Licensed under the Apache License, Version 2.0 (the "License"); ! you may 
+	not use this file except in compliance with the License. ! you may obtain 
+	a copy of the License from ! ! http://www.apache.org/licenses/LICENSE-2.0 
+	! ! Unless required by applicable law or agreed to in writing, software ! 
+	distributed under the License is distributed on an "AS IS" BASIS, ! WITHOUT 
+	WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. ! See the 
+	License for the specific language governing permissions and ! limitations 
+	under the License. ! -->
 <assembly>
 	<id>binary-assembly</id>
 	<formats>
@@ -21,20 +16,30 @@
 	<includeBaseDirectory>false</includeBaseDirectory>
 	<fileSets>
 		<fileSet>
-			<directory>target/appassembler/bin</directory>
+			<directory>src/main/resources/conf</directory>
+			<outputDirectory>conf</outputDirectory>
+			<fileMode>0755</fileMode>
+		</fileSet>
+		<fileSet>
+			<directory>src/main/resources/scripts</directory>
 			<outputDirectory>bin</outputDirectory>
 			<fileMode>0755</fileMode>
 		</fileSet>
 		<fileSet>
 			<directory>target/appassembler/lib</directory>
 			<outputDirectory>lib</outputDirectory>
+			<includes>
+				<include>*.jar</include>
+			</includes>
+			<fileMode>0755</fileMode>
 		</fileSet>
 		<fileSet>
 			<directory>target</directory>
 			<outputDirectory>lib</outputDirectory>
 			<includes>
-				<include>*.jar</include>
+				<include>a-hive-patch.jar</include>
 			</includes>
+			<fileMode>0755</fileMode>
 		</fileSet>
 	</fileSets>
-</assembly>
+</assembly>
\ No newline at end of file
diff --git a/hivesterix/hivesterix-dist/src/main/java/edu/uci/ics/hivesterix/runtime/exec/HyracksExecutionEngine.java b/hivesterix/hivesterix-dist/src/main/java/edu/uci/ics/hivesterix/runtime/exec/HyracksExecutionEngine.java
index d3bcaca..7b88de4 100644
--- a/hivesterix/hivesterix-dist/src/main/java/edu/uci/ics/hivesterix/runtime/exec/HyracksExecutionEngine.java
+++ b/hivesterix/hivesterix-dist/src/main/java/edu/uci/ics/hivesterix/runtime/exec/HyracksExecutionEngine.java
@@ -36,16 +36,20 @@
 import org.apache.hadoop.hive.ql.exec.ConditionalTask;
 import org.apache.hadoop.hive.ql.exec.FileSinkOperator;
 import org.apache.hadoop.hive.ql.exec.MapRedTask;
+import org.apache.hadoop.hive.ql.exec.MoveTask;
 import org.apache.hadoop.hive.ql.exec.Operator;
 import org.apache.hadoop.hive.ql.exec.TableScanOperator;
 import org.apache.hadoop.hive.ql.exec.Task;
+import org.apache.hadoop.hive.ql.exec.Utilities;
 import org.apache.hadoop.hive.ql.plan.DynamicPartitionCtx;
 import org.apache.hadoop.hive.ql.plan.FetchWork;
 import org.apache.hadoop.hive.ql.plan.FileSinkDesc;
 import org.apache.hadoop.hive.ql.plan.MapredLocalWork;
 import org.apache.hadoop.hive.ql.plan.MapredWork;
+import org.apache.hadoop.hive.ql.plan.OperatorDesc;
 import org.apache.hadoop.hive.ql.plan.PartitionDesc;
 import org.apache.hadoop.hive.ql.plan.TableScanDesc;
+import org.apache.hadoop.mapred.Reporter;
 
 import edu.uci.ics.hivesterix.common.config.ConfUtil;
 import edu.uci.ics.hivesterix.logical.expression.HiveExpressionTypeComputer;
@@ -214,7 +218,6 @@
 
         // get all leave Ops
         getLeaves(rootOps, leaveOps);
-
         HiveAlgebricksTranslator translator = new HiveAlgebricksTranslator();
         try {
             translator.translate(rootOps, null, aliasToPath);
@@ -222,7 +225,7 @@
             ILogicalPlan plan = translator.genLogicalPlan();
 
             if (plan.getRoots() != null && plan.getRoots().size() > 0 && plan.getRoots().get(0).getValue() != null) {
-                translator.printOperators();
+                //translator.printOperators();
                 ILogicalPlanAndMetadata planAndMetadata = new HiveLogicalPlanAndMetaData(plan,
                         translator.getMetadataProvider());
 
@@ -238,7 +241,7 @@
                 StringBuilder buffer = new StringBuilder();
                 PlanPrettyPrinter.printPlan(plan, buffer, pvisitor, 0);
                 String planStr = buffer.toString();
-                System.out.println(planStr);
+                LOG.info(planStr);
 
                 if (planPrinter != null)
                     planPrinter.print(planStr);
@@ -377,6 +380,12 @@
                 // remove map-reduce branches in condition task
                 ConditionalTask condition = (ConditionalTask) task;
                 List<Task<? extends Serializable>> branches = condition.getListTasks();
+                for (Task branch : branches) {
+                    if (branch instanceof MoveTask) {
+                        //return articulateMapReduceOperators(branch, rootOps, aliasToPath, rootTasks);
+                        return null;
+                    }
+                }
                 for (int i = branches.size() - 1; i >= 0; i--) {
                     Task branch = branches.get(i);
                     if (branch instanceof MapRedTask) {
@@ -396,7 +405,7 @@
 
         MapRedTask mrtask = (MapRedTask) task;
         MapredWork work = (MapredWork) mrtask.getWork();
-        HashMap<String, Operator<? extends Serializable>> operators = work.getAliasToWork();
+        HashMap<String, Operator<? extends OperatorDesc>> operators = work.getAliasToWork();
 
         Set entries = operators.entrySet();
         Iterator<Entry<String, Operator>> iterator = entries.iterator();
@@ -414,7 +423,7 @@
         // get map local work
         MapredLocalWork localWork = work.getMapLocalWork();
         if (localWork != null) {
-            HashMap<String, Operator<? extends Serializable>> localOperators = localWork.getAliasToWork();
+            HashMap<String, Operator<? extends OperatorDesc>> localOperators = localWork.getAliasToWork();
 
             Set localEntries = localOperators.entrySet();
             Iterator<Entry<String, Operator>> localIterator = localEntries.iterator();
@@ -479,9 +488,9 @@
                 for (Operator childMap : childMapOps) {
                     if (childMap instanceof TableScanOperator) {
                         TableScanDesc topDesc = (TableScanDesc) childMap.getConf();
-                        if (topDesc == null)
+                        if (topDesc == null || topDesc.getAlias() == null) {
                             mapChildren.add(childMap);
-                        else {
+                        } else {
                             rootOps.add(childMap);
                         }
                     } else {
@@ -501,9 +510,14 @@
                 }
                 i = 0;
                 for (Operator child : mapChildren) {
-                    if (child.getParentOperators() == null || child.getParentOperators().size() == 0)
+                    if (child.getParentOperators() == null || child.getParentOperators().size() == 0) {
                         child.setParentOperators(new ArrayList<Operator>());
-                    child.getParentOperators().add(leafs.get(i));
+                    }
+                    if (i < leafs.size()) {
+                        if (child.getParentOperators().size()==0) {
+                            child.getParentOperators().add(leafs.get(i));
+                        }
+                    }
                     i++;
                 }
             }
@@ -603,10 +617,10 @@
             String specPath = desc.getDirName();
             DynamicPartitionCtx dpCtx = desc.getDynPartCtx();
             // for 0.7.0
-            fsOp.mvFileToFinalPath(specPath, conf, true, LOG, dpCtx);
+            //fsOp.mvFileToFinalPath(specPath, conf, true, LOG, dpCtx);
             // for 0.8.0
-            // Utilities.mvFileToFinalPath(specPath, conf, true, LOG, dpCtx,
-            // desc);
+            //Utilities.mvFileToFinalPath(specPath, conf, true, LOG, dpCtx, desc);
+            Utilities.mvFileToFinalPath(specPath, conf, true, LOG, dpCtx, desc, Reporter.NULL);
         }
     }
 }
diff --git a/hivesterix/hivesterix-dist/src/main/java/org/apache/hadoop/hive/ql/Driver.java b/hivesterix/hivesterix-dist/src/main/java/org/apache/hadoop/hive/ql/Driver.java
index 4ef74e9..64a3f12 100644
--- a/hivesterix/hivesterix-dist/src/main/java/org/apache/hadoop/hive/ql/Driver.java
+++ b/hivesterix/hivesterix-dist/src/main/java/org/apache/hadoop/hive/ql/Driver.java
@@ -1,17 +1,3 @@
-/*
- * Copyright 2009-2013 by The Regents of the University of California
- * Licensed under the Apache License, Version 2.0 (the "License");
- * you may not use this file except in compliance with the License.
- * you may obtain a copy of the License from
- * 
- *     http://www.apache.org/licenses/LICENSE-2.0
- * 
- * Unless required by applicable law or agreed to in writing, software
- * distributed under the License is distributed on an "AS IS" BASIS,
- * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- * See the License for the specific language governing permissions and
- * limitations under the License.
- */
 /**
  * Licensed to the Apache Software Foundation (ASF) under one
  * or more contributor license agreements.  See the NOTICE file
@@ -42,11 +28,13 @@
 import java.util.HashMap;
 import java.util.HashSet;
 import java.util.Iterator;
+import java.util.LinkedHashMap;
 import java.util.LinkedList;
 import java.util.List;
 import java.util.Map;
 import java.util.Queue;
 import java.util.Set;
+import java.util.concurrent.ConcurrentLinkedQueue;
 
 import org.apache.commons.lang.StringUtils;
 import org.apache.commons.logging.Log;
@@ -62,6 +50,7 @@
 import org.apache.hadoop.hive.ql.exec.ExecDriver;
 import org.apache.hadoop.hive.ql.exec.FetchTask;
 import org.apache.hadoop.hive.ql.exec.MapRedTask;
+import org.apache.hadoop.hive.ql.exec.MoveTask;
 import org.apache.hadoop.hive.ql.exec.Operator;
 import org.apache.hadoop.hive.ql.exec.StatsTask;
 import org.apache.hadoop.hive.ql.exec.TableScanOperator;
@@ -86,23 +75,22 @@
 import org.apache.hadoop.hive.ql.lockmgr.HiveLockObject;
 import org.apache.hadoop.hive.ql.lockmgr.HiveLockObject.HiveLockObjectData;
 import org.apache.hadoop.hive.ql.lockmgr.LockException;
+import org.apache.hadoop.hive.ql.log.PerfLogger;
 import org.apache.hadoop.hive.ql.metadata.AuthorizationException;
 import org.apache.hadoop.hive.ql.metadata.DummyPartition;
 import org.apache.hadoop.hive.ql.metadata.Hive;
 import org.apache.hadoop.hive.ql.metadata.HiveException;
-import org.apache.hadoop.hive.ql.metadata.HiveUtils;
 import org.apache.hadoop.hive.ql.metadata.Partition;
 import org.apache.hadoop.hive.ql.metadata.Table;
 import org.apache.hadoop.hive.ql.optimizer.ppr.PartitionPruner;
 import org.apache.hadoop.hive.ql.parse.ASTNode;
 import org.apache.hadoop.hive.ql.parse.AbstractSemanticAnalyzerHook;
 import org.apache.hadoop.hive.ql.parse.BaseSemanticAnalyzer;
-import org.apache.hadoop.hive.ql.parse.ErrorMsg;
 import org.apache.hadoop.hive.ql.parse.HiveSemanticAnalyzerHookContext;
 import org.apache.hadoop.hive.ql.parse.HiveSemanticAnalyzerHookContextImpl;
+import org.apache.hadoop.hive.ql.parse.ImportSemanticAnalyzer;
 import org.apache.hadoop.hive.ql.parse.ParseContext;
 import org.apache.hadoop.hive.ql.parse.ParseDriver;
-import org.apache.hadoop.hive.ql.parse.ParseException;
 import org.apache.hadoop.hive.ql.parse.ParseUtils;
 import org.apache.hadoop.hive.ql.parse.PrunedPartitionList;
 import org.apache.hadoop.hive.ql.parse.SemanticAnalyzer;
@@ -112,6 +100,7 @@
 import org.apache.hadoop.hive.ql.plan.ConditionalResolver;
 import org.apache.hadoop.hive.ql.plan.ConditionalResolverMergeFiles;
 import org.apache.hadoop.hive.ql.plan.HiveOperation;
+import org.apache.hadoop.hive.ql.plan.OperatorDesc;
 import org.apache.hadoop.hive.ql.plan.TableDesc;
 import org.apache.hadoop.hive.ql.processors.CommandProcessor;
 import org.apache.hadoop.hive.ql.processors.CommandProcessorResponse;
@@ -127,15 +116,18 @@
 import edu.uci.ics.hivesterix.runtime.exec.HyracksExecutionEngine;
 import edu.uci.ics.hivesterix.runtime.exec.IExecutionEngine;
 
-@SuppressWarnings({ "deprecation", "unused" })
+@SuppressWarnings({ "deprecation", "unchecked", "rawtypes" })
 public class Driver implements CommandProcessor {
 
+    // hivesterix
+    private IExecutionEngine engine;
+    private boolean hivesterix = false;
+    private Set<Task> executedConditionalTsks = new HashSet<Task>();
+
     static final private Log LOG = LogFactory.getLog(Driver.class.getName());
     static final private LogHelper console = new LogHelper(LOG);
 
-    // hive-sterix
-    private IExecutionEngine engine;
-    private boolean hivesterix = false;
+    private static final Object compileMonitor = new Object();
 
     private int maxRows = 100;
     ByteStream.Output bos = new ByteStream.Output();
@@ -152,23 +144,57 @@
 
     // A limit on the number of threads that can be launched
     private int maxthreads;
-    private final int sleeptime = 2000;
-
+    private static final int SLEEP_TIME = 2000;
     protected int tryCount = Integer.MAX_VALUE;
 
-    private int checkLockManager() {
+    /**
+     * for backwards compatibility with current tests
+     */
+    public Driver(HiveConf conf) {
+        this.conf = conf;
+
+    }
+
+    public Driver() {
+        if (SessionState.get() != null) {
+            conf = SessionState.get().getConf();
+        }
+
+        // hivesterix
+        engine = new HyracksExecutionEngine(conf);
+    }
+
+    // hivesterix: plan printer
+    public Driver(HiveConf conf, PrintWriter planPrinter) {
+        this.conf = conf;
+        engine = new HyracksExecutionEngine(conf, planPrinter);
+    }
+
+    public void clear() {
+        this.hivesterix = false;
+        this.executedConditionalTsks.clear();
+    }
+
+    private boolean checkLockManager() {
         boolean supportConcurrency = conf.getBoolVar(HiveConf.ConfVars.HIVE_SUPPORT_CONCURRENCY);
-        if (supportConcurrency && (hiveLockMgr == null)) {
+        if (!supportConcurrency) {
+            return false;
+        }
+        if ((hiveLockMgr == null)) {
             try {
                 setLockManager();
             } catch (SemanticException e) {
                 errorMessage = "FAILED: Error in semantic analysis: " + e.getMessage();
                 SQLState = ErrorMsg.findSQLState(e.getMessage());
                 console.printError(errorMessage, "\n" + org.apache.hadoop.util.StringUtils.stringifyException(e));
-                return (12);
+                return false;
             }
         }
-        return (0);
+        // the reason that we set the lock manager for the cxt here is because each
+        // query has its own ctx object. The hiveLockMgr is shared accross the
+        // same instance of Driver, which can run multiple queries.
+        ctx.setHiveLockMgr(hiveLockMgr);
+        return hiveLockMgr != null;
     }
 
     private void setLockManager() throws SemanticException {
@@ -183,6 +209,16 @@
                 hiveLockMgr = (HiveLockManager) ReflectionUtils.newInstance(conf.getClassByName(lockMgr), conf);
                 hiveLockMgr.setContext(new HiveLockManagerCtx(conf));
             } catch (Exception e) {
+                // set hiveLockMgr to null just in case this invalid manager got set to
+                // next query's ctx.
+                if (hiveLockMgr != null) {
+                    try {
+                        hiveLockMgr.close();
+                    } catch (LockException e1) {
+                        //nothing can do here
+                    }
+                    hiveLockMgr = null;
+                }
                 throw new SemanticException(ErrorMsg.LOCKMGR_NOT_INITIALIZED.getMsg() + e.getMessage());
             }
         }
@@ -230,10 +266,8 @@
         } else if (sem.getFetchTask() != null) {
             FetchTask ft = sem.getFetchTask();
             TableDesc td = ft.getTblDesc();
-            // partitioned tables don't have tableDesc set on the FetchTask.
-            // Instead
-            // they have a list of PartitionDesc objects, each with a table
-            // desc.
+            // partitioned tables don't have tableDesc set on the FetchTask. Instead
+            // they have a list of PartitionDesc objects, each with a table desc.
             // Let's
             // try to fetch the desc for the first partition and use it's
             // deserializer.
@@ -320,59 +354,102 @@
     }
 
     /**
-     * for backwards compatibility with current tests
-     */
-    public Driver(HiveConf conf) {
-        this.conf = conf;
-
-        // hivesterix
-        engine = new HyracksExecutionEngine(conf);
-    }
-
-    public Driver() {
-        if (SessionState.get() != null) {
-            conf = SessionState.get().getConf();
-        }
-
-        // hivesterix
-        engine = new HyracksExecutionEngine(conf);
-    }
-
-    // hivesterix: plan printer
-    public Driver(HiveConf conf, PrintWriter planPrinter) {
-        this.conf = conf;
-        engine = new HyracksExecutionEngine(conf, planPrinter);
-    }
-
-    public void clear() {
-        this.hivesterix = false;
-    }
-
-    /**
-     * Compile a new query. Any currently-planned query associated with this
-     * Driver is discarded.
+     * Compile a new query. Any currently-planned query associated with this Driver is discarded.
      * 
      * @param command
      *            The SQL query to compile.
      */
     public int compile(String command) {
+        return compile(command, true);
+    }
+
+    /**
+     * Hold state variables specific to each query being executed, that may not
+     * be consistent in the overall SessionState
+     */
+    private static class QueryState {
+        private HiveOperation op;
+        private String cmd;
+        private boolean init = false;
+
+        /**
+         * Initialize the queryState with the query state variables
+         */
+        public void init(HiveOperation op, String cmd) {
+            this.op = op;
+            this.cmd = cmd;
+            this.init = true;
+        }
+
+        public boolean isInitialized() {
+            return this.init;
+        }
+
+        public HiveOperation getOp() {
+            return this.op;
+        }
+
+        public String getCmd() {
+            return this.cmd;
+        }
+    }
+
+    public void saveSession(QueryState qs) {
+        SessionState oldss = SessionState.get();
+        if (oldss != null && oldss.getHiveOperation() != null) {
+            qs.init(oldss.getHiveOperation(), oldss.getCmd());
+        }
+    }
+
+    public void restoreSession(QueryState qs) {
+        SessionState ss = SessionState.get();
+        if (ss != null && qs != null && qs.isInitialized()) {
+            ss.setCmd(qs.getCmd());
+            ss.setCommandType(qs.getOp());
+        }
+    }
+
+    /**
+     * Compile a new query, but potentially reset taskID counter. Not resetting task counter
+     * is useful for generating re-entrant QL queries.
+     * 
+     * @param command
+     *            The HiveQL query to compile
+     * @param resetTaskIds
+     *            Resets taskID counter if true.
+     * @return 0 for ok
+     */
+    public int compile(String command, boolean resetTaskIds) {
+        PerfLogger perfLogger = PerfLogger.getPerfLogger();
+        perfLogger.PerfLogBegin(LOG, PerfLogger.COMPILE);
+
+        //holder for parent command type/string when executing reentrant queries
+        QueryState queryState = new QueryState();
+
         if (plan != null) {
             close();
             plan = null;
         }
 
-        TaskFactory.resetId();
+        if (resetTaskIds) {
+            TaskFactory.resetId();
+        }
+        saveSession(queryState);
 
         try {
             command = new VariableSubstitution().substitute(conf, command);
             ctx = new Context(conf);
+            ctx.setTryCount(getTryCount());
+            ctx.setCmd(command);
+            ctx.setHDFSCleanup(true);
 
             ParseDriver pd = new ParseDriver();
             ASTNode tree = pd.parse(command, ctx);
             tree = ParseUtils.findRootNonNullToken(tree);
 
             BaseSemanticAnalyzer sem = SemanticAnalyzerFactory.get(conf, tree);
-            List<AbstractSemanticAnalyzerHook> saHooks = getSemanticAnalyzerHooks();
+            List<AbstractSemanticAnalyzerHook> saHooks = getHooks(HiveConf.ConfVars.SEMANTIC_ANALYZER_HOOK,
+                    AbstractSemanticAnalyzerHook.class);
 
             // Do semantic analysis and plan generation
             if (saHooks != null) {
@@ -382,6 +459,7 @@
                     tree = hook.preAnalyze(hookCtx, tree);
                 }
                 sem.analyze(tree, ctx);
+                hookCtx.update(sem);
                 for (AbstractSemanticAnalyzerHook hook : saHooks) {
                     hook.postAnalyze(hookCtx, sem.getRootTasks());
                 }
@@ -394,19 +472,10 @@
             // validate the plan
             sem.validate();
 
-            plan = new QueryPlan(command, sem);
-            // initialize FetchTask right here
-            if (plan.getFetchTask() != null) {
-                plan.getFetchTask().initialize(conf, plan, null);
-            }
-
-            // get the output schema
-            schema = getSchema(sem, conf);
+            plan = new QueryPlan(command, sem, perfLogger.getStartTime(PerfLogger.DRIVER_RUN));
 
             // test Only - serialize the query plan and deserialize it
-            if (sem instanceof SemanticAnalyzer && command.toLowerCase().indexOf("create") < 0) {
-
-                Thread.currentThread().setContextClassLoader(this.getClass().getClassLoader());
+            if ("true".equalsIgnoreCase(System.getProperty("test.serialize.qplan"))) {
 
                 String queryPlanFileName = ctx.getLocalScratchDir(true) + Path.SEPARATOR_CHAR + "queryplan.xml";
                 LOG.info("query plan = " + queryPlanFileName);
@@ -431,17 +500,24 @@
                 plan.getFetchTask().initialize(conf, plan, null);
             }
 
-            // do the authorization check
+            // get the output schema
+            schema = getSchema(sem, conf);
+
+            //do the authorization check
             if (HiveConf.getBoolVar(conf, HiveConf.ConfVars.HIVE_AUTHORIZATION_ENABLED)) {
                 try {
-                    // doAuthorization(sem);
+                    perfLogger.PerfLogBegin(LOG, PerfLogger.DO_AUTHORIZATION);
+                    doAuthorization(sem);
                 } catch (AuthorizationException authExp) {
                     console.printError("Authorization failed:" + authExp.getMessage()
                             + ". Use show grant to get more details.");
                     return 403;
+                } finally {
+                    perfLogger.PerfLogEnd(LOG, PerfLogger.DO_AUTHORIZATION);
                 }
             }
 
+            //restore state after we're done executing a specific query
             // hyracks run
             if (sem instanceof SemanticAnalyzer && command.toLowerCase().indexOf("create") < 0) {
                 int engineRet = engine.compileJob(sem.getRootTasks());
@@ -450,21 +526,19 @@
                 }
             }
             return 0;
-        } catch (SemanticException e) {
-            errorMessage = "FAILED: Error in semantic analysis: " + e.getMessage();
-            SQLState = ErrorMsg.findSQLState(e.getMessage());
-            console.printError(errorMessage, "\n" + org.apache.hadoop.util.StringUtils.stringifyException(e));
-            return (10);
-        } catch (ParseException e) {
-            errorMessage = "FAILED: Parse Error: " + e.getMessage();
-            SQLState = ErrorMsg.findSQLState(e.getMessage());
-            console.printError(errorMessage, "\n" + org.apache.hadoop.util.StringUtils.stringifyException(e));
-            return (11);
         } catch (Exception e) {
-            errorMessage = "FAILED: Hive Internal Error: " + Utilities.getNameMessage(e);
-            SQLState = ErrorMsg.findSQLState(e.getMessage());
-            console.printError(errorMessage + "\n" + org.apache.hadoop.util.StringUtils.stringifyException(e));
-            return (12);
+            ErrorMsg error = ErrorMsg.getErrorMsg(e.getMessage());
+            errorMessage = "FAILED: " + e.getClass().getSimpleName();
+            if (error != ErrorMsg.GENERIC_ERROR) {
+                errorMessage += " [Error " + error.getErrorCode() + "]:";
+            }
+            errorMessage += " " + e.getMessage();
+            SQLState = error.getSQLState();
+            console.printError(errorMessage, "\n" + org.apache.hadoop.util.StringUtils.stringifyException(e));
+            return error.getErrorCode();
+        } finally {
+            perfLogger.PerfLogEnd(LOG, PerfLogger.COMPILE);
+            restoreSession(queryState);
         }
     }
 
@@ -479,13 +553,13 @@
                 ss.getAuthorizer().authorize(db.getDatabase(db.getCurrentDatabase()), null,
                         HiveOperation.CREATETABLE_AS_SELECT.getOutputRequiredPrivileges());
             } else {
-                // if (op.equals(HiveOperation.IMPORT)) {
-                // ImportSemanticAnalyzer isa = (ImportSemanticAnalyzer) sem;
-                // if (!isa.existsTable()) {
-                ss.getAuthorizer().authorize(db.getDatabase(db.getCurrentDatabase()), null,
-                        HiveOperation.CREATETABLE_AS_SELECT.getOutputRequiredPrivileges());
-                // }
-                // }
+                if (op.equals(HiveOperation.IMPORT)) {
+                    ImportSemanticAnalyzer isa = (ImportSemanticAnalyzer) sem;
+                    if (!isa.existsTable()) {
+                        ss.getAuthorizer().authorize(db.getDatabase(db.getCurrentDatabase()), null,
+                                HiveOperation.CREATETABLE_AS_SELECT.getOutputRequiredPrivileges());
+                    }
+                }
             }
             if (outputs != null && outputs.size() > 0) {
                 for (WriteEntity write : outputs) {
@@ -513,8 +587,8 @@
 
             Map<String, Boolean> tableUsePartLevelAuth = new HashMap<String, Boolean>();
             for (ReadEntity read : inputs) {
-                if (read.getPartition() != null) {
-                    Table tbl = read.getTable();
+                Table tbl = read.getTable();
+                if ((read.getPartition() != null) || (tbl.isPartitioned())) {
                     String tblName = tbl.getTableName();
                     if (tableUsePartLevelAuth.get(tblName) == null) {
                         boolean usePartLevelPriv = (tbl.getParameters().get("PARTITION_LEVEL_PRIVILEGE") != null && ("TRUE"
@@ -533,9 +607,9 @@
                 ParseContext parseCtx = querySem.getParseContext();
                 Map<TableScanOperator, Table> tsoTopMap = parseCtx.getTopToTable();
 
-                for (Map.Entry<String, Operator<? extends Serializable>> topOpMap : querySem.getParseContext()
+                for (Map.Entry<String, Operator<? extends OperatorDesc>> topOpMap : querySem.getParseContext()
                         .getTopOps().entrySet()) {
-                    Operator<? extends Serializable> topOp = topOpMap.getValue();
+                    Operator<? extends OperatorDesc> topOp = topOpMap.getValue();
                     if (topOp instanceof TableScanOperator && tsoTopMap.containsKey(topOp)) {
                         TableScanOperator tableScanOp = (TableScanOperator) topOp;
                         Table tbl = tsoTopMap.get(tableScanOp);
@@ -551,7 +625,10 @@
                                 cols.add(columns.get(i).getName());
                             }
                         }
-                        if (tbl.isPartitioned() && tableUsePartLevelAuth.get(tbl.getTableName())) {
+                        //map may not contain all sources, since input list may have been optimized out
+                        //or non-existent tho such sources may still be referenced by the TableScanOperator
+                        //if it's null then the partition probably doesn't exist so let's use table permission
+                        if (tbl.isPartitioned() && tableUsePartLevelAuth.get(tbl.getTableName()) == Boolean.TRUE) {
                             String alias_id = topOpMap.getKey();
                             PrunedPartitionList partsList = PartitionPruner.prune(parseCtx.getTopToTable().get(topOp),
                                     parseCtx.getOpToPartPruner().get(topOp), parseCtx.getConf(), alias_id,
@@ -582,30 +659,28 @@
             // cache the results for table authorization
             Set<String> tableAuthChecked = new HashSet<String>();
             for (ReadEntity read : inputs) {
-                Table tbl = null;
+                Table tbl = read.getTable();
                 if (read.getPartition() != null) {
-                    tbl = read.getPartition().getTable();
+                    Partition partition = read.getPartition();
+                    tbl = partition.getTable();
                     // use partition level authorization
-                    if (tableUsePartLevelAuth.get(tbl.getTableName())) {
-                        List<String> cols = part2Cols.get(read.getPartition());
+                    if (tableUsePartLevelAuth.get(tbl.getTableName()) == Boolean.TRUE) {
+                        List<String> cols = part2Cols.get(partition);
                         if (cols != null && cols.size() > 0) {
-                            ss.getAuthorizer().authorize(read.getPartition().getTable(), read.getPartition(), cols,
+                            ss.getAuthorizer().authorize(partition.getTable(), partition, cols,
                                     op.getInputRequiredPrivileges(), null);
                         } else {
-                            ss.getAuthorizer().authorize(read.getPartition(), op.getInputRequiredPrivileges(), null);
+                            ss.getAuthorizer().authorize(partition, op.getInputRequiredPrivileges(), null);
                         }
                         continue;
                     }
-                } else if (read.getTable() != null) {
-                    tbl = read.getTable();
                 }
 
-                // if we reach here, it means it needs to do a table
-                // authorization
-                // check, and the table authorization may already happened
-                // because of other
+                // if we reach here, it means it needs to do a table authorization
+                // check, and the table authorization may already happened because of other
                 // partitions
-                if (tbl != null && !tableAuthChecked.contains(tbl.getTableName())) {
+                if (tbl != null && !tableAuthChecked.contains(tbl.getTableName())
+                        && !(tableUsePartLevelAuth.get(tbl.getTableName()) == Boolean.TRUE)) {
                     List<String> cols = tab2Cols.get(tbl);
                     if (cols != null && cols.size() > 0) {
                         ss.getAuthorizer().authorize(tbl, null, cols, op.getInputRequiredPrivileges(), null);
@@ -632,16 +707,15 @@
      * @param p
      *            The partition to be locked
      * @param mode
-     *            The mode of the lock (SHARED/EXCLUSIVE) Get the list of
-     *            objects to be locked. If a partition needs to be locked (in
-     *            any mode), all its parents should also be locked in SHARED
-     *            mode.
+     *            The mode of the lock (SHARED/EXCLUSIVE) Get the list of objects to be locked. If a
+     *            partition needs to be locked (in any mode), all its parents should also be locked in
+     *            SHARED mode.
      **/
     private List<HiveLockObj> getLockObjects(Table t, Partition p, HiveLockMode mode) throws SemanticException {
         List<HiveLockObj> locks = new LinkedList<HiveLockObj>();
 
         HiveLockObjectData lockData = new HiveLockObjectData(plan.getQueryId(), String.valueOf(System
-                .currentTimeMillis()), "IMPLICIT");
+                .currentTimeMillis()), "IMPLICIT", plan.getQueryStr());
 
         if (t != null) {
             locks.add(new HiveLockObj(new HiveLockObject(t, lockData), mode));
@@ -665,16 +739,20 @@
                 name = p.getName().split("@")[2];
             }
 
-            String partName = name;
             String partialName = "";
             String[] partns = name.split("/");
             int len = p instanceof DummyPartition ? partns.length : partns.length - 1;
+            Map<String, String> partialSpec = new LinkedHashMap<String, String>();
             for (int idx = 0; idx < len; idx++) {
                 String partn = partns[idx];
                 partialName += partn;
+                String[] nameValue = partn.split("=");
+                assert (nameValue.length == 2);
+                partialSpec.put(nameValue[0], nameValue[1]);
                 try {
                     locks.add(new HiveLockObj(new HiveLockObject(new DummyPartition(p.getTable(), p.getTable()
-                            .getDbName() + "/" + p.getTable().getTableName() + "/" + partialName), lockData), mode));
+                            .getDbName() + "/" + p.getTable().getTableName() + "/" + partialName, partialSpec),
+                            lockData), mode));
                     partialName += "/";
                 } catch (HiveException e) {
                     throw new SemanticException(e.getMessage());
@@ -688,17 +766,16 @@
     }
 
     /**
-     * Acquire read and write locks needed by the statement. The list of objects
-     * to be locked are obtained from he inputs and outputs populated by the
-     * compiler. The lock acuisition scheme is pretty simple. If all the locks
-     * cannot be obtained, error out. Deadlock is avoided by making sure that
-     * the locks are lexicographically sorted.
+     * Acquire read and write locks needed by the statement. The list of objects to be locked are
+     * obtained from he inputs and outputs populated by the compiler. The lock acuisition scheme is
+     * pretty simple. If all the locks cannot be obtained, error out. Deadlock is avoided by making
+     * sure that the locks are lexicographically sorted.
      **/
     public int acquireReadWriteLocks() {
-        try {
-            int sleepTime = conf.getIntVar(HiveConf.ConfVars.HIVE_LOCK_SLEEP_BETWEEN_RETRIES) * 1000;
-            int numRetries = conf.getIntVar(HiveConf.ConfVars.HIVE_LOCK_NUMRETRIES);
+        PerfLogger perfLogger = PerfLogger.getPerfLogger();
+        perfLogger.PerfLogBegin(LOG, PerfLogger.ACQUIRE_READ_WRITE_LOCKS);
 
+        try {
             boolean supportConcurrency = conf.getBoolVar(HiveConf.ConfVars.HIVE_SUPPORT_CONCURRENCY);
             if (!supportConcurrency) {
                 return 0;
@@ -707,8 +784,7 @@
             List<HiveLockObj> lockObjects = new ArrayList<HiveLockObj>();
 
             // Sort all the inputs, outputs.
-            // If a lock needs to be acquired on any partition, a read lock
-            // needs to be acquired on all
+            // If a lock needs to be acquired on any partition, a read lock needs to be acquired on all
             // its parents also
             for (ReadEntity input : plan.getInputs()) {
                 if (input.getType() == ReadEntity.Type.TABLE) {
@@ -719,16 +795,21 @@
             }
 
             for (WriteEntity output : plan.getOutputs()) {
+                List<HiveLockObj> lockObj = null;
                 if (output.getTyp() == WriteEntity.Type.TABLE) {
-                    lockObjects.addAll(getLockObjects(output.getTable(), null,
-                            output.isComplete() ? HiveLockMode.EXCLUSIVE : HiveLockMode.SHARED));
+                    lockObj = getLockObjects(output.getTable(), null, output.isComplete() ? HiveLockMode.EXCLUSIVE
+                            : HiveLockMode.SHARED);
                 } else if (output.getTyp() == WriteEntity.Type.PARTITION) {
-                    lockObjects.addAll(getLockObjects(null, output.getPartition(), HiveLockMode.EXCLUSIVE));
+                    lockObj = getLockObjects(null, output.getPartition(), HiveLockMode.EXCLUSIVE);
                 }
-                // In case of dynamic queries, it is possible to have incomplete
-                // dummy partitions
+                // In case of dynamic queries, it is possible to have incomplete dummy partitions
                 else if (output.getTyp() == WriteEntity.Type.DUMMYPARTITION) {
-                    lockObjects.addAll(getLockObjects(null, output.getPartition(), HiveLockMode.SHARED));
+                    lockObj = getLockObjects(null, output.getPartition(), HiveLockMode.SHARED);
+                }
+
+                if (lockObj != null) {
+                    lockObjects.addAll(lockObj);
+                    ctx.getOutputLockObjects().put(output, lockObj);
                 }
             }
 
@@ -736,13 +817,8 @@
                 return 0;
             }
 
-            int ret = checkLockManager();
-            if (ret != 0) {
-                return ret;
-            }
-
             HiveLockObjectData lockData = new HiveLockObjectData(plan.getQueryId(), String.valueOf(System
-                    .currentTimeMillis()), "IMPLICIT");
+                    .currentTimeMillis()), "IMPLICIT", plan.getQueryStr());
 
             // Lock the database also
             try {
@@ -753,25 +829,7 @@
                 throw new SemanticException(e.getMessage());
             }
 
-            ctx.setHiveLockMgr(hiveLockMgr);
-            List<HiveLock> hiveLocks = null;
-
-            int tryNum = 1;
-            do {
-
-                // ctx.getHiveLockMgr();
-                // hiveLocks = ctx.getHiveLockMgr().lock(lockObjects, false);
-
-                if (hiveLocks != null) {
-                    break;
-                }
-
-                tryNum++;
-                try {
-                    Thread.sleep(sleepTime);
-                } catch (InterruptedException e) {
-                }
-            } while (tryNum < numRetries);
+            List<HiveLock> hiveLocks = ctx.getHiveLockMgr().lock(lockObjects, false);
 
             if (hiveLocks == null) {
                 throw new SemanticException(ErrorMsg.LOCK_CANNOT_BE_ACQUIRED.getMsg());
@@ -785,138 +843,207 @@
             SQLState = ErrorMsg.findSQLState(e.getMessage());
             console.printError(errorMessage, "\n" + org.apache.hadoop.util.StringUtils.stringifyException(e));
             return (10);
-        } catch (Exception e) {
+        } catch (LockException e) {
             errorMessage = "FAILED: Error in acquiring locks: " + e.getMessage();
             SQLState = ErrorMsg.findSQLState(e.getMessage());
             console.printError(errorMessage, "\n" + org.apache.hadoop.util.StringUtils.stringifyException(e));
             return (10);
-        }
-    }
-
-    /**
-     * Release all the locks acquired implicitly by the statement. Note that the
-     * locks acquired with 'keepAlive' set to True are not released.
-     **/
-    private void releaseLocks() {
-        if (ctx != null && ctx.getHiveLockMgr() != null) {
-            try {
-                ctx.getHiveLockMgr().close();
-                ctx.setHiveLocks(null);
-            } catch (LockException e) {
-            }
+        } finally {
+            perfLogger.PerfLogEnd(LOG, PerfLogger.ACQUIRE_READ_WRITE_LOCKS);
         }
     }
 
     /**
      * @param hiveLocks
-     *            list of hive locks to be released Release all the locks
-     *            specified. If some of the locks have already been released,
-     *            ignore them
+     *            list of hive locks to be released Release all the locks specified. If some of the
+     *            locks have already been released, ignore them
      **/
     private void releaseLocks(List<HiveLock> hiveLocks) {
+        PerfLogger perfLogger = PerfLogger.getPerfLogger();
+        perfLogger.PerfLogBegin(LOG, PerfLogger.RELEASE_LOCKS);
+
         if (hiveLocks != null) {
             ctx.getHiveLockMgr().releaseLocks(hiveLocks);
         }
         ctx.setHiveLocks(null);
+
+        perfLogger.PerfLogEnd(LOG, PerfLogger.RELEASE_LOCKS);
     }
 
-    public CommandProcessorResponse run(String command) {
+    public CommandProcessorResponse run(String command) throws CommandNeedRetryException {
         errorMessage = null;
         SQLState = null;
 
-        int ret = compile(command);
+        if (!validateConfVariables()) {
+            return new CommandProcessorResponse(12, errorMessage, SQLState);
+        }
+
+        HiveDriverRunHookContext hookContext = new HiveDriverRunHookContextImpl(conf, command);
+        // Get all the driver run hooks and pre-execute them.
+        List<HiveDriverRunHook> driverRunHooks;
+        try {
+            driverRunHooks = getHooks(HiveConf.ConfVars.HIVE_DRIVER_RUN_HOOKS, HiveDriverRunHook.class);
+            for (HiveDriverRunHook driverRunHook : driverRunHooks) {
+                driverRunHook.preDriverRun(hookContext);
+            }
+        } catch (Exception e) {
+            errorMessage = "FAILED: Hive Internal Error: " + Utilities.getNameMessage(e);
+            SQLState = ErrorMsg.findSQLState(e.getMessage());
+            console.printError(errorMessage + "\n" + org.apache.hadoop.util.StringUtils.stringifyException(e));
+            return new CommandProcessorResponse(12, errorMessage, SQLState);
+        }
+
+        // Reset the perf logger
+        PerfLogger perfLogger = PerfLogger.getPerfLogger(true);
+        perfLogger.PerfLogBegin(LOG, PerfLogger.DRIVER_RUN);
+        perfLogger.PerfLogBegin(LOG, PerfLogger.TIME_TO_SUBMIT);
+
+        int ret;
+        synchronized (compileMonitor) {
+            ret = compile(command);
+        }
         if (ret != 0) {
-            // releaseLocks(ctx.getHiveLocks());
+            releaseLocks(ctx.getHiveLocks());
             return new CommandProcessorResponse(ret, errorMessage, SQLState);
         }
 
-        // ret = acquireReadWriteLocks();
-        if (ret != 0) {
-            // releaseLocks(ctx.getHiveLocks());
-            return new CommandProcessorResponse(ret, errorMessage, SQLState);
+        boolean requireLock = false;
+        boolean ckLock = checkLockManager();
+
+        if (ckLock) {
+            boolean lockOnlyMapred = HiveConf.getBoolVar(conf, HiveConf.ConfVars.HIVE_LOCK_MAPRED_ONLY);
+            if (lockOnlyMapred) {
+                Queue<Task<? extends Serializable>> taskQueue = new LinkedList<Task<? extends Serializable>>();
+                taskQueue.addAll(plan.getRootTasks());
+                while (taskQueue.peek() != null) {
+                    Task<? extends Serializable> tsk = taskQueue.remove();
+                    requireLock = requireLock || tsk.requireLock();
+                    if (requireLock) {
+                        break;
+                    }
+                    if (tsk instanceof ConditionalTask) {
+                        taskQueue.addAll(((ConditionalTask) tsk).getListTasks());
+                    }
+                    if (tsk.getChildTasks() != null) {
+                        taskQueue.addAll(tsk.getChildTasks());
+                    }
+                    // does not add back up task here, because back up task should be the same
+                    // type of the original task.
+                }
+            } else {
+                requireLock = true;
+            }
+        }
+
+        if (requireLock) {
+            ret = acquireReadWriteLocks();
+            if (ret != 0) {
+                releaseLocks(ctx.getHiveLocks());
+                return new CommandProcessorResponse(ret, errorMessage, SQLState);
+            }
         }
 
         ret = execute();
         if (ret != 0) {
-            // releaseLocks(ctx.getHiveLocks());
+            //if needRequireLock is false, the release here will do nothing because there is no lock
+            releaseLocks(ctx.getHiveLocks());
             return new CommandProcessorResponse(ret, errorMessage, SQLState);
         }
 
-        // releaseLocks(ctx.getHiveLocks());
+        //if needRequireLock is false, the release here will do nothing because there is no lock
+        releaseLocks(ctx.getHiveLocks());
+
+        perfLogger.PerfLogEnd(LOG, PerfLogger.DRIVER_RUN);
+        perfLogger.close(LOG, plan);
+
+        // Take all the driver run hooks and post-execute them.
+        try {
+            for (HiveDriverRunHook driverRunHook : driverRunHooks) {
+                driverRunHook.postDriverRun(hookContext);
+            }
+        } catch (Exception e) {
+            errorMessage = "FAILED: Hive Internal Error: " + Utilities.getNameMessage(e);
+            SQLState = ErrorMsg.findSQLState(e.getMessage());
+            console.printError(errorMessage + "\n" + org.apache.hadoop.util.StringUtils.stringifyException(e));
+            return new CommandProcessorResponse(12, errorMessage, SQLState);
+        }
+
         return new CommandProcessorResponse(ret);
     }
 
-    private List<AbstractSemanticAnalyzerHook> getSemanticAnalyzerHooks() throws Exception {
-        ArrayList<AbstractSemanticAnalyzerHook> saHooks = new ArrayList<AbstractSemanticAnalyzerHook>();
-        String pestr = conf.getVar(HiveConf.ConfVars.SEMANTIC_ANALYZER_HOOK);
-        if (pestr == null) {
-            return saHooks;
+    /**
+     * Validate configuration variables.
+     * 
+     * @return
+     */
+    private boolean validateConfVariables() {
+        boolean valid = true;
+        if ((!conf.getBoolVar(HiveConf.ConfVars.HIVE_HADOOP_SUPPORTS_SUBDIRECTORIES))
+                && ((conf.getBoolVar(HiveConf.ConfVars.HADOOPMAPREDINPUTDIRRECURSIVE))
+                        || (conf.getBoolVar(HiveConf.ConfVars.HIVEOPTLISTBUCKETING)) || ((conf
+                            .getBoolVar(HiveConf.ConfVars.HIVE_OPTIMIZE_UNION_REMOVE))))) {
+            errorMessage = "FAILED: Hive Internal Error: " + ErrorMsg.SUPPORT_DIR_MUST_TRUE_FOR_LIST_BUCKETING.getMsg();
+            SQLState = ErrorMsg.findSQLState(errorMessage);
+            console.printError(errorMessage + "\n");
+            valid = false;
         }
-        pestr = pestr.trim();
-        if (pestr.equals("")) {
-            return saHooks;
-        }
-
-        String[] peClasses = pestr.split(",");
-
-        for (String peClass : peClasses) {
-            try {
-                AbstractSemanticAnalyzerHook hook = HiveUtils.getSemanticAnalyzerHook(conf, peClass);
-                saHooks.add(hook);
-            } catch (HiveException e) {
-                console.printError("Pre Exec Hook Class not found:" + e.getMessage());
-                throw e;
-            }
-        }
-
-        return saHooks;
+        return valid;
     }
 
-    private List<Hook> getPreExecHooks() throws Exception {
-        ArrayList<Hook> pehooks = new ArrayList<Hook>();
-        String pestr = conf.getVar(HiveConf.ConfVars.PREEXECHOOKS);
-        pestr = pestr.trim();
-        if (pestr.equals("")) {
-            return pehooks;
+    /**
+     * Returns a set of hooks specified in a configuration variable.
+     * See getHooks(HiveConf.ConfVars hookConfVar, Class<T> clazz)
+     * 
+     * @param hookConfVar
+     * @return
+     * @throws Exception
+     */
+    private List<Hook> getHooks(HiveConf.ConfVars hookConfVar) throws Exception {
+        return getHooks(hookConfVar, Hook.class);
+    }
+
+    /**
+     * Returns the hooks specified in a configuration variable. The hooks are returned in a list in
+     * the order they were specified in the configuration variable.
+     * 
+     * @param hookConfVar
+     *            The configuration variable specifying a comma separated list of the hook
+     *            class names.
+     * @param clazz
+     *            The super type of the hooks.
+     * @return A list of the hooks cast as the type specified in clazz, in the order
+     *         they are listed in the value of hookConfVar
+     * @throws Exception
+     */
+    private <T extends Hook> List<T> getHooks(HiveConf.ConfVars hookConfVar, Class<T> clazz) throws Exception {
+
+        List<T> hooks = new ArrayList<T>();
+        String csHooks = conf.getVar(hookConfVar);
+        if (csHooks == null) {
+            return hooks;
         }
 
-        String[] peClasses = pestr.split(",");
+        csHooks = csHooks.trim();
+        if (csHooks.equals("")) {
+            return hooks;
+        }
 
-        for (String peClass : peClasses) {
+        String[] hookClasses = csHooks.split(",");
+
+        for (String hookClass : hookClasses) {
             try {
-                pehooks.add((Hook) Class.forName(peClass.trim(), true, JavaUtils.getClassLoader()).newInstance());
+                T hook = (T) Class.forName(hookClass.trim(), true, JavaUtils.getClassLoader()).newInstance();
+                hooks.add(hook);
             } catch (ClassNotFoundException e) {
-                console.printError("Pre Exec Hook Class not found:" + e.getMessage());
+                console.printError(hookConfVar.varname + " Class not found:" + e.getMessage());
                 throw e;
             }
         }
 
-        return pehooks;
+        return hooks;
     }
 
-    private List<Hook> getPostExecHooks() throws Exception {
-        ArrayList<Hook> pehooks = new ArrayList<Hook>();
-        String pestr = conf.getVar(HiveConf.ConfVars.POSTEXECHOOKS);
-        pestr = pestr.trim();
-        if (pestr.equals("")) {
-            return pehooks;
-        }
-
-        String[] peClasses = pestr.split(",");
-
-        for (String peClass : peClasses) {
-            try {
-                pehooks.add((Hook) Class.forName(peClass.trim(), true, JavaUtils.getClassLoader()).newInstance());
-            } catch (ClassNotFoundException e) {
-                console.printError("Post Exec Hook Class not found:" + e.getMessage());
-                throw e;
-            }
-        }
-
-        return pehooks;
-    }
-
-    public int execute() {
+    public int execute() throws CommandNeedRetryException {
         // execute hivesterix plan
         if (hivesterix) {
             hivesterix = false;
@@ -925,6 +1052,9 @@
                 return ret;
         }
 
+        PerfLogger perfLogger = PerfLogger.getPerfLogger();
+        perfLogger.PerfLogBegin(LOG, PerfLogger.DRIVER_EXECUTE);
+
         boolean noName = StringUtils.isEmpty(conf.getVar(HiveConf.ConfVars.HADOOPJOBNAME));
         int maxlen = conf.getIntVar(HiveConf.ConfVars.HIVEJOBNAMELENGTH);
 
@@ -933,6 +1063,10 @@
 
         conf.setVar(HiveConf.ConfVars.HIVEQUERYID, queryId);
         conf.setVar(HiveConf.ConfVars.HIVEQUERYSTRING, queryStr);
+
+        conf.set("mapreduce.workflow.id", "hive_" + queryId);
+        conf.set("mapreduce.workflow.name", queryStr);
+
         maxthreads = HiveConf.getIntVar(conf, HiveConf.ConfVars.EXECPARALLETHREADNUMBER);
 
         try {
@@ -946,14 +1080,23 @@
             }
             resStream = null;
 
-            HookContext hookContext = new HookContext(plan, conf);
+            HookContext hookContext = new HookContext(plan, conf, ctx.getPathToCS());
+            hookContext.setHookType(HookContext.HookType.PRE_EXEC_HOOK);
 
-            for (Hook peh : getPreExecHooks()) {
+            for (Hook peh : getHooks(HiveConf.ConfVars.PREEXECHOOKS)) {
                 if (peh instanceof ExecuteWithHookContext) {
+                    perfLogger.PerfLogBegin(LOG, PerfLogger.PRE_HOOK + peh.getClass().getName());
+
                     ((ExecuteWithHookContext) peh).run(hookContext);
+
+                    perfLogger.PerfLogEnd(LOG, PerfLogger.PRE_HOOK + peh.getClass().getName());
                 } else if (peh instanceof PreExecute) {
+                    perfLogger.PerfLogBegin(LOG, PerfLogger.PRE_HOOK + peh.getClass().getName());
+
                     ((PreExecute) peh).run(SessionState.get(), plan.getInputs(), plan.getOutputs(), ShimLoader
                             .getHadoopShims().getUGIForConf(conf));
+
+                    perfLogger.PerfLogEnd(LOG, PerfLogger.PRE_HOOK + peh.getClass().getName());
                 }
             }
 
@@ -968,32 +1111,36 @@
             }
             String jobname = Utilities.abbreviate(queryStr, maxlen - 6);
 
-            // A runtime that launches runnable tasks as separate Threads
-            // through
+            // A runtime that launches runnable tasks as separate Threads through
             // TaskRunners
             // As soon as a task isRunnable, it is put in a queue
             // At any time, at most maxthreads tasks can be running
-            // The main thread polls the TaskRunners to check if they have
-            // finished.
+            // The main thread polls the TaskRunners to check if they have finished.
 
-            Queue<Task<? extends Serializable>> runnable = new LinkedList<Task<? extends Serializable>>();
+            Queue<Task<? extends Serializable>> runnable = new ConcurrentLinkedQueue<Task<? extends Serializable>>();
             Map<TaskResult, TaskRunner> running = new HashMap<TaskResult, TaskRunner>();
 
             DriverContext driverCxt = new DriverContext(runnable, ctx);
+            ctx.setHDFSCleanup(true);
+
+            SessionState.get().setLastMapRedStatsList(new ArrayList<MapRedStats>());
+            SessionState.get().setStackTraces(new HashMap<String, List<List<String>>>());
+            SessionState.get().setLocalMapRedErrors(new HashMap<String, List<String>>());
 
             // Add root Tasks to runnable
-
             for (Task<? extends Serializable> tsk : plan.getRootTasks()) {
+                // This should never happen, if it does, it's a bug with the potential to produce
+                // incorrect results.
+                assert tsk.getParentTasks() == null || tsk.getParentTasks().isEmpty();
                 driverCxt.addToRunnable(tsk);
             }
 
+            perfLogger.PerfLogEnd(LOG, PerfLogger.TIME_TO_SUBMIT);
             // Loop while you either have tasks running, or tasks queued up
-
             while (running.size() != 0 || runnable.peek() != null) {
                 // Launch upto maxthreads tasks
                 while (runnable.peek() != null && running.size() < maxthreads) {
                     Task<? extends Serializable> tsk = runnable.remove();
-                    console.printInfo("executing task " + tsk.getName());
                     launchTask(tsk, queryId, noName, running, jobname, jobs, driverCxt);
                 }
 
@@ -1005,12 +1152,24 @@
 
                 int exitVal = tskRes.getExitVal();
                 if (exitVal != 0) {
+                    if (tsk.ifRetryCmdWhenFail()) {
+                        if (!running.isEmpty()) {
+                            taskCleanup(running);
+                        }
+                        // in case we decided to run everything in local mode, restore the
+                        // the jobtracker setting to its initial value
+                        ctx.restoreOriginalTracker();
+                        throw new CommandNeedRetryException();
+                    }
                     Task<? extends Serializable> backupTask = tsk.getAndInitBackupTask();
                     if (backupTask != null) {
                         errorMessage = "FAILED: Execution Error, return code " + exitVal + " from "
                                 + tsk.getClass().getName();
+                        ErrorMsg em = ErrorMsg.getErrorMsg(exitVal);
+                        if (em != null) {
+                            errorMessage += ". " + em.getMsg();
+                        }
                         console.printError(errorMessage);
-
                         errorMessage = "ATTEMPT: Execute BackupTask: " + backupTask.getClass().getName();
                         console.printError(errorMessage);
 
@@ -1021,20 +1180,31 @@
                         continue;
 
                     } else {
-                        // TODO: This error messaging is not very informative.
-                        // Fix that.
+                        hookContext.setHookType(HookContext.HookType.ON_FAILURE_HOOK);
+                        // Get all the failure execution hooks and execute them.
+                        for (Hook ofh : getHooks(HiveConf.ConfVars.ONFAILUREHOOKS)) {
+                            perfLogger.PerfLogBegin(LOG, PerfLogger.FAILURE_HOOK + ofh.getClass().getName());
+
+                            ((ExecuteWithHookContext) ofh).run(hookContext);
+
+                            perfLogger.PerfLogEnd(LOG, PerfLogger.FAILURE_HOOK + ofh.getClass().getName());
+                        }
+
                         errorMessage = "FAILED: Execution Error, return code " + exitVal + " from "
                                 + tsk.getClass().getName();
+                        ErrorMsg em = ErrorMsg.getErrorMsg(exitVal);
+                        if (em != null) {
+                            errorMessage += ". " + em.getMsg();
+                        }
                         SQLState = "08S01";
                         console.printError(errorMessage);
-                        if (running.size() != 0) {
-                            taskCleanup();
+                        if (!running.isEmpty()) {
+                            taskCleanup(running);
                         }
-                        // in case we decided to run everything in local mode,
-                        // restore the
+                        // in case we decided to run everything in local mode, restore the
                         // the jobtracker setting to its initial value
                         ctx.restoreOriginalTracker();
-                        return 9;
+                        return exitVal;
                     }
                 }
 
@@ -1047,9 +1217,9 @@
                 if (tsk.getChildTasks() != null) {
                     for (Task<? extends Serializable> child : tsk.getChildTasks()) {
                         // hivesterix: don't check launchable condition
-                        // if (DriverContext.isLaunchable(child)) {
+                        //if(DriverContext.isLaunchable(tsk)){
                         driverCxt.addToRunnable(child);
-                        // }
+                        //}
                     }
                 }
             }
@@ -1059,8 +1229,7 @@
             ctx.restoreOriginalTracker();
 
             // remove incomplete outputs.
-            // Some incomplete outputs may be added at the beginning, for eg:
-            // for dynamic partitions.
+            // Some incomplete outputs may be added at the beginning, for eg: for dynamic partitions.
             // remove them
             HashSet<WriteEntity> remOutputs = new HashSet<WriteEntity>();
             for (WriteEntity output : plan.getOutputs()) {
@@ -1073,15 +1242,24 @@
                 plan.getOutputs().remove(output);
             }
 
+            hookContext.setHookType(HookContext.HookType.POST_EXEC_HOOK);
             // Get all the post execution hooks and execute them.
-            for (Hook peh : getPostExecHooks()) {
+            for (Hook peh : getHooks(HiveConf.ConfVars.POSTEXECHOOKS)) {
                 if (peh instanceof ExecuteWithHookContext) {
+                    perfLogger.PerfLogBegin(LOG, PerfLogger.POST_HOOK + peh.getClass().getName());
+
                     ((ExecuteWithHookContext) peh).run(hookContext);
+
+                    perfLogger.PerfLogEnd(LOG, PerfLogger.POST_HOOK + peh.getClass().getName());
                 } else if (peh instanceof PostExecute) {
+                    perfLogger.PerfLogBegin(LOG, PerfLogger.POST_HOOK + peh.getClass().getName());
+
                     ((PostExecute) peh)
                             .run(SessionState.get(), plan.getInputs(), plan.getOutputs(),
                                     (SessionState.get() != null ? SessionState.get().getLineageState().getLineageInfo()
                                             : null), ShimLoader.getHadoopShims().getUGIForConf(conf));
+
+                    perfLogger.PerfLogEnd(LOG, PerfLogger.POST_HOOK + peh.getClass().getName());
                 }
             }
 
@@ -1089,7 +1267,10 @@
                 SessionState.get().getHiveHistory().setQueryProperty(queryId, Keys.QUERY_RET_CODE, String.valueOf(0));
                 SessionState.get().getHiveHistory().printRowCount(queryId);
             }
+        } catch (CommandNeedRetryException e) {
+            throw e;
         } catch (Exception e) {
+            ctx.restoreOriginalTracker();
             if (SessionState.get() != null) {
                 SessionState.get().getHiveHistory().setQueryProperty(queryId, Keys.QUERY_RET_CODE, String.valueOf(12));
             }
@@ -1105,6 +1286,18 @@
             if (noName) {
                 conf.setVar(HiveConf.ConfVars.HADOOPJOBNAME, "");
             }
+            perfLogger.PerfLogEnd(LOG, PerfLogger.DRIVER_EXECUTE);
+
+            if (SessionState.get().getLastMapRedStatsList() != null
+                    && SessionState.get().getLastMapRedStatsList().size() > 0) {
+                long totalCpu = 0;
+                console.printInfo("MapReduce Jobs Launched: ");
+                for (int i = 0; i < SessionState.get().getLastMapRedStatsList().size(); i++) {
+                    console.printInfo("Job " + i + ": " + SessionState.get().getLastMapRedStatsList().get(i));
+                    totalCpu += SessionState.get().getLastMapRedStatsList().get(i).getCpuMSec();
+                }
+                console.printInfo("Total MapReduce CPU Time Spent: " + Utilities.formatMsecToStr(totalCpu));
+            }
         }
         plan.setDone();
 
@@ -1134,14 +1327,12 @@
      *            name of the task, if it is a map-reduce job
      * @param jobs
      *            number of map-reduce jobs
-     * @param curJobNo
-     *            the sequential number of the next map-reduce job
-     * @return the updated number of last the map-reduce job launched
+     * @param cxt
+     *            the driver context
      */
 
     public void launchTask(Task<? extends Serializable> tsk, String queryId, boolean noName,
             Map<TaskResult, TaskRunner> running, String jobname, int jobs, DriverContext cxt) {
-
         if (SessionState.get() != null) {
             SessionState.get().getHiveHistory().startTask(queryId, tsk, tsk.getClass().getName());
         }
@@ -1149,6 +1340,8 @@
             if (noName) {
                 conf.setVar(HiveConf.ConfVars.HADOOPJOBNAME, jobname + "(" + tsk.getId() + ")");
             }
+            conf.set("mapreduce.workflow.node.name", tsk.getId());
+            Utilities.setWorkflowAdjacencies(conf, plan);
             cxt.incCurJobNo(1);
             console.printInfo("Launching Job " + cxt.getCurJobNo() + " out of " + jobs);
         }
@@ -1156,7 +1349,13 @@
         TaskResult tskRes = new TaskResult();
         TaskRunner tskRun = new TaskRunner(tsk, tskRes);
 
-        // HiveConf.getBoolVar(conf, HiveConf.ConfVars.EXECPARALLEL) &&
+        // Launch Task
+        //if (HiveConf.getBoolVar(conf, HiveConf.ConfVars.EXECPARALLEL) && tsk.isMapRedTask()) {
+        // Launch it in the parallel mode, as a separate thread only for MR tasks
+        //  tskRun.start();
+        //} else {
+        //  tskRun.runSequential();
+        //}
         // Launch Task: hivesterix tweak
         if (tsk instanceof MapRedTask || tsk instanceof StatsTask) {
             // Launch it in the parallel mode, as a separate thread only for MR
@@ -1169,11 +1368,27 @@
             if (crs instanceof ConditionalResolverMergeFiles) {
                 tskRes.setRunning(false);
                 tskRes.setExitVal(0);
-
-                List<Task<? extends Serializable>> children = condTask.getListTasks();
-                for (Task<? extends Serializable> child : children)
-                    if (child instanceof MapRedTask)
-                        cxt.addToRunnable(child);
+                if (!executedConditionalTsks.contains(tsk)) {
+                    List<Task<? extends Serializable>> children = condTask.getListTasks();
+                    Task<? extends Serializable> selectedBranch = null;
+                    for (Task<? extends Serializable> branch : children) {
+                        if (branch instanceof MoveTask) {
+                            selectedBranch = branch;
+                            break;
+                        }
+                    }
+                    if (selectedBranch == null) {
+                        for (int i = children.size() - 1; i >= 0; i--) {
+                            Task<? extends Serializable> child = children.get(i);
+                            if (child instanceof MapRedTask) {
+                                selectedBranch = child;
+                                break;
+                            }
+                        }
+                    }
+                    executedConditionalTsks.add(tsk);
+                    cxt.addToRunnable(selectedBranch);
+                }
             }
         } else {
             tskRun.runSequential();
@@ -1185,12 +1400,18 @@
     /**
      * Cleans up remaining tasks in case of failure
      */
-
-    public void taskCleanup() {
-        // The currently existing Shutdown hooks will be automatically called,
-        // killing the map-reduce processes.
-        // The non MR processes will be killed as well.
-        System.exit(9);
+    public void taskCleanup(Map<TaskResult, TaskRunner> running) {
+        for (Map.Entry<TaskResult, TaskRunner> entry : running.entrySet()) {
+            if (entry.getKey().isRunning()) {
+                Task<?> task = entry.getValue().getTask();
+                try {
+                    task.shutdown();
+                } catch (Exception e) {
+                    console.printError("Exception on shutting down task " + task.getId() + ": " + e);
+                }
+            }
+        }
+        running.clear();
     }
 
     /**
@@ -1214,7 +1435,7 @@
             // In this loop, nothing was found
             // Sleep 10 seconds and restart
             try {
-                Thread.sleep(sleeptime);
+                Thread.sleep(SLEEP_TIME);
             } catch (InterruptedException ie) {
                 // Do Nothing
                 ;
@@ -1223,7 +1444,7 @@
         }
     }
 
-    public boolean getResults(ArrayList<String> res) throws IOException {
+    public boolean getResults(ArrayList<String> res) throws IOException, CommandNeedRetryException {
         if (plan != null && plan.getFetchTask() != null) {
             FetchTask ft = plan.getFetchTask();
             ft.setMaxRows(maxRows);
@@ -1276,6 +1497,14 @@
         return true;
     }
 
+    public int getTryCount() {
+        return tryCount;
+    }
+
+    public void setTryCount(int tryCount) {
+        this.tryCount = tryCount;
+    }
+
     public int close() {
         try {
             if (plan != null) {
@@ -1308,18 +1537,21 @@
     }
 
     public void destroy() {
-        releaseLocks();
+        if (ctx != null) {
+            releaseLocks(ctx.getHiveLocks());
+        }
+
+        if (hiveLockMgr != null) {
+            try {
+                hiveLockMgr.close();
+            } catch (LockException e) {
+                LOG.warn("Exception in closing hive lock manager. "
+                        + org.apache.hadoop.util.StringUtils.stringifyException(e));
+            }
+        }
     }
 
     public org.apache.hadoop.hive.ql.plan.api.Query getQueryPlan() throws IOException {
         return plan.getQueryPlan();
     }
-
-    public int getTryCount() {
-        return tryCount;
-    }
-
-    public void setTryCount(int tryCount) {
-        this.tryCount = tryCount;
-    }
 }
diff --git a/hivesterix/hivesterix-dist/src/main/java/org/apache/hadoop/hive/ql/udf/generic/GenericUDAFCount.java b/hivesterix/hivesterix-dist/src/main/java/org/apache/hadoop/hive/ql/udf/generic/GenericUDAFCount.java
index 2d5191d..1b96259 100644
--- a/hivesterix/hivesterix-dist/src/main/java/org/apache/hadoop/hive/ql/udf/generic/GenericUDAFCount.java
+++ b/hivesterix/hivesterix-dist/src/main/java/org/apache/hadoop/hive/ql/udf/generic/GenericUDAFCount.java
@@ -68,6 +68,7 @@
     @Override
     public GenericUDAFEvaluator getEvaluator(GenericUDAFParameterInfo paramInfo) throws SemanticException {
 
+        @SuppressWarnings("deprecation")
         TypeInfo[] parameters = paramInfo.getParameters();
 
         if (parameters.length == 0) {
diff --git a/hivesterix/hivesterix-dist/src/main/java/org/apache/hadoop/hive/serde2/typeinfo/TypeInfoFactory.java b/hivesterix/hivesterix-dist/src/main/java/org/apache/hadoop/hive/serde2/typeinfo/TypeInfoFactory.java
index 0fea4b9..e26f477 100644
--- a/hivesterix/hivesterix-dist/src/main/java/org/apache/hadoop/hive/serde2/typeinfo/TypeInfoFactory.java
+++ b/hivesterix/hivesterix-dist/src/main/java/org/apache/hadoop/hive/serde2/typeinfo/TypeInfoFactory.java
@@ -36,7 +36,7 @@
 import java.util.List;
 import java.util.concurrent.ConcurrentHashMap;
 
-import org.apache.hadoop.hive.serde.Constants;
+import org.apache.hadoop.hive.serde.serdeConstants;
 import org.apache.hadoop.hive.serde2.objectinspector.primitive.PrimitiveObjectInspectorUtils;
 
 /**
@@ -65,15 +65,18 @@
         return result;
     }
 
-    public static final TypeInfo voidTypeInfo = getPrimitiveTypeInfo(Constants.VOID_TYPE_NAME);
-    public static final TypeInfo booleanTypeInfo = getPrimitiveTypeInfo(Constants.BOOLEAN_TYPE_NAME);
-    public static final TypeInfo intTypeInfo = getPrimitiveTypeInfo(Constants.INT_TYPE_NAME);
-    public static final TypeInfo longTypeInfo = getPrimitiveTypeInfo(Constants.BIGINT_TYPE_NAME);
-    public static final TypeInfo stringTypeInfo = getPrimitiveTypeInfo(Constants.STRING_TYPE_NAME);
-    public static final TypeInfo floatTypeInfo = getPrimitiveTypeInfo(Constants.FLOAT_TYPE_NAME);
-    public static final TypeInfo doubleTypeInfo = getPrimitiveTypeInfo(Constants.DOUBLE_TYPE_NAME);
-    public static final TypeInfo byteTypeInfo = getPrimitiveTypeInfo(Constants.TINYINT_TYPE_NAME);
-    public static final TypeInfo shortTypeInfo = getPrimitiveTypeInfo(Constants.SMALLINT_TYPE_NAME);
+    public static final TypeInfo voidTypeInfo = getPrimitiveTypeInfo(serdeConstants.VOID_TYPE_NAME);
+    public static final TypeInfo booleanTypeInfo = getPrimitiveTypeInfo(serdeConstants.BOOLEAN_TYPE_NAME);
+    public static final TypeInfo intTypeInfo = getPrimitiveTypeInfo(serdeConstants.INT_TYPE_NAME);
+    public static final TypeInfo longTypeInfo = getPrimitiveTypeInfo(serdeConstants.BIGINT_TYPE_NAME);
+    public static final TypeInfo stringTypeInfo = getPrimitiveTypeInfo(serdeConstants.STRING_TYPE_NAME);
+    public static final TypeInfo floatTypeInfo = getPrimitiveTypeInfo(serdeConstants.FLOAT_TYPE_NAME);
+    public static final TypeInfo doubleTypeInfo = getPrimitiveTypeInfo(serdeConstants.DOUBLE_TYPE_NAME);
+    public static final TypeInfo byteTypeInfo = getPrimitiveTypeInfo(serdeConstants.TINYINT_TYPE_NAME);
+    public static final TypeInfo shortTypeInfo = getPrimitiveTypeInfo(serdeConstants.SMALLINT_TYPE_NAME);
+    public static final TypeInfo timestampTypeInfo = getPrimitiveTypeInfo(serdeConstants.TIMESTAMP_TYPE_NAME);
+    public static final TypeInfo binaryTypeInfo = getPrimitiveTypeInfo(serdeConstants.BINARY_TYPE_NAME);
+    public static final TypeInfo decimalTypeInfo = getPrimitiveTypeInfo(serdeConstants.DECIMAL_TYPE_NAME);
 
     public static final TypeInfo unknownTypeInfo = getPrimitiveTypeInfo("unknown");
 
diff --git a/hivesterix/hivesterix-dist/src/main/resources/conf/hive-default.xml b/hivesterix/hivesterix-dist/src/main/resources/conf/hive-default.xml
deleted file mode 100644
index 23a842a..0000000
--- a/hivesterix/hivesterix-dist/src/main/resources/conf/hive-default.xml
+++ /dev/null
@@ -1,773 +0,0 @@
-<?xml version="1.0"?>
-<!--
- ! Copyright 2009-2013 by The Regents of the University of California
- ! Licensed under the Apache License, Version 2.0 (the "License");
- ! you may not use this file except in compliance with the License.
- ! you may obtain a copy of the License from
- ! 
- !     http://www.apache.org/licenses/LICENSE-2.0
- ! 
- ! Unless required by applicable law or agreed to in writing, software
- ! distributed under the License is distributed on an "AS IS" BASIS,
- ! WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- ! See the License for the specific language governing permissions and
- ! limitations under the License.
- !-->
-<?xml-stylesheet type="text/xsl" href="configuration.xsl"?>
-
-<configuration>
-
-	<!-- Hive Configuration can either be stored in this file or in the hadoop 
-		configuration files -->
-	<!-- that are implied by Hadoop setup variables. -->
-	<!-- Aside from Hadoop setup variables - this file is provided as a convenience 
-		so that Hive -->
-	<!-- users do not have to edit hadoop configuration files (that may be managed 
-		as a centralized -->
-	<!-- resource). -->
-
-	<!-- Hive Execution Parameters -->
-	<property>
-		<name>mapred.reduce.tasks</name>
-		<value>-1</value>
-		<description>The default number of reduce tasks per job. Typically set
-			to a prime close to the number of available hosts. Ignored when
-			mapred.job.tracker is "local". Hadoop set this to 1 by default,
-			whereas hive uses -1 as its default value.
-			By setting this property to -1, Hive will automatically figure out what
-			should be the number of reducers.
-        	</description>
-        </property>
-
-        <property>
-		<name>hive.hyracks.connectorpolicy</name>
-		<value>PIPELINING</value>
-        </property>
-
-	<property>
-		<name>hive.hyracks.parrallelism</name>
-		<value>4</value>
-	</property>
-
-	<property>
-		<name>hive.algebricks.groupby.external</name>
-		<value>true</value>
-	</property>
-	
-	<property>
-		<name>hive.algebricks.groupby.external.memory</name>
-		<value>33554432</value>
-	</property>
-	
-	<property>
-		<name>hive.algebricks.sort.memory</name>
-		<value>33554432</value>
-	</property>
-
-	<property>
-		<name>hive.exec.reducers.bytes.per.reducer</name>
-		<value>1000000000</value>
-		<description>size per reducer.The default is 1G, i.e if the input size
-			is 10G, it will use 10 reducers.</description>
-	</property>
-
-	<property>
-		<name>hive.exec.reducers.max</name>
-		<value>999</value>
-		<description>max number of reducers will be used. If the one
-			specified in the configuration parameter mapred.reduce.tasks is
-			negative, hive will use this one as the max number of reducers when
-			automatically determine number of reducers.</description>
-	</property>
-
-	<property>
-		<name>hive.exec.scratchdir</name>
-		<value>/hive-${user.name}</value>
-		<description>Scratch space for Hive jobs</description>
-	</property>
-
-	<property>
-		<name>hive.test.mode</name>
-		<value>false</value>
-		<description>whether hive is running in test mode. If yes, it turns on
-			sampling and prefixes the output tablename</description>
-	</property>
-
-	<property>
-		<name>hive.test.mode.prefix</name>
-		<value>test_</value>
-		<description>if hive is running in test mode, prefixes the output
-			table by this string</description>
-	</property>
-
-	<!-- If the input table is not bucketed, the denominator of the tablesample 
-		is determinied by the parameter below -->
-	<!-- For example, the following query: -->
-	<!-- INSERT OVERWRITE TABLE dest -->
-	<!-- SELECT col1 from src -->
-	<!-- would be converted to -->
-	<!-- INSERT OVERWRITE TABLE test_dest -->
-	<!-- SELECT col1 from src TABLESAMPLE (BUCKET 1 out of 32 on rand(1)) -->
-	<property>
-		<name>hive.test.mode.samplefreq</name>
-		<value>32</value>
-		<description>if hive is running in test mode and table is not
-			bucketed, sampling frequency</description>
-	</property>
-
-	<property>
-		<name>hive.test.mode.nosamplelist</name>
-		<value></value>
-		<description>if hive is running in test mode, dont sample the above
-			comma seperated list of tables</description>
-	</property>
-
-	<property>
-		<name>hive.metastore.local</name>
-		<value>true</value>
-		<description>controls whether to connect to remove metastore server or
-			open a new metastore server in Hive Client JVM</description>
-	</property>
-
-	<property>
-		<name>javax.jdo.option.ConnectionURL</name>
-		<value>jdbc:derby:;databaseName=metastore_db;create=true</value>
-		<description>JDBC connect string for a JDBC metastore</description>
-	</property>
-
-	<property>
-		<name>javax.jdo.option.ConnectionDriverName</name>
-		<value>org.apache.derby.jdbc.EmbeddedDriver</value>
-		<description>Driver class name for a JDBC metastore</description>
-	</property>
-
-	<property>
-		<name>javax.jdo.PersistenceManagerFactoryClass</name>
-		<value>org.datanucleus.jdo.JDOPersistenceManagerFactory</value>
-		<description>class implementing the jdo persistence</description>
-	</property>
-
-	<property>
-		<name>datanucleus.connectionPoolingType</name>
-		<value>DBCP</value>
-		<description>Uses a DBCP connection pool for JDBC metastore
-		</description>
-	</property>
-
-	<property>
-		<name>javax.jdo.option.DetachAllOnCommit</name>
-		<value>true</value>
-		<description>detaches all objects from session so that they can be
-			used after transaction is committed</description>
-	</property>
-
-	<property>
-		<name>javax.jdo.option.NonTransactionalRead</name>
-		<value>true</value>
-		<description>reads outside of transactions</description>
-	</property>
-
-	<property>
-		<name>javax.jdo.option.ConnectionUserName</name>
-		<value>APP</value>
-		<description>username to use against metastore database</description>
-	</property>
-
-	<property>
-		<name>javax.jdo.option.ConnectionPassword</name>
-		<value>mine</value>
-		<description>password to use against metastore database</description>
-	</property>
-
-	<property>
-		<name>datanucleus.validateTables</name>
-		<value>false</value>
-		<description>validates existing schema against code. turn this on if
-			you want to verify existing schema </description>
-	</property>
-
-	<property>
-		<name>datanucleus.validateColumns</name>
-		<value>false</value>
-		<description>validates existing schema against code. turn this on if
-			you want to verify existing schema </description>
-	</property>
-
-	<property>
-		<name>datanucleus.validateConstraints</name>
-		<value>false</value>
-		<description>validates existing schema against code. turn this on if
-			you want to verify existing schema </description>
-	</property>
-
-	<property>
-		<name>datanucleus.storeManagerType</name>
-		<value>rdbms</value>
-		<description>metadata store type</description>
-	</property>
-
-	<property>
-		<name>datanucleus.autoCreateSchema</name>
-		<value>true</value>
-		<description>creates necessary schema on a startup if one doesn't
-			exist. set this to false, after creating it once</description>
-	</property>
-
-	<property>
-		<name>datanucleus.autoStartMechanismMode</name>
-		<value>checked</value>
-		<description>throw exception if metadata tables are incorrect
-		</description>
-	</property>
-
-	<property>
-		<name>datanucleus.transactionIsolation</name>
-		<value>read-committed</value>
-		<description>Default transaction isolation level for identity
-			generation. </description>
-	</property>
-
-	<property>
-		<name>datanucleus.cache.level2</name>
-		<value>false</value>
-		<description>Use a level 2 cache. Turn this off if metadata is changed
-			independently of hive metastore server</description>
-	</property>
-
-	<property>
-		<name>datanucleus.cache.level2.type</name>
-		<value>SOFT</value>
-		<description>SOFT=soft reference based cache, WEAK=weak reference
-			based cache.</description>
-	</property>
-
-	<property>
-		<name>datanucleus.identifierFactory</name>
-		<value>datanucleus</value>
-		<description>Name of the identifier factory to use when generating
-			table/column names etc. 'datanucleus' is used for backward
-			compatibility</description>
-	</property>
-
-	<property>
-		<name>hive.metastore.warehouse.dir</name>
-		<value>/user/hivesterix</value>
-		<description>location of default database for the warehouse
-		</description>
-	</property>
-
-	<property>
-		<name>hive.metastore.connect.retries</name>
-		<value>5</value>
-		<description>Number of retries while opening a connection to metastore
-		</description>
-	</property>
-
-	<property>
-		<name>hive.metastore.rawstore.impl</name>
-		<value>org.apache.hadoop.hive.metastore.ObjectStore</value>
-		<description>Name of the class that implements
-			org.apache.hadoop.hive.metastore.rawstore interface. This class is
-			used to store and retrieval of raw metadata objects such as table,
-			database</description>
-	</property>
-
-	<property>
-		<name>hive.default.fileformat</name>
-		<value>TextFile</value>
-		<description>Default file format for CREATE TABLE statement. Options
-			are TextFile and SequenceFile. Users can explicitly say CREATE TABLE
-			... STORED AS &lt;TEXTFILE|SEQUENCEFILE&gt; to override</description>
-	</property>
-
-	<property>
-		<name>hive.fileformat.check</name>
-		<value>true</value>
-		<description>Whether to check file format or not when loading data
-			files</description>
-	</property>
-
-	<property>
-		<name>hive.map.aggr</name>
-		<value>true</value>
-		<description>Whether to use map-side aggregation in Hive Group By
-			queries</description>
-	</property>
-
-	<property>
-		<name>hive.groupby.skewindata</name>
-		<value>false</value>
-		<description>Whether there is skew in data to optimize group by
-			queries</description>
-	</property>
-
-	<property>
-		<name>hive.groupby.mapaggr.checkinterval</name>
-		<value>100000</value>
-		<description>Number of rows after which size of the grouping
-			keys/aggregation classes is performed</description>
-	</property>
-
-	<property>
-		<name>hive.mapred.local.mem</name>
-		<value>0</value>
-		<description>For local mode, memory of the mappers/reducers
-		</description>
-	</property>
-
-	<property>
-		<name>hive.map.aggr.hash.percentmemory</name>
-		<value>0.5</value>
-		<description>Portion of total memory to be used by map-side grup
-			aggregation hash table</description>
-	</property>
-
-	<property>
-		<name>hive.map.aggr.hash.min.reduction</name>
-		<value>0.5</value>
-		<description>Hash aggregation will be turned off if the ratio between
-			hash
-			table size and input rows is bigger than this number. Set to 1 to make
-			sure
-			hash aggregation is never turned off.</description>
-	</property>
-
-	<property>
-		<name>hive.optimize.cp</name>
-		<value>true</value>
-		<description>Whether to enable column pruner</description>
-	</property>
-
-	<property>
-		<name>hive.optimize.ppd</name>
-		<value>true</value>
-		<description>Whether to enable predicate pushdown</description>
-	</property>
-
-	<property>
-		<name>hive.optimize.pruner</name>
-		<value>true</value>
-		<description>Whether to enable the new partition pruner which depends
-			on predicate pushdown. If this is disabled,
-			the old partition pruner which is based on AST will be enabled.
-		</description>
-	</property>
-
-	<property>
-		<name>hive.optimize.groupby</name>
-		<value>true</value>
-		<description>Whether to enable the bucketed group by from bucketed
-			partitions/tables.</description>
-	</property>
-
-	<property>
-		<name>hive.join.emit.interval</name>
-		<value>1000</value>
-		<description>How many rows in the right-most join operand Hive should
-			buffer before emitting the join result. </description>
-	</property>
-
-	<property>
-		<name>hive.join.cache.size</name>
-		<value>25000</value>
-		<description>How many rows in the joining tables (except the streaming
-			table) should be cached in memory. </description>
-	</property>
-
-	<property>
-		<name>hive.mapjoin.bucket.cache.size</name>
-		<value>100</value>
-		<description>How many values in each keys in the map-joined table
-			should be cached in memory. </description>
-	</property>
-
-	<property>
-		<name>hive.mapjoin.maxsize</name>
-		<value>100000</value>
-		<description>Maximum # of rows of the small table that can be handled
-			by map-side join. If the size is reached and hive.task.progress is
-			set, a fatal error counter is set and the job will be killed.
-		</description>
-	</property>
-
-	<property>
-		<name>hive.mapjoin.cache.numrows</name>
-		<value>25000</value>
-		<description>How many rows should be cached by jdbm for map join.
-		</description>
-	</property>
-
-	<property>
-		<name>hive.optimize.skewjoin</name>
-		<value>false</value>
-		<description>Whether to enable skew join optimization. </description>
-	</property>
-
-	<property>
-		<name>hive.skewjoin.key</name>
-		<value>100000</value>
-		<description>Determine if we get a skew key in join. If we see more
-			than the specified number of rows with the same key in join operator,
-			we think the key as a skew join key. </description>
-	</property>
-
-	<property>
-		<name>hive.skewjoin.mapjoin.map.tasks</name>
-		<value>10000</value>
-		<description> Determine the number of map task used in the follow up
-			map join job
-			for a skew join. It should be used together with
-			hive.skewjoin.mapjoin.min.split
-			to perform a fine grained control.</description>
-	</property>
-
-	<property>
-		<name>hive.skewjoin.mapjoin.min.split</name>
-		<value>33554432</value>
-		<description> Determine the number of map task at most used in the
-			follow up map join job
-			for a skew join by specifying the minimum split size. It should be used
-			together with
-			hive.skewjoin.mapjoin.map.tasks to perform a fine grained control.</description>
-	</property>
-
-	<property>
-		<name>hive.mapred.mode</name>
-		<value>nonstrict</value>
-		<description>The mode in which the hive operations are being
-			performed. In strict mode, some risky queries are not allowed to run
-		</description>
-	</property>
-
-	<property>
-		<name>hive.exec.script.maxerrsize</name>
-		<value>100000</value>
-		<description>Maximum number of bytes a script is allowed to emit to
-			standard error (per map-reduce task). This prevents runaway scripts
-			from filling logs partitions to capacity </description>
-	</property>
-
-	<property>
-		<name>hive.exec.script.allow.partial.consumption</name>
-		<value>false</value>
-		<description> When enabled, this option allows a user script to exit
-			successfully without consuming all the data from the standard input.
-		</description>
-	</property>
-
-	<property>
-		<name>hive.script.operator.id.env.var</name>
-		<value>HIVE_SCRIPT_OPERATOR_ID</value>
-		<description> Name of the environment variable that holds the unique
-			script operator ID in the user's transform function (the custom
-			mapper/reducer that the user has specified in the query)
-		</description>
-	</property>
-
-	<property>
-		<name>hive.exec.compress.output</name>
-		<value>false</value>
-		<description> This controls whether the final outputs of a query (to a
-			local/hdfs file or a hive table) is compressed. The compression codec
-			and other options are determined from hadoop config variables
-			mapred.output.compress* </description>
-	</property>
-
-	<property>
-		<name>hive.exec.compress.intermediate</name>
-		<value>false</value>
-		<description> This controls whether intermediate files produced by
-			hive between multiple map-reduce jobs are compressed. The compression
-			codec and other options are determined from hadoop config variables
-			mapred.output.compress* </description>
-	</property>
-
-	<property>
-		<name>hive.exec.parallel</name>
-		<value>false</value>
-		<description>Whether to execute jobs in parallel</description>
-	</property>
-
-	<property>
-		<name>hive.exec.parallel.thread.number</name>
-		<value>8</value>
-		<description>How many jobs at most can be executed in parallel
-		</description>
-	</property>
-
-	<property>
-		<name>hive.hwi.war.file</name>
-		<value>lib\hive-hwi-0.7.0.war</value>
-		<description>This sets the path to the HWI war file, relative to
-			${HIVE_HOME}. </description>
-	</property>
-
-	<property>
-		<name>hive.hwi.listen.host</name>
-		<value>0.0.0.0</value>
-		<description>This is the host address the Hive Web Interface will
-			listen on</description>
-	</property>
-
-	<property>
-		<name>hive.hwi.listen.port</name>
-		<value>9999</value>
-		<description>This is the port the Hive Web Interface will listen on
-		</description>
-	</property>
-
-	<property>
-		<name>hive.exec.pre.hooks</name>
-		<value></value>
-		<description>Pre Execute Hook for Tests</description>
-	</property>
-
-	<property>
-		<name>hive.merge.mapfiles</name>
-		<value>true</value>
-		<description>Merge small files at the end of a map-only job
-		</description>
-	</property>
-
-	<property>
-		<name>hive.merge.mapredfiles</name>
-		<value>false</value>
-		<description>Merge small files at the end of a map-reduce job
-		</description>
-	</property>
-
-	<property>
-		<name>hive.heartbeat.interval</name>
-		<value>1000</value>
-		<description>Send a heartbeat after this interval - used by mapjoin
-			and filter operators</description>
-	</property>
-
-	<property>
-		<name>hive.merge.size.per.task</name>
-		<value>256000000</value>
-		<description>Size of merged files at the end of the job</description>
-	</property>
-
-	<property>
-		<name>hive.merge.size.smallfiles.avgsize</name>
-		<value>16000000</value>
-		<description>When the average output file size of a job is less than
-			this number, Hive will start an additional map-reduce job to merge
-			the output files into bigger files. This is only done for map-only
-			jobs if hive.merge.mapfiles is true, and for map-reduce jobs if
-			hive.merge.mapredfiles is true.</description>
-	</property>
-
-	<property>
-		<name>hive.script.auto.progress</name>
-		<value>false</value>
-		<description>Whether Hive Tranform/Map/Reduce Clause should
-			automatically send progress information to TaskTracker to avoid the
-			task getting killed because of inactivity. Hive sends progress
-			information when the script is outputting to stderr. This option
-			removes the need of periodically producing stderr messages, but users
-			should be cautious because this may prevent infinite loops in the
-			scripts to be killed by TaskTracker.  </description>
-	</property>
-
-	<property>
-		<name>hive.script.serde</name>
-		<value>org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe</value>
-		<description>The default serde for trasmitting input data to and
-			reading output data from the user scripts. </description>
-	</property>
-
-	<property>
-		<name>hive.script.recordreader</name>
-		<value>org.apache.hadoop.hive.ql.exec.TextRecordReader</value>
-		<description>The default record reader for reading data from the user
-			scripts. </description>
-	</property>
-
-	<property>
-		<name>hive.script.recordwriter</name>
-		<value>org.apache.hadoop.hive.ql.exec.TextRecordWriter</value>
-		<description>The default record writer for writing data to the user
-			scripts. </description>
-	</property>
-
-	<property>
-		<name>hive.input.format</name>
-		<value>org.apache.hadoop.hive.ql.io.HiveInputFormat</value>
-		<description>The default input format, if it is not specified, the
-			system assigns it. It is set to HiveInputFormat for hadoop versions
-			17, 18 and 19, whereas it is set to CombinedHiveInputFormat for
-			hadoop 20. The user can always overwrite it - if there is a bug in
-			CombinedHiveInputFormat, it can always be manually set to
-			HiveInputFormat. </description>
-	</property>
-
-	<property>
-		<name>hive.udtf.auto.progress</name>
-		<value>false</value>
-		<description>Whether Hive should automatically send progress
-			information to TaskTracker when using UDTF's to prevent the task
-			getting killed because of inactivity. Users should be cautious
-			because this may prevent TaskTracker from killing tasks with infinte
-			loops.  </description>
-	</property>
-
-	<property>
-		<name>hive.mapred.reduce.tasks.speculative.execution</name>
-		<value>true</value>
-		<description>Whether speculative execution for reducers should be
-			turned on. </description>
-	</property>
-
-	<property>
-		<name>hive.exec.counters.pull.interval</name>
-		<value>1000</value>
-		<description>The interval with which to poll the JobTracker for the
-			counters the running job. The smaller it is the more load there will
-			be on the jobtracker, the higher it is the less granular the caught
-			will be.</description>
-	</property>
-
-	<property>
-		<name>hive.enforce.bucketing</name>
-		<value>false</value>
-		<description>Whether bucketing is enforced. If true, while inserting
-			into the table, bucketing is enforced. </description>
-	</property>
-
-	<property>
-		<name>hive.enforce.sorting</name>
-		<value>false</value>
-		<description>Whether sorting is enforced. If true, while inserting
-			into the table, sorting is enforced. </description>
-	</property>
-
-	<property>
-		<name>hive.metastore.ds.connection.url.hook</name>
-		<value></value>
-		<description>Name of the hook to use for retriving the JDO connection
-			URL. If empty, the value in javax.jdo.option.ConnectionURL is used
-		</description>
-	</property>
-
-	<property>
-		<name>hive.metastore.ds.retry.attempts</name>
-		<value>1</value>
-		<description>The number of times to retry a metastore call if there
-			were a connection error</description>
-	</property>
-
-	<property>
-		<name>hive.metastore.ds.retry.interval</name>
-		<value>1000</value>
-		<description>The number of miliseconds between metastore retry
-			attempts</description>
-	</property>
-
-	<property>
-		<name>hive.metastore.server.min.threads</name>
-		<value>200</value>
-		<description>Minimum number of worker threads in the Thrift server's
-			pool.</description>
-	</property>
-
-	<property>
-		<name>hive.metastore.server.max.threads</name>
-		<value>100000</value>
-		<description>Maximum number of worker threads in the Thrift server's
-			pool.</description>
-	</property>
-
-	<property>
-		<name>hive.metastore.server.tcp.keepalive</name>
-		<value>true</value>
-		<description>Whether to enable TCP keepalive for the metastore server.
-			Keepalive will prevent accumulation of half-open connections.
-		</description>
-	</property>
-
-	<property>
-		<name>hive.optimize.reducededuplication</name>
-		<value>true</value>
-		<description>Remove extra map-reduce jobs if the data is already
-			clustered by the same key which needs to be used again. This should
-			always be set to true. Since it is a new feature, it has been made
-			configurable.</description>
-	</property>
-
-	<property>
-		<name>hive.exec.dynamic.partition</name>
-		<value>false</value>
-		<description>Whether or not to allow dynamic partitions in DML/DDL.
-		</description>
-	</property>
-
-	<property>
-		<name>hive.exec.dynamic.partition.mode</name>
-		<value>strict</value>
-		<description>In strict mode, the user must specify at least one static
-			partition in case the user accidentally overwrites all partitions.
-		</description>
-	</property>
-
-	<property>
-		<name>hive.exec.max.dynamic.partitions</name>
-		<value>1000</value>
-		<description>Maximum number of dynamic partitions allowed to be
-			created in total.</description>
-	</property>
-
-	<property>
-		<name>hive.exec.max.dynamic.partitions.pernode</name>
-		<value>100</value>
-		<description>Maximum number of dynamic partitions allowed to be
-			created in each mapper/reducer node.</description>
-	</property>
-
-	<property>
-		<name>hive.default.partition.name</name>
-		<value>__HIVE_DEFAULT_PARTITION__</value>
-		<description>The default partition name in case the dynamic partition
-			column value is null/empty string or anyother values that cannot be
-			escaped. This value must not contain any special character used in
-			HDFS URI (e.g., ':', '%', '/' etc). The user has to be aware that the
-			dynamic partition value should not contain this value to avoid
-			confusions.</description>
-	</property>
-
-	<property>
-		<name>fs.har.impl</name>
-		<value>org.apache.hadoop.hive.shims.HiveHarFileSystem</value>
-		<description>The implementation for accessing Hadoop Archives. Note
-			that this won't be applicable to Hadoop vers less than 0.20
-		</description>
-	</property>
-
-	<property>
-		<name>hive.archive.enabled</name>
-		<value>false</value>
-		<description>Whether archiving operations are permitted</description>
-	</property>
-
-	<property>
-		<name>hive.archive.har.parentdir.settable</name>
-		<value>false</value>
-		<description>In new Hadoop versions, the parent directory must be set
-			while
-			creating a HAR. Because this functionality is hard to detect with just
-			version
-			numbers, this conf var needs to be set manually.</description>
-	</property>
-
-	<!-- HBase Storage Handler Parameters -->
-
-	<property>
-		<name>hive.hbase.wal.enabled</name>
-		<value>true</value>
-		<description>Whether writes to HBase should be forced to the
-			write-ahead log. Disabling this improves HBase write performance at
-			the risk of lost writes in case of a crash.</description>
-	</property>
-
-</configuration>
diff --git a/hivesterix/hivesterix-dist/src/main/resources/conf/hive-log4j.properties b/hivesterix/hivesterix-dist/src/main/resources/conf/hive-log4j.properties
index eab38a6..6f195f5 100644
--- a/hivesterix/hivesterix-dist/src/main/resources/conf/hive-log4j.properties
+++ b/hivesterix/hivesterix-dist/src/main/resources/conf/hive-log4j.properties
@@ -31,7 +31,7 @@
 #      FATAL, ERROR, WARN, INFO, DEBUG
 #
 #------------------------------------------------------------------------------
-log4j.rootCategory=INFO, S
+log4j.rootCategory=FATAL, S
 
 log4j.logger.com.dappit.Dapper.parser=ERROR
 log4j.logger.org.w3c.tidy=FATAL
diff --git a/hivesterix/hivesterix-dist/src/main/resources/conf/hive-site.xml b/hivesterix/hivesterix-dist/src/main/resources/conf/hive-site.xml
new file mode 100644
index 0000000..ccfcd74
--- /dev/null
+++ b/hivesterix/hivesterix-dist/src/main/resources/conf/hive-site.xml
@@ -0,0 +1,5189 @@
+<?xml version="1.0"?>
+<!-- ! Copyright 2009-2013 by The Regents of the University of California 
+	! Licensed under the Apache License, Version 2.0 (the "License"); ! you may 
+	not use this file except in compliance with the License. ! you may obtain 
+	a copy of the License from ! ! http://www.apache.org/licenses/LICENSE-2.0 
+	! ! Unless required by applicable law or agreed to in writing, software ! 
+	distributed under the License is distributed on an "AS IS" BASIS, ! WITHOUT 
+	WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. ! See the 
+	License for the specific language governing permissions and ! limitations 
+	under the License. ! -->
+<?xml-stylesheet type="text/xsl" href="configuration.xsl"?>
+
+<configuration>
+
+	<configuration>
+		<!-- Hivesterix Execution Parameters -->
+		<property>
+			<name>hive.hyracks.connectorpolicy</name>
+			<value>PIPELINING</value>
+		</property>
+
+		<property>
+			<name>hive.hyracks.parrallelism</name>
+			<value>4</value>
+		</property>
+
+		<property>
+			<name>hive.algebricks.groupby.external</name>
+			<value>true</value>
+		</property>
+		
+		<property>
+			<name>hive.algebricks.groupby.external.memory</name>
+			<value>33554432</value>
+		</property>
+
+		<property>
+			<name>hive.algebricks.sort.memory</name>
+			<value>33554432</value>
+		</property>
+		
+		<property>
+            <name>hive.algebricks.framesize</name>
+            <value>32768</value>
+        </property>
+
+		<!-- Hive Execution Parameters -->
+		<property>
+			<name>mapred.reduce.tasks</name>
+			<value>-1</value>
+			<description>The default number of reduce tasks per job. Typically
+				set
+				to a prime close to the number of available hosts. Ignored when
+				mapred.job.tracker is "local". Hadoop set this to 1 by default,
+				whereas hive uses -1 as its default value.
+				By setting this property
+				to -1, Hive will automatically figure out
+				what should be the number
+				of reducers.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.reducers.bytes.per.reducer</name>
+			<value>1000000000</value>
+			<description>size per reducer.The default is 1G, i.e if the input
+				size is 10G, it will use 10 reducers.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.reducers.max</name>
+			<value>999</value>
+			<description>max number of reducers will be used. If the one
+				specified in the configuration parameter mapred.reduce.tasks is
+				negative, hive will use this one as the max number of reducers when
+				automatically determine number of reducers.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.cli.print.header</name>
+			<value>false</value>
+			<description>Whether to print the names of the columns in query
+				output.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.cli.print.current.db</name>
+			<value>false</value>
+			<description>Whether to include the current database in the hive
+				prompt.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.cli.prompt</name>
+			<value>hive</value>
+			<description>Command line prompt configuration value. Other hiveconf
+				can be used in
+				this configuration value. Variable substitution will
+				only be invoked at
+				the hive
+				cli startup.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.cli.pretty.output.num.cols</name>
+			<value>-1</value>
+			<description>The number of columns to use when formatting output
+				generated
+				by the DESCRIBE PRETTY table_name command. If the value of
+				this
+				property
+				is -1, then hive will use the auto-detected terminal
+				width.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.scratchdir</name>
+			<value>/tmp/hive-${user.name}</value>
+			<description>Scratch space for Hive jobs</description>
+		</property>
+
+		<property>
+			<name>hive.exec.local.scratchdir</name>
+			<value>/tmp/${user.name}</value>
+			<description>Local scratch space for Hive jobs</description>
+		</property>
+
+		<property>
+			<name>hive.test.mode</name>
+			<value>false</value>
+			<description>whether hive is running in test mode. If yes, it turns
+				on sampling and prefixes the output tablename
+			</description>
+		</property>
+
+		<property>
+			<name>hive.test.mode.prefix</name>
+			<value>test_</value>
+			<description>if hive is running in test mode, prefixes the output
+				table by this string
+			</description>
+		</property>
+
+		<!-- If the input table is not bucketed, the denominator of the tablesample 
+			is determinied by the parameter below -->
+		<!-- For example, the following query: -->
+		<!-- INSERT OVERWRITE TABLE dest -->
+		<!-- SELECT col1 from src -->
+		<!-- would be converted to -->
+		<!-- INSERT OVERWRITE TABLE test_dest -->
+		<!-- SELECT col1 from src TABLESAMPLE (BUCKET 1 out of 32 on rand(1)) -->
+		<property>
+			<name>hive.test.mode.samplefreq</name>
+			<value>32</value>
+			<description>if hive is running in test mode and table is not
+				bucketed, sampling frequency
+			</description>
+		</property>
+
+		<property>
+			<name>hive.test.mode.nosamplelist</name>
+			<value></value>
+			<description>if hive is running in test mode, dont sample the above
+				comma seperated list of tables
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.uris</name>
+			<value></value>
+			<description>Thrift uri for the remote metastore. Used by metastore
+				client to connect to remote metastore.
+			</description>
+		</property>
+
+		<property>
+			<name>javax.jdo.option.ConnectionURL</name>
+			<value>jdbc:derby:;databaseName=metastore_db;create=true</value>
+			<description>JDBC connect string for a JDBC metastore</description>
+		</property>
+
+		<property>
+			<name>javax.jdo.option.ConnectionDriverName</name>
+			<value>org.apache.derby.jdbc.EmbeddedDriver</value>
+			<description>Driver class name for a JDBC metastore</description>
+		</property>
+
+		<property>
+			<name>javax.jdo.PersistenceManagerFactoryClass</name>
+			<value>org.datanucleus.jdo.JDOPersistenceManagerFactory</value>
+			<description>class implementing the jdo persistence</description>
+		</property>
+
+		<property>
+			<name>javax.jdo.option.DetachAllOnCommit</name>
+			<value>true</value>
+			<description>detaches all objects from session so that they can be
+				used after transaction is committed
+			</description>
+		</property>
+
+		<property>
+			<name>javax.jdo.option.NonTransactionalRead</name>
+			<value>true</value>
+			<description>reads outside of transactions</description>
+		</property>
+
+		<property>
+			<name>javax.jdo.option.ConnectionUserName</name>
+			<value>APP</value>
+			<description>username to use against metastore database</description>
+		</property>
+
+		<property>
+			<name>javax.jdo.option.ConnectionPassword</name>
+			<value>mine</value>
+			<description>password to use against metastore database</description>
+		</property>
+
+		<property>
+			<name>javax.jdo.option.Multithreaded</name>
+			<value>true</value>
+			<description>Set this to true if multiple threads access metastore
+				through JDO concurrently.
+			</description>
+		</property>
+
+		<property>
+			<name>datanucleus.connectionPoolingType</name>
+			<value>DBCP</value>
+			<description>Uses a DBCP connection pool for JDBC metastore
+			</description>
+		</property>
+
+		<property>
+			<name>datanucleus.validateTables</name>
+			<value>false</value>
+			<description>validates existing schema against code. turn this on if
+				you want to verify existing schema
+			</description>
+		</property>
+
+		<property>
+			<name>datanucleus.validateColumns</name>
+			<value>false</value>
+			<description>validates existing schema against code. turn this on if
+				you want to verify existing schema
+			</description>
+		</property>
+
+		<property>
+			<name>datanucleus.validateConstraints</name>
+			<value>false</value>
+			<description>validates existing schema against code. turn this on if
+				you want to verify existing schema
+			</description>
+		</property>
+
+		<property>
+			<name>datanucleus.storeManagerType</name>
+			<value>rdbms</value>
+			<description>metadata store type</description>
+		</property>
+
+		<property>
+			<name>datanucleus.autoCreateSchema</name>
+			<value>true</value>
+			<description>creates necessary schema on a startup if one doesn't
+				exist. set this to false, after creating it once
+			</description>
+		</property>
+
+		<property>
+			<name>datanucleus.autoStartMechanismMode</name>
+			<value>checked</value>
+			<description>throw exception if metadata tables are incorrect
+			</description>
+		</property>
+
+		<property>
+			<name>datanucleus.transactionIsolation</name>
+			<value>read-committed</value>
+			<description>Default transaction isolation level for identity
+				generation.
+			</description>
+		</property>
+
+		<property>
+			<name>datanucleus.cache.level2</name>
+			<value>false</value>
+			<description>Use a level 2 cache. Turn this off if metadata is
+				changed independently of hive metastore server
+			</description>
+		</property>
+
+		<property>
+			<name>datanucleus.cache.level2.type</name>
+			<value>SOFT</value>
+			<description>SOFT=soft reference based cache, WEAK=weak reference
+				based cache.
+			</description>
+		</property>
+
+		<property>
+			<name>datanucleus.identifierFactory</name>
+			<value>datanucleus</value>
+			<description>Name of the identifier factory to use when generating
+				table/column names etc. 'datanucleus' is used for backward
+				compatibility
+			</description>
+		</property>
+
+		<property>
+			<name>datanucleus.plugin.pluginRegistryBundleCheck</name>
+			<value>LOG</value>
+			<description>Defines what happens when plugin bundles are found and
+				are duplicated [EXCEPTION|LOG|NONE]
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.warehouse.dir</name>
+			<value>/user/hive/warehouse</value>
+			<description>location of default database for the warehouse
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.execute.setugi</name>
+			<value>false</value>
+			<description>In unsecure mode, setting this property to true will
+				cause the metastore to execute DFS operations using the client's
+				reported user and group permissions. Note that this property must be
+				set on both the client and server sides. Further note that its best
+				effort. If client sets its to true and server sets it to false,
+				client setting will be ignored.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.event.listeners</name>
+			<value></value>
+			<description>list of comma seperated listeners for metastore events.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.partition.inherit.table.properties</name>
+			<value></value>
+			<description>list of comma seperated keys occurring in table
+				properties which will get inherited to newly created partitions. *
+				implies all the keys will get inherited.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metadata.export.location</name>
+			<value></value>
+			<description>When used in conjunction with the
+				org.apache.hadoop.hive.ql.parse.MetaDataExportListener pre event
+				listener, it is the location to which the metadata will be exported.
+				The default is an empty string, which results in the metadata being
+				exported to the current user's home directory on HDFS.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metadata.move.exported.metadata.to.trash</name>
+			<value></value>
+			<description>When used in conjunction with the
+				org.apache.hadoop.hive.ql.parse.MetaDataExportListener pre event
+				listener, this setting determines if the metadata that is exported
+				will subsequently be moved to the user's trash directory alongside
+				the dropped table data. This ensures that the metadata will be
+				cleaned up along with the dropped table data.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.partition.name.whitelist.pattern</name>
+			<value></value>
+			<description>Partition names will be checked against this regex
+				pattern and rejected if not matched.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.end.function.listeners</name>
+			<value></value>
+			<description>list of comma separated listeners for the end of
+				metastore functions.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.event.expiry.duration</name>
+			<value>0</value>
+			<description>Duration after which events expire from events table (in
+				seconds)
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.event.clean.freq</name>
+			<value>0</value>
+			<description>Frequency at which timer task runs to purge expired
+				events in metastore(in seconds).
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.connect.retries</name>
+			<value>5</value>
+			<description>Number of retries while opening a connection to
+				metastore
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.failure.retries</name>
+			<value>3</value>
+			<description>Number of retries upon failure of Thrift metastore calls
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.client.connect.retry.delay</name>
+			<value>1</value>
+			<description>Number of seconds for the client to wait between
+				consecutive connection attempts
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.client.socket.timeout</name>
+			<value>20</value>
+			<description>MetaStore Client socket timeout in seconds</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.rawstore.impl</name>
+			<value>org.apache.hadoop.hive.metastore.ObjectStore</value>
+			<description>Name of the class that implements
+				org.apache.hadoop.hive.metastore.rawstore interface. This class is
+				used to store and retrieval of raw metadata objects such as table,
+				database
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.batch.retrieve.max</name>
+			<value>300</value>
+			<description>Maximum number of objects (tables/partitions) can be
+				retrieved from metastore in one batch. The higher the number, the
+				less the number of round trips is needed to the Hive metastore
+				server, but it may also cause higher memory requirement at the
+				client side.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.batch.retrieve.table.partition.max</name>
+			<value>1000</value>
+			<description>Maximum number of table partitions that metastore
+				internally retrieves in one batch.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.default.fileformat</name>
+			<value>TextFile</value>
+			<description>Default file format for CREATE TABLE statement. Options
+				are TextFile and SequenceFile. Users can explicitly say CREATE TABLE
+				... STORED AS &lt;TEXTFILE|SEQUENCEFILE&gt; to override
+			</description>
+		</property>
+
+		<property>
+			<name>hive.fileformat.check</name>
+			<value>true</value>
+			<description>Whether to check file format or not when loading data
+				files
+			</description>
+		</property>
+
+		<property>
+			<name>hive.map.aggr</name>
+			<value>true</value>
+			<description>Whether to use map-side aggregation in Hive Group By
+				queries
+			</description>
+		</property>
+
+		<property>
+			<name>hive.groupby.skewindata</name>
+			<value>false</value>
+			<description>Whether there is skew in data to optimize group by
+				queries
+			</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.multigroupby.common.distincts</name>
+			<value>true</value>
+			<description>Whether to optimize a multi-groupby query with the same
+				distinct.
+				Consider a query like:
+
+				from src
+				insert overwrite table dest1
+				select col1, count(distinct colx) group by
+				col1
+				insert overwrite table
+				dest2 select col2, count(distinct colx) group by
+				col2;
+
+				With this
+				parameter set to true, first we spray by the distinct value
+				(colx),
+				and then
+				perform the 2 groups bys. This makes sense if map-side
+				aggregation is
+				turned off. However,
+				with maps-side aggregation, it
+				might be useful in some cases to treat
+				the 2 inserts independently,
+				thereby performing the query above in 2MR jobs instead of 3 (due to
+				spraying by distinct key first).
+				If this parameter is turned off, we
+				dont consider the fact that the
+				distinct key is the same across
+				different MR jobs.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.groupby.mapaggr.checkinterval</name>
+			<value>100000</value>
+			<description>Number of rows after which size of the grouping
+				keys/aggregation classes is performed
+			</description>
+		</property>
+
+		<property>
+			<name>hive.mapred.local.mem</name>
+			<value>0</value>
+			<description>For local mode, memory of the mappers/reducers
+			</description>
+		</property>
+
+		<property>
+			<name>hive.mapjoin.followby.map.aggr.hash.percentmemory</name>
+			<value>0.3</value>
+			<description>Portion of total memory to be used by map-side grup
+				aggregation hash table, when this group by is followed by map join
+			</description>
+		</property>
+
+		<property>
+			<name>hive.map.aggr.hash.force.flush.memory.threshold</name>
+			<value>0.9</value>
+			<description>The max memory to be used by map-side grup aggregation
+				hash table, if the memory usage is higher than this number, force to
+				flush data
+			</description>
+		</property>
+
+		<property>
+			<name>hive.map.aggr.hash.percentmemory</name>
+			<value>0.5</value>
+			<description>Portion of total memory to be used by map-side grup
+				aggregation hash table
+			</description>
+		</property>
+
+		<property>
+			<name>hive.map.aggr.hash.min.reduction</name>
+			<value>0.5</value>
+			<description>Hash aggregation will be turned off if the ratio between
+				hash
+				table size and input rows is bigger than this number. Set to 1
+				to make
+				sure
+				hash aggregation is never turned off.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.cp</name>
+			<value>true</value>
+			<description>Whether to enable column pruner</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.index.filter</name>
+			<value>false</value>
+			<description>Whether to enable automatic use of indexes</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.index.groupby</name>
+			<value>false</value>
+			<description>Whether to enable optimization of group-by queries using
+				Aggregate indexes.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.ppd</name>
+			<value>true</value>
+			<description>Whether to enable predicate pushdown</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.ppd.storage</name>
+			<value>true</value>
+			<description>Whether to push predicates down into storage handlers.
+				Ignored when hive.optimize.ppd is false.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.ppd.recognizetransivity</name>
+			<value>true</value>
+			<description>Whether to transitively replicate predicate filters over
+				equijoin conditions.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.groupby</name>
+			<value>true</value>
+			<description>Whether to enable the bucketed group by from bucketed
+				partitions/tables.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.skewjoin.compiletime</name>
+			<value>false</value>
+			<description>Whether to create a separate plan for skewed keys for
+				the tables in the join.
+				This is based on the skewed keys stored in
+				the metadata. At compile
+				time, the plan is broken
+				into different
+				joins: one for the skewed keys, and the other for the
+				remaining keys.
+				And then,
+				a union is performed for the 2 joins generated above. So
+				unless the
+				same skewed key is present
+				in both the joined tables, the
+				join for the skewed key will be
+				performed as a map-side join.
+
+				The main
+				difference between this paramater and hive.optimize.skewjoin
+				is that
+				this parameter
+				uses the skew information stored in the metastore to
+				optimize the plan
+				at compile time itself.
+				If there is no skew
+				information in the metadata, this parameter will
+				not have any affect.
+				Both hive.optimize.skewjoin.compiletime and hive.optimize.skewjoin
+				should be set to true.
+				Ideally, hive.optimize.skewjoin should be
+				renamed as
+				hive.optimize.skewjoin.runtime, but not doing
+				so for
+				backward compatibility.
+
+				If the skew information is correctly stored
+				in the metadata,
+				hive.optimize.skewjoin.compiletime
+				would change the
+				query plan to take care of it, and
+				hive.optimize.skewjoin will be a
+				no-op.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.union.remove</name>
+			<value>false</value>
+			<description>
+				Whether to remove the union and push the operators
+				between union and the
+				filesink above
+				union. This avoids an extra scan
+				of the output by union. This is
+				independently useful for union
+				queries, and specially useful when
+				hive.optimize.skewjoin.compiletime is set
+				to true, since an
+				extra
+				union is inserted.
+
+				The merge is triggered if either of
+				hive.merge.mapfiles or
+				hive.merge.mapredfiles is set to true.
+				If the
+				user has set hive.merge.mapfiles to true and
+				hive.merge.mapredfiles
+				to false, the idea was the
+				number of reducers are few, so the number
+				of files anyway are small.
+				However, with this optimization,
+				we are
+				increasing the number of files possibly by a big margin. So, we
+				merge aggresively.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.mapred.supports.subdirectories</name>
+			<value>false</value>
+			<description>Whether the version of hadoop which is running supports
+				sub-directories for tables/partitions.
+				Many hive optimizations can be
+				applied if the hadoop version supports
+				sub-directories for
+				tables/partitions. It was added by MAPREDUCE-1501
+			</description>
+		</property>
+
+		<property>
+			<name>hive.multigroupby.singlemr</name>
+			<value>true</value>
+			<description>Whether to optimize multi group by query to generate
+				single M/R
+				job plan. If the multi group by query has common group by
+				keys, it will
+				be
+				optimized to generate single M/R job.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.map.groupby.sorted</name>
+			<value>false</value>
+			<description>If the bucketing/sorting properties of the table exactly
+				match the grouping key, whether to
+				perform the group by in the mapper
+				by using BucketizedHiveInputFormat. The
+				only downside to this
+				is that
+				it limits the number of mappers to the number of files.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.map.groupby.sorted.testmode</name>
+			<value>false</value>
+			<description>If the bucketing/sorting properties of the table exactly
+				match the grouping key, whether to
+				perform the group by in the mapper
+				by using BucketizedHiveInputFormat. If
+				the test mode is set, the plan
+				is not converted, but a query property is set to denote the same.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.new.job.grouping.set.cardinality</name>
+			<value>30</value>
+			<description>
+				Whether a new map-reduce job should be launched for
+				grouping
+				sets/rollups/cubes.
+				For a query like: select a, b, c,
+				count(1) from T group by a, b, c with
+				rollup;
+				4 rows are created per
+				row: (a, b, c), (a, b, null), (a, null, null),
+				(null, null, null).
+				This can lead to explosion across map-reduce boundary if the
+				cardinality
+				of T is very high,
+				and map-side aggregation does not do a
+				very good job.
+
+				This parameter decides if hive should add an
+				additional map-reduce job.
+				If the grouping set
+				cardinality (4 in the
+				example above), is more than this value, a new MR job is
+				added under
+				the
+				assumption that the orginal group by will reduce the data size.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.join.emit.interval</name>
+			<value>1000</value>
+			<description>How many rows in the right-most join operand Hive should
+				buffer before emitting the join result.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.join.cache.size</name>
+			<value>25000</value>
+			<description>How many rows in the joining tables (except the
+				streaming table) should be cached in memory.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.mapjoin.bucket.cache.size</name>
+			<value>100</value>
+			<description>How many values in each keys in the map-joined table
+				should be cached in memory.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.mapjoin.cache.numrows</name>
+			<value>25000</value>
+			<description>How many rows should be cached by jdbm for map join.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.skewjoin</name>
+			<value>false</value>
+			<description>Whether to enable skew join optimization.
+				The algorithm
+				is as follows: At runtime, detect the keys with a large
+				skew. Instead
+				of
+				processing those keys, store them temporarily in a hdfs directory.
+				In a
+				follow-up map-reduce
+				job, process those skewed keys. The same key
+				need not be skewed for all
+				the tables, and so,
+				the follow-up
+				map-reduce job (for the skewed keys) would be much
+				faster, since it
+				would be a
+				map-join.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.skewjoin.key</name>
+			<value>100000</value>
+			<description>Determine if we get a skew key in join. If we see more
+				than the specified number of rows with the same key in join
+				operator,
+				we think the key as a skew join key.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.skewjoin.mapjoin.map.tasks</name>
+			<value>10000</value>
+			<description> Determine the number of map task used in the follow up
+				map join job
+				for a skew join. It should be used together with
+				hive.skewjoin.mapjoin.min.split
+				to perform a fine grained control.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.skewjoin.mapjoin.min.split</name>
+			<value>33554432</value>
+			<description> Determine the number of map task at most used in the
+				follow up map join job
+				for a skew join by specifying the minimum
+				split size. It should be used
+				together with
+				hive.skewjoin.mapjoin.map.tasks to perform a fine grained control.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.mapred.mode</name>
+			<value>nonstrict</value>
+			<description>The mode in which the hive operations are being
+				performed.
+				In strict mode, some risky queries are not allowed to run.
+				They
+				include:
+				Cartesian Product.
+				No partition being picked up for a
+				query.
+				Comparing bigints and strings.
+				Comparing bigints and doubles.
+				Orderby without limit.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.enforce.bucketmapjoin</name>
+			<value>false</value>
+			<description>If the user asked for bucketed map-side join, and it
+				cannot be performed,
+				should the query fail or not ? For eg, if the
+				buckets in the tables being
+				joined are
+				not a multiple of each other,
+				bucketed map-side join cannot be
+				performed, and the
+				query will fail if
+				hive.enforce.bucketmapjoin is set to true.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.script.maxerrsize</name>
+			<value>100000</value>
+			<description>Maximum number of bytes a script is allowed to emit to
+				standard error (per map-reduce task). This prevents runaway scripts
+				from filling logs partitions to capacity
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.script.allow.partial.consumption</name>
+			<value>false</value>
+			<description> When enabled, this option allows a user script to exit
+				successfully without consuming all the data from the standard input.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.script.operator.id.env.var</name>
+			<value>HIVE_SCRIPT_OPERATOR_ID</value>
+			<description> Name of the environment variable that holds the unique
+				script operator ID in the user's transform function (the custom
+				mapper/reducer that the user has specified in the query)
+			</description>
+		</property>
+
+		<property>
+			<name>hive.script.operator.truncate.env</name>
+			<value>false</value>
+			<description>Truncate each environment variable for external script
+				in scripts operator to 20KB (to fit system limits)
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.compress.output</name>
+			<value>false</value>
+			<description> This controls whether the final outputs of a query (to
+				a local/hdfs file or a hive table) is compressed. The compression
+				codec and other options are determined from hadoop config variables
+				mapred.output.compress*
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.compress.intermediate</name>
+			<value>false</value>
+			<description> This controls whether intermediate files produced by
+				hive between multiple map-reduce jobs are compressed. The
+				compression codec and other options are determined from hadoop
+				config variables mapred.output.compress*
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.parallel</name>
+			<value>false</value>
+			<description>Whether to execute jobs in parallel</description>
+		</property>
+
+		<property>
+			<name>hive.exec.parallel.thread.number</name>
+			<value>8</value>
+			<description>How many jobs at most can be executed in parallel
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.rowoffset</name>
+			<value>false</value>
+			<description>Whether to provide the row offset virtual column
+			</description>
+		</property>
+
+		<property>
+			<name>hive.task.progress</name>
+			<value>false</value>
+			<description>Whether Hive should periodically update task progress
+				counters during execution. Enabling this allows task progress to be
+				monitored more closely in the job tracker, but may impose a
+				performance penalty. This flag is automatically set to true for jobs
+				with hive.exec.dynamic.partition set to true.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.hwi.war.file</name>
+			<value>lib/hive-hwi-@VERSION@.war</value>
+			<description>This sets the path to the HWI war file, relative to
+				${HIVE_HOME}.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.hwi.listen.host</name>
+			<value>0.0.0.0</value>
+			<description>This is the host address the Hive Web Interface will
+				listen on
+			</description>
+		</property>
+
+		<property>
+			<name>hive.hwi.listen.port</name>
+			<value>9999</value>
+			<description>This is the port the Hive Web Interface will listen on
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.pre.hooks</name>
+			<value></value>
+			<description>Comma-separated list of pre-execution hooks to be
+				invoked for each statement. A pre-execution hook is specified as the
+				name of a Java class which implements the
+				org.apache.hadoop.hive.ql.hooks.ExecuteWithHookContext interface.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.post.hooks</name>
+			<value></value>
+			<description>Comma-separated list of post-execution hooks to be
+				invoked for each statement. A post-execution hook is specified as
+				the name of a Java class which implements the
+				org.apache.hadoop.hive.ql.hooks.ExecuteWithHookContext interface.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.failure.hooks</name>
+			<value></value>
+			<description>Comma-separated list of on-failure hooks to be invoked
+				for each statement. An on-failure hook is specified as the name of
+				Java class which implements the
+				org.apache.hadoop.hive.ql.hooks.ExecuteWithHookContext interface.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.init.hooks</name>
+			<value></value>
+			<description>A comma separated list of hooks to be invoked at the
+				beginning of HMSHandler initialization. Aninit hook is specified as
+				the name of Java class which extends
+				org.apache.hadoop.hive.metastore.MetaStoreInitListener.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.client.stats.publishers</name>
+			<value></value>
+			<description>Comma-separated list of statistics publishers to be
+				invoked on counters on each job. A client stats publisher is
+				specified as the name of a Java class which implements the
+				org.apache.hadoop.hive.ql.stats.ClientStatsPublisher interface.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.client.stats.counters</name>
+			<value></value>
+			<description>Subset of counters that should be of interest for
+				hive.client.stats.publishers (when one wants to limit their
+				publishing). Non-display names should be used
+			</description>
+		</property>
+
+		<property>
+			<name>hive.merge.mapfiles</name>
+			<value>true</value>
+			<description>Merge small files at the end of a map-only job
+			</description>
+		</property>
+
+		<property>
+			<name>hive.merge.mapredfiles</name>
+			<value>false</value>
+			<description>Merge small files at the end of a map-reduce job
+			</description>
+		</property>
+
+		<property>
+			<name>hive.heartbeat.interval</name>
+			<value>1000</value>
+			<description>Send a heartbeat after this interval - used by mapjoin
+				and filter operators
+			</description>
+		</property>
+
+		<property>
+			<name>hive.merge.size.per.task</name>
+			<value>256000000</value>
+			<description>Size of merged files at the end of the job</description>
+		</property>
+
+		<property>
+			<name>hive.merge.smallfiles.avgsize</name>
+			<value>16000000</value>
+			<description>When the average output file size of a job is less than
+				this number, Hive will start an additional map-reduce job to merge
+				the output files into bigger files. This is only done for map-only
+				jobs if hive.merge.mapfiles is true, and for map-reduce jobs if
+				hive.merge.mapredfiles is true.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.mapjoin.smalltable.filesize</name>
+			<value>25000000</value>
+			<description>The threshold for the input file size of the small
+				tables; if the file size is smaller than this threshold, it will try
+				to convert the common join into map join
+			</description>
+		</property>
+
+		<property>
+			<name>hive.ignore.mapjoin.hint</name>
+			<value>true</value>
+			<description>Ignore the mapjoin hint</description>
+		</property>
+
+		<property>
+			<name>hive.mapjoin.localtask.max.memory.usage</name>
+			<value>0.90</value>
+			<description>This number means how much memory the local task can
+				take to hold the key/value into in-memory hash table; If the local
+				task's memory usage is more than this number, the local task will be
+				abort by themself. It means the data of small table is too large to
+				be hold in the memory.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.mapjoin.followby.gby.localtask.max.memory.usage</name>
+			<value>0.55</value>
+			<description>This number means how much memory the local task can
+				take to hold the key/value into in-memory hash table when this map
+				join followed by a group by; If the local task's memory usage is
+				more than this number, the local task will be abort by themself. It
+				means the data of small table is too large to be hold in the memory.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.mapjoin.check.memory.rows</name>
+			<value>100000</value>
+			<description>The number means after how many rows processed it needs
+				to check the memory usage
+			</description>
+		</property>
+
+		<property>
+			<name>hive.auto.convert.join</name>
+			<value>false</value>
+			<description>Whether Hive enable the optimization about converting
+				common join into mapjoin based on the input file size
+			</description>
+		</property>
+
+		<property>
+			<name>hive.auto.convert.join.noconditionaltask</name>
+			<value>true</value>
+			<description>Whether Hive enable the optimization about converting
+				common join into mapjoin based on the input file
+				size. If this
+				paramater is on, and the sum of size for n-1 of the
+				tables/partitions for a n-way join is smaller than the
+				specified
+				size, the join is directly converted to a mapjoin (there is no
+				conditional task).
+			</description>
+		</property>
+
+		<property>
+			<name>hive.auto.convert.join.noconditionaltask.size</name>
+			<value>10000000</value>
+			<description>If hive.auto.convert.join.noconditionaltask is off, this
+				parameter does not take affect. However, if it
+				is on, and the sum of
+				size for n-1 of the tables/partitions for a
+				n-way join is smaller
+				than this size, the join is directly
+				converted to a mapjoin(there is
+				no conditional task). The default is 10MB
+			</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.mapjoin.mapreduce</name>
+			<value>false</value>
+			<description>If hive.auto.convert.join is off, this parameter does
+				not take
+				affect. If it is on, and if there are map-join jobs followed
+				by a
+				map-reduce
+				job (for e.g a group by), each map-only job is merged
+				with the
+				following
+				map-reduce job.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.script.auto.progress</name>
+			<value>false</value>
+			<description>Whether Hive Tranform/Map/Reduce Clause should
+				automatically send progress information to TaskTracker to avoid the
+				task getting killed because of inactivity. Hive sends progress
+				information when the script is outputting to stderr. This option
+				removes the need of periodically producing stderr messages, but
+				users should be cautious because this may prevent infinite loops in
+				the scripts to be killed by TaskTracker.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.script.serde</name>
+			<value>org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe</value>
+			<description>The default serde for trasmitting input data to and
+				reading output data from the user scripts.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.binary.record.max.length</name>
+			<value>1000</value>
+			<description>Read from a binary stream and treat each
+				hive.binary.record.max.length bytes as a record.
+				The last record
+				before the end of stream can have less than
+				hive.binary.record.max.length bytes
+			</description>
+		</property>
+
+
+		<property>
+			<name>hive.script.recordreader</name>
+			<value>org.apache.hadoop.hive.ql.exec.TextRecordReader</value>
+			<description>The default record reader for reading data from the user
+				scripts.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.script.recordwriter</name>
+			<value>org.apache.hadoop.hive.ql.exec.TextRecordWriter</value>
+			<description>The default record writer for writing data to the user
+				scripts.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.input.format</name>
+			<value>org.apache.hadoop.hive.ql.io.CombineHiveInputFormat</value>
+			<description>The default input format. Set this to HiveInputFormat if
+				you encounter problems with CombineHiveInputFormat.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.udtf.auto.progress</name>
+			<value>false</value>
+			<description>Whether Hive should automatically send progress
+				information to TaskTracker when using UDTF's to prevent the task
+				getting killed because of inactivity. Users should be cautious
+				because this may prevent TaskTracker from killing tasks with infinte
+				loops.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.mapred.reduce.tasks.speculative.execution</name>
+			<value>true</value>
+			<description>Whether speculative execution for reducers should be
+				turned on.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.counters.pull.interval</name>
+			<value>1000</value>
+			<description>The interval with which to poll the JobTracker for the
+				counters the running job. The smaller it is the more load there will
+				be on the jobtracker, the higher it is the less granular the caught
+				will be.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.querylog.location</name>
+			<value>/tmp/${user.name}</value>
+			<description>
+				Location of Hive run time structured log file
+			</description>
+		</property>
+
+		<property>
+			<name>hive.querylog.enable.plan.progress</name>
+			<value>true</value>
+			<description>
+				Whether to log the plan's progress every time a job's
+				progress is checked.
+				These logs are written to the location specified
+				by
+				hive.querylog.location
+			</description>
+		</property>
+
+		<property>
+			<name>hive.querylog.plan.progress.interval</name>
+			<value>60000</value>
+			<description>
+				The interval to wait between logging the plan's progress
+				in
+				milliseconds.
+				If there is a whole number percentage change in the
+				progress of the
+				mappers or the reducers,
+				the progress is logged
+				regardless of this value.
+				The actual interval will be the ceiling of
+				(this value divided by the
+				value of
+				hive.exec.counters.pull.interval)
+				multiplied by the value of hive.exec.counters.pull.interval
+				I.e. if
+				it is not divide evenly by the value of
+				hive.exec.counters.pull.interval it will be
+				logged less frequently
+				than specified.
+				This only has an effect if
+				hive.querylog.enable.plan.progress is set to
+				true.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.enforce.bucketing</name>
+			<value>false</value>
+			<description>Whether bucketing is enforced. If true, while inserting
+				into the table, bucketing is enforced.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.enforce.sorting</name>
+			<value>false</value>
+			<description>Whether sorting is enforced. If true, while inserting
+				into the table, sorting is enforced.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.bucketingsorting</name>
+			<value>true</value>
+			<description>If hive.enforce.bucketing or hive.enforce.sorting is
+				true, dont create a reducer for enforcing
+				bucketing/sorting for
+				queries of the form:
+				insert overwrite table T2 select * from T1;
+				where T1 and T2 are bucketed/sorted by the same keys into the same
+				number
+				of buckets.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.enforce.sortmergebucketmapjoin</name>
+			<value>false</value>
+			<description>If the user asked for sort-merge bucketed map-side join,
+				and it cannot be performed,
+				should the query fail or not ?
+			</description>
+		</property>
+
+		<property>
+			<name>hive.auto.convert.sortmerge.join</name>
+			<value>false</value>
+			<description>Will the join be automatically converted to a sort-merge
+				join, if the joined tables pass
+				the criteria for sort-merge join.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.auto.convert.sortmerge.join.bigtable.selection.policy
+			</name>
+			<value>org.apache.hadoop.hive.ql.optimizer.AvgPartitionSizeBasedBigTableSelectorForAutoSMJ
+			</value>
+			<description>The policy to choose the big table for automatic
+				conversion to sort-merge join.
+				By default, the table with the largest
+				partitions is assigned the big
+				table. All policies are:
+				. based on
+				position of the table - the leftmost table is selected
+				org.apache.hadoop.hive.ql.optimizer.LeftmostBigTableSMJ.
+				. based on
+				total size (all the partitions selected in the query) of
+				the table
+				org.apache.hadoop.hive.ql.optimizer.TableSizeBasedBigTableSelectorForAutoSMJ.
+				. based on average size (all the partitions selected in the query)
+				of the table
+				org.apache.hadoop.hive.ql.optimizer.AvgPartitionSizeBasedBigTableSelectorForAutoSMJ.
+				New policies can be added in future.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.ds.connection.url.hook</name>
+			<value></value>
+			<description>Name of the hook to use for retriving the JDO connection
+				URL. If empty, the value in javax.jdo.option.ConnectionURL is used
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.ds.retry.attempts</name>
+			<value>1</value>
+			<description>The number of times to retry a metastore call if there
+				were a connection error
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.ds.retry.interval</name>
+			<value>1000</value>
+			<description>The number of miliseconds between metastore retry
+				attempts
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.server.min.threads</name>
+			<value>200</value>
+			<description>Minimum number of worker threads in the Thrift server's
+				pool.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.server.max.threads</name>
+			<value>100000</value>
+			<description>Maximum number of worker threads in the Thrift server's
+				pool.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.server.tcp.keepalive</name>
+			<value>true</value>
+			<description>Whether to enable TCP keepalive for the metastore
+				server. Keepalive will prevent accumulation of half-open
+				connections.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.sasl.enabled</name>
+			<value>false</value>
+			<description>If true, the metastore thrift interface will be secured
+				with SASL. Clients must authenticate with Kerberos.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.thrift.framed.transport.enabled</name>
+			<value>false</value>
+			<description>If true, the metastore thrift interface will use
+				TFramedTransport. When false (default) a standard TTransport is
+				used.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.kerberos.keytab.file</name>
+			<value></value>
+			<description>The path to the Kerberos Keytab file containing the
+				metastore thrift server's service principal.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.kerberos.principal</name>
+			<value>hive-metastore/_HOST@EXAMPLE.COM</value>
+			<description>The service principal for the metastore thrift server.
+				The special string _HOST will be replaced automatically with the
+				correct host name.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.cluster.delegation.token.store.class</name>
+			<value>org.apache.hadoop.hive.thrift.MemoryTokenStore</value>
+			<description>The delegation token store implementation. Set to
+				org.apache.hadoop.hive.thrift.ZooKeeperTokenStore for load-balanced
+				cluster.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.cluster.delegation.token.store.zookeeper.connectString
+			</name>
+			<value>localhost:2181</value>
+			<description>The ZooKeeper token store connect string.</description>
+		</property>
+
+		<property>
+			<name>hive.cluster.delegation.token.store.zookeeper.znode</name>
+			<value>/hive/cluster/delegation</value>
+			<description>The root path for token store data.</description>
+		</property>
+
+		<property>
+			<name>hive.cluster.delegation.token.store.zookeeper.acl</name>
+			<value>sasl:hive/host1@EXAMPLE.COM:cdrwa,sasl:hive/host2@EXAMPLE.COM:cdrwa
+			</value>
+			<description>ACL for token store entries. List comma separated all
+				server principals for the cluster.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.cache.pinobjtypes</name>
+			<value>Table,StorageDescriptor,SerDeInfo,Partition,Database,Type,FieldSchema,Order
+			</value>
+			<description>List of comma separated metastore object types that
+				should be pinned in the cache
+			</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.reducededuplication</name>
+			<value>true</value>
+			<description>Remove extra map-reduce jobs if the data is already
+				clustered by the same key which needs to be used again. This should
+				always be set to true. Since it is a new feature, it has been made
+				configurable.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.reducededuplication.min.reducer</name>
+			<value>4</value>
+			<description>Reduce deduplication merges two RSs by moving
+				key/parts/reducer-num of the child RS to parent RS.
+				That means if
+				reducer-num of the child RS is fixed (order by or forced
+				bucketing)
+				and small, it can make very slow, single MR.
+				The optimization will be
+				disabled if number of reducers is less than
+				specified value.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.dynamic.partition</name>
+			<value>true</value>
+			<description>Whether or not to allow dynamic partitions in DML/DDL.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.dynamic.partition.mode</name>
+			<value>strict</value>
+			<description>In strict mode, the user must specify at least one
+				static partition in case the user accidentally overwrites all
+				partitions.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.max.dynamic.partitions</name>
+			<value>1000</value>
+			<description>Maximum number of dynamic partitions allowed to be
+				created in total.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.max.dynamic.partitions.pernode</name>
+			<value>100</value>
+			<description>Maximum number of dynamic partitions allowed to be
+				created in each mapper/reducer node.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.max.created.files</name>
+			<value>100000</value>
+			<description>Maximum number of HDFS files created by all
+				mappers/reducers in a MapReduce job.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.default.partition.name</name>
+			<value>__HIVE_DEFAULT_PARTITION__</value>
+			<description>The default partition name in case the dynamic partition
+				column value is null/empty string or anyother values that cannot be
+				escaped. This value must not contain any special character used in
+				HDFS URI (e.g., ':', '%', '/' etc). The user has to be aware that
+				the dynamic partition value should not contain this value to avoid
+				confusions.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.stats.dbclass</name>
+			<value>jdbc:derby</value>
+			<description>The default database that stores temporary hive
+				statistics.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.stats.autogather</name>
+			<value>true</value>
+			<description>A flag to gather statistics automatically during the
+				INSERT OVERWRITE command.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.stats.jdbcdriver</name>
+			<value>org.apache.derby.jdbc.EmbeddedDriver</value>
+			<description>The JDBC driver for the database that stores temporary
+				hive statistics.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.stats.dbconnectionstring</name>
+			<value>jdbc:derby:;databaseName=TempStatsStore;create=true</value>
+			<description>The default connection string for the database that
+				stores temporary hive statistics.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.stats.default.publisher</name>
+			<value></value>
+			<description>The Java class (implementing the StatsPublisher
+				interface) that is used by default if hive.stats.dbclass is not JDBC
+				or HBase.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.stats.default.aggregator</name>
+			<value></value>
+			<description>The Java class (implementing the StatsAggregator
+				interface) that is used by default if hive.stats.dbclass is not JDBC
+				or HBase.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.stats.jdbc.timeout</name>
+			<value>30</value>
+			<description>Timeout value (number of seconds) used by JDBC
+				connection and statements.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.stats.retries.max</name>
+			<value>0</value>
+			<description>Maximum number of retries when stats
+				publisher/aggregator got an exception updating intermediate
+				database. Default is no tries on failures.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.stats.retries.wait</name>
+			<value>3000</value>
+			<description>The base waiting window (in milliseconds) before the
+				next retry. The actual wait time is calculated by baseWindow *
+				failues baseWindow * (failure 1) * (random number between
+				[0.0,1.0]).
+			</description>
+		</property>
+
+		<property>
+			<name>hive.stats.reliable</name>
+			<value>false</value>
+			<description>Whether queries will fail because stats cannot be
+				collected completely accurately.
+				If this is set to true,
+				reading/writing from/into a partition may fail
+				becuase the stats
+				could not be computed accurately.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.stats.collect.tablekeys</name>
+			<value>false</value>
+			<description>Whether join and group by keys on tables are derived and
+				maintained in the QueryPlan.
+				This is useful to identify how tables
+				are accessed and to determine if
+				they should be bucketed.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.stats.collect.scancols</name>
+			<value>false</value>
+			<description>Whether column accesses are tracked in the QueryPlan.
+				This is useful to identify how tables are accessed and to determine
+				if there are wasted columns that can be trimmed.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.stats.ndv.error</name>
+			<value>20.0</value>
+			<description>Standard error expressed in percentage. Provides a
+				tradeoff between accuracy and compute cost.A lower value for error
+				indicates higher accuracy and a higher compute cost.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.stats.key.prefix.max.length</name>
+			<value>200</value>
+			<description>
+				Determines if when the prefix of the key used for
+				intermediate stats
+				collection
+				exceeds a certain length, a hash of the
+				key is used instead. If the value
+				&lt; 0 then hashing
+				is never used,
+				if the value >= 0 then hashing is used only when the
+				key prefixes
+				length
+				exceeds that value. The key prefix is defined as everything
+				preceding the
+				task ID in the key.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.support.concurrency</name>
+			<value>false</value>
+			<description>Whether hive supports concurrency or not. A zookeeper
+				instance must be up and running for the default hive lock manager to
+				support read-write locks.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.lock.numretries</name>
+			<value>100</value>
+			<description>The number of times you want to try to get all the locks
+			</description>
+		</property>
+
+		<property>
+			<name>hive.unlock.numretries</name>
+			<value>10</value>
+			<description>The number of times you want to retry to do one unlock
+			</description>
+		</property>
+
+		<property>
+			<name>hive.lock.sleep.between.retries</name>
+			<value>60</value>
+			<description>The sleep time (in seconds) between various retries
+			</description>
+		</property>
+
+		<property>
+			<name>hive.zookeeper.quorum</name>
+			<value></value>
+			<description>The list of zookeeper servers to talk to. This is only
+				needed for read/write locks.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.zookeeper.client.port</name>
+			<value>2181</value>
+			<description>The port of zookeeper servers to talk to. This is only
+				needed for read/write locks.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.zookeeper.session.timeout</name>
+			<value>600000</value>
+			<description>Zookeeper client's session timeout. The client is
+				disconnected, and as a result, all locks released, if a heartbeat is
+				not sent in the timeout.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.zookeeper.namespace</name>
+			<value>hive_zookeeper_namespace</value>
+			<description>The parent node under which all zookeeper nodes are
+				created.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.zookeeper.clean.extra.nodes</name>
+			<value>false</value>
+			<description>Clean extra nodes at the end of the session.
+			</description>
+		</property>
+
+		<property>
+			<name>fs.har.impl</name>
+			<value>org.apache.hadoop.hive.shims.HiveHarFileSystem</value>
+			<description>The implementation for accessing Hadoop Archives. Note
+				that this won't be applicable to Hadoop vers less than 0.20
+			</description>
+		</property>
+
+		<property>
+			<name>hive.archive.enabled</name>
+			<value>false</value>
+			<description>Whether archiving operations are permitted</description>
+		</property>
+
+		<property>
+			<name>hive.fetch.output.serde</name>
+			<value>org.apache.hadoop.hive.serde2.DelimitedJSONSerDe</value>
+			<description>The serde used by FetchTask to serialize the fetch
+				output.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.mode.local.auto</name>
+			<value>false</value>
+			<description> Let hive determine whether to run in local mode
+				automatically
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.drop.ignorenonexistent</name>
+			<value>true</value>
+			<description>
+				Do not report an error if DROP TABLE/VIEW specifies a
+				non-existent
+				table/view
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.show.job.failure.debug.info</name>
+			<value>true</value>
+			<description>
+				If a job fails, whether to provide a link in the CLI to
+				the task with
+				the
+				most failures, along with debugging hints if
+				applicable.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.auto.progress.timeout</name>
+			<value>0</value>
+			<description>
+				How long to run autoprogressor for the script/UDTF
+				operators (in
+				seconds).
+				Set to 0 for forever.
+			</description>
+		</property>
+
+		<!-- HBase Storage Handler Parameters -->
+
+		<property>
+			<name>hive.hbase.wal.enabled</name>
+			<value>true</value>
+			<description>Whether writes to HBase should be forced to the
+				write-ahead log. Disabling this improves HBase write performance at
+				the risk of lost writes in case of a crash.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.table.parameters.default</name>
+			<value></value>
+			<description>Default property values for newly created tables
+			</description>
+		</property>
+
+		<property>
+			<name>hive.entity.separator</name>
+			<value>@</value>
+			<description>Separator used to construct names of tables and
+				partitions. For example, dbname@tablename@partitionname
+			</description>
+		</property>
+
+		<property>
+			<name>hive.ddl.createtablelike.properties.whitelist</name>
+			<value></value>
+			<description>Table Properties to copy over when executing a Create
+				Table Like.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.variable.substitute</name>
+			<value>true</value>
+			<description>This enables substitution using syntax like ${var}
+				${system:var} and ${env:var}.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.variable.substitute.depth</name>
+			<value>40</value>
+			<description>The maximum replacements the substitution engine will
+				do.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.conf.validation</name>
+			<value>true</value>
+			<description>Eables type checking for registered hive configurations
+			</description>
+		</property>
+
+		<property>
+			<name>hive.security.authorization.enabled</name>
+			<value>false</value>
+			<description>enable or disable the hive client authorization
+			</description>
+		</property>
+
+		<property>
+			<name>hive.security.authorization.createtable.user.grants</name>
+			<value></value>
+			<description>the privileges automatically granted to some users
+				whenever a table gets created.
+				An example like
+				"userX,userY:select;userZ:create" will grant select
+				privilege to
+				userX and userY,
+				and grant create privilege to userZ whenever a new
+				table created.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.security.authorization.createtable.group.grants</name>
+			<value></value>
+			<description>the privileges automatically granted to some groups
+				whenever a table gets created.
+				An example like
+				"groupX,groupY:select;groupZ:create" will grant select
+				privilege to
+				groupX and groupY,
+				and grant create privilege to groupZ whenever a
+				new table created.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.security.authorization.createtable.role.grants</name>
+			<value></value>
+			<description>the privileges automatically granted to some roles
+				whenever a table gets created.
+				An example like
+				"roleX,roleY:select;roleZ:create" will grant select
+				privilege to
+				roleX and roleY,
+				and grant create privilege to roleZ whenever a new
+				table created.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.security.authorization.createtable.owner.grants</name>
+			<value></value>
+			<description>the privileges automatically granted to the owner
+				whenever a table gets created.
+				An example like "select,drop" will
+				grant select and drop privilege to
+				the owner of the table
+			</description>
+		</property>
+
+		<property>
+			<name>hive.metastore.authorization.storage.checks</name>
+			<value>false</value>
+			<description>Should the metastore do authorization checks against the
+				underlying storage
+				for operations like drop-partition (disallow the
+				drop-partition if the
+				user in
+				question doesn't have permissions to
+				delete the corresponding directory
+				on the storage).
+			</description>
+		</property>
+
+		<property>
+			<name>hive.error.on.empty.partition</name>
+			<value>false</value>
+			<description>Whether to throw an excpetion if dynamic partition
+				insert generates empty results.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.index.compact.file.ignore.hdfs</name>
+			<value>false</value>
+			<description>True the hdfs location stored in the index file will be
+				igbored at runtime.
+				If the data got moved or the name of the cluster
+				got changed, the
+				index data should still be usable.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.index.filter.compact.minsize</name>
+			<value>5368709120</value>
+			<description>Minimum size (in bytes) of the inputs on which a compact
+				index is automatically used.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.optimize.index.filter.compact.maxsize</name>
+			<value>-1</value>
+			<description>Maximum size (in bytes) of the inputs on which a compact
+				index is automatically used.
+				A negative number is equivalent to
+				infinity.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.index.compact.query.max.size</name>
+			<value>10737418240</value>
+			<description>The maximum number of bytes that a query using the
+				compact index can read. Negative value is equivalent to infinity.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.index.compact.query.max.entries</name>
+			<value>10000000</value>
+			<description>The maximum number of index entries to read during a
+				query that uses the compact index. Negative value is equivalent to
+				infinity.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.index.compact.binary.search</name>
+			<value>true</value>
+			<description>Whether or not to use a binary search to find the
+				entries in an index table that match the filter, where possible
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exim.uri.scheme.whitelist</name>
+			<value>hdfs,pfile</value>
+			<description>A comma separated list of acceptable URI schemes for
+				import and export.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.lock.mapred.only.operation</name>
+			<value>false</value>
+			<description>This param is to control whether or not only do lock on
+				queries
+				that need to execute at least one mapred job.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.limit.row.max.size</name>
+			<value>100000</value>
+			<description>When trying a smaller subset of data for simple LIMIT,
+				how much size we need to guarantee
+				each row to have at least.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.limit.optimize.limit.file</name>
+			<value>10</value>
+			<description>When trying a smaller subset of data for simple LIMIT,
+				maximum number of files we can
+				sample.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.limit.optimize.enable</name>
+			<value>false</value>
+			<description>Whether to enable to optimization to trying a smaller
+				subset of data for simple LIMIT first.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.limit.optimize.fetch.max</name>
+			<value>50000</value>
+			<description>Maximum number of rows allowed for a smaller subset of
+				data for simple LIMIT, if it is a fetch query.
+				Insert queries are not
+				restricted by this limit.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.rework.mapredwork</name>
+			<value>false</value>
+			<description>should rework the mapred work or not.
+				This is first
+				introduced by SymlinkTextInputFormat to replace symlink
+				files with
+				real paths at compile time.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.concatenate.check.index</name>
+			<value>true</value>
+			<description>If this sets to true, hive will throw error when doing
+				'alter table tbl_name [partSpec] concatenate' on a table/partition
+				that has indexes on it. The reason the user want to set this to true
+				is because it can help user to avoid handling all index drop,
+				recreation,
+				rebuild work. This is very helpful for tables with
+				thousands of partitions.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.sample.seednumber</name>
+			<value>0</value>
+			<description>A number used to percentage sampling. By changing this
+				number, user will change the subsets
+				of data sampled.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.io.exception.handlers</name>
+			<value></value>
+			<description>A list of io exception handler class names. This is used
+				to construct a list exception handlers to handle exceptions thrown
+				by record readers
+			</description>
+		</property>
+
+		<property>
+			<name>hive.autogen.columnalias.prefix.label</name>
+			<value>_c</value>
+			<description>String used as a prefix when auto generating column
+				alias.
+				By default the prefix label will be appended with a column
+				position
+				number to form the column alias. Auto generation would
+				happen if an
+				aggregate function is used in a select clause without an
+				explicit
+				alias.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.autogen.columnalias.prefix.includefuncname</name>
+			<value>false</value>
+			<description>Whether to include function name in the column alias
+				auto generated by hive.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.perf.logger</name>
+			<value>org.apache.hadoop.hive.ql.log.PerfLogger</value>
+			<description>The class responsible logging client side performance
+				metrics. Must be a subclass of
+				org.apache.hadoop.hive.ql.log.PerfLogger
+			</description>
+		</property>
+
+		<property>
+			<name>hive.start.cleanup.scratchdir</name>
+			<value>false</value>
+			<description>To cleanup the hive scratchdir while starting the hive
+				server
+			</description>
+		</property>
+
+		<property>
+			<name>hive.output.file.extension</name>
+			<value></value>
+			<description>String used as a file extension for output files. If not
+				set, defaults to the codec extension for text files (e.g. ".gz"), or
+				no extension otherwise.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.insert.into.multilevel.dirs</name>
+			<value>false</value>
+			<description>Where to insert into multilevel directories like
+				"insert
+				directory '/HIVEFT25686/chinna/' from table"
+			</description>
+		</property>
+
+		<property>
+			<name>hive.warehouse.subdir.inherit.perms</name>
+			<value>false</value>
+			<description>Set this to true if the the table directories should
+				inherit the
+				permission of the warehouse or database directory instead
+				of being created
+				with the permissions derived from dfs umask
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.job.debug.capture.stacktraces</name>
+			<value>true</value>
+			<description>Whether or not stack traces parsed from the task logs of
+				a sampled failed task for
+				each failed job should be stored in the
+				SessionState
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.driver.run.hooks</name>
+			<value></value>
+			<description>A comma separated list of hooks which implement
+				HiveDriverRunHook and will be run at the
+				beginning and end of
+				Driver.run, these will be run in the order specified
+			</description>
+		</property>
+
+		<property>
+			<name>hive.ddl.output.format</name>
+			<value>text</value>
+			<description>
+				The data format to use for DDL output. One of "text"
+				(for human
+				readable text) or "json" (for a json object).
+			</description>
+		</property>
+
+		<property>
+			<name>hive.transform.escape.input</name>
+			<value>false</value>
+			<description>
+				This adds an option to escape special chars (newlines,
+				carriage returns
+				and
+				tabs) when they are passed to the user script.
+				This is useful if the hive
+				tables
+				can contain data that contains
+				special characters.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.rcfile.use.explicit.header</name>
+			<value>true</value>
+			<description>
+				If this is set the header for RC Files will simply be
+				RCF. If this is
+				not
+				set the header will be that borrowed from sequence
+				files, e.g. SEQ-
+				followed
+				by the input and output RC File formats.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.multi.insert.move.tasks.share.dependencies</name>
+			<value>false</value>
+			<description>
+				If this is set all move tasks for tables/partitions (not
+				directories)
+				at the end of a
+				multi-insert query will only begin once
+				the dependencies for all these move
+				tasks have been
+				met.
+				Advantages: If
+				concurrency is enabled, the locks will only be released once the
+				query has
+				finished, so with this config enabled, the time when the
+				table/partition is
+				generated will be much closer to when the lock on
+				it is released.
+				Disadvantages: If concurrency is not enabled, with
+				this disabled,
+				the tables/partitions which
+				are produced by this query
+				and finish earlier will be available for
+				querying
+				much earlier. Since
+				the locks are only released once the query finishes,
+				this
+				does not
+				apply if concurrency is enabled.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.fetch.task.conversion</name>
+			<value>minimal</value>
+			<description>
+				Some select queries can be converted to single FETCH
+				task minimizing
+				latency.
+				Currently the query should be single sourced
+				not having any subquery and
+				should not have
+				any aggregations or
+				distincts (which incurrs RS), lateral views and
+				joins.
+				1. minimal :
+				SELECT STAR, FILTER on partition columns, LIMIT only
+				2. more :
+				SELECT, FILTER, LIMIT only (TABLESAMPLE, virtual columns)
+			</description>
+		</property>
+
+		<property>
+			<name>hive.hmshandler.retry.attempts</name>
+			<value>1</value>
+			<description>The number of times to retry a HMSHandler call if there
+				were a connection error
+			</description>
+		</property>
+
+		<property>
+			<name>hive.hmshandler.retry.interval</name>
+			<value>1000</value>
+			<description>The number of miliseconds between HMSHandler retry
+				attempts
+			</description>
+		</property>
+
+		<property>
+			<name>hive.server.read.socket.timeout</name>
+			<value>10</value>
+			<description>Timeout for the HiveServer to close the connection if no
+				response from the client in N seconds, defaults to 10 seconds.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.server.tcp.keepalive</name>
+			<value>true</value>
+			<description>Whether to enable TCP keepalive for the Hive server.
+				Keepalive will prevent accumulation of half-open connections.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.decode.partition.name</name>
+			<value>false</value>
+			<description>Whether to show the unquoted partition names in query
+				results.
+			</description>
+		</property>
+
+		<property>
+			<name>hive.log4j.file</name>
+			<value></value>
+			<description>Hive log4j configuration file.
+				If the property is not
+				set, then logging will be initialized using
+				hive-log4j.properties
+				found on the classpath.
+				If the property is set, the value must be a
+				valid URI (java.net.URI,
+				e.g. "file:///tmp/my-logging.properties"),
+				which you can then
+				extract a URL from and pass to
+				PropertyConfigurator.configure(URL).
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.log4j.file</name>
+			<value></value>
+			<description>Hive log4j configuration file for execution mode(sub
+				command).
+				If the property is not set, then logging will be
+				initialized using
+				hive-exec-log4j.properties found on the classpath.
+				If the property is set, the value must be a valid URI (java.net.URI,
+				e.g. "file:///tmp/my-logging.properties"), which you can then
+				extract a URL from and pass to PropertyConfigurator.configure(URL).
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.infer.bucket.sort</name>
+			<value>false</value>
+			<description>
+				If this is set, when writing partitions, the metadata
+				will include the
+				bucketing/sorting
+				properties with which the data was
+				written if any (this will not overwrite the
+				metadata
+				inherited from
+				the table if the table is bucketed/sorted)
+			</description>
+		</property>
+
+		<property>
+			<name>hive.exec.infer.bucket.sort.num.buckets.power.two</name>
+			<value>false</value>
+			<description>
+				If this is set, when setting the number of reducers for
+				the map reduce
+				task which writes the
+				final output files, it will
+				choose a number which is a power of two,
+				unless the user specifies
+				the number of reducers to use using mapred.reduce.tasks. The number
+				of
+				reducers
+				may be set to a power of two, only to be followed by a
+				merge task
+				meaning preventing
+				anything from being inferred.
+				With
+				hive.exec.infer.bucket.sort set to true:
+				Advantages: If this is not
+				set, the number of buckets for partitions will seem
+				arbitrary,
+				which
+				means that the number of mappers used for optimized joins, for
+				example, will
+				be very low. With this set, since the number of buckets
+				used for any
+				partition is
+				a power of two, the number of mappers used
+				for optimized joins will
+				be the least
+				number of buckets used by any
+				partition being joined.
+				Disadvantages: This may mean a much larger or
+				much smaller number of reducers
+				being used in the
+				final map reduce
+				job, e.g. if a job was originally going to take 257
+				reducers,
+				it will
+				now take 512 reducers, similarly if the max number of reducers
+				is
+				511,
+				and a job was going to use this many, it will now use 256
+				reducers.
+
+			</description>
+		</property>
+
+		<property>
+			<name>hive.groupby.orderby.position.alias</name>
+			<value>false</value>
+			<description>Whether to enable using Column Position Alias in Group
+				By or Order By
+			</description>
+		</property>
+
+		<property>
+			<name>hive.server2.thrift.min.worker.threads</name>
+			<value>5</value>
+			<description>Minimum number of Thrift worker threads</description>
+		</property>
+
+		<property>
+			<name>hive.server2.thrift.max.worker.threads</name>
+			<value>100</value>
+			<description>Maximum number of Thrift worker threads</description>
+		</property>
+
+		<property>
+			<name>hive.server2.thrift.port</name>
+			<value>10000</value>
+			<description>Port number of HiveServer2 Thrift interface.
+				Can be
+				overridden by setting $HIVE_SERVER2_THRIFT_PORT
+			</description>
+		</property>
+
+		<property>
+			<name>hive.server2.thrift.bind.host</name>
+			<value>localhost</value>
+			<description>Bind host on which to run the HiveServer2 Thrift
+				interface.
+				Can be overridden by setting
+				$HIVE_SERVER2_THRIFT_BIND_HOST
+			</description>
+		</property>
+
+		<property>
+			<name>hive.server2.authentication</name>
+			<value>NONE</value>
+			<description>
+				Client authentication types.
+				NONE: no authentication
+				check
+				LDAP: LDAP/AD based authentication
+				KERBEROS: Kerberos/GSSAPI
+				authentication
+				CUSTOM: Custom authentication provider
+				(Use with
+				property hive.server2.custom.authentication.class)
+			</description>
+		</property>
+
+		<property>
+			<name>hive.server2.custom.authentication.class</name>
+			<value></value>
+			<description>
+				Custom authentication class. Used when property
+				'hive.server2.authentication' is set to 'CUSTOM'. Provided class
+				must be a proper implementation of the interface
+				org.apache.hive.service.auth.PasswdAuthenticationProvider.
+				HiveServer2
+				will call its Authenticate(user, passed) method to
+				authenticate
+				requests.
+				The implementation may optionally extend the
+				Hadoop's
+				org.apache.hadoop.conf.Configured class to grab Hive's
+				Configuration
+				object.
+			</description>
+		</property>
+
+		<property>
+			<name>>hive.server2.authentication.kerberos.principal</name>
+			<value></value>
+			<description>
+				Kerberos server principal
+			</description>
+		</property>
+
+		<property>
+			<name>>hive.server2.authentication.kerberos.keytab</name>
+			<value></value>
+			<description>
+				Kerberos keytab file for server principal
+			</description>
+		</property>
+
+		<property>
+			<name>hive.server2.authentication.ldap.url</name>
+			<value></value>
+			<description>
+				LDAP connection URL
+			</description>
+		</property>
+
+		<property>
+			<name>hive.server2.authentication.ldap.baseDN</name>
+			<value></value>
+			<description>
+				LDAP base DN
+			</description>
+		</property>
+
+		<property>
+			<name>hive.server2.enable.doAs</name>
+			<value>true</value>
+			<description>
+				Setting this property to true will have hive server2
+				execute
+				hive operations as the user making the calls to it.
+			</description>
+		</property>
+
+
+	</configuration>
+
+	<!-- Hive Execution Parameters -->
+	<property>
+		<name>mapred.reduce.tasks</name>
+		<value>-1</value>
+		<description>The default number of reduce tasks per job. Typically set
+			to a prime close to the number of available hosts. Ignored when
+			mapred.job.tracker is "local". Hadoop set this to 1 by default,
+			whereas hive uses -1 as its default value.
+			By setting this property to
+			-1, Hive will automatically figure out what
+			should be the number of
+			reducers.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.reducers.bytes.per.reducer</name>
+		<value>1000000000</value>
+		<description>size per reducer.The default is 1G, i.e if the input size
+			is 10G, it will use 10 reducers.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.reducers.max</name>
+		<value>999</value>
+		<description>max number of reducers will be used. If the one
+			specified
+			in the configuration parameter mapred.reduce.tasks is
+			negative, hive
+			will use this one as the max number of reducers when
+			automatically
+			determine number of reducers.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.cli.print.header</name>
+		<value>false</value>
+		<description>Whether to print the names of the columns in query
+			output.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.cli.print.current.db</name>
+		<value>false</value>
+		<description>Whether to include the current database in the hive
+			prompt.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.cli.prompt</name>
+		<value>hive</value>
+		<description>Command line prompt configuration value. Other hiveconf
+			can be used in
+			this configuration value. Variable substitution will
+			only be invoked at
+			the hive
+			cli startup.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.cli.pretty.output.num.cols</name>
+		<value>-1</value>
+		<description>The number of columns to use when formatting output
+			generated
+			by the DESCRIBE PRETTY table_name command. If the value of
+			this
+			property
+			is -1, then hive will use the auto-detected terminal
+			width.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.scratchdir</name>
+		<value>/tmp/hive-${user.name}</value>
+		<description>Scratch space for Hive jobs</description>
+	</property>
+
+	<property>
+		<name>hive.exec.local.scratchdir</name>
+		<value>/tmp/${user.name}</value>
+		<description>Local scratch space for Hive jobs</description>
+	</property>
+
+	<property>
+		<name>hive.test.mode</name>
+		<value>false</value>
+		<description>whether hive is running in test mode. If yes, it turns on
+			sampling and prefixes the output tablename
+		</description>
+	</property>
+
+	<property>
+		<name>hive.test.mode.prefix</name>
+		<value>test_</value>
+		<description>if hive is running in test mode, prefixes the output
+			table by this string
+		</description>
+	</property>
+
+	<!-- If the input table is not bucketed, the denominator of the tablesample 
+		is determinied by the parameter below -->
+	<!-- For example, the following query: -->
+	<!-- INSERT OVERWRITE TABLE dest -->
+	<!-- SELECT col1 from src -->
+	<!-- would be converted to -->
+	<!-- INSERT OVERWRITE TABLE test_dest -->
+	<!-- SELECT col1 from src TABLESAMPLE (BUCKET 1 out of 32 on rand(1)) -->
+	<property>
+		<name>hive.test.mode.samplefreq</name>
+		<value>32</value>
+		<description>if hive is running in test mode and table is not
+			bucketed, sampling frequency
+		</description>
+	</property>
+
+	<property>
+		<name>hive.test.mode.nosamplelist</name>
+		<value></value>
+		<description>if hive is running in test mode, dont sample the above
+			comma seperated list of tables
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.uris</name>
+		<value></value>
+		<description>Thrift uri for the remote metastore. Used by metastore
+			client to connect to remote metastore.
+		</description>
+	</property>
+
+	<property>
+		<name>javax.jdo.option.ConnectionURL</name>
+		<value>jdbc:derby:;databaseName=metastore_db;create=true</value>
+		<description>JDBC connect string for a JDBC metastore</description>
+	</property>
+
+	<property>
+		<name>javax.jdo.option.ConnectionDriverName</name>
+		<value>org.apache.derby.jdbc.EmbeddedDriver</value>
+		<description>Driver class name for a JDBC metastore</description>
+	</property>
+
+	<property>
+		<name>javax.jdo.PersistenceManagerFactoryClass</name>
+		<value>org.datanucleus.jdo.JDOPersistenceManagerFactory</value>
+		<description>class implementing the jdo persistence</description>
+	</property>
+
+	<property>
+		<name>javax.jdo.option.DetachAllOnCommit</name>
+		<value>true</value>
+		<description>detaches all objects from session so that they can be
+			used after transaction is committed
+		</description>
+	</property>
+
+	<property>
+		<name>javax.jdo.option.NonTransactionalRead</name>
+		<value>true</value>
+		<description>reads outside of transactions</description>
+	</property>
+
+	<property>
+		<name>javax.jdo.option.ConnectionUserName</name>
+		<value>APP</value>
+		<description>username to use against metastore database</description>
+	</property>
+
+	<property>
+		<name>javax.jdo.option.ConnectionPassword</name>
+		<value>mine</value>
+		<description>password to use against metastore database</description>
+	</property>
+
+	<property>
+		<name>javax.jdo.option.Multithreaded</name>
+		<value>true</value>
+		<description>Set this to true if multiple threads access metastore
+			through JDO concurrently.
+		</description>
+	</property>
+
+	<property>
+		<name>datanucleus.connectionPoolingType</name>
+		<value>DBCP</value>
+		<description>Uses a DBCP connection pool for JDBC metastore
+		</description>
+	</property>
+
+	<property>
+		<name>datanucleus.validateTables</name>
+		<value>false</value>
+		<description>validates existing schema against code. turn this on if
+			you want to verify existing schema
+		</description>
+	</property>
+
+	<property>
+		<name>datanucleus.validateColumns</name>
+		<value>false</value>
+		<description>validates existing schema against code. turn this on if
+			you want to verify existing schema
+		</description>
+	</property>
+
+	<property>
+		<name>datanucleus.validateConstraints</name>
+		<value>false</value>
+		<description>validates existing schema against code. turn this on if
+			you want to verify existing schema
+		</description>
+	</property>
+
+	<property>
+		<name>datanucleus.storeManagerType</name>
+		<value>rdbms</value>
+		<description>metadata store type</description>
+	</property>
+
+	<property>
+		<name>datanucleus.autoCreateSchema</name>
+		<value>true</value>
+		<description>creates necessary schema on a startup if one doesn't
+			exist. set this to false, after creating it once
+		</description>
+	</property>
+
+	<property>
+		<name>datanucleus.autoStartMechanismMode</name>
+		<value>checked</value>
+		<description>throw exception if metadata tables are incorrect
+		</description>
+	</property>
+
+	<property>
+		<name>datanucleus.transactionIsolation</name>
+		<value>read-committed</value>
+		<description>Default transaction isolation level for identity
+			generation.
+		</description>
+	</property>
+
+	<property>
+		<name>datanucleus.cache.level2</name>
+		<value>false</value>
+		<description>Use a level 2 cache. Turn this off if metadata is changed
+			independently of hive metastore server
+		</description>
+	</property>
+
+	<property>
+		<name>datanucleus.cache.level2.type</name>
+		<value>SOFT</value>
+		<description>SOFT=soft reference based cache, WEAK=weak reference
+			based cache.
+		</description>
+	</property>
+
+	<property>
+		<name>datanucleus.identifierFactory</name>
+		<value>datanucleus</value>
+		<description>Name of the identifier factory to use when generating
+			table/column names etc. 'datanucleus' is used for backward
+			compatibility
+		</description>
+	</property>
+
+	<property>
+		<name>datanucleus.plugin.pluginRegistryBundleCheck</name>
+		<value>LOG</value>
+		<description>Defines what happens when plugin bundles are found and
+			are duplicated [EXCEPTION|LOG|NONE]
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.warehouse.dir</name>
+		<value>/user/hive/warehouse</value>
+		<description>location of default database for the warehouse
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.execute.setugi</name>
+		<value>false</value>
+		<description>In unsecure mode, setting this property to true will
+			cause the metastore to execute DFS operations using the client's
+			reported user and group permissions. Note that this property must be
+			set on both the client and server sides. Further note that its best
+			effort. If client sets its to true and server sets it to false,
+			client setting will be ignored.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.event.listeners</name>
+		<value></value>
+		<description>list of comma seperated listeners for metastore events.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.partition.inherit.table.properties</name>
+		<value></value>
+		<description>list of comma seperated keys occurring in table
+			properties which will get inherited to newly created partitions. *
+			implies all the keys will get inherited.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metadata.export.location</name>
+		<value></value>
+		<description>When used in conjunction with the
+			org.apache.hadoop.hive.ql.parse.MetaDataExportListener pre event
+			listener, it is the location to which the metadata will be exported.
+			The default is an empty string, which results in the metadata being
+			exported to the current user's home directory on HDFS.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metadata.move.exported.metadata.to.trash</name>
+		<value></value>
+		<description>When used in conjunction with the
+			org.apache.hadoop.hive.ql.parse.MetaDataExportListener pre event
+			listener, this setting determines if the metadata that is exported
+			will subsequently be moved to the user's trash directory alongside
+			the dropped table data. This ensures that the metadata will be
+			cleaned up along with the dropped table data.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.partition.name.whitelist.pattern</name>
+		<value></value>
+		<description>Partition names will be checked against this regex
+			pattern and rejected if not matched.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.end.function.listeners</name>
+		<value></value>
+		<description>list of comma separated listeners for the end of
+			metastore functions.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.event.expiry.duration</name>
+		<value>0</value>
+		<description>Duration after which events expire from events table (in
+			seconds)
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.event.clean.freq</name>
+		<value>0</value>
+		<description>Frequency at which timer task runs to purge expired
+			events in metastore(in seconds).
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.connect.retries</name>
+		<value>5</value>
+		<description>Number of retries while opening a connection to metastore
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.failure.retries</name>
+		<value>3</value>
+		<description>Number of retries upon failure of Thrift metastore calls
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.client.connect.retry.delay</name>
+		<value>1</value>
+		<description>Number of seconds for the client to wait between
+			consecutive connection attempts
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.client.socket.timeout</name>
+		<value>20</value>
+		<description>MetaStore Client socket timeout in seconds</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.rawstore.impl</name>
+		<value>org.apache.hadoop.hive.metastore.ObjectStore</value>
+		<description>Name of the class that implements
+			org.apache.hadoop.hive.metastore.rawstore interface. This class is
+			used to store and retrieval of raw metadata objects such as table,
+			database
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.batch.retrieve.max</name>
+		<value>300</value>
+		<description>Maximum number of objects (tables/partitions) can be
+			retrieved from metastore in one batch. The higher the number, the
+			less the number of round trips is needed to the Hive metastore
+			server, but it may also cause higher memory requirement at the client
+			side.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.batch.retrieve.table.partition.max</name>
+		<value>1000</value>
+		<description>Maximum number of table partitions that metastore
+			internally retrieves in one batch.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.default.fileformat</name>
+		<value>TextFile</value>
+		<description>Default file format for CREATE TABLE statement. Options
+			are TextFile and SequenceFile. Users can explicitly say CREATE TABLE
+			... STORED AS &lt;TEXTFILE|SEQUENCEFILE&gt; to override</description>
+	</property>
+
+	<property>
+		<name>hive.fileformat.check</name>
+		<value>true</value>
+		<description>Whether to check file format or not when loading data
+			files
+		</description>
+	</property>
+
+	<property>
+		<name>hive.map.aggr</name>
+		<value>true</value>
+		<description>Whether to use map-side aggregation in Hive Group By
+			queries
+		</description>
+	</property>
+
+	<property>
+		<name>hive.groupby.skewindata</name>
+		<value>false</value>
+		<description>Whether there is skew in data to optimize group by
+			queries
+		</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.multigroupby.common.distincts</name>
+		<value>true</value>
+		<description>Whether to optimize a multi-groupby query with the same
+			distinct.
+			Consider a query like:
+
+			from src
+			insert overwrite table dest1
+			select col1, count(distinct colx) group by
+			col1
+			insert overwrite table
+			dest2 select col2, count(distinct colx) group by
+			col2;
+
+			With this
+			parameter set to true, first we spray by the distinct value
+			(colx),
+			and then
+			perform the 2 groups bys. This makes sense if map-side
+			aggregation is turned
+			off. However,
+			with maps-side aggregation, it
+			might be useful in some cases to treat the
+			2 inserts independently,
+			thereby performing the query above in 2MR jobs instead of 3 (due to
+			spraying
+			by distinct key first).
+			If this parameter is turned off, we
+			dont consider the fact that the
+			distinct key is the same across
+			different MR jobs.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.groupby.mapaggr.checkinterval</name>
+		<value>100000</value>
+		<description>Number of rows after which size of the grouping
+			keys/aggregation classes is performed
+		</description>
+	</property>
+
+	<property>
+		<name>hive.mapred.local.mem</name>
+		<value>0</value>
+		<description>For local mode, memory of the mappers/reducers
+		</description>
+	</property>
+
+	<property>
+		<name>hive.mapjoin.followby.map.aggr.hash.percentmemory</name>
+		<value>0.3</value>
+		<description>Portion of total memory to be used by map-side grup
+			aggregation hash table, when this group by is followed by map join
+		</description>
+	</property>
+
+	<property>
+		<name>hive.map.aggr.hash.force.flush.memory.threshold</name>
+		<value>0.9</value>
+		<description>The max memory to be used by map-side grup aggregation
+			hash table, if the memory usage is higher than this number, force to
+			flush data
+		</description>
+	</property>
+
+	<property>
+		<name>hive.map.aggr.hash.percentmemory</name>
+		<value>0.5</value>
+		<description>Portion of total memory to be used by map-side grup
+			aggregation hash table
+		</description>
+	</property>
+
+	<property>
+		<name>hive.map.aggr.hash.min.reduction</name>
+		<value>0.5</value>
+		<description>Hash aggregation will be turned off if the ratio between
+			hash
+			table size and input rows is bigger than this number. Set to 1 to
+			make
+			sure
+			hash aggregation is never turned off.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.cp</name>
+		<value>true</value>
+		<description>Whether to enable column pruner</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.index.filter</name>
+		<value>false</value>
+		<description>Whether to enable automatic use of indexes</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.index.groupby</name>
+		<value>false</value>
+		<description>Whether to enable optimization of group-by queries using
+			Aggregate indexes.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.ppd</name>
+		<value>true</value>
+		<description>Whether to enable predicate pushdown</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.ppd.storage</name>
+		<value>true</value>
+		<description>Whether to push predicates down into storage handlers.
+			Ignored when hive.optimize.ppd is false.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.ppd.recognizetransivity</name>
+		<value>true</value>
+		<description>Whether to transitively replicate predicate filters over
+			equijoin conditions.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.groupby</name>
+		<value>true</value>
+		<description>Whether to enable the bucketed group by from bucketed
+			partitions/tables.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.skewjoin.compiletime</name>
+		<value>false</value>
+		<description>Whether to create a separate plan for skewed keys for the
+			tables in the join.
+			This is based on the skewed keys stored in the
+			metadata. At compile time,
+			the plan is broken
+			into different joins: one
+			for the skewed keys, and the other for the
+			remaining keys. And then,
+			a
+			union is performed for the 2 joins generated above. So unless the
+			same skewed key is present
+			in both the joined tables, the join for the
+			skewed key will be
+			performed as a map-side join.
+
+			The main difference
+			between this paramater and hive.optimize.skewjoin is
+			that this
+			parameter
+			uses the skew information stored in the metastore to
+			optimize the plan at
+			compile time itself.
+			If there is no skew
+			information in the metadata, this parameter will
+			not have any affect.
+			Both hive.optimize.skewjoin.compiletime and hive.optimize.skewjoin
+			should
+			be set to true.
+			Ideally, hive.optimize.skewjoin should be
+			renamed as
+			hive.optimize.skewjoin.runtime, but not doing
+			so for
+			backward compatibility.
+
+			If the skew information is correctly stored in
+			the metadata,
+			hive.optimize.skewjoin.compiletime
+			would change the query
+			plan to take care of it, and hive.optimize.skewjoin
+			will be a no-op.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.union.remove</name>
+		<value>false</value>
+		<description>
+			Whether to remove the union and push the operators
+			between union and the
+			filesink above
+			union. This avoids an extra scan
+			of the output by union. This is
+			independently useful for union
+			queries, and specially useful when hive.optimize.skewjoin.compiletime
+			is set
+			to true, since an
+			extra union is inserted.
+
+			The merge is triggered
+			if either of hive.merge.mapfiles or
+			hive.merge.mapredfiles is set to
+			true.
+			If the user has set hive.merge.mapfiles to true and
+			hive.merge.mapredfiles to false, the idea was the
+			number of reducers
+			are few, so the number of files anyway are small.
+			However, with this
+			optimization,
+			we are increasing the number of files possibly by a big
+			margin. So, we
+			merge aggresively.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.mapred.supports.subdirectories</name>
+		<value>false</value>
+		<description>Whether the version of hadoop which is running supports
+			sub-directories for tables/partitions.
+			Many hive optimizations can be
+			applied if the hadoop version supports
+			sub-directories for
+			tables/partitions. It was added by MAPREDUCE-1501
+		</description>
+	</property>
+
+	<property>
+		<name>hive.multigroupby.singlemr</name>
+		<value>false</value>
+		<description>Whether to optimize multi group by query to generate
+			single M/R
+			job plan. If the multi group by query has common group by
+			keys, it will
+			be
+			optimized to generate single M/R job.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.map.groupby.sorted</name>
+		<value>false</value>
+		<description>If the bucketing/sorting properties of the table exactly
+			match the grouping key, whether to
+			perform the group by in the mapper
+			by using BucketizedHiveInputFormat. The
+			only downside to this
+			is that
+			it limits the number of mappers to the number of files.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.map.groupby.sorted.testmode</name>
+		<value>false</value>
+		<description>If the bucketing/sorting properties of the table exactly
+			match the grouping key, whether to
+			perform the group by in the mapper
+			by using BucketizedHiveInputFormat. If
+			the test mode is set, the plan
+			is not converted, but a query property is set to denote the same.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.new.job.grouping.set.cardinality</name>
+		<value>30</value>
+		<description>
+			Whether a new map-reduce job should be launched for
+			grouping
+			sets/rollups/cubes.
+			For a query like: select a, b, c, count(1)
+			from T group by a, b, c with
+			rollup;
+			4 rows are created per row: (a, b,
+			c), (a, b, null), (a, null, null),
+			(null, null, null).
+			This can lead to
+			explosion across map-reduce boundary if the cardinality
+			of T is very
+			high,
+			and map-side aggregation does not do a very good job.
+
+			This
+			parameter decides if hive should add an additional map-reduce job.
+			If
+			the grouping set
+			cardinality (4 in the example above), is more than
+			this value, a new MR job is
+			added under the
+			assumption that the orginal
+			group by will reduce the data size.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.join.emit.interval</name>
+		<value>1000</value>
+		<description>How many rows in the right-most join operand Hive should
+			buffer before emitting the join result.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.join.cache.size</name>
+		<value>25000</value>
+		<description>How many rows in the joining tables (except the streaming
+			table) should be cached in memory.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.mapjoin.bucket.cache.size</name>
+		<value>100</value>
+		<description>How many values in each keys in the map-joined table
+			should be cached in memory.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.mapjoin.cache.numrows</name>
+		<value>25000</value>
+		<description>How many rows should be cached by jdbm for map join.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.skewjoin</name>
+		<value>false</value>
+		<description>Whether to enable skew join optimization.
+			The algorithm is
+			as follows: At runtime, detect the keys with a large
+			skew. Instead of
+			processing those keys, store them temporarily in a hdfs directory. In
+			a
+			follow-up map-reduce
+			job, process those skewed keys. The same key
+			need not be skewed for all
+			the tables, and so,
+			the follow-up map-reduce
+			job (for the skewed keys) would be much faster,
+			since it would be a
+			map-join.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.skewjoin.key</name>
+		<value>100000</value>
+		<description>Determine if we get a skew key in join. If we see more
+			than the specified number of rows with the same key in join operator,
+			we think the key as a skew join key.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.skewjoin.mapjoin.map.tasks</name>
+		<value>10000</value>
+		<description> Determine the number of map task used in the follow up
+			map join job
+			for a skew join. It should be used together with
+			hive.skewjoin.mapjoin.min.split
+			to perform a fine grained control.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.skewjoin.mapjoin.min.split</name>
+		<value>33554432</value>
+		<description> Determine the number of map task at most used in the
+			follow up map join job
+			for a skew join by specifying the minimum split
+			size. It should be used
+			together with
+			hive.skewjoin.mapjoin.map.tasks
+			to perform a fine grained control.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.mapred.mode</name>
+		<value>nonstrict</value>
+		<description>The mode in which the hive operations are being
+			performed.
+			In strict mode, some risky queries are not allowed to run.
+			They
+			include:
+			Cartesian Product.
+			No partition being picked up for a
+			query.
+			Comparing bigints and strings.
+			Comparing bigints and doubles.
+			Orderby without limit.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.enforce.bucketmapjoin</name>
+		<value>false</value>
+		<description>If the user asked for bucketed map-side join, and it
+			cannot be performed,
+			should the query fail or not ? For eg, if the
+			buckets in the tables being
+			joined are
+			not a multiple of each other,
+			bucketed map-side join cannot be
+			performed, and the
+			query will fail if
+			hive.enforce.bucketmapjoin is set to true.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.script.maxerrsize</name>
+		<value>100000</value>
+		<description>Maximum number of bytes a script is allowed to emit to
+			standard error (per map-reduce task). This prevents runaway scripts
+			from filling logs partitions to capacity
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.script.allow.partial.consumption</name>
+		<value>false</value>
+		<description> When enabled, this option allows a user script to exit
+			successfully without consuming all the data from the standard input.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.script.operator.id.env.var</name>
+		<value>HIVE_SCRIPT_OPERATOR_ID</value>
+		<description> Name of the environment variable that holds the unique
+			script operator ID in the user's transform function (the custom
+			mapper/reducer that the user has specified in the query)
+		</description>
+	</property>
+
+	<property>
+		<name>hive.script.operator.truncate.env</name>
+		<value>false</value>
+		<description>Truncate each environment variable for external script in
+			scripts operator to 20KB (to fit system limits)
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.compress.output</name>
+		<value>false</value>
+		<description> This controls whether the final outputs of a query (to a
+			local/hdfs file or a hive table) is compressed. The compression codec
+			and other options are determined from hadoop config variables
+			mapred.output.compress*
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.compress.intermediate</name>
+		<value>false</value>
+		<description> This controls whether intermediate files produced by
+			hive between multiple map-reduce jobs are compressed. The compression
+			codec and other options are determined from hadoop config variables
+			mapred.output.compress*
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.parallel</name>
+		<value>false</value>
+		<description>Whether to execute jobs in parallel</description>
+	</property>
+
+	<property>
+		<name>hive.exec.parallel.thread.number</name>
+		<value>8</value>
+		<description>How many jobs at most can be executed in parallel
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.rowoffset</name>
+		<value>false</value>
+		<description>Whether to provide the row offset virtual column
+		</description>
+	</property>
+
+	<property>
+		<name>hive.task.progress</name>
+		<value>false</value>
+		<description>Whether Hive should periodically update task progress
+			counters during execution. Enabling this allows task progress to be
+			monitored more closely in the job tracker, but may impose a
+			performance penalty. This flag is automatically set to true for jobs
+			with hive.exec.dynamic.partition set to true.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.hwi.war.file</name>
+		<value>lib/hive-hwi-@VERSION@.war</value>
+		<description>This sets the path to the HWI war file, relative to
+			${HIVE_HOME}.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.hwi.listen.host</name>
+		<value>0.0.0.0</value>
+		<description>This is the host address the Hive Web Interface will
+			listen on
+		</description>
+	</property>
+
+	<property>
+		<name>hive.hwi.listen.port</name>
+		<value>9999</value>
+		<description>This is the port the Hive Web Interface will listen on
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.pre.hooks</name>
+		<value></value>
+		<description>Comma-separated list of pre-execution hooks to be invoked
+			for each statement. A pre-execution hook is specified as the name of
+			a Java class which implements the
+			org.apache.hadoop.hive.ql.hooks.ExecuteWithHookContext interface.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.post.hooks</name>
+		<value></value>
+		<description>Comma-separated list of post-execution hooks to be
+			invoked for each statement. A post-execution hook is specified as the
+			name of a Java class which implements the
+			org.apache.hadoop.hive.ql.hooks.ExecuteWithHookContext interface.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.failure.hooks</name>
+		<value></value>
+		<description>Comma-separated list of on-failure hooks to be invoked
+			for each statement. An on-failure hook is specified as the name of
+			Java class which implements the
+			org.apache.hadoop.hive.ql.hooks.ExecuteWithHookContext interface.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.init.hooks</name>
+		<value></value>
+		<description>A comma separated list of hooks to be invoked at the
+			beginning of HMSHandler initialization. Aninit hook is specified as
+			the name of Java class which extends
+			org.apache.hadoop.hive.metastore.MetaStoreInitListener.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.client.stats.publishers</name>
+		<value></value>
+		<description>Comma-separated list of statistics publishers to be
+			invoked on counters on each job. A client stats publisher is
+			specified as the name of a Java class which implements the
+			org.apache.hadoop.hive.ql.stats.ClientStatsPublisher interface.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.client.stats.counters</name>
+		<value></value>
+		<description>Subset of counters that should be of interest for
+			hive.client.stats.publishers (when one wants to limit their
+			publishing). Non-display names should be used
+		</description>
+	</property>
+
+	<property>
+		<name>hive.merge.mapfiles</name>
+		<value>true</value>
+		<description>Merge small files at the end of a map-only job
+		</description>
+	</property>
+
+	<property>
+		<name>hive.merge.mapredfiles</name>
+		<value>false</value>
+		<description>Merge small files at the end of a map-reduce job
+		</description>
+	</property>
+
+	<property>
+		<name>hive.heartbeat.interval</name>
+		<value>1000</value>
+		<description>Send a heartbeat after this interval - used by mapjoin
+			and filter operators
+		</description>
+	</property>
+
+	<property>
+		<name>hive.merge.size.per.task</name>
+		<value>256000000</value>
+		<description>Size of merged files at the end of the job</description>
+	</property>
+
+	<property>
+		<name>hive.merge.smallfiles.avgsize</name>
+		<value>16000000</value>
+		<description>When the average output file size of a job is less than
+			this number, Hive will start an additional map-reduce job to merge
+			the output files into bigger files. This is only done for map-only
+			jobs if hive.merge.mapfiles is true, and for map-reduce jobs if
+			hive.merge.mapredfiles is true.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.mapjoin.smalltable.filesize</name>
+		<value>25000000</value>
+		<description>The threshold for the input file size of the small
+			tables; if the file size is smaller than this threshold, it will try
+			to convert the common join into map join
+		</description>
+	</property>
+
+	<property>
+		<name>hive.ignore.mapjoin.hint</name>
+		<value>true</value>
+		<description>Ignore the mapjoin hint</description>
+	</property>
+
+	<property>
+		<name>hive.mapjoin.localtask.max.memory.usage</name>
+		<value>0.90</value>
+		<description>This number means how much memory the local task can take
+			to hold the key/value into in-memory hash table; If the local task's
+			memory usage is more than this number, the local task will be abort
+			by themself. It means the data of small table is too large to be hold
+			in the memory.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.mapjoin.followby.gby.localtask.max.memory.usage</name>
+		<value>0.55</value>
+		<description>This number means how much memory the local task can take
+			to hold the key/value into in-memory hash table when this map join
+			followed by a group by; If the local task's memory usage is more than
+			this number, the local task will be abort by themself. It means the
+			data of small table is too large to be hold in the memory.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.mapjoin.check.memory.rows</name>
+		<value>100000</value>
+		<description>The number means after how many rows processed it needs
+			to check the memory usage
+		</description>
+	</property>
+
+	<property>
+		<name>hive.auto.convert.join</name>
+		<value>false</value>
+		<description>Whether Hive enable the optimization about converting
+			common join into mapjoin based on the input file size
+		</description>
+	</property>
+
+	<property>
+		<name>hive.auto.convert.join.noconditionaltask</name>
+		<value>true</value>
+		<description>Whether Hive enable the optimization about converting
+			common join into mapjoin based on the input file
+			size. If this
+			paramater is on, and the sum of size for n-1 of the
+			tables/partitions
+			for a n-way join is smaller than the
+			specified size, the join is
+			directly converted to a mapjoin (there is no
+			conditional task).
+		</description>
+	</property>
+
+	<property>
+		<name>hive.auto.convert.join.noconditionaltask.size</name>
+		<value>10000000</value>
+		<description>If hive.auto.convert.join.noconditionaltask is off, this
+			parameter does not take affect. However, if it
+			is on, and the sum of
+			size for n-1 of the tables/partitions for a n-way
+			join is smaller than
+			this size, the join is directly
+			converted to a mapjoin(there is no
+			conditional task). The default is 10MB
+		</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.mapjoin.mapreduce</name>
+		<value>false</value>
+		<description>If hive.auto.convert.join is off, this parameter does not
+			take
+			affect. If it is on, and if there are map-join jobs followed by a
+			map-reduce
+			job (for e.g a group by), each map-only job is merged with
+			the
+			following
+			map-reduce job.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.script.auto.progress</name>
+		<value>false</value>
+		<description>Whether Hive Tranform/Map/Reduce Clause should
+			automatically send progress information to TaskTracker to avoid the
+			task getting killed because of inactivity. Hive sends progress
+			information when the script is outputting to stderr. This option
+			removes the need of periodically producing stderr messages, but users
+			should be cautious because this may prevent infinite loops in the
+			scripts to be killed by TaskTracker.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.script.serde</name>
+		<value>org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe</value>
+		<description>The default serde for trasmitting input data to and
+			reading output data from the user scripts.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.binary.record.max.length</name>
+		<value>1000</value>
+		<description>Read from a binary stream and treat each
+			hive.binary.record.max.length bytes as a record.
+			The last record
+			before the end of stream can have less than
+			hive.binary.record.max.length bytes
+		</description>
+	</property>
+
+
+	<property>
+		<name>hive.script.recordreader</name>
+		<value>org.apache.hadoop.hive.ql.exec.TextRecordReader</value>
+		<description>The default record reader for reading data from the user
+			scripts.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.script.recordwriter</name>
+		<value>org.apache.hadoop.hive.ql.exec.TextRecordWriter</value>
+		<description>The default record writer for writing data to the user
+			scripts.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.input.format</name>
+		<value>org.apache.hadoop.hive.ql.io.CombineHiveInputFormat</value>
+		<description>The default input format. Set this to HiveInputFormat if
+			you encounter problems with CombineHiveInputFormat.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.udtf.auto.progress</name>
+		<value>false</value>
+		<description>Whether Hive should automatically send progress
+			information to TaskTracker when using UDTF's to prevent the task
+			getting killed because of inactivity. Users should be cautious
+			because this may prevent TaskTracker from killing tasks with infinte
+			loops.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.mapred.reduce.tasks.speculative.execution</name>
+		<value>true</value>
+		<description>Whether speculative execution for reducers should be
+			turned on.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.counters.pull.interval</name>
+		<value>1000</value>
+		<description>The interval with which to poll the JobTracker for the
+			counters the running job. The smaller it is the more load there will
+			be on the jobtracker, the higher it is the less granular the caught
+			will be.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.querylog.location</name>
+		<value>/tmp/${user.name}</value>
+		<description>
+			Location of Hive run time structured log file
+		</description>
+	</property>
+
+	<property>
+		<name>hive.querylog.enable.plan.progress</name>
+		<value>true</value>
+		<description>
+			Whether to log the plan's progress every time a job's
+			progress is checked.
+			These logs are written to the location specified
+			by
+			hive.querylog.location
+		</description>
+	</property>
+
+	<property>
+		<name>hive.querylog.plan.progress.interval</name>
+		<value>60000</value>
+		<description>
+			The interval to wait between logging the plan's progress
+			in
+			milliseconds.
+			If there is a whole number percentage change in the
+			progress of the
+			mappers or the reducers,
+			the progress is logged
+			regardless of this value.
+			The actual interval will be the ceiling of
+			(this value divided by the
+			value of
+			hive.exec.counters.pull.interval)
+			multiplied by the value of hive.exec.counters.pull.interval
+			I.e. if it
+			is not divide evenly by the value of
+			hive.exec.counters.pull.interval
+			it will be
+			logged less frequently than specified.
+			This only has an
+			effect if hive.querylog.enable.plan.progress is set to
+			true.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.enforce.bucketing</name>
+		<value>false</value>
+		<description>Whether bucketing is enforced. If true, while inserting
+			into the table, bucketing is enforced.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.enforce.sorting</name>
+		<value>false</value>
+		<description>Whether sorting is enforced. If true, while inserting
+			into the table, sorting is enforced.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.bucketingsorting</name>
+		<value>true</value>
+		<description>If hive.enforce.bucketing or hive.enforce.sorting is
+			true, dont create a reducer for enforcing
+			bucketing/sorting for
+			queries of the form:
+			insert overwrite table T2 select * from T1;
+			where
+			T1 and T2 are bucketed/sorted by the same keys into the same number
+			of buckets.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.enforce.sortmergebucketmapjoin</name>
+		<value>false</value>
+		<description>If the user asked for sort-merge bucketed map-side join,
+			and it cannot be performed,
+			should the query fail or not ?
+		</description>
+	</property>
+
+	<property>
+		<name>hive.auto.convert.sortmerge.join</name>
+		<value>false</value>
+		<description>Will the join be automatically converted to a sort-merge
+			join, if the joined tables pass
+			the criteria for sort-merge join.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.auto.convert.sortmerge.join.bigtable.selection.policy
+		</name>
+		<value>org.apache.hadoop.hive.ql.optimizer.AvgPartitionSizeBasedBigTableSelectorForAutoSMJ
+		</value>
+		<description>The policy to choose the big table for automatic
+			conversion to sort-merge join.
+			By default, the table with the largest
+			partitions is assigned the big
+			table. All policies are:
+			. based on
+			position of the table - the leftmost table is selected
+			org.apache.hadoop.hive.ql.optimizer.LeftmostBigTableSMJ.
+			. based on
+			total size (all the partitions selected in the query) of
+			the table
+			org.apache.hadoop.hive.ql.optimizer.TableSizeBasedBigTableSelectorForAutoSMJ.
+			. based on average size (all the partitions selected in the query) of
+			the table
+			org.apache.hadoop.hive.ql.optimizer.AvgPartitionSizeBasedBigTableSelectorForAutoSMJ.
+			New policies can be added in future.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.ds.connection.url.hook</name>
+		<value></value>
+		<description>Name of the hook to use for retriving the JDO connection
+			URL. If empty, the value in javax.jdo.option.ConnectionURL is used
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.ds.retry.attempts</name>
+		<value>1</value>
+		<description>The number of times to retry a metastore call if there
+			were a connection error
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.ds.retry.interval</name>
+		<value>1000</value>
+		<description>The number of miliseconds between metastore retry
+			attempts
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.server.min.threads</name>
+		<value>200</value>
+		<description>Minimum number of worker threads in the Thrift server's
+			pool.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.server.max.threads</name>
+		<value>100000</value>
+		<description>Maximum number of worker threads in the Thrift server's
+			pool.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.server.tcp.keepalive</name>
+		<value>true</value>
+		<description>Whether to enable TCP keepalive for the metastore server.
+			Keepalive will prevent accumulation of half-open connections.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.sasl.enabled</name>
+		<value>false</value>
+		<description>If true, the metastore thrift interface will be secured
+			with SASL. Clients must authenticate with Kerberos.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.thrift.framed.transport.enabled</name>
+		<value>false</value>
+		<description>If true, the metastore thrift interface will use
+			TFramedTransport. When false (default) a standard TTransport is used.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.kerberos.keytab.file</name>
+		<value></value>
+		<description>The path to the Kerberos Keytab file containing the
+			metastore thrift server's service principal.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.kerberos.principal</name>
+		<value>hive-metastore/_HOST@EXAMPLE.COM</value>
+		<description>The service principal for the metastore thrift server.
+			The special string _HOST will be replaced automatically with the
+			correct host name.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.cluster.delegation.token.store.class</name>
+		<value>org.apache.hadoop.hive.thrift.MemoryTokenStore</value>
+		<description>The delegation token store implementation. Set to
+			org.apache.hadoop.hive.thrift.ZooKeeperTokenStore for load-balanced
+			cluster.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.cluster.delegation.token.store.zookeeper.connectString
+		</name>
+		<value>localhost:2181</value>
+		<description>The ZooKeeper token store connect string.</description>
+	</property>
+
+	<property>
+		<name>hive.cluster.delegation.token.store.zookeeper.znode</name>
+		<value>/hive/cluster/delegation</value>
+		<description>The root path for token store data.</description>
+	</property>
+
+	<property>
+		<name>hive.cluster.delegation.token.store.zookeeper.acl</name>
+		<value>sasl:hive/host1@EXAMPLE.COM:cdrwa,sasl:hive/host2@EXAMPLE.COM:cdrwa
+		</value>
+		<description>ACL for token store entries. List comma separated all
+			server principals for the cluster.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.cache.pinobjtypes</name>
+		<value>Table,StorageDescriptor,SerDeInfo,Partition,Database,Type,FieldSchema,Order
+		</value>
+		<description>List of comma separated metastore object types that
+			should be pinned in the cache
+		</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.reducededuplication</name>
+		<value>true</value>
+		<description>Remove extra map-reduce jobs if the data is already
+			clustered by the same key which needs to be used again. This should
+			always be set to true. Since it is a new feature, it has been made
+			configurable.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.reducededuplication.min.reducer</name>
+		<value>4</value>
+		<description>Reduce deduplication merges two RSs by moving
+			key/parts/reducer-num of the child RS to parent RS.
+			That means if
+			reducer-num of the child RS is fixed (order by or forced
+			bucketing)
+			and small, it can make very slow, single MR.
+			The optimization will be
+			disabled if number of reducers is less than
+			specified value.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.dynamic.partition</name>
+		<value>true</value>
+		<description>Whether or not to allow dynamic partitions in DML/DDL.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.dynamic.partition.mode</name>
+		<value>strict</value>
+		<description>In strict mode, the user must specify at least one static
+			partition in case the user accidentally overwrites all partitions.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.max.dynamic.partitions</name>
+		<value>1000</value>
+		<description>Maximum number of dynamic partitions allowed to be
+			created in total.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.max.dynamic.partitions.pernode</name>
+		<value>100</value>
+		<description>Maximum number of dynamic partitions allowed to be
+			created in each mapper/reducer node.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.max.created.files</name>
+		<value>100000</value>
+		<description>Maximum number of HDFS files created by all
+			mappers/reducers in a MapReduce job.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.default.partition.name</name>
+		<value>__HIVE_DEFAULT_PARTITION__</value>
+		<description>The default partition name in case the dynamic partition
+			column value is null/empty string or anyother values that cannot be
+			escaped. This value must not contain any special character used in
+			HDFS URI (e.g., ':', '%', '/' etc). The user has to be aware that the
+			dynamic partition value should not contain this value to avoid
+			confusions.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.stats.dbclass</name>
+		<value>jdbc:derby</value>
+		<description>The default database that stores temporary hive
+			statistics.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.stats.autogather</name>
+		<value>true</value>
+		<description>A flag to gather statistics automatically during the
+			INSERT OVERWRITE command.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.stats.jdbcdriver</name>
+		<value>org.apache.derby.jdbc.EmbeddedDriver</value>
+		<description>The JDBC driver for the database that stores temporary
+			hive statistics.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.stats.dbconnectionstring</name>
+		<value>jdbc:derby:;databaseName=TempStatsStore;create=true</value>
+		<description>The default connection string for the database that
+			stores temporary hive statistics.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.stats.default.publisher</name>
+		<value></value>
+		<description>The Java class (implementing the StatsPublisher
+			interface) that is used by default if hive.stats.dbclass is not JDBC
+			or HBase.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.stats.default.aggregator</name>
+		<value></value>
+		<description>The Java class (implementing the StatsAggregator
+			interface) that is used by default if hive.stats.dbclass is not JDBC
+			or HBase.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.stats.jdbc.timeout</name>
+		<value>30</value>
+		<description>Timeout value (number of seconds) used by JDBC connection
+			and statements.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.stats.retries.max</name>
+		<value>0</value>
+		<description>Maximum number of retries when stats publisher/aggregator
+			got an exception updating intermediate database. Default is no tries
+			on failures.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.stats.retries.wait</name>
+		<value>3000</value>
+		<description>The base waiting window (in milliseconds) before the next
+			retry. The actual wait time is calculated by baseWindow * failues
+			baseWindow * (failure 1) * (random number between [0.0,1.0]).
+		</description>
+	</property>
+
+	<property>
+		<name>hive.stats.reliable</name>
+		<value>false</value>
+		<description>Whether queries will fail because stats cannot be
+			collected completely accurately.
+			If this is set to true,
+			reading/writing from/into a partition may fail
+			becuase the stats
+			could
+			not be computed accurately.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.stats.collect.tablekeys</name>
+		<value>false</value>
+		<description>Whether join and group by keys on tables are derived and
+			maintained in the QueryPlan.
+			This is useful to identify how tables are
+			accessed and to determine if
+			they should be bucketed.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.stats.collect.scancols</name>
+		<value>false</value>
+		<description>Whether column accesses are tracked in the QueryPlan.
+			This is useful to identify how tables are accessed and to determine
+			if there are wasted columns that can be trimmed.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.stats.ndv.error</name>
+		<value>20.0</value>
+		<description>Standard error expressed in percentage. Provides a
+			tradeoff between accuracy and compute cost.A lower value for error
+			indicates higher accuracy and a higher compute cost.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.stats.key.prefix.max.length</name>
+		<value>200</value>
+		<description>
+			Determines if when the prefix of the key used for
+			intermediate stats collection
+			exceeds a certain length, a hash of the
+			key is used instead. If the
+			value &lt; 0 then hashing
+			is never used, if
+			the value >= 0 then hashing is used only when the key
+			prefixes length
+			exceeds that value. The key prefix is defined as everything preceding
+			the
+			task ID in the key.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.support.concurrency</name>
+		<value>false</value>
+		<description>Whether hive supports concurrency or not. A zookeeper
+			instance must be up and running for the default hive lock manager to
+			support read-write locks.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.lock.numretries</name>
+		<value>100</value>
+		<description>The number of times you want to try to get all the locks
+		</description>
+	</property>
+
+	<property>
+		<name>hive.unlock.numretries</name>
+		<value>10</value>
+		<description>The number of times you want to retry to do one unlock
+		</description>
+	</property>
+
+	<property>
+		<name>hive.lock.sleep.between.retries</name>
+		<value>60</value>
+		<description>The sleep time (in seconds) between various retries
+		</description>
+	</property>
+
+	<property>
+		<name>hive.zookeeper.quorum</name>
+		<value></value>
+		<description>The list of zookeeper servers to talk to. This is only
+			needed for read/write locks.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.zookeeper.client.port</name>
+		<value>2181</value>
+		<description>The port of zookeeper servers to talk to. This is only
+			needed for read/write locks.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.zookeeper.session.timeout</name>
+		<value>600000</value>
+		<description>Zookeeper client's session timeout. The client is
+			disconnected, and as a result, all locks released, if a heartbeat is
+			not sent in the timeout.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.zookeeper.namespace</name>
+		<value>hive_zookeeper_namespace</value>
+		<description>The parent node under which all zookeeper nodes are
+			created.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.zookeeper.clean.extra.nodes</name>
+		<value>false</value>
+		<description>Clean extra nodes at the end of the session.
+		</description>
+	</property>
+
+	<property>
+		<name>fs.har.impl</name>
+		<value>org.apache.hadoop.hive.shims.HiveHarFileSystem</value>
+		<description>The implementation for accessing Hadoop Archives. Note
+			that this won't be applicable to Hadoop vers less than 0.20
+		</description>
+	</property>
+
+	<property>
+		<name>hive.archive.enabled</name>
+		<value>false</value>
+		<description>Whether archiving operations are permitted</description>
+	</property>
+
+	<property>
+		<name>hive.fetch.output.serde</name>
+		<value>org.apache.hadoop.hive.serde2.DelimitedJSONSerDe</value>
+		<description>The serde used by FetchTask to serialize the fetch
+			output.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.mode.local.auto</name>
+		<value>false</value>
+		<description> Let hive determine whether to run in local mode
+			automatically
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.drop.ignorenonexistent</name>
+		<value>true</value>
+		<description>
+			Do not report an error if DROP TABLE/VIEW specifies a
+			non-existent
+			table/view
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.show.job.failure.debug.info</name>
+		<value>true</value>
+		<description>
+			If a job fails, whether to provide a link in the CLI to
+			the task with
+			the
+			most failures, along with debugging hints if
+			applicable.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.auto.progress.timeout</name>
+		<value>0</value>
+		<description>
+			How long to run autoprogressor for the script/UDTF
+			operators (in
+			seconds).
+			Set to 0 for forever.
+		</description>
+	</property>
+
+	<!-- HBase Storage Handler Parameters -->
+
+	<property>
+		<name>hive.hbase.wal.enabled</name>
+		<value>true</value>
+		<description>Whether writes to HBase should be forced to the
+			write-ahead log. Disabling this improves HBase write performance at
+			the risk of lost writes in case of a crash.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.table.parameters.default</name>
+		<value></value>
+		<description>Default property values for newly created tables
+		</description>
+	</property>
+
+	<property>
+		<name>hive.entity.separator</name>
+		<value>@</value>
+		<description>Separator used to construct names of tables and
+			partitions. For example, dbname@tablename@partitionname
+		</description>
+	</property>
+
+	<property>
+		<name>hive.ddl.createtablelike.properties.whitelist</name>
+		<value></value>
+		<description>Table Properties to copy over when executing a Create
+			Table Like.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.variable.substitute</name>
+		<value>true</value>
+		<description>This enables substitution using syntax like ${var}
+			${system:var} and ${env:var}.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.variable.substitute.depth</name>
+		<value>40</value>
+		<description>The maximum replacements the substitution engine will do.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.conf.validation</name>
+		<value>true</value>
+		<description>Eables type checking for registered hive configurations
+		</description>
+	</property>
+
+	<property>
+		<name>hive.security.authorization.enabled</name>
+		<value>false</value>
+		<description>enable or disable the hive client authorization
+		</description>
+	</property>
+
+	<property>
+		<name>hive.security.authorization.createtable.user.grants</name>
+		<value></value>
+		<description>the privileges automatically granted to some users
+			whenever a table gets created.
+			An example like
+			"userX,userY:select;userZ:create" will grant select
+			privilege to userX
+			and userY,
+			and grant create privilege to userZ whenever a new table
+			created.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.security.authorization.createtable.group.grants</name>
+		<value></value>
+		<description>the privileges automatically granted to some groups
+			whenever a table gets created.
+			An example like
+			"groupX,groupY:select;groupZ:create" will grant select
+			privilege to
+			groupX and groupY,
+			and grant create privilege to groupZ whenever a new
+			table created.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.security.authorization.createtable.role.grants</name>
+		<value></value>
+		<description>the privileges automatically granted to some roles
+			whenever a table gets created.
+			An example like
+			"roleX,roleY:select;roleZ:create" will grant select
+			privilege to roleX
+			and roleY,
+			and grant create privilege to roleZ whenever a new table
+			created.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.security.authorization.createtable.owner.grants</name>
+		<value></value>
+		<description>the privileges automatically granted to the owner
+			whenever a table gets created.
+			An example like "select,drop" will
+			grant select and drop privilege to
+			the owner of the table
+		</description>
+	</property>
+
+	<property>
+		<name>hive.metastore.authorization.storage.checks</name>
+		<value>false</value>
+		<description>Should the metastore do authorization checks against the
+			underlying storage
+			for operations like drop-partition (disallow the
+			drop-partition if the
+			user in
+			question doesn't have permissions to
+			delete the corresponding directory
+			on the storage).
+		</description>
+	</property>
+
+	<property>
+		<name>hive.error.on.empty.partition</name>
+		<value>false</value>
+		<description>Whether to throw an excpetion if dynamic partition insert
+			generates empty results.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.index.compact.file.ignore.hdfs</name>
+		<value>false</value>
+		<description>True the hdfs location stored in the index file will be
+			igbored at runtime.
+			If the data got moved or the name of the cluster
+			got changed, the index
+			data should still be usable.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.index.filter.compact.minsize</name>
+		<value>5368709120</value>
+		<description>Minimum size (in bytes) of the inputs on which a compact
+			index is automatically used.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.optimize.index.filter.compact.maxsize</name>
+		<value>-1</value>
+		<description>Maximum size (in bytes) of the inputs on which a compact
+			index is automatically used.
+			A negative number is equivalent to
+			infinity.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.index.compact.query.max.size</name>
+		<value>10737418240</value>
+		<description>The maximum number of bytes that a query using the
+			compact index can read. Negative value is equivalent to infinity.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.index.compact.query.max.entries</name>
+		<value>10000000</value>
+		<description>The maximum number of index entries to read during a
+			query that uses the compact index. Negative value is equivalent to
+			infinity.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.index.compact.binary.search</name>
+		<value>true</value>
+		<description>Whether or not to use a binary search to find the entries
+			in an index table that match the filter, where possible
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exim.uri.scheme.whitelist</name>
+		<value>hdfs,pfile</value>
+		<description>A comma separated list of acceptable URI schemes for
+			import and export.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.lock.mapred.only.operation</name>
+		<value>false</value>
+		<description>This param is to control whether or not only do lock on
+			queries
+			that need to execute at least one mapred job.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.limit.row.max.size</name>
+		<value>100000</value>
+		<description>When trying a smaller subset of data for simple LIMIT,
+			how much size we need to guarantee
+			each row to have at least.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.limit.optimize.limit.file</name>
+		<value>10</value>
+		<description>When trying a smaller subset of data for simple LIMIT,
+			maximum number of files we can
+			sample.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.limit.optimize.enable</name>
+		<value>false</value>
+		<description>Whether to enable to optimization to trying a smaller
+			subset of data for simple LIMIT first.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.limit.optimize.fetch.max</name>
+		<value>50000</value>
+		<description>Maximum number of rows allowed for a smaller subset of
+			data for simple LIMIT, if it is a fetch query.
+			Insert queries are not
+			restricted by this limit.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.rework.mapredwork</name>
+		<value>false</value>
+		<description>should rework the mapred work or not.
+			This is first
+			introduced by SymlinkTextInputFormat to replace symlink
+			files with
+			real paths at compile time.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.concatenate.check.index</name>
+		<value>true</value>
+		<description>If this sets to true, hive will throw error when doing
+			'alter table tbl_name [partSpec] concatenate' on a table/partition
+			that has indexes on it. The reason the user want to set this to true
+			is because it can help user to avoid handling all index drop,
+			recreation,
+			rebuild work. This is very helpful for tables with
+			thousands of partitions.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.sample.seednumber</name>
+		<value>0</value>
+		<description>A number used to percentage sampling. By changing this
+			number, user will change the subsets
+			of data sampled.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.io.exception.handlers</name>
+		<value></value>
+		<description>A list of io exception handler class names. This is used
+			to construct a list exception handlers to handle exceptions thrown
+			by
+			record readers
+		</description>
+	</property>
+
+	<property>
+		<name>hive.autogen.columnalias.prefix.label</name>
+		<value>_c</value>
+		<description>String used as a prefix when auto generating column
+			alias.
+			By default the prefix label will be appended with a column
+			position
+			number to form the column alias. Auto generation would happen
+			if an
+			aggregate function is used in a select clause without an
+			explicit
+			alias.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.autogen.columnalias.prefix.includefuncname</name>
+		<value>false</value>
+		<description>Whether to include function name in the column alias auto
+			generated by hive.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.perf.logger</name>
+		<value>org.apache.hadoop.hive.ql.log.PerfLogger</value>
+		<description>The class responsible logging client side performance
+			metrics. Must be a subclass of
+			org.apache.hadoop.hive.ql.log.PerfLogger
+		</description>
+	</property>
+
+	<property>
+		<name>hive.start.cleanup.scratchdir</name>
+		<value>false</value>
+		<description>To cleanup the hive scratchdir while starting the hive
+			server
+		</description>
+	</property>
+
+	<property>
+		<name>hive.output.file.extension</name>
+		<value></value>
+		<description>String used as a file extension for output files. If not
+			set, defaults to the codec extension for text files (e.g. ".gz"), or
+			no extension otherwise.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.insert.into.multilevel.dirs</name>
+		<value>false</value>
+		<description>Where to insert into multilevel directories like
+			"insert
+			directory '/HIVEFT25686/chinna/' from table"
+		</description>
+	</property>
+
+	<property>
+		<name>hive.warehouse.subdir.inherit.perms</name>
+		<value>false</value>
+		<description>Set this to true if the the table directories should
+			inherit the
+			permission of the warehouse or database directory instead
+			of being created
+			with the permissions derived from dfs umask
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.job.debug.capture.stacktraces</name>
+		<value>true</value>
+		<description>Whether or not stack traces parsed from the task logs of
+			a sampled failed task for
+			each failed job should be stored in the
+			SessionState
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.driver.run.hooks</name>
+		<value></value>
+		<description>A comma separated list of hooks which implement
+			HiveDriverRunHook and will be run at the
+			beginning and end of
+			Driver.run, these will be run in the order specified
+		</description>
+	</property>
+
+	<property>
+		<name>hive.ddl.output.format</name>
+		<value>text</value>
+		<description>
+			The data format to use for DDL output. One of "text" (for
+			human
+			readable text) or "json" (for a json object).
+		</description>
+	</property>
+
+	<property>
+		<name>hive.transform.escape.input</name>
+		<value>false</value>
+		<description>
+			This adds an option to escape special chars (newlines,
+			carriage returns
+			and
+			tabs) when they are passed to the user script.
+			This is useful if the hive
+			tables
+			can contain data that contains
+			special characters.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.rcfile.use.explicit.header</name>
+		<value>true</value>
+		<description>
+			If this is set the header for RC Files will simply be
+			RCF. If this is
+			not
+			set the header will be that borrowed from sequence
+			files, e.g. SEQ-
+			followed
+			by the input and output RC File formats.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.multi.insert.move.tasks.share.dependencies</name>
+		<value>false</value>
+		<description>
+			If this is set all move tasks for tables/partitions (not
+			directories)
+			at the end of a
+			multi-insert query will only begin once
+			the dependencies for all these move tasks
+			have been
+			met.
+			Advantages: If
+			concurrency is enabled, the locks will only be released once the
+			query has
+			finished, so with this config enabled, the time when the
+			table/partition is
+			generated will be much closer to when the lock on
+			it is released.
+			Disadvantages: If concurrency is not enabled, with
+			this disabled, the
+			tables/partitions which
+			are produced by this query
+			and finish earlier will be available for
+			querying
+			much earlier. Since
+			the locks are only released once the query finishes,
+			this
+			does not
+			apply if concurrency is enabled.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.fetch.task.conversion</name>
+		<value>minimal</value>
+		<description>
+			Some select queries can be converted to single FETCH task
+			minimizing
+			latency.
+			Currently the query should be single sourced not
+			having any subquery and
+			should not have
+			any aggregations or distincts
+			(which incurrs RS), lateral views and
+			joins.
+			1. minimal : SELECT STAR,
+			FILTER on partition columns, LIMIT only
+			2. more : SELECT, FILTER,
+			LIMIT only (TABLESAMPLE, virtual columns)
+		</description>
+	</property>
+
+	<property>
+		<name>hive.hmshandler.retry.attempts</name>
+		<value>1</value>
+		<description>The number of times to retry a HMSHandler call if there
+			were a connection error
+		</description>
+	</property>
+
+	<property>
+		<name>hive.hmshandler.retry.interval</name>
+		<value>1000</value>
+		<description>The number of miliseconds between HMSHandler retry
+			attempts
+		</description>
+	</property>
+
+	<property>
+		<name>hive.server.read.socket.timeout</name>
+		<value>10</value>
+		<description>Timeout for the HiveServer to close the connection if no
+			response from the client in N seconds, defaults to 10 seconds.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.server.tcp.keepalive</name>
+		<value>true</value>
+		<description>Whether to enable TCP keepalive for the Hive server.
+			Keepalive will prevent accumulation of half-open connections.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.decode.partition.name</name>
+		<value>false</value>
+		<description>Whether to show the unquoted partition names in query
+			results.
+		</description>
+	</property>
+
+	<property>
+		<name>hive.log4j.file</name>
+		<value></value>
+		<description>Hive log4j configuration file.
+			If the property is not set,
+			then logging will be initialized using
+			hive-log4j.properties found on
+			the classpath.
+			If the property is set, the value must be a valid URI
+			(java.net.URI,
+			e.g. "file:///tmp/my-logging.properties"), which you
+			can then extract
+			a URL from and pass to
+			PropertyConfigurator.configure(URL).
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.log4j.file</name>
+		<value></value>
+		<description>Hive log4j configuration file for execution mode(sub
+			command).
+			If the property is not set, then logging will be initialized
+			using
+			hive-exec-log4j.properties found on the classpath.
+			If the
+			property is set, the value must be a valid URI (java.net.URI,
+			e.g.
+			"file:///tmp/my-logging.properties"), which you can then extract
+			a URL
+			from and pass to PropertyConfigurator.configure(URL).
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.infer.bucket.sort</name>
+		<value>false</value>
+		<description>
+			If this is set, when writing partitions, the metadata
+			will include the
+			bucketing/sorting
+			properties with which the data was
+			written if any (this will not overwrite the
+			metadata
+			inherited from the
+			table if the table is bucketed/sorted)
+		</description>
+	</property>
+
+	<property>
+		<name>hive.exec.infer.bucket.sort.num.buckets.power.two</name>
+		<value>false</value>
+		<description>
+			If this is set, when setting the number of reducers for
+			the map reduce
+			task which writes the
+			final output files, it will choose
+			a number which is a power of two,
+			unless the user specifies
+			the number
+			of reducers to use using mapred.reduce.tasks. The number of
+			reducers
+			may be set to a power of two, only to be followed by a merge task
+			meaning preventing
+			anything from being inferred.
+			With
+			hive.exec.infer.bucket.sort set to true:
+			Advantages: If this is not
+			set, the number of buckets for partitions will seem
+			arbitrary,
+			which
+			means that the number of mappers used for optimized joins, for
+			example, will
+			be very low. With this set, since the number of buckets
+			used for any
+			partition is
+			a power of two, the number of mappers used
+			for optimized joins will be
+			the least
+			number of buckets used by any
+			partition being joined.
+			Disadvantages: This may mean a much larger or
+			much smaller number of reducers being
+			used in the
+			final map reduce job,
+			e.g. if a job was originally going to take 257
+			reducers,
+			it will now
+			take 512 reducers, similarly if the max number of reducers
+			is 511,
+			and
+			a job was going to use this many, it will now use 256 reducers.
+
+		</description>
+	</property>
+
+	<property>
+		<name>hive.groupby.orderby.position.alias</name>
+		<value>false</value>
+		<description>Whether to enable using Column Position Alias in Group By
+			or Order By
+		</description>
+	</property>
+
+	<property>
+		<name>hive.server2.thrift.min.worker.threads</name>
+		<value>5</value>
+		<description>Minimum number of Thrift worker threads</description>
+	</property>
+
+	<property>
+		<name>hive.server2.thrift.max.worker.threads</name>
+		<value>100</value>
+		<description>Maximum number of Thrift worker threads</description>
+	</property>
+
+	<property>
+		<name>hive.server2.thrift.port</name>
+		<value>10000</value>
+		<description>Port number of HiveServer2 Thrift interface.
+			Can be
+			overridden by setting $HIVE_SERVER2_THRIFT_PORT
+		</description>
+	</property>
+
+	<property>
+		<name>hive.server2.thrift.bind.host</name>
+		<value>localhost</value>
+		<description>Bind host on which to run the HiveServer2 Thrift
+			interface.
+			Can be overridden by setting $HIVE_SERVER2_THRIFT_BIND_HOST
+		</description>
+	</property>
+
+	<property>
+		<name>hive.server2.authentication</name>
+		<value>NONE</value>
+		<description>
+			Client authentication types.
+			NONE: no authentication check
+			LDAP: LDAP/AD based authentication
+			KERBEROS: Kerberos/GSSAPI
+			authentication
+			CUSTOM: Custom authentication provider
+			(Use with
+			property hive.server2.custom.authentication.class)
+		</description>
+	</property>
+
+	<property>
+		<name>hive.server2.custom.authentication.class</name>
+		<value></value>
+		<description>
+			Custom authentication class. Used when property
+			'hive.server2.authentication' is set to 'CUSTOM'. Provided class
+			must
+			be a proper implementation of the interface
+			org.apache.hive.service.auth.PasswdAuthenticationProvider.
+			HiveServer2
+			will call its Authenticate(user, passed) method to
+			authenticate requests.
+			The implementation may optionally extend the
+			Hadoop's
+			org.apache.hadoop.conf.Configured class to grab Hive's
+			Configuration
+			object.
+		</description>
+	</property>
+
+	<property>
+		<name>>hive.server2.authentication.kerberos.principal</name>
+		<value></value>
+		<description>
+			Kerberos server principal
+		</description>
+	</property>
+
+	<property>
+		<name>>hive.server2.authentication.kerberos.keytab</name>
+		<value></value>
+		<description>
+			Kerberos keytab file for server principal
+		</description>
+	</property>
+
+	<property>
+		<name>hive.server2.authentication.ldap.url</name>
+		<value></value>
+		<description>
+			LDAP connection URL
+		</description>
+	</property>
+
+	<property>
+		<name>hive.server2.authentication.ldap.baseDN</name>
+		<value></value>
+		<description>
+			LDAP base DN
+		</description>
+	</property>
+
+	<property>
+		<name>hive.server2.enable.doAs</name>
+		<value>true</value>
+		<description>
+			Setting this property to true will have hive server2
+			execute
+			hive operations as the user making the calls to it.
+		</description>
+	</property>
+
+
+</configuration>
diff --git a/hivesterix/hivesterix-dist/src/main/resources/scripts/copylog.sh b/hivesterix/hivesterix-dist/src/main/resources/scripts/copylog.sh
new file mode 100644
index 0000000..7767b2d
--- /dev/null
+++ b/hivesterix/hivesterix-dist/src/main/resources/scripts/copylog.sh
@@ -0,0 +1,7 @@
+. conf/cluster.properties
+
+NODEID=`hostname | cut -d '.' -f 1`
+#echo $NODEID
+
+#echo "rsync ${NCLOGS_DIR}/${NODEID}.log ${1}:${2}"
+rsync ${NCLOGS_DIR}/${NODEID}.log ${1}:${2}
diff --git a/hivesterix/hivesterix-dist/src/main/resources/scripts/dumpAll.sh b/hivesterix/hivesterix-dist/src/main/resources/scripts/dumpAll.sh
new file mode 100644
index 0000000..e7d45e8
--- /dev/null
+++ b/hivesterix/hivesterix-dist/src/main/resources/scripts/dumpAll.sh
@@ -0,0 +1,12 @@
+. conf/cluster.properties
+PREGELIX_PATH=`pwd`
+LOG_PATH=$PREGELIX_PATH/logs/
+rm -rf $LOG_PATH
+mkdir $LOG_PATH
+ccname=`hostname`
+
+for i in `cat conf/slaves`
+do
+   ssh $i "cd ${PREGELIX_PATH}; bin/dumptrace.sh; bin/copylog.sh ${ccname} ${LOG_PATH}"
+done
+
diff --git a/hivesterix/hivesterix-dist/src/main/resources/scripts/dumptrace.sh b/hivesterix/hivesterix-dist/src/main/resources/scripts/dumptrace.sh
new file mode 100644
index 0000000..9fe55f0
--- /dev/null
+++ b/hivesterix/hivesterix-dist/src/main/resources/scripts/dumptrace.sh
@@ -0,0 +1,15 @@
+echo `hostname`
+#Kill process
+PID=`ps -ef|grep ${USER}|grep java|grep 'Dapp.name=pregelixnc'|awk '{print $2}'`
+
+if [ "$PID" == "" ]; then
+  PID=`ps -ef|grep ${USER}|grep java|grep 'hyracks'|awk '{print $2}'`
+fi
+
+if [ "$PID" == "" ]; then
+  USERID=`id | sed 's/^uid=//;s/(.*$//'`
+  PID=`ps -ef|grep ${USERID}|grep java|grep 'Dapp.name=pregelixnc'|awk '{print $2}'`
+fi
+
+echo $PID
+kill -QUIT $PID
diff --git a/hivesterix/hivesterix-dist/src/main/resources/scripts/hive b/hivesterix/hivesterix-dist/src/main/resources/scripts/hive
index 38a9e33..d2ef909 100755
--- a/hivesterix/hivesterix-dist/src/main/resources/scripts/hive
+++ b/hivesterix/hivesterix-dist/src/main/resources/scripts/hive
@@ -175,13 +175,6 @@
     exit 5
 fi
 
-if [ $hadoop_minor_ver -ne 20 -o $hadoop_patch_ver -eq 0 ]; then
-    echo "Hive requires Hadoop 0.20.x (x >= 1)."
-    echo "'hadoop version' returned:"
-    echo `$HADOOP version`
-    exit 6
-fi
-
 if [ "${AUX_PARAM}" != "" ]; then
   HIVE_OPTS="$HIVE_OPTS -hiveconf hive.aux.jars.path=${AUX_PARAM}"
   AUX_JARS_CMD_LINE="-libjars ${AUX_PARAM}"