You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hive.apache.org by ha...@apache.org on 2013/11/08 16:56:31 UTC

svn commit: r1540094 [1/8] - in /hive/trunk: common/src/java/org/apache/hadoop/hive/common/ hbase-handler/src/test/results/positive/ metastore/src/java/org/apache/hadoop/hive/metastore/ ql/src/java/org/apache/hadoop/hive/ql/exec/ ql/src/java/org/apache...

Author: hashutosh
Date: Fri Nov  8 15:56:27 2013
New Revision: 1540094

URL: http://svn.apache.org/r1540094
Log:
HIVE-3777 : add a property in the partition to figure out if stats are accurate (Ashutosh Chauhan via Thejas Nair)

Added:
    hive/trunk/ql/src/test/queries/clientpositive/stats_invalidation.q
    hive/trunk/ql/src/test/results/clientpositive/stats_invalidation.q.out
Modified:
    hive/trunk/common/src/java/org/apache/hadoop/hive/common/StatsSetupConst.java
    hive/trunk/hbase-handler/src/test/results/positive/hbase_stats.q.out
    hive/trunk/hbase-handler/src/test/results/positive/hbase_stats2.q.out
    hive/trunk/hbase-handler/src/test/results/positive/hbase_stats3.q.out
    hive/trunk/hbase-handler/src/test/results/positive/hbase_stats_empty_partition.q.out
    hive/trunk/metastore/src/java/org/apache/hadoop/hive/metastore/MetaStoreUtils.java
    hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/exec/StatsTask.java
    hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/metadata/Hive.java
    hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/optimizer/StatsOptimizer.java
    hive/trunk/ql/src/test/queries/clientpositive/stats_only_null.q
    hive/trunk/ql/src/test/results/clientnegative/stats_partialscan_autogether.q.out
    hive/trunk/ql/src/test/results/clientnegative/unset_table_property.q.out
    hive/trunk/ql/src/test/results/clientpositive/alter_numbuckets_partitioned_table.q.out
    hive/trunk/ql/src/test/results/clientpositive/alter_numbuckets_partitioned_table2.q.out
    hive/trunk/ql/src/test/results/clientpositive/alter_partition_clusterby_sortby.q.out
    hive/trunk/ql/src/test/results/clientpositive/alter_partition_coltype.q.out
    hive/trunk/ql/src/test/results/clientpositive/alter_skewed_table.q.out
    hive/trunk/ql/src/test/results/clientpositive/alter_table_not_sorted.q.out
    hive/trunk/ql/src/test/results/clientpositive/alter_table_serde2.q.out
    hive/trunk/ql/src/test/results/clientpositive/auto_join_reordering_values.q.out
    hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_1.q.out
    hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_11.q.out
    hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_12.q.out
    hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_2.q.out
    hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_3.q.out
    hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_4.q.out
    hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_5.q.out
    hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_7.q.out
    hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_8.q.out
    hive/trunk/ql/src/test/results/clientpositive/binary_output_format.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucket1.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucket2.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucket3.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucket4.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucket5.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucket_map_join_1.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucket_map_join_2.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketcontext_1.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketcontext_2.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketcontext_3.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketcontext_4.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketcontext_5.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketcontext_6.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketcontext_7.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketcontext_8.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketizedhiveinputformat.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketmapjoin1.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketmapjoin10.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketmapjoin11.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketmapjoin12.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketmapjoin13.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketmapjoin2.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketmapjoin3.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketmapjoin4.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketmapjoin5.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketmapjoin7.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketmapjoin8.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketmapjoin9.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketmapjoin_negative.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketmapjoin_negative2.q.out
    hive/trunk/ql/src/test/results/clientpositive/bucketmapjoin_negative3.q.out
    hive/trunk/ql/src/test/results/clientpositive/columnstats_partlvl.q.out
    hive/trunk/ql/src/test/results/clientpositive/columnstats_tbllvl.q.out
    hive/trunk/ql/src/test/results/clientpositive/combine2_hadoop20.q.out
    hive/trunk/ql/src/test/results/clientpositive/create_alter_list_bucketing_table1.q.out
    hive/trunk/ql/src/test/results/clientpositive/ctas_colname.q.out
    hive/trunk/ql/src/test/results/clientpositive/ctas_hadoop20.q.out
    hive/trunk/ql/src/test/results/clientpositive/ctas_uses_database_location.q.out
    hive/trunk/ql/src/test/results/clientpositive/describe_comment_nonascii.q.out
    hive/trunk/ql/src/test/results/clientpositive/describe_table.q.out
    hive/trunk/ql/src/test/results/clientpositive/disable_merge_for_bucketing.q.out
    hive/trunk/ql/src/test/results/clientpositive/dynamic_partition_skip_default.q.out
    hive/trunk/ql/src/test/results/clientpositive/filter_join_breaktask.q.out
    hive/trunk/ql/src/test/results/clientpositive/groupby_map_ppr.q.out
    hive/trunk/ql/src/test/results/clientpositive/groupby_map_ppr_multi_distinct.q.out
    hive/trunk/ql/src/test/results/clientpositive/groupby_ppr.q.out
    hive/trunk/ql/src/test/results/clientpositive/groupby_ppr_multi_distinct.q.out
    hive/trunk/ql/src/test/results/clientpositive/groupby_sort_1.q.out
    hive/trunk/ql/src/test/results/clientpositive/groupby_sort_6.q.out
    hive/trunk/ql/src/test/results/clientpositive/groupby_sort_skew_1.q.out
    hive/trunk/ql/src/test/results/clientpositive/infer_bucket_sort.q.out
    hive/trunk/ql/src/test/results/clientpositive/infer_bucket_sort_bucketed_table.q.out
    hive/trunk/ql/src/test/results/clientpositive/infer_bucket_sort_convert_join.q.out
    hive/trunk/ql/src/test/results/clientpositive/infer_bucket_sort_dyn_part.q.out
    hive/trunk/ql/src/test/results/clientpositive/infer_bucket_sort_grouping_operators.q.out
    hive/trunk/ql/src/test/results/clientpositive/infer_bucket_sort_map_operators.q.out
    hive/trunk/ql/src/test/results/clientpositive/infer_bucket_sort_merge.q.out
    hive/trunk/ql/src/test/results/clientpositive/infer_bucket_sort_multi_insert.q.out
    hive/trunk/ql/src/test/results/clientpositive/infer_bucket_sort_num_buckets.q.out
    hive/trunk/ql/src/test/results/clientpositive/infer_bucket_sort_reducers_power_two.q.out
    hive/trunk/ql/src/test/results/clientpositive/input23.q.out
    hive/trunk/ql/src/test/results/clientpositive/input42.q.out
    hive/trunk/ql/src/test/results/clientpositive/input_part1.q.out
    hive/trunk/ql/src/test/results/clientpositive/input_part2.q.out
    hive/trunk/ql/src/test/results/clientpositive/input_part7.q.out
    hive/trunk/ql/src/test/results/clientpositive/input_part9.q.out
    hive/trunk/ql/src/test/results/clientpositive/join17.q.out
    hive/trunk/ql/src/test/results/clientpositive/join26.q.out
    hive/trunk/ql/src/test/results/clientpositive/join32.q.out
    hive/trunk/ql/src/test/results/clientpositive/join32_lessSize.q.out
    hive/trunk/ql/src/test/results/clientpositive/join33.q.out
    hive/trunk/ql/src/test/results/clientpositive/join34.q.out
    hive/trunk/ql/src/test/results/clientpositive/join35.q.out
    hive/trunk/ql/src/test/results/clientpositive/join9.q.out
    hive/trunk/ql/src/test/results/clientpositive/join_filters_overlap.q.out
    hive/trunk/ql/src/test/results/clientpositive/join_map_ppr.q.out
    hive/trunk/ql/src/test/results/clientpositive/list_bucket_dml_10.q.out
    hive/trunk/ql/src/test/results/clientpositive/load_dyn_part8.q.out
    hive/trunk/ql/src/test/results/clientpositive/louter_join_ppr.q.out
    hive/trunk/ql/src/test/results/clientpositive/merge3.q.out
    hive/trunk/ql/src/test/results/clientpositive/metadata_only_queries.q.out
    hive/trunk/ql/src/test/results/clientpositive/outer_join_ppr.q.out
    hive/trunk/ql/src/test/results/clientpositive/parallel_orderby.q.out
    hive/trunk/ql/src/test/results/clientpositive/pcr.q.out
    hive/trunk/ql/src/test/results/clientpositive/ppd_join_filter.q.out
    hive/trunk/ql/src/test/results/clientpositive/ppd_union_view.q.out
    hive/trunk/ql/src/test/results/clientpositive/ppd_vc.q.out
    hive/trunk/ql/src/test/results/clientpositive/ppr_allchildsarenull.q.out
    hive/trunk/ql/src/test/results/clientpositive/push_or.q.out
    hive/trunk/ql/src/test/results/clientpositive/rand_partitionpruner1.q.out
    hive/trunk/ql/src/test/results/clientpositive/rand_partitionpruner2.q.out
    hive/trunk/ql/src/test/results/clientpositive/rand_partitionpruner3.q.out
    hive/trunk/ql/src/test/results/clientpositive/rcfile_default_format.q.out
    hive/trunk/ql/src/test/results/clientpositive/reduce_deduplicate.q.out
    hive/trunk/ql/src/test/results/clientpositive/regexp_extract.q.out
    hive/trunk/ql/src/test/results/clientpositive/router_join_ppr.q.out
    hive/trunk/ql/src/test/results/clientpositive/sample1.q.out
    hive/trunk/ql/src/test/results/clientpositive/sample10.q.out
    hive/trunk/ql/src/test/results/clientpositive/sample2.q.out
    hive/trunk/ql/src/test/results/clientpositive/sample4.q.out
    hive/trunk/ql/src/test/results/clientpositive/sample5.q.out
    hive/trunk/ql/src/test/results/clientpositive/sample6.q.out
    hive/trunk/ql/src/test/results/clientpositive/sample7.q.out
    hive/trunk/ql/src/test/results/clientpositive/sample8.q.out
    hive/trunk/ql/src/test/results/clientpositive/sample9.q.out
    hive/trunk/ql/src/test/results/clientpositive/serde_user_properties.q.out
    hive/trunk/ql/src/test/results/clientpositive/show_create_table_alter.q.out
    hive/trunk/ql/src/test/results/clientpositive/show_tblproperties.q.out
    hive/trunk/ql/src/test/results/clientpositive/smb_mapjoin_11.q.out
    hive/trunk/ql/src/test/results/clientpositive/smb_mapjoin_12.q.out
    hive/trunk/ql/src/test/results/clientpositive/smb_mapjoin_13.q.out
    hive/trunk/ql/src/test/results/clientpositive/smb_mapjoin_15.q.out
    hive/trunk/ql/src/test/results/clientpositive/sort_merge_join_desc_5.q.out
    hive/trunk/ql/src/test/results/clientpositive/sort_merge_join_desc_6.q.out
    hive/trunk/ql/src/test/results/clientpositive/sort_merge_join_desc_7.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats0.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats1.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats10.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats11.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats12.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats13.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats14.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats15.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats16.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats18.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats19.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats3.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats4.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats5.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats6.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats7.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats8.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats9.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats_empty_partition.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats_noscan_1.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats_noscan_2.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats_only_null.q.out
    hive/trunk/ql/src/test/results/clientpositive/stats_partscan_1.q.out
    hive/trunk/ql/src/test/results/clientpositive/transform_ppr1.q.out
    hive/trunk/ql/src/test/results/clientpositive/transform_ppr2.q.out
    hive/trunk/ql/src/test/results/clientpositive/truncate_column.q.out
    hive/trunk/ql/src/test/results/clientpositive/udf_explode.q.out
    hive/trunk/ql/src/test/results/clientpositive/udf_reflect2.q.out
    hive/trunk/ql/src/test/results/clientpositive/udtf_explode.q.out
    hive/trunk/ql/src/test/results/clientpositive/union22.q.out
    hive/trunk/ql/src/test/results/clientpositive/union24.q.out
    hive/trunk/ql/src/test/results/clientpositive/union_ppr.q.out
    hive/trunk/ql/src/test/results/clientpositive/unset_table_view_property.q.out
    hive/trunk/ql/src/test/results/compiler/plan/case_sensitivity.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/cast1.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/groupby1.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/groupby2.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/groupby3.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/groupby4.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/groupby5.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/groupby6.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/input1.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/input2.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/input20.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/input3.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/input4.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/input5.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/input6.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/input7.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/input8.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/input9.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/input_part1.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/input_testsequencefile.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/input_testxpath.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/input_testxpath2.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/join1.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/join2.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/join3.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/join4.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/join5.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/join6.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/join7.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/join8.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/sample1.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/sample2.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/sample3.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/sample4.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/sample5.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/sample6.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/sample7.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/subq.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/udf1.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/udf6.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/udf_case.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/udf_when.q.xml
    hive/trunk/ql/src/test/results/compiler/plan/union.q.xml

Modified: hive/trunk/common/src/java/org/apache/hadoop/hive/common/StatsSetupConst.java
URL: http://svn.apache.org/viewvc/hive/trunk/common/src/java/org/apache/hadoop/hive/common/StatsSetupConst.java?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/common/src/java/org/apache/hadoop/hive/common/StatsSetupConst.java (original)
+++ hive/trunk/common/src/java/org/apache/hadoop/hive/common/StatsSetupConst.java Fri Nov  8 15:56:27 2013
@@ -17,8 +17,8 @@
  */
 package org.apache.hadoop.hive.common;
 
-import java.util.ArrayList;
-import java.util.List;
+import java.util.Map;
+
 
 
 /**
@@ -66,33 +66,34 @@ public class StatsSetupConst {
   /**
    * @return List of all supported statistics
    */
-  public static List<String> getSupportedStats() {
-    List<String> supportedStats = new ArrayList<String>();
-    supportedStats.add(NUM_FILES);
-    supportedStats.add(ROW_COUNT);
-    supportedStats.add(TOTAL_SIZE);
-    supportedStats.add(RAW_DATA_SIZE);
-    return supportedStats;
-  }
+  public static final String[] supportedStats = new String[] 
+	{NUM_FILES,ROW_COUNT,TOTAL_SIZE,RAW_DATA_SIZE};
 
   /**
    * @return List of all statistics that need to be collected during query execution. These are
    * statistics that inherently require a scan of the data.
    */
-  public static List<String> getStatsToBeCollected() {
-    List<String> collectableStats = new ArrayList<String>();
-    collectableStats.add(ROW_COUNT);
-    collectableStats.add(RAW_DATA_SIZE);
-    return collectableStats;
-  }
+  public static final String[] statsRequireCompute = new String[] {ROW_COUNT,RAW_DATA_SIZE};
 
   /**
    * @return List of statistics that can be collected quickly without requiring a scan of the data.
    */
-  public static List<String> getStatsFastCollection() {
-    List<String> fastStats = new ArrayList<String>();
-    fastStats.add(NUM_FILES);
-    fastStats.add(TOTAL_SIZE);
-    return fastStats;
+  public static final String[] fastStats = new String[] {NUM_FILES,TOTAL_SIZE};
+  
+  // This string constant is used by stats task to indicate to AlterHandler that
+  // alterPartition/alterTable is happening via statsTask.
+  public static final String STATS_GENERATED_VIA_STATS_TASK = "STATS_GENERATED_VIA_STATS_TASK";
+
+  // This string constant will be persisted in metastore to indicate whether corresponding
+  // table or partition's statistics are accurate or not.
+  public static final String COLUMN_STATS_ACCURATE = "COLUMN_STATS_ACCURATE";
+  
+  public static final String TRUE = "true";
+  
+  public static final String FALSE = "false";
+  
+  public static boolean areStatsUptoDate(Map<String,String> params) {
+	String statsAcc = params.get(COLUMN_STATS_ACCURATE);
+	return statsAcc == null ? false : statsAcc.equals(TRUE);
   }
 }

Modified: hive/trunk/hbase-handler/src/test/results/positive/hbase_stats.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/hbase-handler/src/test/results/positive/hbase_stats.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/hbase-handler/src/test/results/positive/hbase_stats.q.out (original)
+++ hive/trunk/hbase-handler/src/test/results/positive/hbase_stats.q.out Fri Nov  8 15:56:27 2013
@@ -42,6 +42,7 @@ Retention:          	0                  
 #### A masked pattern was here ####
 Table Type:         	MANAGED_TABLE       	 
 Table Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -217,6 +218,7 @@ Table:              	stats_part         
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -264,6 +266,7 @@ Table:              	stats_part         
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                

Modified: hive/trunk/hbase-handler/src/test/results/positive/hbase_stats2.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/hbase-handler/src/test/results/positive/hbase_stats2.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/hbase-handler/src/test/results/positive/hbase_stats2.q.out (original)
+++ hive/trunk/hbase-handler/src/test/results/positive/hbase_stats2.q.out Fri Nov  8 15:56:27 2013
@@ -42,6 +42,7 @@ Retention:          	0                  
 #### A masked pattern was here ####
 Table Type:         	MANAGED_TABLE       	 
 Table Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -217,6 +218,7 @@ Table:              	stats_part         
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -264,6 +266,7 @@ Table:              	stats_part         
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                

Modified: hive/trunk/hbase-handler/src/test/results/positive/hbase_stats3.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/hbase-handler/src/test/results/positive/hbase_stats3.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/hbase-handler/src/test/results/positive/hbase_stats3.q.out (original)
+++ hive/trunk/hbase-handler/src/test/results/positive/hbase_stats3.q.out Fri Nov  8 15:56:27 2013
@@ -40,6 +40,7 @@ Table:              	stats_part         
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -97,6 +98,7 @@ Table:              	stats_part         
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -158,6 +160,7 @@ Table:              	stats_part         
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -227,6 +230,7 @@ Table:              	stats_part         
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -296,6 +300,7 @@ Table:              	stats_part         
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -369,6 +374,7 @@ Table:              	stats_part         
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                

Modified: hive/trunk/hbase-handler/src/test/results/positive/hbase_stats_empty_partition.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/hbase-handler/src/test/results/positive/hbase_stats_empty_partition.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/hbase-handler/src/test/results/positive/hbase_stats_empty_partition.q.out (original)
+++ hive/trunk/hbase-handler/src/test/results/positive/hbase_stats_empty_partition.q.out Fri Nov  8 15:56:27 2013
@@ -43,6 +43,7 @@ Table:              	tmptable           
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	0                   
 	rawDataSize         	0                   

Modified: hive/trunk/metastore/src/java/org/apache/hadoop/hive/metastore/MetaStoreUtils.java
URL: http://svn.apache.org/viewvc/hive/trunk/metastore/src/java/org/apache/hadoop/hive/metastore/MetaStoreUtils.java?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/metastore/src/java/org/apache/hadoop/hive/metastore/MetaStoreUtils.java (original)
+++ hive/trunk/metastore/src/java/org/apache/hadoop/hive/metastore/MetaStoreUtils.java Fri Nov  8 15:56:27 2013
@@ -148,8 +148,7 @@ public class MetaStoreUtils {
    * @return True if the passed Parameters Map contains values for all "Fast Stats".
    */
   public static boolean containsAllFastStats(Map<String, String> partParams) {
-    List<String> fastStats = StatsSetupConst.getStatsFastCollection();
-    for (String stat : fastStats) {
+    for (String stat : StatsSetupConst.fastStats) {
       if (!partParams.containsKey(stat)) {
         return false;
       }
@@ -157,11 +156,6 @@ public class MetaStoreUtils {
     return true;
   }
 
-  public static boolean updateUnpartitionedTableStatsFast(Database db, Table tbl, Warehouse wh)
-      throws MetaException {
-    return updateUnpartitionedTableStatsFast(db, tbl, wh, false, false);
-  }
-
   public static boolean updateUnpartitionedTableStatsFast(Database db, Table tbl, Warehouse wh,
       boolean madeDir) throws MetaException {
     return updateUnpartitionedTableStatsFast(db, tbl, wh, madeDir, false);
@@ -200,9 +194,15 @@ public class MetaStoreUtils {
         }
         params.put(StatsSetupConst.TOTAL_SIZE, Long.toString(tableSize));
         LOG.info("Updated size of table " + tbl.getTableName() +" to "+ Long.toString(tableSize));
-        if (params.containsKey(StatsSetupConst.ROW_COUNT) ||
-            params.containsKey(StatsSetupConst.RAW_DATA_SIZE)) {
-          // TODO: Add a MetaStore flag indicating accuracy of these stats and update it here.
+        if(!params.containsKey(StatsSetupConst.STATS_GENERATED_VIA_STATS_TASK)) {
+          // invalidate stats requiring scan since this is a regular ddl alter case
+          for (String stat : StatsSetupConst.statsRequireCompute) {
+            params.put(stat, "-1");
+          }
+          params.put(StatsSetupConst.COLUMN_STATS_ACCURATE, StatsSetupConst.FALSE);
+        } else {
+          params.remove(StatsSetupConst.STATS_GENERATED_VIA_STATS_TASK);	
+          params.put(StatsSetupConst.COLUMN_STATS_ACCURATE, StatsSetupConst.TRUE);
         }
       }
       tbl.setParameters(params);
@@ -215,10 +215,6 @@ public class MetaStoreUtils {
   public static boolean requireCalStats(Configuration hiveConf, Partition oldPart,
     Partition newPart, Table tbl) {
 
-    if (!HiveConf.getBoolVar(hiveConf, HiveConf.ConfVars.HIVESTATSAUTOGATHER)) {
-      return false;
-    }
-
     if (MetaStoreUtils.isView(tbl)) {
       return false;
     }
@@ -233,9 +229,13 @@ public class MetaStoreUtils {
       return true;
     }
 
+    if(newPart.getParameters().containsKey(StatsSetupConst.STATS_GENERATED_VIA_STATS_TASK)) {
+      return true;
+    }
+    
     // requires to calculate stats if new and old have different fast stats
     if ((oldPart != null) && (oldPart.getParameters() != null)) {
-      for (String stat : StatsSetupConst.getStatsFastCollection()) {
+      for (String stat : StatsSetupConst.fastStats) {
         if (oldPart.getParameters().containsKey(stat)) {
           Long oldStat = Long.parseLong(oldPart.getParameters().get(stat));
           Long newStat = Long.parseLong(newPart.getParameters().get(stat));
@@ -290,11 +290,15 @@ public class MetaStoreUtils {
         }
         params.put(StatsSetupConst.TOTAL_SIZE, Long.toString(partSize));
         LOG.warn("Updated size to " + Long.toString(partSize));
-        if (params.containsKey(StatsSetupConst.ROW_COUNT) ||
-            params.containsKey(StatsSetupConst.RAW_DATA_SIZE)) {
-          // The accuracy of these "collectable" stats at this point is suspect unless we know that
-          // StatsTask was just run before this MetaStore call and populated them.
-          // TODO: Add a MetaStore flag indicating accuracy of these stats and update it here.
+        if(!params.containsKey(StatsSetupConst.STATS_GENERATED_VIA_STATS_TASK)) {
+          // invalidate stats requiring scan since this is a regular ddl alter case
+          for (String stat : StatsSetupConst.statsRequireCompute) {
+            params.put(stat, "-1");
+          }
+          params.put(StatsSetupConst.COLUMN_STATS_ACCURATE, StatsSetupConst.FALSE);
+        } else {
+          params.remove(StatsSetupConst.STATS_GENERATED_VIA_STATS_TASK);
+          params.put(StatsSetupConst.COLUMN_STATS_ACCURATE, StatsSetupConst.TRUE);
         }
       }
       part.setParameters(params);

Modified: hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/exec/StatsTask.java
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/exec/StatsTask.java?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/exec/StatsTask.java (original)
+++ hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/exec/StatsTask.java Fri Nov  8 15:56:27 2013
@@ -66,9 +66,6 @@ public class StatsTask extends Task<Stat
   private Table table;
   private List<LinkedHashMap<String, String>> dpPartSpecs;
 
-  private static final List<String> collectableStats = StatsSetupConst.getStatsToBeCollected();
-  private static final List<String> supportedStats = StatsSetupConst.getSupportedStats();
-
   public StatsTask() {
     super();
     dpPartSpecs = null;
@@ -84,7 +81,7 @@ public class StatsTask extends Task<Stat
 
     public Statistics() {
       stats = new HashMap<String, LongWritable>();
-      for (String statType : supportedStats) {
+      for (String statType : StatsSetupConst.supportedStats) {
         stats.put(statType, new LongWritable(0L));
       }
     }
@@ -108,7 +105,7 @@ public class StatsTask extends Task<Stat
 
     @Override
     public String toString() {
-      return org.apache.commons.lang.StringUtils.join(supportedStats, ", ");
+      return org.apache.commons.lang.StringUtils.join(StatsSetupConst.supportedStats, ", ");
     }
   }
 
@@ -207,7 +204,7 @@ public class StatsTask extends Task<Stat
 
       boolean tableStatsExist = this.existStats(parameters);
 
-      for (String statType : supportedStats) {
+      for (String statType : StatsSetupConst.supportedStats) {
         if (parameters.containsKey(statType)) {
           tblStats.setStat(statType, Long.parseLong(parameters.get(statType)));
         }
@@ -226,14 +223,14 @@ public class StatsTask extends Task<Stat
         // In case of a non-partitioned table, the key for stats temporary store is "rootDir"
         if (statsAggregator != null) {
           String aggKey = Utilities.getHashedStatsPrefix(work.getAggKey(), maxPrefixLength);
-          updateStats(collectableStats, tblStats, statsAggregator, parameters,
+          updateStats(StatsSetupConst.statsRequireCompute, tblStats, statsAggregator, parameters,
               aggKey, atomic);
           statsAggregator.cleanUp(aggKey);
         }
         // The collectable stats for the aggregator needs to be cleared.
         // For eg. if a file is being loaded, the old number of rows are not valid
         else if (work.isClearAggregatorStats()) {
-          for (String statType : collectableStats) {
+          for (String statType : StatsSetupConst.statsRequireCompute) {
             if (parameters.containsKey(statType)) {
               tblStats.setStat(statType, 0L);
             }
@@ -242,9 +239,10 @@ public class StatsTask extends Task<Stat
 
         // write table stats to metastore
         parameters = tTable.getParameters();
-        for (String statType : collectableStats) {
+        for (String statType : StatsSetupConst.statsRequireCompute) {
           parameters.put(statType, Long.toString(tblStats.getStat(statType)));
         }
+        parameters.put(StatsSetupConst.STATS_GENERATED_VIA_STATS_TASK, StatsSetupConst.TRUE);
         tTable.setParameters(parameters);
 
         String tableFullName = table.getDbName() + "." + table.getTableName();
@@ -269,7 +267,7 @@ public class StatsTask extends Task<Stat
           }
 
           Map<String, Long> currentValues = new HashMap<String, Long>();
-          for (String statType : supportedStats) {
+          for (String statType : StatsSetupConst.supportedStats) {
             Long val = parameters.containsKey(statType) ? Long.parseLong(parameters.get(statType))
                 : 0L;
             currentValues.put(statType, val);
@@ -288,11 +286,11 @@ public class StatsTask extends Task<Stat
           LOG.info("Stats aggregator : " + partitionID);
 
           if (statsAggregator != null) {
-            updateStats(collectableStats, newPartStats, statsAggregator,
+            updateStats(StatsSetupConst.statsRequireCompute, newPartStats, statsAggregator,
                 parameters, partitionID, atomic);
             statsAggregator.cleanUp(partitionID);
           } else {
-            for (String statType : collectableStats) {
+            for (String statType : StatsSetupConst.statsRequireCompute) {
               // The collectable stats for the aggregator needs to be cleared.
               // For eg. if a file is being loaded, the old number of rows are not valid
               if (work.isClearAggregatorStats()) {
@@ -320,13 +318,14 @@ public class StatsTask extends Task<Stat
           //
           // update the metastore
           //
-          for (String statType : supportedStats) {
+          for (String statType : StatsSetupConst.supportedStats) {
             long statValue = newPartStats.getStat(statType);
             if (statValue >= 0) {
               parameters.put(statType, Long.toString(newPartStats.getStat(statType)));
             }
           }
 
+          parameters.put(StatsSetupConst.STATS_GENERATED_VIA_STATS_TASK, StatsSetupConst.TRUE);
           tPart.setParameters(parameters);
           String tableFullName = table.getDbName() + "." + table.getTableName();
           db.alterPartition(tableFullName, new Partition(table, tPart));
@@ -364,7 +363,7 @@ public class StatsTask extends Task<Stat
         || parameters.containsKey(StatsSetupConst.NUM_PARTITIONS);
   }
 
-  private void updateStats(List<String> statsList, Statistics stats,
+  private void updateStats(String[] statsList, Statistics stats,
       StatsAggregator statsAggregator, Map<String, String> parameters,
       String aggKey, boolean atomic) throws HiveException {
 

Modified: hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/metadata/Hive.java
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/metadata/Hive.java?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/metadata/Hive.java (original)
+++ hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/metadata/Hive.java Fri Nov  8 15:56:27 2013
@@ -1567,8 +1567,8 @@ private void constructOneLBLocationMap(F
     if (tpartParams == null) {
       return;
     }
-    List<String> statTypes = StatsSetupConst.getSupportedStats();
-    for (String statType : statTypes) {
+
+    for (String statType : StatsSetupConst.supportedStats) {
       tpartParams.remove(statType);
     }
   }

Modified: hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/optimizer/StatsOptimizer.java
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/optimizer/StatsOptimizer.java?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/optimizer/StatsOptimizer.java (original)
+++ hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/optimizer/StatsOptimizer.java Fri Nov  8 15:56:27 2013
@@ -4,6 +4,7 @@ import java.util.ArrayList;
 import java.util.LinkedHashMap;
 import java.util.List;
 import java.util.Map;
+import java.util.Set;
 import java.util.Stack;
 
 import org.apache.commons.logging.Log;
@@ -216,13 +217,15 @@ public class StatsOptimizer implements T
               String colName = desc.getColumn();
               StatType type = getType(desc.getTypeString());
               if(!tbl.isPartitioned()) {
-                rowCnt = Long.parseLong(tbl.getProperty(StatsSetupConst.ROW_COUNT));
+                if (!StatsSetupConst.areStatsUptoDate(tbl.getParameters())) {
+                  Log.debug("Stats for table : " + tbl.getTableName() + " are not upto date.");
+                  return null;
+                  }
+            	rowCnt = Long.parseLong(tbl.getProperty(StatsSetupConst.ROW_COUNT));
                 if (rowCnt < 1) {
                   Log.debug("Table doesn't have upto date stats " + tbl.getTableName());
                   return null;
                 }
-                //TODO: After HIVE-3777 use the property to figure out if following
-                // stats is fresh or not.
                 ColumnStatisticsData statData = hive.getMSC().getTableColumnStatistics(
                     tbl.getDbName(),tbl.getTableName(),colName).
                     getStatsObjIterator().next().getStatsData();
@@ -236,18 +239,20 @@ public class StatsOptimizer implements T
                 }
               } else {
                 for (Partition part : hive.getAllPartitionsOf(tbl)) {
-                  Long partRowCnt = Long.parseLong(part.getParameters()
+                  if (!StatsSetupConst.areStatsUptoDate(part.getParameters())) {
+                    Log.debug("Stats for part : " + part.getSpec() + " are not upto date.");
+                    return null;
+                    }
+                	Long partRowCnt = Long.parseLong(part.getParameters()
                     .get(StatsSetupConst.ROW_COUNT));
                   if (partRowCnt < 1) {
                     Log.debug("Partition doesn't have upto date stats " + part.getSpec());
                     return null;
                   }
                   rowCnt += partRowCnt;
-                  //TODO: After HIVE-3777 use the property to figure out if following
-                  // stats is fresh or not.
                   ColumnStatisticsData statData = hive.getMSC().getPartitionColumnStatistics(
-                      tbl.getDbName(), tbl.getTableName(),part.getName(), colName)
-                      .getStatsObjIterator().next().getStatsData();
+                    tbl.getDbName(), tbl.getTableName(),part.getName(), colName)
+                    .getStatsObjIterator().next().getStatsData();
                   Long nullCnt = getNullcountFor(type, statData);
                   if(nullCnt == null) {
                     Log.debug("Unsupported type: " + desc.getTypeString() + " encountered in " +
@@ -268,11 +273,13 @@ public class StatsOptimizer implements T
             String colName = colDesc.getColumn();
             StatType type = getType(colDesc.getTypeString());
             if(!tbl.isPartitioned()) {
-              //TODO: After HIVE-3777 use the property to figure out if following
-              // stats is fresh or not.
+              if (!StatsSetupConst.areStatsUptoDate(tbl.getParameters())) {
+                  Log.debug("Stats for table : " + tbl.getTableName() + " are not upto date.");
+                  return null;
+                  }
               ColumnStatisticsData statData = hive.getMSC().getTableColumnStatistics(
-                  tbl.getDbName(),tbl.getTableName(),colName).
-                  getStatsObjIterator().next().getStatsData();
+                tbl.getDbName(),tbl.getTableName(),colName).
+                getStatsObjIterator().next().getStatsData();
               switch (type) {
               case Integeral:
                 oneRow.add(statData.getLongStats().getHighValue());
@@ -291,16 +298,17 @@ public class StatsOptimizer implements T
                 return null;
               }
             } else {
-              List<String> parts = hive.getMSC().listPartitionNames(tbl.getDbName(),
-                  tbl.getTableName(), (short)-1);
+              Set<Partition> parts = hive.getAllPartitionsOf(tbl);
               switch(type) {
               case Integeral: {
                 long maxVal = Long.MIN_VALUE;
-                for (String part : parts) {
-                  //TODO: After HIVE-3777 use the property to figure out if following
-                  // stats is fresh or not.
-                  ColumnStatisticsData statData = hive.getMSC().getPartitionColumnStatistics(
-                      tbl.getDbName(),tbl.getTableName(), part, colName).
+                for (Partition part : parts) {
+                  if (!StatsSetupConst.areStatsUptoDate(part.getParameters())) {
+                    Log.debug("Stats for part : " + part.getSpec() + " are not upto date.");
+                    return null;
+                    }
+                	ColumnStatisticsData statData = hive.getMSC().getPartitionColumnStatistics(
+                      tbl.getDbName(),tbl.getTableName(), part.getName(), colName).
                       getStatsObjIterator().next().getStatsData();
                   maxVal = Math.max(maxVal,statData.getLongStats().getHighValue());
                 }
@@ -311,12 +319,14 @@ public class StatsOptimizer implements T
               }
               case Double: {
                 double maxVal = Double.MIN_VALUE;
-                for (String part : parts) {
-                  //TODO: After HIVE-3777 use the property to figure out if following
-                  // stats is fresh or not.
+                for (Partition part : parts) {
+                  if (!StatsSetupConst.areStatsUptoDate(part.getParameters())) {
+                    Log.debug("Stats for part : " + part.getSpec() + " are not upto date.");
+                    return null;
+                    }
                   ColumnStatisticsData statData = hive.getMSC().getPartitionColumnStatistics(
-                      tbl.getDbName(),tbl.getTableName(), part, colName).
-                      getStatsObjIterator().next().getStatsData();
+                    tbl.getDbName(),tbl.getTableName(), part.getName(), colName).
+                    getStatsObjIterator().next().getStatsData();
                   maxVal = Math.max(maxVal,statData.getDoubleStats().getHighValue());
                 }
                 oneRow.add(maxVal);
@@ -336,11 +346,13 @@ public class StatsOptimizer implements T
             String colName = colDesc.getColumn();
             StatType type = getType(colDesc.getTypeString());
             if (!tbl.isPartitioned()) {
-              //TODO: After HIVE-3777 use the property to figure out if following
-              // stats is fresh or not.
+              if (!StatsSetupConst.areStatsUptoDate(tbl.getParameters())) {
+                Log.debug("Stats for table : " + tbl.getTableName() + " are not upto date.");
+                return null;
+                }
               ColumnStatisticsData statData = hive.getMSC().getTableColumnStatistics(
-                  tbl.getDbName(),tbl.getTableName(),colName).
-                  getStatsObjIterator().next().getStatsData();
+                tbl.getDbName(),tbl.getTableName(),colName).
+                getStatsObjIterator().next().getStatsData();
               switch (type) {
               case Integeral:
                 oneRow.add(statData.getLongStats().getLowValue());
@@ -358,17 +370,18 @@ public class StatsOptimizer implements T
                 return null;
               }
             } else {
-              List<String> parts = hive.getMSC().listPartitionNames(tbl.getDbName(),
-                  tbl.getTableName(), (short)-1);
+              Set<Partition> parts = hive.getAllPartitionsOf(tbl);
               switch(type) {
               case Integeral: {
                 long minVal = Long.MAX_VALUE;
-                for (String part : parts) {
-                  //TODO: After HIVE-3777 use the property to figure out if following
-                  // stats is fresh or not.
+                for (Partition part : parts) {
+                  if (!StatsSetupConst.areStatsUptoDate(part.getParameters())) {
+                    Log.debug("Stats for part : " + part.getSpec() + " are not upto date.");
+                    return null;
+                    }
                   ColumnStatisticsData statData = hive.getMSC().getPartitionColumnStatistics(
-                      tbl.getDbName(),tbl.getTableName(), part, colName).
-                      getStatsObjIterator().next().getStatsData();
+                    tbl.getDbName(),tbl.getTableName(), part.getName(), colName).
+                    getStatsObjIterator().next().getStatsData();
                   minVal = Math.min(minVal,statData.getLongStats().getLowValue());
                 }
                 oneRow.add(minVal);
@@ -378,12 +391,14 @@ public class StatsOptimizer implements T
               }
               case Double: {
                 double minVal = Double.MAX_VALUE;
-                for (String part : parts) {
-                  //TODO: After HIVE-3777 use the property to figure out if following
-                  // stats is fresh or not.
+                for (Partition part : parts) {
+                  if (!StatsSetupConst.areStatsUptoDate(part.getParameters())) {
+                    Log.debug("Stats for part : " + part.getSpec() + " are not upto date.");
+                    return null;
+                    }
                   ColumnStatisticsData statData = hive.getMSC().getPartitionColumnStatistics(
-                      tbl.getDbName(),tbl.getTableName(), part, colName).
-                      getStatsObjIterator().next().getStatsData();
+                    tbl.getDbName(),tbl.getTableName(), part.getName(), colName).
+                    getStatsObjIterator().next().getStatsData();
                   minVal = Math.min(minVal,statData.getDoubleStats().getLowValue());
                 }
                 oneRow.add(minVal);

Added: hive/trunk/ql/src/test/queries/clientpositive/stats_invalidation.q
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/queries/clientpositive/stats_invalidation.q?rev=1540094&view=auto
==============================================================================
--- hive/trunk/ql/src/test/queries/clientpositive/stats_invalidation.q (added)
+++ hive/trunk/ql/src/test/queries/clientpositive/stats_invalidation.q Fri Nov  8 15:56:27 2013
@@ -0,0 +1,15 @@
+set hive.stats.autogather=true;
+
+CREATE TABLE stats_invalid (key string, value string);
+
+insert overwrite table stats_invalid
+select * from src;
+
+analyze table stats_invalid compute statistics for columns key,value;
+
+desc formatted stats_invalid;
+alter table stats_invalid add  columns (new_col string);
+
+desc formatted stats_invalid;
+drop table stats_invalid;
+

Modified: hive/trunk/ql/src/test/queries/clientpositive/stats_only_null.q
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/queries/clientpositive/stats_only_null.q?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/queries/clientpositive/stats_only_null.q (original)
+++ hive/trunk/ql/src/test/queries/clientpositive/stats_only_null.q Fri Nov  8 15:56:27 2013
@@ -9,9 +9,9 @@ CREATE TABLE stats_null_part(a double, b
 LOAD DATA LOCAL INPATH '../../data/files/null.txt' INTO TABLE temps_null;
 
 insert overwrite table stats_null select * from temps_null;
-insert overwrite table stats_null_part partition(dt='2010') select * from temps_null where d <=5;
+insert into table stats_null_part partition(dt='2010') select * from temps_null where d <=5;
 
-insert overwrite table stats_null_part partition(dt='2011') select * from temps_null where d > 5;
+insert into table stats_null_part partition(dt='2011') select * from temps_null where d > 5;
 explain 
 select count(*), count(a), count(b), count(c), count(d) from stats_null;
 explain 
@@ -22,6 +22,9 @@ analyze table stats_null compute statist
 analyze table stats_null_part partition(dt='2010') compute statistics for columns a,b,c,d;
 analyze table stats_null_part partition(dt='2011') compute statistics for columns a,b,c,d;
 
+describe formatted stats_null_part partition (dt='2010');
+describe formatted stats_null_part partition (dt='2011');
+
 explain 
 select count(*), count(a), count(b), count(c), count(d) from stats_null;
 explain 

Modified: hive/trunk/ql/src/test/results/clientnegative/stats_partialscan_autogether.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientnegative/stats_partialscan_autogether.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientnegative/stats_partialscan_autogether.q.out (original)
+++ hive/trunk/ql/src/test/results/clientnegative/stats_partialscan_autogether.q.out Fri Nov  8 15:56:27 2013
@@ -71,7 +71,10 @@ Table:              	analyze_srcpart_par
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	false               
 	numFiles            	1                   
+	numRows             	-1                  
+	rawDataSize         	-1                  
 	totalSize           	5293                
 #### A masked pattern was here ####
 	 	 

Modified: hive/trunk/ql/src/test/results/clientnegative/unset_table_property.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientnegative/unset_table_property.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientnegative/unset_table_property.q.out (original)
+++ hive/trunk/ql/src/test/results/clientnegative/unset_table_property.q.out Fri Nov  8 15:56:27 2013
@@ -21,8 +21,11 @@ numFiles	0
 c	3
 #### A masked pattern was here ####
 a	1
+COLUMN_STATS_ACCURATE	false
 #### A masked pattern was here ####
+numRows	-1
 totalSize	0
+rawDataSize	-1
 FAILED: SemanticException [Error 10215]: Please use the following syntax if not sure whether the property existed or not:
 ALTER TABLE tableName UNSET TBLPROPERTIES IF EXISTS (key1, key2, ...)
  The following property z does not exist in testtable

Modified: hive/trunk/ql/src/test/results/clientpositive/alter_numbuckets_partitioned_table.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/alter_numbuckets_partitioned_table.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/alter_numbuckets_partitioned_table.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/alter_numbuckets_partitioned_table.q.out Fri Nov  8 15:56:27 2013
@@ -79,6 +79,7 @@ Table:              	tst1               
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -147,6 +148,7 @@ Table:              	tst1               
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -280,6 +282,7 @@ Table:              	tst1               
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -372,10 +375,11 @@ Table:              	tst1               
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	false               
 #### A masked pattern was here ####
 	numFiles            	1                   
-	numRows             	500                 
-	rawDataSize         	5312                
+	numRows             	-1                  
+	rawDataSize         	-1                  
 	totalSize           	5812                
 #### A masked pattern was here ####
 	 	 

Modified: hive/trunk/ql/src/test/results/clientpositive/alter_numbuckets_partitioned_table2.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/alter_numbuckets_partitioned_table2.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/alter_numbuckets_partitioned_table2.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/alter_numbuckets_partitioned_table2.q.out Fri Nov  8 15:56:27 2013
@@ -77,6 +77,7 @@ Table:              	tst1               
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -179,6 +180,7 @@ Table:              	tst1               
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -289,6 +291,7 @@ Table:              	tst1               
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -407,6 +410,7 @@ Table:              	tst1               
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -533,6 +537,7 @@ Table:              	tst1               
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -667,6 +672,7 @@ Table:              	tst1               
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -809,6 +815,7 @@ Table:              	tst1               
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -959,6 +966,7 @@ Table:              	tst1               
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -1117,6 +1125,7 @@ Table:              	tst1               
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                

Modified: hive/trunk/ql/src/test/results/clientpositive/alter_partition_clusterby_sortby.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/alter_partition_clusterby_sortby.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/alter_partition_clusterby_sortby.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/alter_partition_clusterby_sortby.q.out Fri Nov  8 15:56:27 2013
@@ -45,8 +45,11 @@ Table:              	alter_table_partiti
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	false               
 #### A masked pattern was here ####
 	numFiles            	0                   
+	numRows             	-1                  
+	rawDataSize         	-1                  
 	totalSize           	0                   
 #### A masked pattern was here ####
 	 	 
@@ -95,8 +98,11 @@ Table:              	alter_table_partiti
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	false               
 #### A masked pattern was here ####
 	numFiles            	0                   
+	numRows             	-1                  
+	rawDataSize         	-1                  
 	totalSize           	0                   
 #### A masked pattern was here ####
 	 	 
@@ -145,8 +151,11 @@ Table:              	alter_table_partiti
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	false               
 #### A masked pattern was here ####
 	numFiles            	0                   
+	numRows             	-1                  
+	rawDataSize         	-1                  
 	totalSize           	0                   
 #### A masked pattern was here ####
 	 	 

Modified: hive/trunk/ql/src/test/results/clientpositive/alter_partition_coltype.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/alter_partition_coltype.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/alter_partition_coltype.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/alter_partition_coltype.q.out Fri Nov  8 15:56:27 2013
@@ -138,6 +138,7 @@ STAGE PLANS:
               dt 100x
               ts 6:30pm
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count -1
               columns key,value
               columns.types string:string
@@ -333,6 +334,7 @@ STAGE PLANS:
               dt 100x
               ts 6:30pm
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count -1
               columns key,value
               columns.types string:string
@@ -486,6 +488,7 @@ STAGE PLANS:
               dt 10
               ts 3.0
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count -1
               columns key,value
               columns.types string:string
@@ -527,6 +530,7 @@ STAGE PLANS:
               dt 100x
               ts 3.0
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count -1
               columns key,value
               columns.types string:string
@@ -568,6 +572,7 @@ STAGE PLANS:
               dt 100x
               ts 6:30pm
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count -1
               columns key,value
               columns.types string:string
@@ -808,6 +813,7 @@ STAGE PLANS:
               dt 10
               ts 3.0
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count -1
               columns key,value
               columns.types string:string
@@ -849,6 +855,7 @@ STAGE PLANS:
               dt 100x
               ts 3.0
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count -1
               columns key,value
               columns.types string:string
@@ -890,6 +897,7 @@ STAGE PLANS:
               dt 100x
               ts 6:30pm
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count -1
               columns key,value
               columns.types string:string
@@ -1026,6 +1034,7 @@ STAGE PLANS:
               dt 10
               ts 3.0
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count -1
               columns key,value
               columns.types string:string
@@ -1067,6 +1076,7 @@ STAGE PLANS:
               dt 100x
               ts 3.0
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count -1
               columns key,value
               columns.types string:string
@@ -1108,6 +1118,7 @@ STAGE PLANS:
               dt 100x
               ts 6:30pm
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count -1
               columns key,value
               columns.types string:string

Modified: hive/trunk/ql/src/test/results/clientpositive/alter_skewed_table.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/alter_skewed_table.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/alter_skewed_table.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/alter_skewed_table.q.out Fri Nov  8 15:56:27 2013
@@ -57,8 +57,11 @@ Retention:          	0                  
 #### A masked pattern was here ####
 Table Type:         	MANAGED_TABLE       	 
 Table Parameters:	 	 
+	COLUMN_STATS_ACCURATE	false               
 #### A masked pattern was here ####
 	numFiles            	0                   
+	numRows             	-1                  
+	rawDataSize         	-1                  
 	totalSize           	0                   
 #### A masked pattern was here ####
 	 	 
@@ -141,8 +144,11 @@ Retention:          	0                  
 #### A masked pattern was here ####
 Table Type:         	MANAGED_TABLE       	 
 Table Parameters:	 	 
+	COLUMN_STATS_ACCURATE	false               
 #### A masked pattern was here ####
 	numFiles            	0                   
+	numRows             	-1                  
+	rawDataSize         	-1                  
 	totalSize           	0                   
 #### A masked pattern was here ####
 	 	 
@@ -227,8 +233,11 @@ Retention:          	0                  
 #### A masked pattern was here ####
 Table Type:         	MANAGED_TABLE       	 
 Table Parameters:	 	 
+	COLUMN_STATS_ACCURATE	false               
 #### A masked pattern was here ####
 	numFiles            	0                   
+	numRows             	-1                  
+	rawDataSize         	-1                  
 	totalSize           	0                   
 #### A masked pattern was here ####
 	 	 

Modified: hive/trunk/ql/src/test/results/clientpositive/alter_table_not_sorted.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/alter_table_not_sorted.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/alter_table_not_sorted.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/alter_table_not_sorted.q.out Fri Nov  8 15:56:27 2013
@@ -58,9 +58,12 @@ Retention:          	0                  
 #### A masked pattern was here ####
 Table Type:         	MANAGED_TABLE       	 
 Table Parameters:	 	 
+	COLUMN_STATS_ACCURATE	false               
 	SORTBUCKETCOLSPREFIX	TRUE                
 #### A masked pattern was here ####
 	numFiles            	0                   
+	numRows             	-1                  
+	rawDataSize         	-1                  
 	totalSize           	0                   
 #### A masked pattern was here ####
 	 	 

Modified: hive/trunk/ql/src/test/results/clientpositive/alter_table_serde2.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/alter_table_serde2.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/alter_table_serde2.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/alter_table_serde2.q.out Fri Nov  8 15:56:27 2013
@@ -77,6 +77,7 @@ Table:              	tst1               
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                
@@ -182,6 +183,7 @@ Table:              	tst1               
 Protect Mode:       	None                	 
 #### A masked pattern was here ####
 Partition Parameters:	 	 
+	COLUMN_STATS_ACCURATE	true                
 	numFiles            	1                   
 	numRows             	500                 
 	rawDataSize         	5312                

Modified: hive/trunk/ql/src/test/results/clientpositive/auto_join_reordering_values.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/auto_join_reordering_values.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/auto_join_reordering_values.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/auto_join_reordering_values.q.out Fri Nov  8 15:56:27 2013
@@ -153,6 +153,7 @@ STAGE PLANS:
             input format: org.apache.hadoop.mapred.TextInputFormat
             output format: org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count -1
               columns dealid,date,time,cityid,userid
               columns.types int:string:string:int:int
@@ -171,6 +172,7 @@ STAGE PLANS:
               input format: org.apache.hadoop.mapred.TextInputFormat
               output format: org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat
               properties:
+                COLUMN_STATS_ACCURATE true
                 bucket_count -1
                 columns dealid,date,time,cityid,userid
                 columns.types int:string:string:int:int
@@ -284,6 +286,7 @@ STAGE PLANS:
             input format: org.apache.hadoop.mapred.TextInputFormat
             output format: org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count -1
               columns dealid,date,time,cityid,userid
               columns.types int:string:string:int:int
@@ -302,6 +305,7 @@ STAGE PLANS:
               input format: org.apache.hadoop.mapred.TextInputFormat
               output format: org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat
               properties:
+                COLUMN_STATS_ACCURATE true
                 bucket_count -1
                 columns dealid,date,time,cityid,userid
                 columns.types int:string:string:int:int
@@ -413,6 +417,7 @@ STAGE PLANS:
             input format: org.apache.hadoop.mapred.TextInputFormat
             output format: org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count -1
               columns dealid,date,time,cityid,userid
               columns.types int:string:string:int:int
@@ -431,6 +436,7 @@ STAGE PLANS:
               input format: org.apache.hadoop.mapred.TextInputFormat
               output format: org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat
               properties:
+                COLUMN_STATS_ACCURATE true
                 bucket_count -1
                 columns dealid,date,time,cityid,userid
                 columns.types int:string:string:int:int
@@ -540,6 +546,7 @@ STAGE PLANS:
             input format: org.apache.hadoop.mapred.TextInputFormat
             output format: org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count -1
               columns userid
               columns.types int
@@ -558,6 +565,7 @@ STAGE PLANS:
               input format: org.apache.hadoop.mapred.TextInputFormat
               output format: org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat
               properties:
+                COLUMN_STATS_ACCURATE true
                 bucket_count -1
                 columns userid
                 columns.types int

Modified: hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_1.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_1.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_1.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_1.q.out Fri Nov  8 15:56:27 2013
@@ -133,6 +133,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -176,6 +177,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-09
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -324,6 +326,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -367,6 +370,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-09
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -494,6 +498,7 @@ STAGE PLANS:
                   partition values:
                     ds 2008-04-08
                   properties:
+                    COLUMN_STATS_ACCURATE true
                     bucket_count 2
                     bucket_field_name key
                     columns key,value
@@ -588,6 +593,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -631,6 +637,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-09
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -673,6 +680,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value
@@ -759,6 +767,7 @@ STAGE PLANS:
                   partition values:
                     ds 2008-04-08
                   properties:
+                    COLUMN_STATS_ACCURATE true
                     bucket_count 4
                     bucket_field_name key
                     columns key,value
@@ -801,6 +810,7 @@ STAGE PLANS:
                   partition values:
                     ds 2008-04-09
                   properties:
+                    COLUMN_STATS_ACCURATE true
                     bucket_count 4
                     bucket_field_name key
                     columns key,value
@@ -895,6 +905,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -938,6 +949,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-09
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -980,6 +992,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value
@@ -1093,6 +1106,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -1136,6 +1150,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-09
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value

Modified: hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_11.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_11.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_11.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_11.q.out Fri Nov  8 15:56:27 2013
@@ -102,6 +102,7 @@ STAGE PLANS:
                   partition values:
                     ds 2008-04-08
                   properties:
+                    COLUMN_STATS_ACCURATE true
                     bucket_count 2
                     bucket_field_name key
                     columns key,value
@@ -195,6 +196,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -237,6 +239,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-09
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -279,6 +282,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value
@@ -407,6 +411,7 @@ STAGE PLANS:
                   partition values:
                     ds 2008-04-08
                   properties:
+                    COLUMN_STATS_ACCURATE true
                     bucket_count 2
                     bucket_field_name key
                     columns key,value
@@ -500,6 +505,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -542,6 +548,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-09
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -584,6 +591,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value
@@ -705,6 +713,7 @@ STAGE PLANS:
                   partition values:
                     ds 2008-04-08
                   properties:
+                    COLUMN_STATS_ACCURATE true
                     bucket_count 2
                     bucket_field_name key
                     columns key,value
@@ -806,6 +815,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -848,6 +858,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-09
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value

Modified: hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_12.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_12.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_12.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_12.q.out Fri Nov  8 15:56:27 2013
@@ -130,6 +130,7 @@ STAGE PLANS:
                   partition values:
                     ds 2008-04-08
                   properties:
+                    COLUMN_STATS_ACCURATE true
                     bucket_count 2
                     bucket_field_name key
                     columns key,value
@@ -176,6 +177,7 @@ STAGE PLANS:
                   partition values:
                     ds 2008-04-08
                   properties:
+                    COLUMN_STATS_ACCURATE true
                     bucket_count 3
                     bucket_field_name key
                     columns key,value
@@ -222,6 +224,7 @@ STAGE PLANS:
                   partition values:
                     ds 2008-04-08
                   properties:
+                    COLUMN_STATS_ACCURATE true
                     bucket_count 3
                     bucket_field_name key
                     columns key,value
@@ -360,6 +363,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -403,6 +407,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-09
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -446,6 +451,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 3
               bucket_field_name key
               columns key,value
@@ -489,6 +495,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value

Modified: hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_2.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_2.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_2.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_2.q.out Fri Nov  8 15:56:27 2013
@@ -117,6 +117,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value
@@ -160,6 +161,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-09
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value
@@ -289,6 +291,7 @@ STAGE PLANS:
                   partition values:
                     ds 2008-04-08
                   properties:
+                    COLUMN_STATS_ACCURATE true
                     bucket_count 4
                     bucket_field_name key
                     columns key,value
@@ -383,6 +386,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value
@@ -426,6 +430,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-09
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value
@@ -468,6 +473,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -554,6 +560,7 @@ STAGE PLANS:
                   partition values:
                     ds 2008-04-08
                   properties:
+                    COLUMN_STATS_ACCURATE true
                     bucket_count 2
                     bucket_field_name key
                     columns key,value
@@ -596,6 +603,7 @@ STAGE PLANS:
                   partition values:
                     ds 2008-04-09
                   properties:
+                    COLUMN_STATS_ACCURATE true
                     bucket_count 2
                     bucket_field_name key
                     columns key,value
@@ -690,6 +698,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value
@@ -733,6 +742,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-09
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value
@@ -775,6 +785,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -888,6 +899,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value
@@ -931,6 +943,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-09
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value

Modified: hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_3.q.out
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_3.q.out?rev=1540094&r1=1540093&r2=1540094&view=diff
==============================================================================
--- hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_3.q.out (original)
+++ hive/trunk/ql/src/test/results/clientpositive/auto_sortmerge_join_3.q.out Fri Nov  8 15:56:27 2013
@@ -117,6 +117,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -264,6 +265,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -390,6 +392,7 @@ STAGE PLANS:
                   partition values:
                     ds 2008-04-08
                   properties:
+                    COLUMN_STATS_ACCURATE true
                     bucket_count 2
                     bucket_field_name key
                     columns key,value
@@ -431,6 +434,7 @@ STAGE PLANS:
                   partition values:
                     ds 2008-04-08
                   properties:
+                    COLUMN_STATS_ACCURATE true
                     bucket_count 2
                     bucket_field_name key
                     columns key,value
@@ -525,6 +529,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -567,6 +572,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value
@@ -609,6 +615,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value
@@ -694,6 +701,7 @@ STAGE PLANS:
                   partition values:
                     ds 2008-04-08
                   properties:
+                    COLUMN_STATS_ACCURATE true
                     bucket_count 4
                     bucket_field_name key
                     columns key,value
@@ -788,6 +796,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value
@@ -830,6 +839,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value
@@ -872,6 +882,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 2
               bucket_field_name key
               columns key,value
@@ -986,6 +997,7 @@ STAGE PLANS:
             partition values:
               ds 2008-04-08
             properties:
+              COLUMN_STATS_ACCURATE true
               bucket_count 4
               bucket_field_name key
               columns key,value