You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hive.apache.org by ct...@apache.org on 2017/05/09 13:53:48 UTC

hive git commit: HIVE-16572: Rename a partition should not drop its column stats (Chaoyu Tang, reviewed by Yongzhi Chen)

Repository: hive
Updated Branches:
  refs/heads/master 5835503d4 -> f3310a37b


HIVE-16572: Rename a partition should not drop its column stats (Chaoyu Tang, reviewed by Yongzhi Chen)


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

Branch: refs/heads/master
Commit: f3310a37b836b708fbfdd8665ade84043d282391
Parents: 5835503
Author: Chaoyu Tang <ct...@cloudera.com>
Authored: Tue May 9 09:53:24 2017 -0400
Committer: Chaoyu Tang <ct...@cloudera.com>
Committed: Tue May 9 09:53:24 2017 -0400

----------------------------------------------------------------------
 .../hadoop/hive/metastore/HiveAlterHandler.java | 143 +++---
 .../clientpositive/alter_table_column_stats.q   |  68 ++-
 .../rename_external_partition_location.q        |  11 +
 .../alter_table_column_stats.q.out              | 486 ++++++++++++++++---
 .../rename_external_partition_location.q.out    | 208 ++++++++
 5 files changed, 759 insertions(+), 157 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/hive/blob/f3310a37/metastore/src/java/org/apache/hadoop/hive/metastore/HiveAlterHandler.java
----------------------------------------------------------------------
diff --git a/metastore/src/java/org/apache/hadoop/hive/metastore/HiveAlterHandler.java b/metastore/src/java/org/apache/hadoop/hive/metastore/HiveAlterHandler.java
index d8af7a7..7978a40 100644
--- a/metastore/src/java/org/apache/hadoop/hive/metastore/HiveAlterHandler.java
+++ b/metastore/src/java/org/apache/hadoop/hive/metastore/HiveAlterHandler.java
@@ -376,13 +376,7 @@ public class HiveAlterHandler implements AlterHandler {
     EnvironmentContext environmentContext, HMSHandler handler)
       throws InvalidOperationException, InvalidObjectException, AlreadyExistsException, MetaException {
     boolean success = false;
-    Path srcPath = null;
-    Path destPath = null;
-    FileSystem srcFs = null;
-    FileSystem destFs;
     Partition oldPart = null;
-    String oldPartLoc = null;
-    String newPartLoc = null;
     List<MetaStoreEventListener> transactionalListeners = null;
     if (handler != null) {
       transactionalListeners = handler.getTransactionalListeners();
@@ -445,6 +439,13 @@ public class HiveAlterHandler implements AlterHandler {
     }
 
     //rename partition
+    String oldPartLoc = null;
+    String newPartLoc = null;
+    Path srcPath = null;
+    Path destPath = null;
+    FileSystem srcFs = null;
+    FileSystem destFs = null;
+    boolean dataWasMoved = false;
     try {
       msdb.openTransaction();
       try {
@@ -468,20 +469,11 @@ public class HiveAlterHandler implements AlterHandler {
             new_part.getValues());
       }
 
-      // if the external partition is renamed, the file should not change
-      if (tbl.getTableType().equals(TableType.EXTERNAL_TABLE.toString())) {
-        new_part.getSd().setLocation(oldPart.getSd().getLocation());
-        String oldPartName = Warehouse.makePartName(tbl.getPartitionKeys(), oldPart.getValues());
-        try {
-          //existing partition column stats is no longer valid, remove
-          msdb.deletePartitionColumnStatistics(dbname, name, oldPartName, oldPart.getValues(), null);
-        } catch (NoSuchObjectException nsoe) {
-          //ignore
-        } catch (InvalidInputException iie) {
-          throw new InvalidOperationException("Unable to update partition stats in table rename." + iie);
-        }
-        msdb.alterPartition(dbname, name, part_vals, new_part);
-      } else {
+      // when renaming a partition, we should update
+      // 1) partition SD Location
+      // 2) partition column stats if there are any because of part_name field in HMS table PART_COL_STATS
+      // 3) rename the partition directory if it is not an external table
+      if (!tbl.getTableType().equals(TableType.EXTERNAL_TABLE.toString())) {
         try {
           // if tbl location is available use it
           // else derive the tbl location from database location
@@ -510,34 +502,54 @@ public class HiveAlterHandler implements AlterHandler {
           }
 
           try {
-            srcFs.exists(srcPath);
-            if (newPartLoc.compareTo(oldPartLoc) != 0 && destFs.exists(destPath)) {
-              throw new InvalidOperationException("New location for this table "
-                + tbl.getDbName() + "." + tbl.getTableName()
-                + " already exists : " + destPath);
+            if (srcFs.exists(srcPath)) {
+              if (newPartLoc.compareTo(oldPartLoc) != 0 && destFs.exists(destPath)) {
+                throw new InvalidOperationException("New location for this table "
+                  + tbl.getDbName() + "." + tbl.getTableName()
+                  + " already exists : " + destPath);
+              }
+              //if destPath's parent path doesn't exist, we should mkdir it
+              Path destParentPath = destPath.getParent();
+              if (!wh.mkdirs(destParentPath)) {
+                  throw new MetaException("Unable to create path " + destParentPath);
+              }
+
+              //rename the data directory
+              wh.renameDir(srcPath, destPath);
+              LOG.info("Partition directory rename from " + srcPath + " to " + destPath + " done.");
+              dataWasMoved = true;
             }
           } catch (IOException e) {
-            throw new InvalidOperationException("Unable to access new location "
-              + destPath + " for partition " + tbl.getDbName() + "."
-              + tbl.getTableName() + " " + new_part.getValues());
+            LOG.error("Cannot rename partition directory from " + srcPath + " to " + destPath, e);
+            throw new InvalidOperationException("Unable to access src or dest location for partition "
+                + tbl.getDbName() + "." + tbl.getTableName() + " " + new_part.getValues());
+          } catch (MetaException me) {
+            LOG.error("Cannot rename partition directory from " + srcPath + " to " + destPath, me);
+            throw me;
           }
 
           new_part.getSd().setLocation(newPartLoc);
-          if (MetaStoreUtils.requireCalStats(hiveConf, oldPart, new_part, tbl, environmentContext)) {
-            MetaStoreUtils.updatePartitionStatsFast(new_part, wh, false, true, environmentContext);
-          }
+        }
+      } else {
+        new_part.getSd().setLocation(oldPart.getSd().getLocation());
+      }
 
-          String oldPartName = Warehouse.makePartName(tbl.getPartitionKeys(), oldPart.getValues());
-          try {
-            //existing partition column stats is no longer valid, remove
-            msdb.deletePartitionColumnStatistics(dbname, name, oldPartName, oldPart.getValues(), null);
-          } catch (NoSuchObjectException nsoe) {
-            //ignore
-          } catch (InvalidInputException iie) {
-            throw new InvalidOperationException("Unable to update partition stats in table rename." + iie);
-          }
+      if (MetaStoreUtils.requireCalStats(hiveConf, oldPart, new_part, tbl, environmentContext)) {
+        MetaStoreUtils.updatePartitionStatsFast(new_part, wh, false, true, environmentContext);
+      }
 
-          msdb.alterPartition(dbname, name, part_vals, new_part);
+      String newPartName = Warehouse.makePartName(tbl.getPartitionKeys(), new_part.getValues());
+      ColumnStatistics cs = updateOrGetPartitionColumnStats(msdb, dbname, name, oldPart.getValues(),
+          oldPart.getSd().getCols(), tbl, new_part);
+      msdb.alterPartition(dbname, name, part_vals, new_part);
+      if (cs != null) {
+        cs.getStatsDesc().setPartName(newPartName);
+        try {
+          msdb.updatePartitionColumnStatistics(cs, new_part.getValues());
+        } catch (InvalidInputException iie) {
+          throw new InvalidOperationException("Unable to update partition stats in table rename." + iie);
+        } catch (NoSuchObjectException nsoe) {
+          // It is ok, ignore
         }
       }
 
@@ -551,48 +563,21 @@ public class HiveAlterHandler implements AlterHandler {
       success = msdb.commitTransaction();
     } finally {
       if (!success) {
+        LOG.error("Failed to rename a partition. Rollback transaction");
         msdb.rollbackTransaction();
-      }
-
-      if (success && newPartLoc != null && newPartLoc.compareTo(oldPartLoc) != 0) {
-        //rename the data directory
-        try{
-          if (srcFs.exists(srcPath)) {
-            //if destPath's parent path doesn't exist, we should mkdir it
-            Path destParentPath = destPath.getParent();
-            if (!wh.mkdirs(destParentPath)) {
-                throw new IOException("Unable to create path " + destParentPath);
-            }
-
-            wh.renameDir(srcPath, destPath);
-            LOG.info("Partition directory rename from " + srcPath + " to " + destPath + " done.");
-          }
-        } catch (IOException ex) {
-          LOG.error("Cannot rename partition directory from " + srcPath + " to " +
-              destPath, ex);
-          boolean revertMetaDataTransaction = false;
+        if (dataWasMoved) {
+          LOG.error("Revert the data move in renaming a partition.");
           try {
-            msdb.openTransaction();
-            msdb.alterPartition(dbname, name, new_part.getValues(), oldPart);
-            if (transactionalListeners != null && !transactionalListeners.isEmpty()) {
-              MetaStoreListenerNotifier.notifyEvent(transactionalListeners,
-                                                    EventMessage.EventType.ALTER_PARTITION,
-                                                    new AlterPartitionEvent(new_part, oldPart, tbl, false, success, handler),
-                                                    environmentContext);
-            }
-
-            revertMetaDataTransaction = msdb.commitTransaction();
-          } catch (Exception ex2) {
-            LOG.error("Attempt to revert partition metadata change failed. The revert was attempted " +
-                "because associated filesystem rename operation failed with exception " + ex.getMessage(), ex2);
-            if (!revertMetaDataTransaction) {
-              msdb.rollbackTransaction();
+            if (destFs.exists(destPath)) {
+              wh.renameDir(destPath, srcPath);
             }
+          } catch (MetaException me) {
+            LOG.error("Failed to restore partition data from " + destPath + " to " + srcPath
+                +  " in alter partition failure. Manual restore is needed.");
+          } catch (IOException ioe) {
+            LOG.error("Failed to restore partition data from " + destPath + " to " + srcPath
+                +  " in alter partition failure. Manual restore is needed.");
           }
-
-          throw new InvalidOperationException("Unable to access old location "
-              + srcPath + " for partition " + tbl.getDbName() + "."
-              + tbl.getTableName() + " " + part_vals);
         }
       }
     }

http://git-wip-us.apache.org/repos/asf/hive/blob/f3310a37/ql/src/test/queries/clientpositive/alter_table_column_stats.q
----------------------------------------------------------------------
diff --git a/ql/src/test/queries/clientpositive/alter_table_column_stats.q b/ql/src/test/queries/clientpositive/alter_table_column_stats.q
index 39dfb0c..8892d3f 100644
--- a/ql/src/test/queries/clientpositive/alter_table_column_stats.q
+++ b/ql/src/test/queries/clientpositive/alter_table_column_stats.q
@@ -76,14 +76,26 @@ describe formatted statsdb1.testpart1 partition (part = 'part2') col1;
 describe formatted statsdb1.testpart1 partition (part = 'part2') col2;
 describe formatted statsdb1.testpart1 partition (part = 'part2') col3;
 
+-- rename a partition should not change its table, partition, and column stats
+alter table statsdb1.testpart1 partition (part = 'part1') rename to partition (part = 'part11');
+describe formatted statsdb1.testpart1;
+describe formatted statsdb1.testpart1 partition (part = 'part11');
+describe formatted statsdb1.testpart1 partition (part = 'part11') col1;
+describe formatted statsdb1.testpart1 partition (part = 'part11') col2;
+describe formatted statsdb1.testpart1 partition (part = 'part11') col3;
+describe formatted statsdb1.testpart1 partition (part = 'part2');
+describe formatted statsdb1.testpart1 partition (part = 'part2') col1;
+describe formatted statsdb1.testpart1 partition (part = 'part2') col2;
+describe formatted statsdb1.testpart1 partition (part = 'part2') col3;
+
 -- when cascade replacing columns in a partitioned table, the table and partition stats should not change,
 -- but the stats of the changed columns are removed
 alter table statsdb1.testpart1 replace columns (col1 int, col2 string, col4 string) cascade;
 describe formatted statsdb1.testpart1;
-describe formatted statsdb1.testpart1 partition (part = 'part1');
-describe formatted statsdb1.testpart1 partition (part = 'part1') col1;
-describe formatted statsdb1.testpart1 partition (part = 'part1') col2;
-describe formatted statsdb1.testpart1 partition (part = 'part1') col4;
+describe formatted statsdb1.testpart1 partition (part = 'part11');
+describe formatted statsdb1.testpart1 partition (part = 'part11') col1;
+describe formatted statsdb1.testpart1 partition (part = 'part11') col2;
+describe formatted statsdb1.testpart1 partition (part = 'part11') col4;
 describe formatted statsdb1.testpart1 partition (part = 'part2');
 describe formatted statsdb1.testpart1 partition (part = 'part2') col1;
 describe formatted statsdb1.testpart1 partition (part = 'part2') col2;
@@ -93,10 +105,10 @@ describe formatted statsdb1.testpart1 partition (part = 'part2') col4;
 -- but the stats of the type-changed columns are removed
 alter table statsdb1.testpart1 change column col1 col1 string cascade;
 describe formatted statsdb1.testpart1;
-describe formatted statsdb1.testpart1 partition (part = 'part1');
-describe formatted statsdb1.testpart1 partition (part = 'part1') col1;
-describe formatted statsdb1.testpart1 partition (part = 'part1') col2;
-describe formatted statsdb1.testpart1 partition (part = 'part1') col4;
+describe formatted statsdb1.testpart1 partition (part = 'part11');
+describe formatted statsdb1.testpart1 partition (part = 'part11') col1;
+describe formatted statsdb1.testpart1 partition (part = 'part11') col2;
+describe formatted statsdb1.testpart1 partition (part = 'part11') col4;
 describe formatted statsdb1.testpart1 partition (part = 'part2');
 describe formatted statsdb1.testpart1 partition (part = 'part2') col1;
 describe formatted statsdb1.testpart1 partition (part = 'part2') col2;
@@ -105,9 +117,9 @@ describe formatted statsdb1.testpart1 partition (part = 'part2') col4;
 -- change database of a partition should not change table, partition and columns stats
 alter table statsdb1.testpart1 rename to statsdb2.testpart2;
 describe formatted statsdb2.testpart2;
-describe formatted statsdb2.testpart2 partition (part = 'part1') col1;
-describe formatted statsdb2.testpart2 partition (part = 'part1') col2;
-describe formatted statsdb2.testpart2 partition (part = 'part1') col4;
+describe formatted statsdb2.testpart2 partition (part = 'part11') col1;
+describe formatted statsdb2.testpart2 partition (part = 'part11') col2;
+describe formatted statsdb2.testpart2 partition (part = 'part11') col4;
 describe formatted statsdb2.testpart2 partition (part = 'part2') col1;
 describe formatted statsdb2.testpart2 partition (part = 'part2') col2;
 describe formatted statsdb2.testpart2 partition (part = 'part2') col4;
@@ -196,14 +208,26 @@ describe formatted statsdb1.testpart1 partition (part = 'part2') col1;
 describe formatted statsdb1.testpart1 partition (part = 'part2') col2;
 describe formatted statsdb1.testpart1 partition (part = 'part2') col3;
 
+-- rename a partition should not change its table, partition, and column stats
+alter table statsdb1.testpart1 partition (part = 'part1') rename to partition (part = 'part11');
+describe formatted statsdb1.testpart1;
+describe formatted statsdb1.testpart1 partition (part = 'part11');
+describe formatted statsdb1.testpart1 partition (part = 'part11') col1;
+describe formatted statsdb1.testpart1 partition (part = 'part11') col2;
+describe formatted statsdb1.testpart1 partition (part = 'part11') col3;
+describe formatted statsdb1.testpart1 partition (part = 'part2');
+describe formatted statsdb1.testpart1 partition (part = 'part2') col1;
+describe formatted statsdb1.testpart1 partition (part = 'part2') col2;
+describe formatted statsdb1.testpart1 partition (part = 'part2') col3;
+
 -- when cascade replacing columns in a partitioned table, the table and partition stats should not change,
 -- but the stats of the changed columns are removed
 alter table statsdb1.testpart1 replace columns (col1 int, col2 string, col4 string) cascade;
 describe formatted statsdb1.testpart1;
-describe formatted statsdb1.testpart1 partition (part = 'part1');
-describe formatted statsdb1.testpart1 partition (part = 'part1') col1;
-describe formatted statsdb1.testpart1 partition (part = 'part1') col2;
-describe formatted statsdb1.testpart1 partition (part = 'part1') col4;
+describe formatted statsdb1.testpart1 partition (part = 'part11');
+describe formatted statsdb1.testpart1 partition (part = 'part11') col1;
+describe formatted statsdb1.testpart1 partition (part = 'part11') col2;
+describe formatted statsdb1.testpart1 partition (part = 'part11') col4;
 describe formatted statsdb1.testpart1 partition (part = 'part2');
 describe formatted statsdb1.testpart1 partition (part = 'part2') col1;
 describe formatted statsdb1.testpart1 partition (part = 'part2') col2;
@@ -213,10 +237,10 @@ describe formatted statsdb1.testpart1 partition (part = 'part2') col4;
 -- but the stats of the type-changed columns are removed
 alter table statsdb1.testpart1 change column col1 col1 string cascade;
 describe formatted statsdb1.testpart1;
-describe formatted statsdb1.testpart1 partition (part = 'part1');
-describe formatted statsdb1.testpart1 partition (part = 'part1') col1;
-describe formatted statsdb1.testpart1 partition (part = 'part1') col2;
-describe formatted statsdb1.testpart1 partition (part = 'part1') col4;
+describe formatted statsdb1.testpart1 partition (part = 'part11');
+describe formatted statsdb1.testpart1 partition (part = 'part11') col1;
+describe formatted statsdb1.testpart1 partition (part = 'part11') col2;
+describe formatted statsdb1.testpart1 partition (part = 'part11') col4;
 describe formatted statsdb1.testpart1 partition (part = 'part2');
 describe formatted statsdb1.testpart1 partition (part = 'part2') col1;
 describe formatted statsdb1.testpart1 partition (part = 'part2') col2;
@@ -225,9 +249,9 @@ describe formatted statsdb1.testpart1 partition (part = 'part2') col4;
 -- change database of a partition should not change table, partition and columns stats
 alter table statsdb1.testpart1 rename to statsdb2.testpart2;
 describe formatted statsdb2.testpart2;
-describe formatted statsdb2.testpart2 partition (part = 'part1') col1;
-describe formatted statsdb2.testpart2 partition (part = 'part1') col2;
-describe formatted statsdb2.testpart2 partition (part = 'part1') col4;
+describe formatted statsdb2.testpart2 partition (part = 'part11') col1;
+describe formatted statsdb2.testpart2 partition (part = 'part11') col2;
+describe formatted statsdb2.testpart2 partition (part = 'part11') col4;
 describe formatted statsdb2.testpart2 partition (part = 'part2') col1;
 describe formatted statsdb2.testpart2 partition (part = 'part2') col2;
 describe formatted statsdb2.testpart2 partition (part = 'part2') col4;

http://git-wip-us.apache.org/repos/asf/hive/blob/f3310a37/ql/src/test/queries/clientpositive/rename_external_partition_location.q
----------------------------------------------------------------------
diff --git a/ql/src/test/queries/clientpositive/rename_external_partition_location.q b/ql/src/test/queries/clientpositive/rename_external_partition_location.q
index be93bd4..44500d3 100644
--- a/ql/src/test/queries/clientpositive/rename_external_partition_location.q
+++ b/ql/src/test/queries/clientpositive/rename_external_partition_location.q
@@ -16,6 +16,12 @@ SELECT * from ex_table where part='part1' ORDER BY key;
 dfs -ls ${system:test.tmp.dir}/ex_table/part=part1;
 dfs -cat ${system:test.tmp.dir}/ex_table/part=part1/000000_0;
 
+ANALYZE TABLE ex_table PARTITION (part='part1') COMPUTE STATISTICS FOR COLUMNS;
+DESCRIBE FORMATTED ex_table;
+DESCRIBE FORMATTED ex_table PARTITION (part='part1');
+DESCRIBE FORMATTED ex_table PARTITION (part='part1') key;
+DESCRIBE FORMATTED ex_table PARTITION (part='part1') value;
+
 ALTER TABLE ex_table PARTITION (part='part1') RENAME TO PARTITION (part='part2');
 
 SHOW PARTITIONS ex_table;
@@ -23,3 +29,8 @@ SELECT * from ex_table where part='part2' ORDER BY key;
 
 dfs -ls ${system:test.tmp.dir}/ex_table/part=part1;
 dfs -cat ${system:test.tmp.dir}/ex_table/part=part1/000000_0;
+
+DESCRIBE FORMATTED ex_table;
+DESCRIBE FORMATTED ex_table PARTITION (part='part2');
+DESCRIBE FORMATTED ex_table PARTITION (part='part2') key;
+DESCRIBE FORMATTED ex_table PARTITION (part='part2') value;

http://git-wip-us.apache.org/repos/asf/hive/blob/f3310a37/ql/src/test/results/clientpositive/alter_table_column_stats.q.out
----------------------------------------------------------------------
diff --git a/ql/src/test/results/clientpositive/alter_table_column_stats.q.out b/ql/src/test/results/clientpositive/alter_table_column_stats.q.out
index 8739bfe..2cc7cbc 100644
--- a/ql/src/test/results/clientpositive/alter_table_column_stats.q.out
+++ b/ql/src/test/results/clientpositive/alter_table_column_stats.q.out
@@ -823,17 +823,204 @@ POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
 	 	 	 	 	 	 	 	 	 	 
 col3                	string              	                    	                    	0                   	1                   	4.0                 	4                   	                    	                    	from deserializer   
+PREHOOK: query: alter table statsdb1.testpart1 partition (part = 'part1') rename to partition (part = 'part11')
+PREHOOK: type: ALTERTABLE_RENAMEPART
+PREHOOK: Input: statsdb1@testpart1
+PREHOOK: Output: statsdb1@testpart1@part=part1
+POSTHOOK: query: alter table statsdb1.testpart1 partition (part = 'part1') rename to partition (part = 'part11')
+POSTHOOK: type: ALTERTABLE_RENAMEPART
+POSTHOOK: Input: statsdb1@testpart1
+POSTHOOK: Input: statsdb1@testpart1@part=part1
+POSTHOOK: Output: statsdb1@testpart1@part=part1
+POSTHOOK: Output: statsdb1@testpart1@part=part11
+PREHOOK: query: describe formatted statsdb1.testpart1
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	comment             
+	 	 
+col1                	int                 	                    
+col2                	string              	                    
+col3                	string              	                    
+	 	 
+# Partition Information	 	 
+# col_name            	data_type           	comment             
+	 	 
+part                	string              	                    
+	 	 
+# Detailed Table Information	 	 
+Database:           	statsdb1            	 
+#### A masked pattern was here ####
+Retention:          	0                   	 
+#### A masked pattern was here ####
+Table Type:         	MANAGED_TABLE       	 
+Table Parameters:	 	 
+	COLUMN_STATS_ACCURATE	{\"BASIC_STATS\":\"true\"}
+#### A masked pattern was here ####
+	numFiles            	2                   
+	numPartitions       	2                   
+	numRows             	30                  
+	rawDataSize         	466                 
+	totalSize           	496                 
+#### A masked pattern was here ####
+	 	 
+# Storage Information	 	 
+SerDe Library:      	org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe	 
+InputFormat:        	org.apache.hadoop.mapred.TextInputFormat	 
+OutputFormat:       	org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat	 
+Compressed:         	No                  	 
+Num Buckets:        	-1                  	 
+Bucket Columns:     	[]                  	 
+Sort Columns:       	[]                  	 
+Storage Desc Params:	 	 
+	serialization.format	1                   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11')
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11')
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	comment             
+	 	 
+col1                	int                 	                    
+col2                	string              	                    
+col3                	string              	                    
+	 	 
+# Partition Information	 	 
+# col_name            	data_type           	comment             
+	 	 
+part                	string              	                    
+	 	 
+# Detailed Partition Information	 	 
+Partition Value:    	[part11]            	 
+Database:           	statsdb1            	 
+Table:              	testpart1           	 
+#### A masked pattern was here ####
+Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	{\"BASIC_STATS\":\"true\",\"COLUMN_STATS\":{\"col1\":\"true\",\"col2\":\"true\",\"col3\":\"true\"}}
+	numFiles            	1                   
+	numRows             	10                  
+	rawDataSize         	154                 
+	totalSize           	164                 
+#### A masked pattern was here ####
+	 	 
+# Storage Information	 	 
+SerDe Library:      	org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe	 
+InputFormat:        	org.apache.hadoop.mapred.TextInputFormat	 
+OutputFormat:       	org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat	 
+Compressed:         	No                  	 
+Num Buckets:        	-1                  	 
+Bucket Columns:     	[]                  	 
+Sort Columns:       	[]                  	 
+Storage Desc Params:	 	 
+	serialization.format	1                   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col1
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col1
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+col1                	int                 	27                  	484                 	0                   	8                   	                    	                    	                    	                    	from deserializer   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col2
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col2
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+col2                	string              	                    	                    	0                   	12                  	6.7                 	7                   	                    	                    	from deserializer   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col3
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col3
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+col3                	string              	                    	                    	0                   	1                   	4.0                 	4                   	                    	                    	from deserializer   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2')
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2')
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	comment             
+	 	 
+col1                	int                 	                    
+col2                	string              	                    
+col3                	string              	                    
+	 	 
+# Partition Information	 	 
+# col_name            	data_type           	comment             
+	 	 
+part                	string              	                    
+	 	 
+# Detailed Partition Information	 	 
+Partition Value:    	[part2]             	 
+Database:           	statsdb1            	 
+Table:              	testpart1           	 
+#### A masked pattern was here ####
+Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	{\"BASIC_STATS\":\"true\",\"COLUMN_STATS\":{\"col1\":\"true\",\"col2\":\"true\",\"col3\":\"true\"}}
+	numFiles            	1                   
+	numRows             	20                  
+	rawDataSize         	312                 
+	totalSize           	332                 
+#### A masked pattern was here ####
+	 	 
+# Storage Information	 	 
+SerDe Library:      	org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe	 
+InputFormat:        	org.apache.hadoop.mapred.TextInputFormat	 
+OutputFormat:       	org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat	 
+Compressed:         	No                  	 
+Num Buckets:        	-1                  	 
+Bucket Columns:     	[]                  	 
+Sort Columns:       	[]                  	 
+Storage Desc Params:	 	 
+	serialization.format	1                   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2') col1
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2') col1
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+col1                	int                 	27                  	484                 	0                   	18                  	                    	                    	                    	                    	from deserializer   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2') col2
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2') col2
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+col2                	string              	                    	                    	0                   	18                  	6.8                 	7                   	                    	                    	from deserializer   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2') col3
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2') col3
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+col3                	string              	                    	                    	0                   	1                   	4.0                 	4                   	                    	                    	from deserializer   
 PREHOOK: query: alter table statsdb1.testpart1 replace columns (col1 int, col2 string, col4 string) cascade
 PREHOOK: type: ALTERTABLE_REPLACECOLS
 PREHOOK: Input: statsdb1@testpart1
 PREHOOK: Output: statsdb1@testpart1
-PREHOOK: Output: statsdb1@testpart1@part=part1
+PREHOOK: Output: statsdb1@testpart1@part=part11
 PREHOOK: Output: statsdb1@testpart1@part=part2
 POSTHOOK: query: alter table statsdb1.testpart1 replace columns (col1 int, col2 string, col4 string) cascade
 POSTHOOK: type: ALTERTABLE_REPLACECOLS
 POSTHOOK: Input: statsdb1@testpart1
 POSTHOOK: Output: statsdb1@testpart1
-POSTHOOK: Output: statsdb1@testpart1@part=part1
+POSTHOOK: Output: statsdb1@testpart1@part=part11
 POSTHOOK: Output: statsdb1@testpart1@part=part2
 PREHOOK: query: describe formatted statsdb1.testpart1
 PREHOOK: type: DESCTABLE
@@ -878,10 +1065,10 @@ Bucket Columns:     	[]
 Sort Columns:       	[]                  	 
 Storage Desc Params:	 	 
 	serialization.format	1                   
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1')
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11')
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1')
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11')
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	comment             
@@ -896,7 +1083,7 @@ col4                	string
 part                	string              	                    
 	 	 
 # Detailed Partition Information	 	 
-Partition Value:    	[part1]             	 
+Partition Value:    	[part11]            	 
 Database:           	statsdb1            	 
 Table:              	testpart1           	 
 #### A masked pattern was here ####
@@ -918,28 +1105,28 @@ Bucket Columns:     	[]
 Sort Columns:       	[]                  	 
 Storage Desc Params:	 	 
 	serialization.format	1                   
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col1
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col1
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col1
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
 	 	 	 	 	 	 	 	 	 	 
 col1                	int                 	27                  	484                 	0                   	8                   	                    	                    	                    	                    	from deserializer   
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col2
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col2
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col2
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col2
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
 	 	 	 	 	 	 	 	 	 	 
 col2                	string              	                    	                    	0                   	12                  	6.7                 	7                   	                    	                    	from deserializer   
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col4
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col4
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col4
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col4
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	comment             	 	 	 	 	 	 	 	 
@@ -1016,13 +1203,13 @@ PREHOOK: query: alter table statsdb1.testpart1 change column col1 col1 string ca
 PREHOOK: type: ALTERTABLE_RENAMECOL
 PREHOOK: Input: statsdb1@testpart1
 PREHOOK: Output: statsdb1@testpart1
-PREHOOK: Output: statsdb1@testpart1@part=part1
+PREHOOK: Output: statsdb1@testpart1@part=part11
 PREHOOK: Output: statsdb1@testpart1@part=part2
 POSTHOOK: query: alter table statsdb1.testpart1 change column col1 col1 string cascade
 POSTHOOK: type: ALTERTABLE_RENAMECOL
 POSTHOOK: Input: statsdb1@testpart1
 POSTHOOK: Output: statsdb1@testpart1
-POSTHOOK: Output: statsdb1@testpart1@part=part1
+POSTHOOK: Output: statsdb1@testpart1@part=part11
 POSTHOOK: Output: statsdb1@testpart1@part=part2
 PREHOOK: query: describe formatted statsdb1.testpart1
 PREHOOK: type: DESCTABLE
@@ -1067,10 +1254,10 @@ Bucket Columns:     	[]
 Sort Columns:       	[]                  	 
 Storage Desc Params:	 	 
 	serialization.format	1                   
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1')
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11')
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1')
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11')
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	comment             
@@ -1085,7 +1272,7 @@ col4                	string
 part                	string              	                    
 	 	 
 # Detailed Partition Information	 	 
-Partition Value:    	[part1]             	 
+Partition Value:    	[part11]            	 
 Database:           	statsdb1            	 
 Table:              	testpart1           	 
 #### A masked pattern was here ####
@@ -1107,28 +1294,28 @@ Bucket Columns:     	[]
 Sort Columns:       	[]                  	 
 Storage Desc Params:	 	 
 	serialization.format	1                   
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col1
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col1
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col1
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	comment             	 	 	 	 	 	 	 	 
 	 	 	 	 	 	 	 	 	 	 
 col1                	string              	from deserializer   	 	 	 	 	 	 	 	 
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col2
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col2
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col2
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col2
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
 	 	 	 	 	 	 	 	 	 	 
 col2                	string              	                    	                    	0                   	12                  	6.7                 	7                   	                    	                    	from deserializer   
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col4
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col4
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col4
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col4
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	comment             	 	 	 	 	 	 	 	 
@@ -1253,28 +1440,28 @@ Bucket Columns:     	[]
 Sort Columns:       	[]                  	 
 Storage Desc Params:	 	 
 	serialization.format	1                   
-PREHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part1') col1
+PREHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part11') col1
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb2@testpart2
-POSTHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part1') col1
+POSTHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part11') col1
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb2@testpart2
 # col_name            	data_type           	comment             	 	 	 	 	 	 	 	 
 	 	 	 	 	 	 	 	 	 	 
 col1                	string              	from deserializer   	 	 	 	 	 	 	 	 
-PREHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part1') col2
+PREHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part11') col2
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb2@testpart2
-POSTHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part1') col2
+POSTHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part11') col2
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb2@testpart2
 # col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
 	 	 	 	 	 	 	 	 	 	 
 col2                	string              	                    	                    	0                   	12                  	6.7                 	7                   	                    	                    	from deserializer   
-PREHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part1') col4
+PREHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part11') col4
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb2@testpart2
-POSTHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part1') col4
+POSTHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part11') col4
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb2@testpart2
 # col_name            	data_type           	comment             	 	 	 	 	 	 	 	 
@@ -2176,17 +2363,204 @@ POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
 	 	 	 	 	 	 	 	 	 	 
 col3                	string              	                    	                    	0                   	1                   	4.0                 	4                   	                    	                    	from deserializer   
+PREHOOK: query: alter table statsdb1.testpart1 partition (part = 'part1') rename to partition (part = 'part11')
+PREHOOK: type: ALTERTABLE_RENAMEPART
+PREHOOK: Input: statsdb1@testpart1
+PREHOOK: Output: statsdb1@testpart1@part=part1
+POSTHOOK: query: alter table statsdb1.testpart1 partition (part = 'part1') rename to partition (part = 'part11')
+POSTHOOK: type: ALTERTABLE_RENAMEPART
+POSTHOOK: Input: statsdb1@testpart1
+POSTHOOK: Input: statsdb1@testpart1@part=part1
+POSTHOOK: Output: statsdb1@testpart1@part=part1
+POSTHOOK: Output: statsdb1@testpart1@part=part11
+PREHOOK: query: describe formatted statsdb1.testpart1
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	comment             
+	 	 
+col1                	int                 	                    
+col2                	string              	                    
+col3                	string              	                    
+	 	 
+# Partition Information	 	 
+# col_name            	data_type           	comment             
+	 	 
+part                	string              	                    
+	 	 
+# Detailed Table Information	 	 
+Database:           	statsdb1            	 
+#### A masked pattern was here ####
+Retention:          	0                   	 
+#### A masked pattern was here ####
+Table Type:         	MANAGED_TABLE       	 
+Table Parameters:	 	 
+	COLUMN_STATS_ACCURATE	{\"BASIC_STATS\":\"true\"}
+#### A masked pattern was here ####
+	numFiles            	2                   
+	numPartitions       	2                   
+	numRows             	30                  
+	rawDataSize         	466                 
+	totalSize           	496                 
+#### A masked pattern was here ####
+	 	 
+# Storage Information	 	 
+SerDe Library:      	org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe	 
+InputFormat:        	org.apache.hadoop.mapred.TextInputFormat	 
+OutputFormat:       	org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat	 
+Compressed:         	No                  	 
+Num Buckets:        	-1                  	 
+Bucket Columns:     	[]                  	 
+Sort Columns:       	[]                  	 
+Storage Desc Params:	 	 
+	serialization.format	1                   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11')
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11')
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	comment             
+	 	 
+col1                	int                 	                    
+col2                	string              	                    
+col3                	string              	                    
+	 	 
+# Partition Information	 	 
+# col_name            	data_type           	comment             
+	 	 
+part                	string              	                    
+	 	 
+# Detailed Partition Information	 	 
+Partition Value:    	[part11]            	 
+Database:           	statsdb1            	 
+Table:              	testpart1           	 
+#### A masked pattern was here ####
+Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	{\"BASIC_STATS\":\"true\",\"COLUMN_STATS\":{\"col1\":\"true\",\"col2\":\"true\",\"col3\":\"true\"}}
+	numFiles            	1                   
+	numRows             	10                  
+	rawDataSize         	154                 
+	totalSize           	164                 
+#### A masked pattern was here ####
+	 	 
+# Storage Information	 	 
+SerDe Library:      	org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe	 
+InputFormat:        	org.apache.hadoop.mapred.TextInputFormat	 
+OutputFormat:       	org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat	 
+Compressed:         	No                  	 
+Num Buckets:        	-1                  	 
+Bucket Columns:     	[]                  	 
+Sort Columns:       	[]                  	 
+Storage Desc Params:	 	 
+	serialization.format	1                   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col1
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col1
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+col1                	int                 	27                  	484                 	0                   	8                   	                    	                    	                    	                    	from deserializer   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col2
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col2
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+col2                	string              	                    	                    	0                   	12                  	6.7                 	7                   	                    	                    	from deserializer   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col3
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col3
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+col3                	string              	                    	                    	0                   	1                   	4.0                 	4                   	                    	                    	from deserializer   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2')
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2')
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	comment             
+	 	 
+col1                	int                 	                    
+col2                	string              	                    
+col3                	string              	                    
+	 	 
+# Partition Information	 	 
+# col_name            	data_type           	comment             
+	 	 
+part                	string              	                    
+	 	 
+# Detailed Partition Information	 	 
+Partition Value:    	[part2]             	 
+Database:           	statsdb1            	 
+Table:              	testpart1           	 
+#### A masked pattern was here ####
+Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	{\"BASIC_STATS\":\"true\",\"COLUMN_STATS\":{\"col1\":\"true\",\"col2\":\"true\",\"col3\":\"true\"}}
+	numFiles            	1                   
+	numRows             	20                  
+	rawDataSize         	312                 
+	totalSize           	332                 
+#### A masked pattern was here ####
+	 	 
+# Storage Information	 	 
+SerDe Library:      	org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe	 
+InputFormat:        	org.apache.hadoop.mapred.TextInputFormat	 
+OutputFormat:       	org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat	 
+Compressed:         	No                  	 
+Num Buckets:        	-1                  	 
+Bucket Columns:     	[]                  	 
+Sort Columns:       	[]                  	 
+Storage Desc Params:	 	 
+	serialization.format	1                   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2') col1
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2') col1
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+col1                	int                 	27                  	484                 	0                   	18                  	                    	                    	                    	                    	from deserializer   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2') col2
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2') col2
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+col2                	string              	                    	                    	0                   	18                  	6.8                 	7                   	                    	                    	from deserializer   
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2') col3
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: statsdb1@testpart1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part2') col3
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: statsdb1@testpart1
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+col3                	string              	                    	                    	0                   	1                   	4.0                 	4                   	                    	                    	from deserializer   
 PREHOOK: query: alter table statsdb1.testpart1 replace columns (col1 int, col2 string, col4 string) cascade
 PREHOOK: type: ALTERTABLE_REPLACECOLS
 PREHOOK: Input: statsdb1@testpart1
 PREHOOK: Output: statsdb1@testpart1
-PREHOOK: Output: statsdb1@testpart1@part=part1
+PREHOOK: Output: statsdb1@testpart1@part=part11
 PREHOOK: Output: statsdb1@testpart1@part=part2
 POSTHOOK: query: alter table statsdb1.testpart1 replace columns (col1 int, col2 string, col4 string) cascade
 POSTHOOK: type: ALTERTABLE_REPLACECOLS
 POSTHOOK: Input: statsdb1@testpart1
 POSTHOOK: Output: statsdb1@testpart1
-POSTHOOK: Output: statsdb1@testpart1@part=part1
+POSTHOOK: Output: statsdb1@testpart1@part=part11
 POSTHOOK: Output: statsdb1@testpart1@part=part2
 PREHOOK: query: describe formatted statsdb1.testpart1
 PREHOOK: type: DESCTABLE
@@ -2231,10 +2605,10 @@ Bucket Columns:     	[]
 Sort Columns:       	[]                  	 
 Storage Desc Params:	 	 
 	serialization.format	1                   
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1')
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11')
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1')
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11')
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	comment             
@@ -2249,7 +2623,7 @@ col4                	string
 part                	string              	                    
 	 	 
 # Detailed Partition Information	 	 
-Partition Value:    	[part1]             	 
+Partition Value:    	[part11]            	 
 Database:           	statsdb1            	 
 Table:              	testpart1           	 
 #### A masked pattern was here ####
@@ -2271,28 +2645,28 @@ Bucket Columns:     	[]
 Sort Columns:       	[]                  	 
 Storage Desc Params:	 	 
 	serialization.format	1                   
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col1
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col1
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col1
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
 	 	 	 	 	 	 	 	 	 	 
 col1                	int                 	27                  	484                 	0                   	8                   	                    	                    	                    	                    	from deserializer   
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col2
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col2
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col2
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col2
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
 	 	 	 	 	 	 	 	 	 	 
 col2                	string              	                    	                    	0                   	12                  	6.7                 	7                   	                    	                    	from deserializer   
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col4
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col4
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col4
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col4
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	comment             	 	 	 	 	 	 	 	 
@@ -2369,13 +2743,13 @@ PREHOOK: query: alter table statsdb1.testpart1 change column col1 col1 string ca
 PREHOOK: type: ALTERTABLE_RENAMECOL
 PREHOOK: Input: statsdb1@testpart1
 PREHOOK: Output: statsdb1@testpart1
-PREHOOK: Output: statsdb1@testpart1@part=part1
+PREHOOK: Output: statsdb1@testpart1@part=part11
 PREHOOK: Output: statsdb1@testpart1@part=part2
 POSTHOOK: query: alter table statsdb1.testpart1 change column col1 col1 string cascade
 POSTHOOK: type: ALTERTABLE_RENAMECOL
 POSTHOOK: Input: statsdb1@testpart1
 POSTHOOK: Output: statsdb1@testpart1
-POSTHOOK: Output: statsdb1@testpart1@part=part1
+POSTHOOK: Output: statsdb1@testpart1@part=part11
 POSTHOOK: Output: statsdb1@testpart1@part=part2
 PREHOOK: query: describe formatted statsdb1.testpart1
 PREHOOK: type: DESCTABLE
@@ -2420,10 +2794,10 @@ Bucket Columns:     	[]
 Sort Columns:       	[]                  	 
 Storage Desc Params:	 	 
 	serialization.format	1                   
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1')
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11')
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1')
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11')
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	comment             
@@ -2438,7 +2812,7 @@ col4                	string
 part                	string              	                    
 	 	 
 # Detailed Partition Information	 	 
-Partition Value:    	[part1]             	 
+Partition Value:    	[part11]            	 
 Database:           	statsdb1            	 
 Table:              	testpart1           	 
 #### A masked pattern was here ####
@@ -2460,28 +2834,28 @@ Bucket Columns:     	[]
 Sort Columns:       	[]                  	 
 Storage Desc Params:	 	 
 	serialization.format	1                   
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col1
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col1
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col1
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col1
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	comment             	 	 	 	 	 	 	 	 
 	 	 	 	 	 	 	 	 	 	 
 col1                	string              	from deserializer   	 	 	 	 	 	 	 	 
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col2
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col2
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col2
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col2
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
 	 	 	 	 	 	 	 	 	 	 
 col2                	string              	                    	                    	0                   	12                  	6.7                 	7                   	                    	                    	from deserializer   
-PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col4
+PREHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col4
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb1@testpart1
-POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part1') col4
+POSTHOOK: query: describe formatted statsdb1.testpart1 partition (part = 'part11') col4
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb1@testpart1
 # col_name            	data_type           	comment             	 	 	 	 	 	 	 	 
@@ -2606,28 +2980,28 @@ Bucket Columns:     	[]
 Sort Columns:       	[]                  	 
 Storage Desc Params:	 	 
 	serialization.format	1                   
-PREHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part1') col1
+PREHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part11') col1
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb2@testpart2
-POSTHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part1') col1
+POSTHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part11') col1
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb2@testpart2
 # col_name            	data_type           	comment             	 	 	 	 	 	 	 	 
 	 	 	 	 	 	 	 	 	 	 
 col1                	string              	from deserializer   	 	 	 	 	 	 	 	 
-PREHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part1') col2
+PREHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part11') col2
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb2@testpart2
-POSTHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part1') col2
+POSTHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part11') col2
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb2@testpart2
 # col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
 	 	 	 	 	 	 	 	 	 	 
 col2                	string              	                    	                    	0                   	12                  	6.7                 	7                   	                    	                    	from deserializer   
-PREHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part1') col4
+PREHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part11') col4
 PREHOOK: type: DESCTABLE
 PREHOOK: Input: statsdb2@testpart2
-POSTHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part1') col4
+POSTHOOK: query: describe formatted statsdb2.testpart2 partition (part = 'part11') col4
 POSTHOOK: type: DESCTABLE
 POSTHOOK: Input: statsdb2@testpart2
 # col_name            	data_type           	comment             	 	 	 	 	 	 	 	 

http://git-wip-us.apache.org/repos/asf/hive/blob/f3310a37/ql/src/test/results/clientpositive/rename_external_partition_location.q.out
----------------------------------------------------------------------
diff --git a/ql/src/test/results/clientpositive/rename_external_partition_location.q.out b/ql/src/test/results/clientpositive/rename_external_partition_location.q.out
index 1670b4e..ec4076f 100644
--- a/ql/src/test/results/clientpositive/rename_external_partition_location.q.out
+++ b/ql/src/test/results/clientpositive/rename_external_partition_location.q.out
@@ -65,6 +65,115 @@ Found 1 items
 2val_2
 5val_5
 9val_9
+PREHOOK: query: ANALYZE TABLE ex_table PARTITION (part='part1') COMPUTE STATISTICS FOR COLUMNS
+PREHOOK: type: QUERY
+PREHOOK: Input: default@ex_table
+PREHOOK: Input: default@ex_table@part=part1
+#### A masked pattern was here ####
+POSTHOOK: query: ANALYZE TABLE ex_table PARTITION (part='part1') COMPUTE STATISTICS FOR COLUMNS
+POSTHOOK: type: QUERY
+POSTHOOK: Input: default@ex_table
+POSTHOOK: Input: default@ex_table@part=part1
+#### A masked pattern was here ####
+PREHOOK: query: DESCRIBE FORMATTED ex_table
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: default@ex_table
+POSTHOOK: query: DESCRIBE FORMATTED ex_table
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: default@ex_table
+# col_name            	data_type           	comment             
+	 	 
+key                 	int                 	                    
+value               	string              	                    
+	 	 
+# Partition Information	 	 
+# col_name            	data_type           	comment             
+	 	 
+part                	string              	                    
+	 	 
+# Detailed Table Information	 	 
+Database:           	default             	 
+#### A masked pattern was here ####
+Retention:          	0                   	 
+#### A masked pattern was here ####
+Table Type:         	EXTERNAL_TABLE      	 
+Table Parameters:	 	 
+	COLUMN_STATS_ACCURATE	{\"BASIC_STATS\":\"true\"}
+	EXTERNAL            	TRUE                
+	numFiles            	1                   
+	numPartitions       	1                   
+	numRows             	10                  
+	rawDataSize         	70                  
+	totalSize           	80                  
+#### A masked pattern was here ####
+	 	 
+# Storage Information	 	 
+SerDe Library:      	org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe	 
+InputFormat:        	org.apache.hadoop.mapred.TextInputFormat	 
+OutputFormat:       	org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat	 
+Compressed:         	No                  	 
+Num Buckets:        	-1                  	 
+Bucket Columns:     	[]                  	 
+Sort Columns:       	[]                  	 
+Storage Desc Params:	 	 
+	serialization.format	1                   
+PREHOOK: query: DESCRIBE FORMATTED ex_table PARTITION (part='part1')
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: default@ex_table
+POSTHOOK: query: DESCRIBE FORMATTED ex_table PARTITION (part='part1')
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: default@ex_table
+# col_name            	data_type           	comment             
+	 	 
+key                 	int                 	                    
+value               	string              	                    
+	 	 
+# Partition Information	 	 
+# col_name            	data_type           	comment             
+	 	 
+part                	string              	                    
+	 	 
+# Detailed Partition Information	 	 
+Partition Value:    	[part1]             	 
+Database:           	default             	 
+Table:              	ex_table            	 
+#### A masked pattern was here ####
+Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	{\"BASIC_STATS\":\"true\",\"COLUMN_STATS\":{\"key\":\"true\",\"value\":\"true\"}}
+	numFiles            	1                   
+	numRows             	10                  
+	rawDataSize         	70                  
+	totalSize           	80                  
+#### A masked pattern was here ####
+	 	 
+# Storage Information	 	 
+SerDe Library:      	org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe	 
+InputFormat:        	org.apache.hadoop.mapred.TextInputFormat	 
+OutputFormat:       	org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat	 
+Compressed:         	No                  	 
+Num Buckets:        	-1                  	 
+Bucket Columns:     	[]                  	 
+Sort Columns:       	[]                  	 
+Storage Desc Params:	 	 
+	serialization.format	1                   
+PREHOOK: query: DESCRIBE FORMATTED ex_table PARTITION (part='part1') key
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: default@ex_table
+POSTHOOK: query: DESCRIBE FORMATTED ex_table PARTITION (part='part1') key
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: default@ex_table
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+key                 	int                 	0                   	9                   	0                   	6                   	                    	                    	                    	                    	from deserializer   
+PREHOOK: query: DESCRIBE FORMATTED ex_table PARTITION (part='part1') value
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: default@ex_table
+POSTHOOK: query: DESCRIBE FORMATTED ex_table PARTITION (part='part1') value
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: default@ex_table
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+value               	string              	                    	                    	0                   	5                   	5.0                 	5                   	                    	                    	from deserializer   
 PREHOOK: query: ALTER TABLE ex_table PARTITION (part='part1') RENAME TO PARTITION (part='part2')
 PREHOOK: type: ALTERTABLE_RENAMEPART
 PREHOOK: Input: default@ex_table
@@ -114,3 +223,102 @@ Found 1 items
 2val_2
 5val_5
 9val_9
+PREHOOK: query: DESCRIBE FORMATTED ex_table
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: default@ex_table
+POSTHOOK: query: DESCRIBE FORMATTED ex_table
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: default@ex_table
+# col_name            	data_type           	comment             
+	 	 
+key                 	int                 	                    
+value               	string              	                    
+	 	 
+# Partition Information	 	 
+# col_name            	data_type           	comment             
+	 	 
+part                	string              	                    
+	 	 
+# Detailed Table Information	 	 
+Database:           	default             	 
+#### A masked pattern was here ####
+Retention:          	0                   	 
+#### A masked pattern was here ####
+Table Type:         	EXTERNAL_TABLE      	 
+Table Parameters:	 	 
+	COLUMN_STATS_ACCURATE	{\"BASIC_STATS\":\"true\"}
+	EXTERNAL            	TRUE                
+	numFiles            	1                   
+	numPartitions       	1                   
+	numRows             	10                  
+	rawDataSize         	70                  
+	totalSize           	80                  
+#### A masked pattern was here ####
+	 	 
+# Storage Information	 	 
+SerDe Library:      	org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe	 
+InputFormat:        	org.apache.hadoop.mapred.TextInputFormat	 
+OutputFormat:       	org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat	 
+Compressed:         	No                  	 
+Num Buckets:        	-1                  	 
+Bucket Columns:     	[]                  	 
+Sort Columns:       	[]                  	 
+Storage Desc Params:	 	 
+	serialization.format	1                   
+PREHOOK: query: DESCRIBE FORMATTED ex_table PARTITION (part='part2')
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: default@ex_table
+POSTHOOK: query: DESCRIBE FORMATTED ex_table PARTITION (part='part2')
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: default@ex_table
+# col_name            	data_type           	comment             
+	 	 
+key                 	int                 	                    
+value               	string              	                    
+	 	 
+# Partition Information	 	 
+# col_name            	data_type           	comment             
+	 	 
+part                	string              	                    
+	 	 
+# Detailed Partition Information	 	 
+Partition Value:    	[part2]             	 
+Database:           	default             	 
+Table:              	ex_table            	 
+#### A masked pattern was here ####
+Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	{\"BASIC_STATS\":\"true\",\"COLUMN_STATS\":{\"key\":\"true\",\"value\":\"true\"}}
+	numFiles            	1                   
+	numRows             	10                  
+	rawDataSize         	70                  
+	totalSize           	80                  
+#### A masked pattern was here ####
+	 	 
+# Storage Information	 	 
+SerDe Library:      	org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe	 
+InputFormat:        	org.apache.hadoop.mapred.TextInputFormat	 
+OutputFormat:       	org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat	 
+Compressed:         	No                  	 
+Num Buckets:        	-1                  	 
+Bucket Columns:     	[]                  	 
+Sort Columns:       	[]                  	 
+Storage Desc Params:	 	 
+	serialization.format	1                   
+PREHOOK: query: DESCRIBE FORMATTED ex_table PARTITION (part='part2') key
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: default@ex_table
+POSTHOOK: query: DESCRIBE FORMATTED ex_table PARTITION (part='part2') key
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: default@ex_table
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+key                 	int                 	0                   	9                   	0                   	6                   	                    	                    	                    	                    	from deserializer   
+PREHOOK: query: DESCRIBE FORMATTED ex_table PARTITION (part='part2') value
+PREHOOK: type: DESCTABLE
+PREHOOK: Input: default@ex_table
+POSTHOOK: query: DESCRIBE FORMATTED ex_table PARTITION (part='part2') value
+POSTHOOK: type: DESCTABLE
+POSTHOOK: Input: default@ex_table
+# col_name            	data_type           	min                 	max                 	num_nulls           	distinct_count      	avg_col_len         	max_col_len         	num_trues           	num_falses          	comment             
+	 	 	 	 	 	 	 	 	 	 
+value               	string              	                    	                    	0                   	5                   	5.0                 	5                   	                    	                    	from deserializer