You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hive.apache.org by br...@apache.org on 2014/08/13 02:25:34 UTC

svn commit: r1617633 [1/6] - in /hive/branches/spark: ./ ant/ ant/src/org/apache/hadoop/hive/ant/ beeline/ beeline/src/java/org/apache/hive/beeline/ beeline/src/main/resources/ cli/src/java/org/apache/hadoop/hive/cli/ common/src/java/org/apache/hadoop/...

Author: brock
Date: Wed Aug 13 00:25:32 2014
New Revision: 1617633

URL: http://svn.apache.org/r1617633
Log:
Merge from trunk into spark

Added:
    hive/branches/spark/itests/src/
      - copied from r1617632, hive/trunk/itests/src/
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/processors/ListResourceProcessor.java
      - copied unchanged from r1617632, hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/processors/ListResourceProcessor.java
    hive/branches/spark/ql/src/test/queries/clientpositive/alter_rename_table.q
      - copied unchanged from r1617632, hive/trunk/ql/src/test/queries/clientpositive/alter_rename_table.q
    hive/branches/spark/ql/src/test/queries/clientpositive/bucket6.q
      - copied unchanged from r1617632, hive/trunk/ql/src/test/queries/clientpositive/bucket6.q
    hive/branches/spark/ql/src/test/results/clientpositive/alter_rename_table.q.out
      - copied unchanged from r1617632, hive/trunk/ql/src/test/results/clientpositive/alter_rename_table.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/bucket6.q.out
      - copied unchanged from r1617632, hive/trunk/ql/src/test/results/clientpositive/bucket6.q.out
Removed:
    hive/branches/spark/itests/qtest/testconfiguration.properties
Modified:
    hive/branches/spark/   (props changed)
    hive/branches/spark/ant/pom.xml
    hive/branches/spark/ant/src/org/apache/hadoop/hive/ant/QTestGenTask.java
    hive/branches/spark/beeline/pom.xml
    hive/branches/spark/beeline/src/java/org/apache/hive/beeline/BeeLine.java
    hive/branches/spark/beeline/src/java/org/apache/hive/beeline/BeeLineOpts.java
    hive/branches/spark/beeline/src/java/org/apache/hive/beeline/SeparatedValuesOutputFormat.java
    hive/branches/spark/beeline/src/main/resources/BeeLine.properties
    hive/branches/spark/cli/src/java/org/apache/hadoop/hive/cli/CliDriver.java
    hive/branches/spark/common/src/java/org/apache/hadoop/hive/conf/HiveConf.java
    hive/branches/spark/itests/hive-unit/src/test/java/org/apache/hadoop/hive/ql/security/authorization/plugin/TestHiveAuthorizerCheckInvocation.java
    hive/branches/spark/itests/hive-unit/src/test/java/org/apache/hive/beeline/TestBeeLineWithArgs.java
    hive/branches/spark/itests/qtest/pom.xml
    hive/branches/spark/itests/src/test/resources/testconfiguration.properties
    hive/branches/spark/itests/util/src/main/java/org/apache/hadoop/hive/ql/hooks/CheckColumnAccessHook.java
    hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/HiveAlterHandler.java
    hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/HiveMetaStore.java
    hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/MetaStoreUtils.java
    hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/ObjectStore.java
    hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/Warehouse.java
    hive/branches/spark/pom.xml
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/Driver.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/DDLTask.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/HashTableSinkOperator.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/Utilities.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/persistence/BytesBytesMultiHashMap.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/persistence/HashMapWrapper.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/persistence/MapJoinBytesTableContainer.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/tez/HashTableLoader.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/vector/VectorizationContext.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/vector/expressions/ConstantVectorExpression.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/vector/expressions/VectorExpressionWriterFactory.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/index/AggregateIndexHandler.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/index/TableBasedIndexHandler.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/metadata/Hive.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/AbstractBucketJoinProc.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/AbstractSMBJoinProc.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/BucketingSortingReduceSinkOptimizer.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/ColumnPrunerProcFactory.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/ConstantPropagate.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/GroupByOptimizer.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/IndexUtils.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/ReduceSinkMapJoinProc.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/index/RewriteCanApplyCtx.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/index/RewriteCanApplyProcFactory.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/index/RewriteGBUsingIndex.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/index/RewriteQueryUsingAggregateIndex.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/index/RewriteQueryUsingAggregateIndexCtx.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/physical/NullScanOptimizer.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/physical/Vectorizer.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/physical/index/IndexWhereProcessor.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/physical/index/IndexWhereTaskDispatcher.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/ppr/PartitionPruner.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/optimizer/stats/annotation/StatsRulesProcFactory.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/parse/BaseSemanticAnalyzer.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/parse/ColumnAccessAnalyzer.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/parse/ColumnAccessInfo.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/parse/DDLSemanticAnalyzer.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/parse/HiveParser.g
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/parse/IndexUpdater.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/parse/ParseContext.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/parse/SemanticAnalyzer.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/parse/authorization/HiveAuthorizationTaskFactoryImpl.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/plan/AlterIndexDesc.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/plan/AlterTableAlterPartDesc.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/plan/AlterTableSimpleDesc.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/plan/MapJoinDesc.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/plan/PrivilegeObjectDesc.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/plan/RenamePartitionDesc.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/plan/ShowColumnsDesc.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/plan/ShowGrantDesc.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/plan/Statistics.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/processors/CommandProcessor.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/processors/CommandProcessorFactory.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/processors/HiveCommand.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/security/authorization/AuthorizationUtils.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/security/authorization/plugin/HivePrivilegeObject.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/security/authorization/plugin/HiveV1Authorizer.java
    hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/udf/generic/GenericUDAFAverage.java
    hive/branches/spark/ql/src/test/org/apache/hadoop/hive/ql/exec/vector/TestVectorizationContext.java
    hive/branches/spark/ql/src/test/org/apache/hadoop/hive/ql/metadata/TestHive.java
    hive/branches/spark/ql/src/test/org/apache/hadoop/hive/ql/parse/TestQBCompact.java
    hive/branches/spark/ql/src/test/org/apache/hadoop/hive/ql/parse/authorization/PrivilegesTestBase.java
    hive/branches/spark/ql/src/test/org/apache/hadoop/hive/ql/parse/authorization/TestHiveAuthorizationTaskFactory.java
    hive/branches/spark/ql/src/test/org/apache/hadoop/hive/ql/parse/authorization/TestPrivilegesV1.java
    hive/branches/spark/ql/src/test/org/apache/hadoop/hive/ql/parse/authorization/TestPrivilegesV2.java
    hive/branches/spark/ql/src/test/queries/clientpositive/having.q
    hive/branches/spark/ql/src/test/queries/clientpositive/mapjoin_mapjoin.q
    hive/branches/spark/ql/src/test/queries/clientpositive/vector_coalesce.q
    hive/branches/spark/ql/src/test/results/clientnegative/alter_concatenate_indexed_table.q.out
    hive/branches/spark/ql/src/test/results/clientnegative/alter_view_failure6.q.out
    hive/branches/spark/ql/src/test/results/clientnegative/merge_negative_1.q.out
    hive/branches/spark/ql/src/test/results/clientnegative/merge_negative_2.q.out
    hive/branches/spark/ql/src/test/results/clientnegative/show_columns3.q.out
    hive/branches/spark/ql/src/test/results/clientnegative/show_tableproperties1.q.out
    hive/branches/spark/ql/src/test/results/clientnegative/temp_table_index.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/annotate_stats_join.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/drop_multi_partitions.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/having.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/input3.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/insert2_overwrite_partitions.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/mapjoin_mapjoin.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/ql_rewrite_gbtoidx.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/show_create_table_db_table.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/show_tblproperties.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/temp_table_names.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/temp_table_precedence.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/tez/having.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/tez/mapjoin_mapjoin.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/tez/vector_cast_constant.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/tez/vectorization_14.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/tez/vectorization_15.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/tez/vectorization_9.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/tez/vectorization_short_regress.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/union20.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/vector_between_in.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/vector_cast_constant.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/vector_coalesce.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/vector_decimal_mapjoin.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/vector_decimal_math_funcs.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/vector_elt.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/vectorization_14.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/vectorization_15.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/vectorization_16.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/vectorization_9.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/vectorization_div0.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/vectorization_short_regress.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/vectorized_math_funcs.q.out
    hive/branches/spark/ql/src/test/results/clientpositive/vectorized_parquet.q.out

Propchange: hive/branches/spark/
------------------------------------------------------------------------------
  Merged /hive/trunk:r1617008-1617632

Modified: hive/branches/spark/ant/pom.xml
URL: http://svn.apache.org/viewvc/hive/branches/spark/ant/pom.xml?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/ant/pom.xml (original)
+++ hive/branches/spark/ant/pom.xml Wed Aug 13 00:25:32 2014
@@ -39,6 +39,11 @@
       <artifactId>commons-lang</artifactId>
       <version>${commons-lang.version}</version>
     </dependency>
+      <dependency>
+        <groupId>com.google.guava</groupId>
+        <artifactId>guava</artifactId>
+        <version>${guava.version}</version>
+      </dependency>
     <dependency>
       <groupId>org.apache.ant</groupId>
       <artifactId>ant</artifactId>

Modified: hive/branches/spark/ant/src/org/apache/hadoop/hive/ant/QTestGenTask.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/ant/src/org/apache/hadoop/hive/ant/QTestGenTask.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/ant/src/org/apache/hadoop/hive/ant/QTestGenTask.java (original)
+++ hive/branches/spark/ant/src/org/apache/hadoop/hive/ant/QTestGenTask.java Wed Aug 13 00:25:32 2014
@@ -32,10 +32,11 @@ import java.util.ArrayList;
 import java.util.regex.Pattern;
 import java.util.HashMap;
 
+import com.google.common.base.Splitter;
+import com.google.common.collect.Sets;
 import org.apache.commons.lang.StringUtils;
 import org.apache.tools.ant.BuildException;
 import org.apache.tools.ant.Task;
-
 import org.apache.velocity.app.VelocityEngine;
 import org.apache.velocity.Template;
 import org.apache.velocity.VelocityContext;
@@ -45,6 +46,9 @@ import org.apache.velocity.exception.Res
 import org.apache.velocity.runtime.RuntimeConstants;
 
 public class QTestGenTask extends Task {
+   private static final Splitter CSV_SPLITTER = Splitter.on(',')
+       .trimResults()
+       .omitEmptyStrings();
 
   public class IncludeFilter implements FileFilter {
 
@@ -74,9 +78,8 @@ public class QTestGenTask extends Task {
       }
       return true;
     }
-    
   }
-  
+
   public class DisabledQFileFilter extends IncludeFilter {
     public DisabledQFileFilter(Set<String> includeOnly) {
       super(includeOnly);
@@ -87,17 +90,16 @@ public class QTestGenTask extends Task {
         return false;
       }
       return !fpath.isDirectory() && fpath.getName().endsWith(".q.disabled");
-    }  
+    }
   }
-  
+
   public class QFileRegexFilter extends QFileFilter {
     Pattern filterPattern;
-    
     public QFileRegexFilter(String filter, Set<String> includeOnly) {
       super(includeOnly);
       filterPattern = Pattern.compile(filter);
     }
-    
+
     public boolean accept(File filePath) {
       if (!super.accept(filePath)) {
         return false;
@@ -110,17 +112,17 @@ public class QTestGenTask extends Task {
   private List<String> templatePaths = new ArrayList<String>();
 
   private String hiveRootDirectory;
-  
+
   private String outputDirectory;
- 
+
   private String queryDirectory;
- 
+
   private String queryFile;
 
   private String includeQueryFile;
 
   private String excludeQueryFile;
-  
+
   private String queryFileRegex;
 
   private String resultsDirectory;
@@ -138,7 +140,7 @@ public class QTestGenTask extends Task {
   private String hiveConfDir;
 
   private String runDisabled;
-  
+
   private String hadoopVersion;
 
   private String initScript;
@@ -160,7 +162,7 @@ public class QTestGenTask extends Task {
   public String getHiveConfDir() {
     return hiveConfDir;
   }
-  
+
   public void setClusterMode(String clusterMode) {
     this.clusterMode = clusterMode;
   }
@@ -228,10 +230,10 @@ public class QTestGenTask extends Task {
   public String getHiveRootDirectory() {
     return hiveRootDirectory;
   }
-  
+
   public void setTemplatePath(String templatePath) throws Exception {
     templatePaths.clear();
-    for (String relativePath : templatePath.split(",")) {
+    for (String relativePath : CSV_SPLITTER.split(templatePath)) {
       templatePaths.add(project.resolveFile(relativePath).getCanonicalPath());
     }
     System.out.println("Template Path:" + getTemplatePath());
@@ -336,7 +338,7 @@ public class QTestGenTask extends Task {
 
     Set<String> includeOnly = null;
     if (includeQueryFile != null && !includeQueryFile.isEmpty()) {
-      includeOnly = new HashSet<String>(Arrays.asList(includeQueryFile.split(",")));
+      includeOnly = Sets.<String>newHashSet(CSV_SPLITTER.split(includeQueryFile));
     }
 
     List<File> qFiles;
@@ -346,7 +348,7 @@ public class QTestGenTask extends Task {
     File outDir = null;
     File resultsDir = null;
     File logDir = null;
-    
+
     try {
       // queryDirectory should not be null
       queryDir = new File(queryDirectory);
@@ -355,7 +357,7 @@ public class QTestGenTask extends Task {
       Set<File> testFiles = new HashSet<File>();
       if (queryFile != null && !queryFile.equals("")) {
         // The user may have passed a list of files - comma separated
-        for (String qFile : queryFile.split(",")) {
+        for (String qFile : CSV_SPLITTER.split(queryFile)) {
           if (includeOnly != null && !includeOnly.contains(qFile)) {
             continue;
           }
@@ -366,7 +368,7 @@ public class QTestGenTask extends Task {
           }
         }
       } else if (queryFileRegex != null && !queryFileRegex.equals("")) {
-        for (String regex : queryFileRegex.split(",")) {
+        for (String regex : CSV_SPLITTER.split(queryFileRegex)) {
           testFiles.addAll(Arrays.asList(queryDir.listFiles(
               new QFileRegexFilter(regex, includeOnly))));
         }
@@ -378,7 +380,7 @@ public class QTestGenTask extends Task {
 
       if (excludeQueryFile != null && !excludeQueryFile.equals("")) {
         // Exclude specified query files, comma separated
-        for (String qFile : excludeQueryFile.split(",")) {
+        for (String qFile : CSV_SPLITTER.split(excludeQueryFile)) {
           if (null != queryDir) {
             testFiles.remove(new File(queryDir, qFile));
           } else {
@@ -488,9 +490,11 @@ public class QTestGenTask extends Task {
       throw new BuildException("Generation failed", e);
     }
   }
+
   private String relativePath(File hiveRootDir, File file) {
     return escapePath(hiveRootDir.toURI().relativize(file.toURI()).getPath());
-  }  
+  }
+
   private static String escapePath(String path) {
     if (System.getProperty("os.name").toLowerCase().startsWith("win")) {
       // Escape the backward slash in CanonicalPath if the unit test runs on windows

Modified: hive/branches/spark/beeline/pom.xml
URL: http://svn.apache.org/viewvc/hive/branches/spark/beeline/pom.xml?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/beeline/pom.xml (original)
+++ hive/branches/spark/beeline/pom.xml Wed Aug 13 00:25:32 2014
@@ -80,6 +80,11 @@
       <artifactId>libthrift</artifactId>
       <version>${libthrift.version}</version>
     </dependency>
+    <dependency>
+      <groupId>net.sf.supercsv</groupId>
+      <artifactId>super-csv</artifactId>
+      <version>${super-csv.version}</version>
+    </dependency>
     <!-- test intra-project -->
     <dependency>
       <groupId>org.apache.hive</groupId>

Modified: hive/branches/spark/beeline/src/java/org/apache/hive/beeline/BeeLine.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/beeline/src/java/org/apache/hive/beeline/BeeLine.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/beeline/src/java/org/apache/hive/beeline/BeeLine.java (original)
+++ hive/branches/spark/beeline/src/java/org/apache/hive/beeline/BeeLine.java Wed Aug 13 00:25:32 2014
@@ -79,13 +79,13 @@ import jline.ConsoleReader;
 import jline.FileNameCompletor;
 import jline.History;
 import jline.SimpleCompletor;
-import org.apache.hadoop.io.IOUtils;
 
 import org.apache.commons.cli.CommandLine;
 import org.apache.commons.cli.GnuParser;
 import org.apache.commons.cli.OptionBuilder;
 import org.apache.commons.cli.Options;
 import org.apache.commons.cli.ParseException;
+import org.apache.hadoop.io.IOUtils;
 
 
 /**
@@ -148,6 +148,7 @@ public class BeeLine implements Closeabl
       "table", new TableOutputFormat(this),
       "csv", new SeparatedValuesOutputFormat(this, ','),
       "tsv", new SeparatedValuesOutputFormat(this, '\t'),
+      "dsv", new SeparatedValuesOutputFormat(this, BeeLineOpts.DEFAULT_DELIMITER_FOR_DSV),
       "xmlattr", new XMLAttributeOutputFormat(this),
       "xmlelements", new XMLElementOutputFormat(this),
   });

Modified: hive/branches/spark/beeline/src/java/org/apache/hive/beeline/BeeLineOpts.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/beeline/src/java/org/apache/hive/beeline/BeeLineOpts.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/beeline/src/java/org/apache/hive/beeline/BeeLineOpts.java (original)
+++ hive/branches/spark/beeline/src/java/org/apache/hive/beeline/BeeLineOpts.java Wed Aug 13 00:25:32 2014
@@ -53,6 +53,7 @@ class BeeLineOpts implements Completor {
   public static final String PROPERTY_NAME_EXIT =
       PROPERTY_PREFIX + "system.exit";
   public static final String DEFAULT_NULL_STRING = "NULL";
+  public static final char DEFAULT_DELIMITER_FOR_DSV = '|';
 
   private final BeeLine beeLine;
   private boolean autosave = false;
@@ -90,7 +91,7 @@ class BeeLineOpts implements Completor {
   private String scriptFile = null;
   private String initFile = null;
   private String authType = null;
-
+  private char delimiterForDSV = DEFAULT_DELIMITER_FOR_DSV;
 
   private Map<String, String> hiveVariables = new HashMap<String, String>();
   private Map<String, String> hiveConfVariables = new HashMap<String, String>();
@@ -500,5 +501,13 @@ class BeeLineOpts implements Completor {
   public void setTruncateTable(boolean truncateTable) {
     this.truncateTable = truncateTable;
   }
+
+  public char getDelimiterForDSV() {
+    return delimiterForDSV;
+  }
+
+  public void setDelimiterForDSV(char delimiterForDSV) {
+    this.delimiterForDSV = delimiterForDSV;
+  }
 }
 

Modified: hive/branches/spark/beeline/src/java/org/apache/hive/beeline/SeparatedValuesOutputFormat.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/beeline/src/java/org/apache/hive/beeline/SeparatedValuesOutputFormat.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/beeline/src/java/org/apache/hive/beeline/SeparatedValuesOutputFormat.java (original)
+++ hive/branches/spark/beeline/src/java/org/apache/hive/beeline/SeparatedValuesOutputFormat.java Wed Aug 13 00:25:32 2014
@@ -22,25 +22,43 @@
  */
 package org.apache.hive.beeline;
 
+import java.io.IOException;
+import java.io.StringWriter;
+
+import org.apache.hadoop.io.IOUtils;
+import org.supercsv.io.CsvListWriter;
+import org.supercsv.prefs.CsvPreference;
+
 /**
  * OutputFormat for values separated by a delimiter.
- *
- * <strong>TODO</strong>: Handle character escaping
- *
  */
 class SeparatedValuesOutputFormat implements OutputFormat {
   /**
    *
    */
   private final BeeLine beeLine;
-  private char separator;
+  private CsvPreference csvPreference;
 
-  public SeparatedValuesOutputFormat(BeeLine beeLine, char separator) {
+  SeparatedValuesOutputFormat(BeeLine beeLine, char separator) {
     this.beeLine = beeLine;
-    setSeparator(separator);
+    csvPreference = new CsvPreference.Builder('"', separator, "").build();
+  }
+
+  private void updateCsvPreference() {
+    if (beeLine.getOpts().getOutputFormat().equals("dsv")) {
+      // check whether delimiter changed by user
+      char curDel = (char) csvPreference.getDelimiterChar();
+      char newDel = beeLine.getOpts().getDelimiterForDSV();
+      // if delimiter changed, rebuild the csv preference
+      if (newDel != curDel) {
+        csvPreference = new CsvPreference.Builder('"', newDel, "").build();
+      }
+    }
   }
 
   public int print(Rows rows) {
+    updateCsvPreference();
+
     int count = 0;
     while (rows.hasNext()) {
       printRow(rows, (Rows.Row) rows.next());
@@ -49,23 +67,24 @@ class SeparatedValuesOutputFormat implem
     return count - 1; // sans header row
   }
 
-  public void printRow(Rows rows, Rows.Row row) {
-    String[] vals = row.values;
-    StringBuilder buf = new StringBuilder();
-    for (int i = 0; i < vals.length; i++) {
-      buf.append(buf.length() == 0 ? "" : "" + getSeparator())
-          .append('\'')
-          .append(vals[i] == null ? "" : vals[i])
-          .append('\'');
+  private String getFormattedStr(String[] vals) {
+    StringWriter strWriter = new StringWriter();
+    CsvListWriter writer = new CsvListWriter(strWriter, csvPreference);
+    if (vals.length > 0) {
+      try {
+        writer.write(vals);
+      } catch (IOException e) {
+        beeLine.error(e);
+      } finally {
+        IOUtils.closeStream(writer);
+      }
     }
-    beeLine.output(buf.toString());
-  }
-
-  public void setSeparator(char separator) {
-    this.separator = separator;
+    return strWriter.toString();
   }
 
-  public char getSeparator() {
-    return this.separator;
+  public void printRow(Rows rows, Rows.Row row) {
+    String[] vals = row.values;
+    String formattedStr = getFormattedStr(vals);
+    beeLine.output(formattedStr);
   }
 }

Modified: hive/branches/spark/beeline/src/main/resources/BeeLine.properties
URL: http://svn.apache.org/viewvc/hive/branches/spark/beeline/src/main/resources/BeeLine.properties?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/beeline/src/main/resources/BeeLine.properties (original)
+++ hive/branches/spark/beeline/src/main/resources/BeeLine.properties Wed Aug 13 00:25:32 2014
@@ -68,7 +68,8 @@ help-procedures: List all the procedures
 help-tables: List all the tables in the database
 help-columns: List all the columns for the specified table
 help-properties: Connect to the database specified in the properties file(s)
-help-outputformat: Set the output format for displaying results (table,vertical,csv,tsv,xmlattrs,xmlelements)
+help-outputformat: Set the output format for displaying results (table,vertical,csv,dsv,tsv,xmlattrs,xmlelements)
+help-delimiterForDSV: Set the delimiter for dsv output format
 help-nullemptystring: Set to true to get historic behavior of printing null as empty string. Default is false.
 
 jline-missing: The JLine jar was not found. Please ensure it is installed.
@@ -166,8 +167,9 @@ cmd-usage: Usage: java org.apache.hive.c
 \  --maxColumnWidth=MAXCOLWIDTH    the maximum width to use when displaying columns\n \
 \  --silent=[true/false]           be more silent\n \
 \  --autosave=[true/false]         automatically save preferences\n \
-\  --outputformat=[table/vertical/csv/tsv]   format mode for result display\n \
+\  --outputformat=[table/vertical/csv/tsv/dsv]   format mode for result display\n \
 \  --truncateTable=[true/false]    truncate table column when it exceeds length\n \
+\  --delimiterForDSV=DELIMITER     specify the delimiter for delimiter-separated values output format (default: |)\n \
 \  --isolation=LEVEL               set the transaction isolation level\n \
 \  --nullemptystring=[true/false]  set to true to get historic behavior of printing null as empty string\n \
 \  --help                          display this message

Modified: hive/branches/spark/cli/src/java/org/apache/hadoop/hive/cli/CliDriver.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/cli/src/java/org/apache/hadoop/hive/cli/CliDriver.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/cli/src/java/org/apache/hadoop/hive/cli/CliDriver.java (original)
+++ hive/branches/spark/cli/src/java/org/apache/hadoop/hive/cli/CliDriver.java Wed Aug 13 00:25:32 2014
@@ -32,7 +32,6 @@ import java.util.ArrayList;
 import java.util.Arrays;
 import java.util.List;
 import java.util.Map;
-import java.util.Set;
 
 import jline.ArgumentCompletor;
 import jline.ArgumentCompletor.AbstractArgumentDelimiter;
@@ -150,24 +149,6 @@ public class CliDriver {
             stringifyException(e));
         ret = 1;
       }
-    } else if (tokens[0].toLowerCase().equals("list")) {
-
-      SessionState.ResourceType t;
-      if (tokens.length < 2 || (t = SessionState.find_resource_type(tokens[1])) == null) {
-        console.printError("Usage: list ["
-            + StringUtils.join(SessionState.ResourceType.values(), "|") + "] [<value> [<value>]*]");
-        ret = 1;
-      } else {
-        List<String> filter = null;
-        if (tokens.length >= 3) {
-          System.arraycopy(tokens, 2, tokens, 0, tokens.length - 2);
-          filter = Arrays.asList(tokens);
-        }
-        Set<String> s = ss.list_resource(t, filter);
-        if (s != null && !s.isEmpty()) {
-          ss.out.println(StringUtils.join(s, "\n"));
-        }
-      }
     } else if (ss.isRemoteMode()) { // remote mode -- connecting to remote hive server
       HiveClient client = ss.getClient();
       PrintStream out = ss.out;

Modified: hive/branches/spark/common/src/java/org/apache/hadoop/hive/conf/HiveConf.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/common/src/java/org/apache/hadoop/hive/conf/HiveConf.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/common/src/java/org/apache/hadoop/hive/conf/HiveConf.java (original)
+++ hive/branches/spark/common/src/java/org/apache/hadoop/hive/conf/HiveConf.java Wed Aug 13 00:25:32 2014
@@ -897,7 +897,12 @@ public class HiveConf extends Configurat
         "This controls how many partitions can be scanned for each partitioned table.\n" +
         "The default value \"-1\" means no limit."),
 
-    HIVEHASHTABLETHRESHOLD("hive.hashtable.initialCapacity", 100000, ""),
+    HIVEHASHTABLEKEYCOUNTADJUSTMENT("hive.hashtable.key.count.adjustment", 1.0f,
+        "Adjustment to mapjoin hashtable size derived from table and column statistics; the estimate" +
+        " of the number of keys is divided by this value. If the value is 0, statistics are not used" +
+        "and hive.hashtable.initialCapacity is used instead."),
+    HIVEHASHTABLETHRESHOLD("hive.hashtable.initialCapacity", 100000, "Initial capacity of " +
+        "mapjoin hashtable if statistics are absent, or if hive.hashtable.stats.key.estimate.adjustment is set to 0"),
     HIVEHASHTABLELOADFACTOR("hive.hashtable.loadfactor", (float) 0.75, ""),
     HIVEHASHTABLEFOLLOWBYGBYMAXMEMORYUSAGE("hive.mapjoin.followby.gby.localtask.max.memory.usage", (float) 0.55,
         "This number means how much memory the local task can take to hold the key/value into an in-memory hash table \n" +
@@ -1551,7 +1556,7 @@ public class HiveConf extends Configurat
     HIVE_SERVER2_SSL_KEYSTORE_PATH("hive.server2.keystore.path", "", ""),
     HIVE_SERVER2_SSL_KEYSTORE_PASSWORD("hive.server2.keystore.password", "", ""),
 
-    HIVE_SECURITY_COMMAND_WHITELIST("hive.security.command.whitelist", "set,reset,dfs,add,delete,compile",
+    HIVE_SECURITY_COMMAND_WHITELIST("hive.security.command.whitelist", "set,reset,dfs,add,list,delete,compile",
         "Comma separated list of non-SQL Hive commands users are authorized to execute"),
 
     HIVE_CONF_RESTRICTED_LIST("hive.conf.restricted.list",

Modified: hive/branches/spark/itests/hive-unit/src/test/java/org/apache/hadoop/hive/ql/security/authorization/plugin/TestHiveAuthorizerCheckInvocation.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/itests/hive-unit/src/test/java/org/apache/hadoop/hive/ql/security/authorization/plugin/TestHiveAuthorizerCheckInvocation.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/itests/hive-unit/src/test/java/org/apache/hadoop/hive/ql/security/authorization/plugin/TestHiveAuthorizerCheckInvocation.java (original)
+++ hive/branches/spark/itests/hive-unit/src/test/java/org/apache/hadoop/hive/ql/security/authorization/plugin/TestHiveAuthorizerCheckInvocation.java Wed Aug 13 00:25:32 2014
@@ -29,7 +29,6 @@ import java.util.ArrayList;
 import java.util.Arrays;
 import java.util.Collections;
 import java.util.List;
-import java.util.Set;
 
 import org.apache.hadoop.hive.conf.HiveConf;
 import org.apache.hadoop.hive.conf.HiveConf.ConfVars;
@@ -110,7 +109,7 @@ public class TestHiveAuthorizerCheckInvo
         getSortedList(tableObj.getColumns()));
   }
 
-  private List<String> getSortedList(Set<String> columns) {
+  private List<String> getSortedList(List<String> columns) {
     List<String> sortedCols = new ArrayList<String>(columns);
     Collections.sort(sortedCols);
     return sortedCols;

Modified: hive/branches/spark/itests/hive-unit/src/test/java/org/apache/hive/beeline/TestBeeLineWithArgs.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/itests/hive-unit/src/test/java/org/apache/hive/beeline/TestBeeLineWithArgs.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/itests/hive-unit/src/test/java/org/apache/hive/beeline/TestBeeLineWithArgs.java (original)
+++ hive/branches/spark/itests/hive-unit/src/test/java/org/apache/hive/beeline/TestBeeLineWithArgs.java Wed Aug 13 00:25:32 2014
@@ -362,7 +362,7 @@ public class TestBeeLineWithArgs {
     final String TEST_NAME = "testNullNonDefault";
     final String SCRIPT_TEXT = "set hive.support.concurrency = false;\n" +
                 "!set nullemptystring true\n select 'abc',null,'def' from " + tableName + " limit 1 ;\n";
-    final String EXPECTED_PATTERN = "'abc','','def'";
+    final String EXPECTED_PATTERN = "abc,,def";
 
     List<String> argList = getBaseArgs(JDBC_URL);
     argList.add("--outputformat=csv");
@@ -382,7 +382,7 @@ public class TestBeeLineWithArgs {
     final String SCRIPT_TEXT = "set hive.support.concurrency = false;\n" +
                 "select 'abc',null,'def' from " + tableName + " limit 1 ;\n";
     //final String EXPECTED_PATTERN = "| abc  |      | def  |";
-    final String EXPECTED_PATTERN = "'abc','','def'";
+    final String EXPECTED_PATTERN = "abc,,def";
 
     List<String> argList = getBaseArgs(JDBC_URL);
     argList.add("--nullemptystring=true");

Modified: hive/branches/spark/itests/qtest/pom.xml
URL: http://svn.apache.org/viewvc/hive/branches/spark/itests/qtest/pom.xml?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/itests/qtest/pom.xml (original)
+++ hive/branches/spark/itests/qtest/pom.xml Wed Aug 13 00:25:32 2014
@@ -373,7 +373,7 @@
             </goals>
             <configuration>
               <files>
-                <file>${basedir}/testconfiguration.properties</file>
+                <file>${basedir}/../src/test/resources/testconfiguration.properties</file>
               </files>
             </configuration>
           </execution>

Modified: hive/branches/spark/itests/src/test/resources/testconfiguration.properties
URL: http://svn.apache.org/viewvc/hive/branches/spark/itests/src/test/resources/testconfiguration.properties?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/itests/src/test/resources/testconfiguration.properties (original)
+++ hive/branches/spark/itests/src/test/resources/testconfiguration.properties Wed Aug 13 00:25:32 2014
@@ -309,3 +309,5 @@ minimr.query.negative.files=cluster_task
   mapreduce_stack_trace_turnoff_hadoop20.q,\
   minimr_broken_pipe.q,\
   udf_local_resource.q
+
+spark.query.files=spark_test.q

Modified: hive/branches/spark/itests/util/src/main/java/org/apache/hadoop/hive/ql/hooks/CheckColumnAccessHook.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/itests/util/src/main/java/org/apache/hadoop/hive/ql/hooks/CheckColumnAccessHook.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/itests/util/src/main/java/org/apache/hadoop/hive/ql/hooks/CheckColumnAccessHook.java (original)
+++ hive/branches/spark/itests/util/src/main/java/org/apache/hadoop/hive/ql/hooks/CheckColumnAccessHook.java Wed Aug 13 00:25:32 2014
@@ -18,9 +18,9 @@
 package org.apache.hadoop.hive.ql.hooks;
 
 import java.util.Arrays;
+import java.util.List;
 import java.util.Map;
 import java.util.HashMap;
-import java.util.Set;
 
 import org.apache.commons.lang.StringUtils;
 import org.apache.hadoop.hive.conf.HiveConf;
@@ -34,7 +34,7 @@ import org.mortbay.log.Log;
 /*
  * This hook is used for verifying the column access information
  * that is generated and maintained in the QueryPlan object by the
- * ColumnAccessAnalyer. All the hook does is print out the columns
+ * ColumnAccessAnalyzer. All the hook does is print out the columns
  * accessed from each table as recorded in the ColumnAccessInfo
  * in the QueryPlan.
  */
@@ -58,14 +58,14 @@ public class CheckColumnAccessHook imple
     }
 
     LogHelper console = SessionState.getConsole();
-    Map<String, Set<String>> tableToColumnAccessMap =
+    Map<String, List<String>> tableToColumnAccessMap =
       columnAccessInfo.getTableToColumnAccessMap();
 
     // We need a new map to ensure output is always produced in the same order.
     // This makes tests that use this hook deterministic.
     Map<String, String> outputOrderedMap = new HashMap<String, String>();
 
-    for (Map.Entry<String, Set<String>> tableAccess : tableToColumnAccessMap.entrySet()) {
+    for (Map.Entry<String, List<String>> tableAccess : tableToColumnAccessMap.entrySet()) {
       StringBuilder perTableInfo = new StringBuilder();
       perTableInfo.append("Table:").append(tableAccess.getKey()).append("\n");
       // Sort columns to make output deterministic

Modified: hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/HiveAlterHandler.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/HiveAlterHandler.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/HiveAlterHandler.java (original)
+++ hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/HiveAlterHandler.java Wed Aug 13 00:25:32 2014
@@ -30,6 +30,7 @@ import org.apache.hadoop.conf.Configurat
 import org.apache.hadoop.fs.FileSystem;
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.hive.common.FileUtils;
+import org.apache.hadoop.hive.common.ObjectPair;
 import org.apache.hadoop.hive.conf.HiveConf;
 import org.apache.hadoop.hive.metastore.api.AlreadyExistsException;
 import org.apache.hadoop.hive.metastore.api.Database;
@@ -81,11 +82,11 @@ public class HiveAlterHandler implements
     FileSystem destFs = null;
 
     boolean success = false;
-    String oldTblLoc = null;
-    String newTblLoc = null;
     boolean moveData = false;
     boolean rename = false;
     Table oldt = null;
+    List<ObjectPair<Partition, String>> altps = new ArrayList<ObjectPair<Partition, String>>();
+
     try {
       msdb.openTransaction();
       name = name.toLowerCase();
@@ -132,30 +133,30 @@ public class HiveAlterHandler implements
 
       // if this alter is a rename, the table is not a virtual view, the user
       // didn't change the default location (or new location is empty), and
-      // table is not an external table, that means useris asking metastore to
+      // table is not an external table, that means user is asking metastore to
       // move data to the new location corresponding to the new name
       if (rename
           && !oldt.getTableType().equals(TableType.VIRTUAL_VIEW.toString())
           && (oldt.getSd().getLocation().compareTo(newt.getSd().getLocation()) == 0
             || StringUtils.isEmpty(newt.getSd().getLocation()))
           && !MetaStoreUtils.isExternalTable(oldt)) {
+
+        srcPath = new Path(oldt.getSd().getLocation());
+        srcFs = wh.getFs(srcPath);
+
         // that means user is asking metastore to move data to new location
         // corresponding to the new name
         // get new location
-        newTblLoc = wh.getTablePath(msdb.getDatabase(newt.getDbName()),
-            newt.getTableName()).toString();
-        Path newTblPath = constructRenamedPath(new Path(newTblLoc),
-            new Path(newt.getSd().getLocation()));
-        newTblLoc = newTblPath.toString();
-        newt.getSd().setLocation(newTblLoc);
-        oldTblLoc = oldt.getSd().getLocation();
+        Path databasePath = constructRenamedPath(
+            wh.getDefaultDatabasePath(newt.getDbName()), srcPath);
+        destPath = new Path(databasePath, newt.getTableName());
+        destFs = wh.getFs(destPath);
+
+        newt.getSd().setLocation(destPath.toString());
         moveData = true;
+
         // check that destination does not exist otherwise we will be
         // overwriting data
-        srcPath = new Path(oldTblLoc);
-        srcFs = wh.getFs(srcPath);
-        destPath = new Path(newTblLoc);
-        destFs = wh.getFs(destPath);
         // check that src and dest are on the same file system
         if (!FileUtils.equalsFileSystem(srcFs, destFs)) {
           throw new InvalidOperationException("table new location " + destPath
@@ -177,22 +178,18 @@ public class HiveAlterHandler implements
               + destPath + " for table " + newt.getDbName() + "."
               + newt.getTableName());
         }
+        String oldTblLocPath = srcPath.toUri().getPath();
+        String newTblLocPath = destPath.toUri().getPath();
+
         // also the location field in partition
         List<Partition> parts = msdb.getPartitions(dbname, name, -1);
         for (Partition part : parts) {
           String oldPartLoc = part.getSd().getLocation();
-          Path oldPartLocPath = new Path(oldPartLoc);
-          String oldTblLocPath = new Path(oldTblLoc).toUri().getPath();
-          String newTblLocPath = new Path(newTblLoc).toUri().getPath();
           if (oldPartLoc.contains(oldTblLocPath)) {
-            Path newPartLocPath = null;
-            URI oldUri = oldPartLocPath.toUri();
-            String newPath = oldUri.getPath().replace(oldTblLocPath,
-                                                      newTblLocPath);
-
-            newPartLocPath = new Path(oldUri.getScheme(),
-                                      oldUri.getAuthority(),
-                                      newPath);
+            URI oldUri = new Path(oldPartLoc).toUri();
+            String newPath = oldUri.getPath().replace(oldTblLocPath, newTblLocPath);
+            Path newPartLocPath = new Path(oldUri.getScheme(), oldUri.getAuthority(), newPath);
+            altps.add(ObjectPair.create(part, part.getSd().getLocation()));
             part.getSd().setLocation(newPartLocPath.toString());
             msdb.alterPartition(dbname, name, part.getValues(), part);
           }
@@ -235,9 +232,23 @@ public class HiveAlterHandler implements
           try {
             msdb.openTransaction();
             msdb.alterTable(dbname, newt.getTableName(), oldt);
+            for (ObjectPair<Partition, String> pair : altps) {
+              Partition part = pair.getFirst();
+              part.getSd().setLocation(pair.getSecond());
+              msdb.alterPartition(dbname, name, part.getValues(), part);
+            }
             revertMetaDataTransaction = msdb.commitTransaction();
           } catch (Exception e1) {
-            LOG.error("Reverting metadata opeation failed During HDFS operation failed", e1);
+            // we should log this for manual rollback by administrator
+            LOG.error("Reverting metadata by HDFS operation failure failed During HDFS operation failed", e1);
+            LOG.error("Table " + Warehouse.getQualifiedName(newt) +
+                " should be renamed to " + Warehouse.getQualifiedName(oldt));
+            LOG.error("Table " + Warehouse.getQualifiedName(newt) +
+                " should have path " + srcPath);
+            for (ObjectPair<Partition, String> pair : altps) {
+              LOG.error("Partition " + Warehouse.getQualifiedName(pair.getFirst()) +
+                  " should have path " + pair.getSecond());
+            }
             if (!revertMetaDataTransaction) {
               msdb.rollbackTransaction();
             }

Modified: hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/HiveMetaStore.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/HiveMetaStore.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/HiveMetaStore.java (original)
+++ hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/HiveMetaStore.java Wed Aug 13 00:25:32 2014
@@ -3269,6 +3269,9 @@ public class HiveMetaStore extends Thrif
 
       boolean success = false, indexTableCreated = false;
 
+      String[] qualified =
+          MetaStoreUtils.getQualifiedName(index.getDbName(), index.getIndexTableName());
+
       try {
         ms.openTransaction();
         Index old_index = null;
@@ -3291,7 +3294,7 @@ public class HiveMetaStore extends Thrif
         Table indexTbl = indexTable;
         if (indexTbl != null) {
           try {
-            indexTbl = ms.getTable(index.getDbName(), index.getIndexTableName());
+            indexTbl = ms.getTable(qualified[0], qualified[1]);
           } catch (Exception e) {
           }
           if (indexTbl != null) {
@@ -3312,7 +3315,7 @@ public class HiveMetaStore extends Thrif
         if (!success) {
           if (indexTableCreated) {
             try {
-              this.drop_table(index.getDbName(), index.getIndexTableName(), false);
+              drop_table(qualified[0], qualified[1], false);
             } catch (Exception e) {
             }
           }
@@ -3366,8 +3369,8 @@ public class HiveMetaStore extends Thrif
 
         String idxTblName = index.getIndexTableName();
         if (idxTblName != null) {
-          Table tbl = null;
-          tbl = this.get_table(dbName, idxTblName);
+          String[] qualified = MetaStoreUtils.getQualifiedName(index.getDbName(), idxTblName);
+          Table tbl = get_table(qualified[0], qualified[1]);
           if (tbl.getSd() == null) {
             throw new MetaException("Table metadata is corrupted");
           }

Modified: hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/MetaStoreUtils.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/MetaStoreUtils.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/MetaStoreUtils.java (original)
+++ hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/MetaStoreUtils.java Wed Aug 13 00:25:32 2014
@@ -1538,4 +1538,12 @@ public class MetaStoreUtils {
       return part.getValues().size();
     }
   }
+
+  public static String[] getQualifiedName(String defaultDbName, String tableName) {
+    String[] names = tableName.split("\\.");
+    if (names.length == 1) {
+      return new String[] { defaultDbName, tableName};
+    }
+    return new String[] {names[0], names[1]};
+  }
 }

Modified: hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/ObjectStore.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/ObjectStore.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/ObjectStore.java (original)
+++ hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/ObjectStore.java Wed Aug 13 00:25:32 2014
@@ -2847,7 +2847,8 @@ public class ObjectStore implements RawS
           "Original table does not exist for the given index.");
     }
 
-    MTable indexTable = getMTable(index.getDbName(), index.getIndexTableName());
+    String[] qualified = MetaStoreUtils.getQualifiedName(index.getDbName(), index.getIndexTableName());
+    MTable indexTable = getMTable(qualified[0], qualified[1]);
     if (indexTable == null) {
       throw new InvalidObjectException(
           "Underlying index table does not exist for the given index.");

Modified: hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/Warehouse.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/Warehouse.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/Warehouse.java (original)
+++ hive/branches/spark/metastore/src/java/org/apache/hadoop/hive/metastore/Warehouse.java Wed Aug 13 00:25:32 2014
@@ -51,6 +51,7 @@ import org.apache.hadoop.hive.conf.HiveC
 import org.apache.hadoop.hive.metastore.api.Database;
 import org.apache.hadoop.hive.metastore.api.FieldSchema;
 import org.apache.hadoop.hive.metastore.api.MetaException;
+import org.apache.hadoop.hive.metastore.api.Partition;
 import org.apache.hadoop.hive.metastore.api.StorageDescriptor;
 import org.apache.hadoop.hive.metastore.api.Table;
 import org.apache.hadoop.hive.shims.ShimLoader;
@@ -186,6 +187,14 @@ public class Warehouse {
     return getDnsPath(new Path(getDatabasePath(db), tableName.toLowerCase()));
   }
 
+  public static String getQualifiedName(Table table) {
+    return table.getDbName() + "." + table.getTableName();
+  }
+
+  public static String getQualifiedName(Partition partition) {
+    return partition.getDbName() + "." + partition.getTableName() + partition.getValues();
+  }
+
   public boolean mkdirs(Path f, boolean inheritPermCandidate) throws MetaException {
     boolean inheritPerms = HiveConf.getBoolVar(conf,
       HiveConf.ConfVars.HIVE_WAREHOUSE_SUBDIR_INHERIT_PERMS) && inheritPermCandidate;

Modified: hive/branches/spark/pom.xml
URL: http://svn.apache.org/viewvc/hive/branches/spark/pom.xml?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/pom.xml (original)
+++ hive/branches/spark/pom.xml Wed Aug 13 00:25:32 2014
@@ -146,6 +146,7 @@
     <stax.version>1.0.1</stax.version>
     <slf4j.version>1.7.5</slf4j.version>
     <ST4.version>4.0.4</ST4.version>
+    <super-csv.version>2.2.0</super-csv.version>
     <tez.version>0.4.0-incubating</tez.version>
     <spark.version>1.1.0-SNAPSHOT</spark.version>
     <scala.binary.version>2.10</scala.binary.version>

Modified: hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/Driver.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/Driver.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/Driver.java (original)
+++ hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/Driver.java Wed Aug 13 00:25:32 2014
@@ -507,7 +507,7 @@ public class Driver implements CommandPr
       // get mapping of tables to columns used
       ColumnAccessInfo colAccessInfo = sem.getColumnAccessInfo();
       // colAccessInfo is set only in case of SemanticAnalyzer
-      Map<String, Set<String>> tab2Cols = colAccessInfo != null ? colAccessInfo
+      Map<String, List<String>> tab2Cols = colAccessInfo != null ? colAccessInfo
           .getTableToColumnAccessMap() : null;
       doAuthorizationV2(ss, op, inputs, outputs, command, tab2Cols);
      return;
@@ -700,7 +700,7 @@ public class Driver implements CommandPr
   }
 
   private static void doAuthorizationV2(SessionState ss, HiveOperation op, HashSet<ReadEntity> inputs,
-      HashSet<WriteEntity> outputs, String command, Map<String, Set<String>> tab2cols) throws HiveException {
+      HashSet<WriteEntity> outputs, String command, Map<String, List<String>> tab2cols) throws HiveException {
 
     HiveAuthzContext.Builder authzContextBuilder = new HiveAuthzContext.Builder();
 
@@ -711,36 +711,14 @@ public class Driver implements CommandPr
     authzContextBuilder.setCommandString(command);
 
     HiveOperationType hiveOpType = getHiveOperationType(op);
-    List<HivePrivilegeObject> inputsHObjs = getHivePrivObjects(inputs);
-    updateInputColumnInfo(inputsHObjs, tab2cols);
+    List<HivePrivilegeObject> inputsHObjs = getHivePrivObjects(inputs, tab2cols);
+    List<HivePrivilegeObject> outputHObjs = getHivePrivObjects(outputs, null);
 
-    List<HivePrivilegeObject> outputHObjs = getHivePrivObjects(outputs);
     ss.getAuthorizerV2().checkPrivileges(hiveOpType, inputsHObjs, outputHObjs, authzContextBuilder.build());
-    return;
   }
 
-  /**
-   * Add column information for input table objects
-   * @param inputsHObjs input HivePrivilegeObject
-   * @param map table to used input columns mapping
-   */
-  private static void updateInputColumnInfo(List<HivePrivilegeObject> inputsHObjs,
-      Map<String, Set<String>> tableName2Cols) {
-    if(tableName2Cols == null) {
-      return;
-    }
-    for(HivePrivilegeObject inputObj : inputsHObjs){
-      if(inputObj.getType() != HivePrivilegeObjectType.TABLE_OR_VIEW){
-        // input columns are relevant only for tables or views
-        continue;
-      }
-      Set<String> cols = tableName2Cols.get(Table.getCompleteName(inputObj.getDbname(),
-          inputObj.getObjectName()));
-      inputObj.setColumns(cols);
-    }
-  }
-
-  private static List<HivePrivilegeObject> getHivePrivObjects(HashSet<? extends Entity> privObjects) {
+  private static List<HivePrivilegeObject> getHivePrivObjects(
+      HashSet<? extends Entity> privObjects, Map<String, List<String>> tableName2Cols) {
     List<HivePrivilegeObject> hivePrivobjs = new ArrayList<HivePrivilegeObject>();
     if(privObjects == null){
       return hivePrivobjs;
@@ -764,13 +742,17 @@ public class Driver implements CommandPr
       //support for authorization on partitions needs to be added
       String dbname = null;
       String objName = null;
+      List<String> partKeys = null;
+      List<String> columns = null;
       switch(privObject.getType()){
       case DATABASE:
-        dbname = privObject.getDatabase() == null ? null : privObject.getDatabase().getName();
+        dbname = privObject.getDatabase().getName();
         break;
       case TABLE:
-        dbname = privObject.getTable() == null ? null : privObject.getTable().getDbName();
-        objName = privObject.getTable() == null ? null : privObject.getTable().getTableName();
+        dbname = privObject.getTable().getDbName();
+        objName = privObject.getTable().getTableName();
+        columns = tableName2Cols == null ? null :
+            tableName2Cols.get(Table.getCompleteName(dbname, objName));
         break;
       case DFS_DIR:
       case LOCAL_DIR:
@@ -788,7 +770,7 @@ public class Driver implements CommandPr
       }
       HivePrivObjectActionType actionType = AuthorizationUtils.getActionType(privObject);
       HivePrivilegeObject hPrivObject = new HivePrivilegeObject(privObjType, dbname, objName,
-          actionType);
+          partKeys, columns, actionType, null);
       hivePrivobjs.add(hPrivObject);
     }
     return hivePrivobjs;

Modified: hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/DDLTask.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/DDLTask.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/DDLTask.java (original)
+++ hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/DDLTask.java Wed Aug 13 00:25:32 2014
@@ -35,7 +35,6 @@ import java.util.Arrays;
 import java.util.Collections;
 import java.util.Comparator;
 import java.util.HashMap;
-import java.util.HashSet;
 import java.util.Iterator;
 import java.util.List;
 import java.util.Map;
@@ -600,13 +599,9 @@ public class DDLTask extends Task<DDLWor
 
     HiveAuthorizer authorizer = getSessionAuthorizer();
     try {
-      Set<String> colSet = showGrantDesc.getColumns() != null ? new HashSet<String>(
-          showGrantDesc.getColumns()) : null;
       List<HivePrivilegeInfo> privInfos = authorizer.showPrivileges(
           AuthorizationUtils.getHivePrincipal(showGrantDesc.getPrincipalDesc()),
-          AuthorizationUtils.getHivePrivilegeObject(showGrantDesc.getHiveObj(),
-              colSet
-              ));
+          AuthorizationUtils.getHivePrivilegeObject(showGrantDesc.getHiveObj()));
       boolean testMode = conf.getBoolVar(HiveConf.ConfVars.HIVE_IN_TEST);
       writeToFile(writeGrantInfo(privInfos, testMode), showGrantDesc.getResFile());
     } catch (IOException e) {
@@ -625,7 +620,7 @@ public class DDLTask extends Task<DDLWor
     //Convert to object types used by the authorization plugin interface
     List<HivePrincipal> hivePrincipals = AuthorizationUtils.getHivePrincipals(principals);
     List<HivePrivilege> hivePrivileges = AuthorizationUtils.getHivePrivileges(privileges);
-    HivePrivilegeObject hivePrivObject = AuthorizationUtils.getHivePrivilegeObject(privSubjectDesc, null);
+    HivePrivilegeObject hivePrivObject = AuthorizationUtils.getHivePrivilegeObject(privSubjectDesc);
 
     HivePrincipal grantorPrincipal = new HivePrincipal(
         grantor, AuthorizationUtils.getHivePrincipalType(grantorType));
@@ -754,8 +749,7 @@ public class DDLTask extends Task<DDLWor
   }
 
   private int dropIndex(Hive db, DropIndexDesc dropIdx) throws HiveException {
-    db.dropIndex(SessionState.get().getCurrentDatabase(), dropIdx.getTableName(),
-        dropIdx.getIndexName(), true);
+    db.dropIndex(dropIdx.getTableName(), dropIdx.getIndexName(), true);
     return 0;
   }
 
@@ -765,11 +759,7 @@ public class DDLTask extends Task<DDLWor
       validateSerDe(crtIndex.getSerde());
     }
 
-    String indexTableName =
-      crtIndex.getIndexTableName() != null ? crtIndex.getIndexTableName() :
-        MetaStoreUtils.getIndexTableName(SessionState.get().getCurrentDatabase(),
-             crtIndex.getTableName(), crtIndex.getIndexName());
-
+    String indexTableName = crtIndex.getIndexTableName();
     if (!Utilities.isDefaultNameNode(conf)) {
       // If location is specified - ensure that it is a full qualified name
       makeLocationQualified(crtIndex, indexTableName);
@@ -792,10 +782,9 @@ public class DDLTask extends Task<DDLWor
   }
 
   private int alterIndex(Hive db, AlterIndexDesc alterIndex) throws HiveException {
-    String dbName = alterIndex.getDbName();
     String baseTableName = alterIndex.getBaseTableName();
     String indexName = alterIndex.getIndexName();
-    Index idx = db.getIndex(dbName, baseTableName, indexName);
+    Index idx = db.getIndex(baseTableName, indexName);
 
     switch(alterIndex.getOp()) {
     case ADDPROPS:
@@ -806,8 +795,7 @@ public class DDLTask extends Task<DDLWor
         Map<String, String> props = new HashMap<String, String>();
         Map<Map<String, String>, Long> basePartTs = new HashMap<Map<String, String>, Long>();
 
-        Table baseTbl = db.getTable(SessionState.get().getCurrentDatabase(),
-            baseTableName);
+        Table baseTbl = db.getTable(baseTableName);
 
         if (baseTbl.isPartitioned()) {
           List<Partition> baseParts;
@@ -854,7 +842,7 @@ public class DDLTask extends Task<DDLWor
     }
 
     try {
-      db.alterIndex(dbName, baseTableName, indexName, idx);
+      db.alterIndex(baseTableName, indexName, idx);
     } catch (InvalidOperationException e) {
       console.printError("Invalid alter operation: " + e.getMessage());
       LOG.info("alter index: " + stringifyException(e));
@@ -896,7 +884,7 @@ public class DDLTask extends Task<DDLWor
    */
   private int renamePartition(Hive db, RenamePartitionDesc renamePartitionDesc) throws HiveException {
 
-    Table tbl = db.getTable(renamePartitionDesc.getDbName(), renamePartitionDesc.getTableName());
+    Table tbl = db.getTable(renamePartitionDesc.getTableName());
 
     Partition oldPart = db.getPartition(tbl, renamePartitionDesc.getOldPartSpec(), false);
     Partition part = db.getPartition(tbl, renamePartitionDesc.getOldPartSpec(), false);
@@ -923,7 +911,7 @@ public class DDLTask extends Task<DDLWor
   private int alterTableAlterPart(Hive db, AlterTableAlterPartDesc alterPartitionDesc)
       throws HiveException {
 
-    Table tbl = db.getTable(alterPartitionDesc.getDbName(), alterPartitionDesc.getTableName());
+    Table tbl = db.getTable(alterPartitionDesc.getTableName(), true);
     String tabName = alterPartitionDesc.getTableName();
 
     // This is checked by DDLSemanticAnalyzer
@@ -1015,14 +1003,11 @@ public class DDLTask extends Task<DDLWor
   private int touch(Hive db, AlterTableSimpleDesc touchDesc)
       throws HiveException {
 
-    String dbName = touchDesc.getDbName();
-    String tblName = touchDesc.getTableName();
-
-    Table tbl = db.getTable(dbName, tblName);
+    Table tbl = db.getTable(touchDesc.getTableName());
 
     if (touchDesc.getPartSpec() == null) {
       try {
-        db.alterTable(tblName, tbl);
+        db.alterTable(touchDesc.getTableName(), tbl);
       } catch (InvalidOperationException e) {
         throw new HiveException("Uable to update table");
       }
@@ -1034,7 +1019,7 @@ public class DDLTask extends Task<DDLWor
         throw new HiveException("Specified partition does not exist");
       }
       try {
-        db.alterPartition(tblName, part);
+        db.alterPartition(touchDesc.getTableName(), part);
       } catch (InvalidOperationException e) {
         throw new HiveException(e);
       }
@@ -1173,10 +1158,8 @@ public class DDLTask extends Task<DDLWor
   private int archive(Hive db, AlterTableSimpleDesc simpleDesc,
       DriverContext driverContext)
           throws HiveException {
-    String dbName = simpleDesc.getDbName();
-    String tblName = simpleDesc.getTableName();
 
-    Table tbl = db.getTable(dbName, tblName);
+    Table tbl = db.getTable(simpleDesc.getTableName());
 
     if (tbl.getTableType() != TableType.MANAGED_TABLE) {
       throw new HiveException("ARCHIVE can only be performed on managed tables");
@@ -1378,7 +1361,7 @@ public class DDLTask extends Task<DDLWor
             authority.toString(),
             harPartitionDir.getPath()); // make in Path to ensure no slash at the end
         setArchived(p, harPath, partSpecInfo.values.size());
-        db.alterPartition(tblName, p);
+        db.alterPartition(simpleDesc.getTableName(), p);
       }
     } catch (Exception e) {
       throw new HiveException("Unable to change the partition info for HAR", e);
@@ -1399,10 +1382,8 @@ public class DDLTask extends Task<DDLWor
 
   private int unarchive(Hive db, AlterTableSimpleDesc simpleDesc)
       throws HiveException {
-    String dbName = simpleDesc.getDbName();
-    String tblName = simpleDesc.getTableName();
 
-    Table tbl = db.getTable(dbName, tblName);
+    Table tbl = db.getTable(simpleDesc.getTableName());
 
     // Means user specified a table, not a partition
     if (simpleDesc.getPartSpec() == null) {
@@ -1587,7 +1568,7 @@ public class DDLTask extends Task<DDLWor
     for(Partition p: partitions) {
       setUnArchived(p);
       try {
-        db.alterPartition(tblName, p);
+        db.alterPartition(simpleDesc.getTableName(), p);
       } catch (InvalidOperationException e) {
         throw new HiveException(e);
       }
@@ -1636,10 +1617,7 @@ public class DDLTask extends Task<DDLWor
 
   private int compact(Hive db, AlterTableSimpleDesc desc) throws HiveException {
 
-    String dbName = desc.getDbName();
-    String tblName = desc.getTableName();
-
-    Table tbl = db.getTable(dbName, tblName);
+    Table tbl = db.getTable(desc.getTableName());
 
     String partName = null;
     if (desc.getPartSpec() == null) {
@@ -1852,7 +1830,7 @@ public class DDLTask extends Task<DDLWor
     final String ROW_FORMAT = "row_format";
     final String TBL_LOCATION = "tbl_location";
     final String TBL_PROPERTIES = "tbl_properties";
-    boolean isHbaseTable = false;
+    boolean needsLocation = true;
     StringBuilder createTab_str = new StringBuilder();
 
     String tableName = showCreateTbl.getTableName();
@@ -1864,9 +1842,7 @@ public class DDLTask extends Task<DDLWor
       FileSystem fs = resFile.getFileSystem(conf);
       outStream = fs.create(resFile);
 
-      if (tbl.getStorageHandler() != null) {
-        isHbaseTable = tbl.getStorageHandler().toString().equals("org.apache.hadoop.hive.hbase.HBaseStorageHandler");
-      }
+      needsLocation = doesTableNeedLocation(tbl);
 
       if (tbl.isView()) {
         String createTab_stmt = "CREATE VIEW `" + tableName + "` AS " + tbl.getViewExpandedText();
@@ -1883,7 +1859,7 @@ public class DDLTask extends Task<DDLWor
       createTab_str.append("<" + LIST_PARTITIONS + ">\n");
       createTab_str.append("<" + SORT_BUCKET + ">\n");
       createTab_str.append("<" + ROW_FORMAT + ">\n");
-      if (!isHbaseTable) {
+      if (needsLocation) {
         createTab_str.append("LOCATION\n");
         createTab_str.append("<" + TBL_LOCATION + ">\n");
       }
@@ -2065,7 +2041,7 @@ public class DDLTask extends Task<DDLWor
       createTab_stmt.add(SORT_BUCKET, tbl_sort_bucket);
       createTab_stmt.add(ROW_FORMAT, tbl_row_format);
       // Table location should not be printed with hbase backed tables
-      if (!isHbaseTable) {
+      if (needsLocation) {
         createTab_stmt.add(TBL_LOCATION, tbl_location);
       }
       createTab_stmt.add(TBL_PROPERTIES, tbl_properties);
@@ -2233,15 +2209,7 @@ public class DDLTask extends Task<DDLWor
   public int showColumns(Hive db, ShowColumnsDesc showCols)
       throws HiveException {
 
-    String dbName = showCols.getDbName();
-    String tableName = showCols.getTableName();
-    Table table = null;
-    if (dbName == null) {
-      table = db.getTable(tableName);
-    }
-    else {
-      table = db.getTable(dbName, tableName);
-    }
+    Table table = db.getTable(showCols.getTableName());
 
     // write the results in the file
     DataOutputStream outStream = null;
@@ -3280,7 +3248,8 @@ public class DDLTask extends Task<DDLWor
     Table oldTbl = tbl.copy();
 
     if (alterTbl.getOp() == AlterTableDesc.AlterTableTypes.RENAME) {
-      tbl.setTableName(alterTbl.getNewName());
+      tbl.setDbName(Utilities.getDatabaseName(alterTbl.getNewName()));
+      tbl.setTableName(Utilities.getTableName(alterTbl.getNewName()));
     } else if (alterTbl.getOp() == AlterTableDesc.AlterTableTypes.ADDCOLS) {
       List<FieldSchema> newCols = alterTbl.getNewCols();
       List<FieldSchema> oldCols = tbl.getCols();
@@ -3933,7 +3902,7 @@ public class DDLTask extends Task<DDLWor
       tbl.getTTable().getSd().setOutputFormat(tbl.getOutputFormatClass().getName());
     }
 
-    if (!Utilities.isDefaultNameNode(conf) && tbl.getTTable().getSd().isSetLocation()) {
+    if (!Utilities.isDefaultNameNode(conf) && doesTableNeedLocation(tbl)) {
       // If location is specified - ensure that it is a full qualified name
       makeLocationQualified(tbl.getDbName(), tbl.getTTable().getSd(), tbl.getTableName());
     }
@@ -4309,10 +4278,12 @@ public class DDLTask extends Task<DDLWor
     if (crtIndex.getLocation() == null) {
       // Location is not set, leave it as-is if index doesn't belong to default DB
       // Currently all indexes are created in current DB only
-      if (db.getDatabaseCurrent().getName().equalsIgnoreCase(MetaStoreUtils.DEFAULT_DATABASE_NAME)) {
+      if (Utilities.getDatabaseName(name).equalsIgnoreCase(MetaStoreUtils.DEFAULT_DATABASE_NAME)) {
         // Default database name path is always ignored, use METASTOREWAREHOUSE and object name
         // instead
-        path = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.METASTOREWAREHOUSE), name.toLowerCase());
+        String warehouse = HiveConf.getVar(conf, ConfVars.METASTOREWAREHOUSE);
+        String tableName = Utilities.getTableName(name);
+        path = new Path(warehouse, tableName.toLowerCase());
       }
     }
     else {
@@ -4341,4 +4312,15 @@ public class DDLTask extends Task<DDLWor
               database.getName().toLowerCase() + ".db")));
     }
   }
+
+  private static boolean doesTableNeedLocation(Table tbl) {
+    // If we are ok with breaking compatibility of existing 3rd party StorageHandlers,
+    // this method could be moved to the HiveStorageHandler interface.
+    boolean retval = true;
+    if (tbl.getStorageHandler() != null) {
+      retval = !tbl.getStorageHandler().toString().equals(
+          "org.apache.hadoop.hive.hbase.HBaseStorageHandler");
+    }
+    return retval;
+  }
 }

Modified: hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/HashTableSinkOperator.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/HashTableSinkOperator.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/HashTableSinkOperator.java (original)
+++ hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/HashTableSinkOperator.java Wed Aug 13 00:25:32 2014
@@ -181,7 +181,7 @@ public class HashTableSinkOperator exten
         if (pos == posBigTableAlias) {
           continue;
         }
-        mapJoinTables[pos] = new HashMapWrapper(hconf);
+        mapJoinTables[pos] = new HashMapWrapper(hconf, -1);
         TableDesc valueTableDesc = conf.getValueTblFilteredDescs().get(pos);
         SerDe valueSerDe = (SerDe) ReflectionUtils.newInstance(valueTableDesc.getDeserializerClass(), null);
         SerDeUtils.initializeSerDe(valueSerDe, null, valueTableDesc.getProperties(), null);

Modified: hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/Utilities.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/Utilities.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/Utilities.java (original)
+++ hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/Utilities.java Wed Aug 13 00:25:32 2014
@@ -96,6 +96,7 @@ import org.apache.hadoop.fs.permission.F
 import org.apache.hadoop.hive.common.HiveInterruptCallback;
 import org.apache.hadoop.hive.common.HiveInterruptUtils;
 import org.apache.hadoop.hive.common.HiveStatsUtils;
+import org.apache.hadoop.hive.common.JavaUtils;
 import org.apache.hadoop.hive.conf.HiveConf;
 import org.apache.hadoop.hive.conf.HiveConf.ConfVars;
 import org.apache.hadoop.hive.metastore.Warehouse;
@@ -1389,9 +1390,8 @@ public final class Utilities {
   public static RCFile.Writer createRCFileWriter(JobConf jc, FileSystem fs, Path file,
       boolean isCompressed, Progressable progressable) throws IOException {
     CompressionCodec codec = null;
-    Class<?> codecClass = null;
     if (isCompressed) {
-      codecClass = FileOutputFormat.getOutputCompressorClass(jc, DefaultCodec.class);
+      Class<?> codecClass = FileOutputFormat.getOutputCompressorClass(jc, DefaultCodec.class);
       codec = (CompressionCodec) ReflectionUtils.newInstance(codecClass, jc);
     }
     return new RCFile.Writer(fs, jc, file, progressable, codec);
@@ -1980,6 +1980,7 @@ public final class Utilities {
         newPath.remove(oneurl);
       }
     }
+    JavaUtils.closeClassLoader(loader);
 
     loader = new URLClassLoader(newPath.toArray(new URL[0]));
     curThread.setContextClassLoader(loader);
@@ -2054,19 +2055,53 @@ public final class Utilities {
    * @return String array with two elements, first is db name, second is table name
    * @throws HiveException
    */
-  public static String[] getDbTableName(String dbtable) throws HiveException{
-    if(dbtable == null){
+  public static String[] getDbTableName(String dbtable) throws SemanticException {
+    return getDbTableName(SessionState.get().getCurrentDatabase(), dbtable);
+  }
+
+  public static String[] getDbTableName(String defaultDb, String dbtable) throws SemanticException {
+    if (dbtable == null) {
       return new String[2];
     }
     String[] names =  dbtable.split("\\.");
     switch (names.length) {
-    case 2:
-      return names;
-    case 1:
-      return new String [] {SessionState.get().getCurrentDatabase(), dbtable};
-    default:
-      throw new HiveException(ErrorMsg.INVALID_TABLE_NAME, dbtable);
+      case 2:
+        return names;
+      case 1:
+        return new String [] {defaultDb, dbtable};
+      default:
+        throw new SemanticException(ErrorMsg.INVALID_TABLE_NAME, dbtable);
+    }
+  }
+
+  /**
+   * Accepts qualified name which is in the form of dbname.tablename and returns dbname from it
+   *
+   * @param dbTableName
+   * @return dbname
+   * @throws SemanticException input string is not qualified name
+   */
+  public static String getDatabaseName(String dbTableName) throws SemanticException {
+    String[] split = dbTableName.split("\\.");
+    if (split.length != 2) {
+      throw new SemanticException(ErrorMsg.INVALID_TABLE_NAME, dbTableName);
+    }
+    return split[0];
+  }
+
+  /**
+   * Accepts qualified name which is in the form of dbname.tablename and returns tablename from it
+   *
+   * @param dbTableName
+   * @return tablename
+   * @throws SemanticException input string is not qualified name
+   */
+  public static String getTableName(String dbTableName) throws SemanticException {
+    String[] split = dbTableName.split("\\.");
+    if (split.length != 2) {
+      throw new SemanticException(ErrorMsg.INVALID_TABLE_NAME, dbTableName);
     }
+    return split[1];
   }
 
   public static void validateColumnNames(List<String> colNames, List<String> checkCols)

Modified: hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/persistence/BytesBytesMultiHashMap.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/persistence/BytesBytesMultiHashMap.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/persistence/BytesBytesMultiHashMap.java (original)
+++ hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/persistence/BytesBytesMultiHashMap.java Wed Aug 13 00:25:32 2014
@@ -145,8 +145,7 @@ public final class BytesBytesMultiHashMa
   private long[] refs;
   private int startingHashBitCount, hashBitCount;
 
-  private int metricPutConflict = 0, metricSameBitsDiffKey = 0,
-      metricSameBitsSameKey = 0, metricDiffBits = 0;
+  private int metricPutConflict = 0, metricExpands = 0, metricExpandsUs = 0;
 
   /** We have 39 bits to store list pointer from the first record; this is size limit */
   final static long MAX_WB_SIZE = ((long)1) << 38;
@@ -430,16 +429,13 @@ public final class BytesBytesMultiHashMa
    */
   private boolean isSameKey(byte[] key, int length, long ref, int hashCode) {
     if (!compareHashBits(ref, hashCode)) {
-      ++metricDiffBits;
       return false;  // Hash bits don't match.
     }
     writeBuffers.setReadPoint(getFirstRecordLengthsOffset(ref));
     int valueLength = (int)writeBuffers.readVLong(), keyLength = (int)writeBuffers.readVLong();
     long keyOffset = Ref.getOffset(ref) - (valueLength + keyLength);
     // See the comment in the other isSameKey
-    boolean result = writeBuffers.isEqual(key, length, keyOffset, keyLength);
-    if (result) { ++metricSameBitsSameKey; } else { ++metricSameBitsDiffKey; }
-    return result;
+    return writeBuffers.isEqual(key, length, keyOffset, keyLength);
   }
 
   private boolean compareHashBits(long ref, int hashCode) {
@@ -461,6 +457,7 @@ public final class BytesBytesMultiHashMa
   }
 
   private void expandAndRehash() {
+    long expandTime = System.nanoTime();
     final long[] oldRefs = refs;
     long capacity = refs.length << 1;
     validateCapacity(capacity);
@@ -492,6 +489,9 @@ public final class BytesBytesMultiHashMa
     this.largestNumberOfSteps = maxSteps;
     this.hashBitCount = newHashBitCount;
     this.resizeThreshold = (int)(capacity * loadFactor);
+    metricExpandsUs += (System.nanoTime() - expandTime);
+    ++metricExpands;
+
   }
 
   /**
@@ -703,11 +703,9 @@ public final class BytesBytesMultiHashMa
   }
 
   public void debugDumpMetrics() {
-    if (LOG.isDebugEnabled()) {
-      LOG.debug("Map metrics: keys " + this.keysAssigned + ", write conflict " + metricPutConflict
-          + ", write max dist " + largestNumberOfSteps + ", read neq " + metricDiffBits
-          + ", read eq-eq " + metricSameBitsSameKey + ", read eq-neq " + metricSameBitsDiffKey);
-    }
+    LOG.info("Map metrics: keys allocated " + this.refs.length +", keys assigned " + keysAssigned
+        + ", write conflict " + metricPutConflict  + ", write max dist " + largestNumberOfSteps
+        + ", expanded " + metricExpands + " times in " + metricExpandsUs + "us");
   }
 
   private void debugDumpKeyProbe(long keyOffset, int keyLength, int hashCode, int finalSlot) {

Modified: hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/persistence/HashMapWrapper.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/persistence/HashMapWrapper.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/persistence/HashMapWrapper.java (original)
+++ hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/persistence/HashMapWrapper.java Wed Aug 13 00:25:32 2014
@@ -71,25 +71,39 @@ public class HashMapWrapper extends Abst
   }
 
   public HashMapWrapper() {
-    this(HiveConf.ConfVars.HIVEHASHTABLETHRESHOLD.defaultIntVal,
-        HiveConf.ConfVars.HIVEHASHTABLELOADFACTOR.defaultFloatVal, false, false);
+    this(HiveConf.ConfVars.HIVEHASHTABLEKEYCOUNTADJUSTMENT.defaultFloatVal,
+        HiveConf.ConfVars.HIVEHASHTABLETHRESHOLD.defaultIntVal,
+        HiveConf.ConfVars.HIVEHASHTABLELOADFACTOR.defaultFloatVal, false, false, -1);
   }
 
-  public HashMapWrapper(Configuration hconf) {
-    this(HiveConf.getIntVar(hconf, HiveConf.ConfVars.HIVEHASHTABLETHRESHOLD),
+  public HashMapWrapper(Configuration hconf, long keyCount) {
+    this(HiveConf.getFloatVar(hconf, HiveConf.ConfVars.HIVEHASHTABLEKEYCOUNTADJUSTMENT),
+        HiveConf.getIntVar(hconf, HiveConf.ConfVars.HIVEHASHTABLETHRESHOLD),
         HiveConf.getFloatVar(hconf, HiveConf.ConfVars.HIVEHASHTABLELOADFACTOR),
         HiveConf.getBoolVar(hconf, HiveConf.ConfVars.HIVEMAPJOINLAZYHASHTABLE),
-        HiveConf.getBoolVar(hconf, HiveConf.ConfVars.HIVEMAPJOINUSEOPTIMIZEDKEYS));
+        HiveConf.getBoolVar(hconf, HiveConf.ConfVars.HIVEMAPJOINUSEOPTIMIZEDKEYS), keyCount);
   }
 
-  private HashMapWrapper(
-      int threshold, float loadFactor, boolean useLazyRows, boolean useOptimizedKeys) {
+  private HashMapWrapper(float keyCountAdj, int threshold, float loadFactor,
+      boolean useLazyRows, boolean useOptimizedKeys, long keyCount) {
     super(createConstructorMetaData(threshold, loadFactor));
+    threshold = calculateTableSize(keyCountAdj, threshold, loadFactor, keyCount);
     mHash = new HashMap<MapJoinKey, MapJoinRowContainer>(threshold, loadFactor);
     this.useLazyRows = useLazyRows;
     this.useOptimizedKeys = useOptimizedKeys;
   }
 
+  public static int calculateTableSize(
+      float keyCountAdj, int threshold, float loadFactor, long keyCount) {
+    if (keyCount >= 0 && keyCountAdj != 0) {
+      // We have statistics for the table. Size appropriately.
+      threshold = (int)Math.ceil(keyCount / (keyCountAdj * loadFactor));
+    }
+    LOG.info("Key count from statistics is " + keyCount + "; setting map size to " + threshold);
+    return threshold;
+  }
+
+
   @Override
   public MapJoinRowContainer get(MapJoinKey key) {
     return mHash.get(key);

Modified: hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/persistence/MapJoinBytesTableContainer.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/persistence/MapJoinBytesTableContainer.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/persistence/MapJoinBytesTableContainer.java (original)
+++ hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/persistence/MapJoinBytesTableContainer.java Wed Aug 13 00:25:32 2014
@@ -59,15 +59,17 @@ public class MapJoinBytesTableContainer 
 
   private List<Object> EMPTY_LIST = new ArrayList<Object>(0);
 
-  public MapJoinBytesTableContainer(Configuration hconf, MapJoinObjectSerDeContext valCtx)
-      throws SerDeException {
-    this(HiveConf.getIntVar(hconf, HiveConf.ConfVars.HIVEHASHTABLETHRESHOLD),
+  public MapJoinBytesTableContainer(Configuration hconf,
+      MapJoinObjectSerDeContext valCtx, long keyCount) throws SerDeException {
+    this(HiveConf.getFloatVar(hconf, HiveConf.ConfVars.HIVEHASHTABLEKEYCOUNTADJUSTMENT),
+        HiveConf.getIntVar(hconf, HiveConf.ConfVars.HIVEHASHTABLETHRESHOLD),
         HiveConf.getFloatVar(hconf, HiveConf.ConfVars.HIVEHASHTABLELOADFACTOR),
-        HiveConf.getIntVar(hconf, HiveConf.ConfVars.HIVEHASHTABLEWBSIZE), valCtx);
+        HiveConf.getIntVar(hconf, HiveConf.ConfVars.HIVEHASHTABLEWBSIZE), valCtx, keyCount);
   }
 
-  private MapJoinBytesTableContainer(int threshold, float loadFactor, int wbSize,
-      MapJoinObjectSerDeContext valCtx) throws SerDeException {
+  private MapJoinBytesTableContainer(float keyCountAdj, int threshold, float loadFactor,
+      int wbSize, MapJoinObjectSerDeContext valCtx, long keyCount) throws SerDeException {
+    threshold = HashMapWrapper.calculateTableSize(keyCountAdj, threshold, loadFactor, keyCount);
     hashMap = new BytesBytesMultiHashMap(threshold, loadFactor, wbSize);
   }
 

Modified: hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/tez/HashTableLoader.java
URL: http://svn.apache.org/viewvc/hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/tez/HashTableLoader.java?rev=1617633&r1=1617632&r2=1617633&view=diff
==============================================================================
--- hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/tez/HashTableLoader.java (original)
+++ hive/branches/spark/ql/src/java/org/apache/hadoop/hive/ql/exec/tez/HashTableLoader.java Wed Aug 13 00:25:32 2014
@@ -88,6 +88,7 @@ public class HashTableLoader implements 
 
     TezContext tezContext = (TezContext) MapredContext.get();
     Map<Integer, String> parentToInput = desc.getParentToInput();
+    Map<Integer, Long> parentKeyCounts = desc.getParentKeyCounts();
 
     boolean useOptimizedTables = HiveConf.getBoolVar(
         hconf, HiveConf.ConfVars.HIVEMAPJOINUSEOPTIMIZEDTABLE);
@@ -117,8 +118,11 @@ public class HashTableLoader implements 
           }
         }
         isFirstKey = false;
+        Long keyCountObj = parentKeyCounts.get(pos);
+        long keyCount = (keyCountObj == null) ? -1 : keyCountObj.longValue();
         MapJoinTableContainer tableContainer = useOptimizedTables
-            ? new MapJoinBytesTableContainer(hconf, valCtx) : new HashMapWrapper(hconf);
+            ? new MapJoinBytesTableContainer(hconf, valCtx, keyCount)
+            : new HashMapWrapper(hconf, keyCount);
 
         while (kvReader.next()) {
           lastKey = tableContainer.putRow(keyCtx, (Writable)kvReader.getCurrentKey(),