You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@impala.apache.org by jo...@apache.org on 2020/03/18 20:51:46 UTC

[impala] branch master updated: IMPALA-5308: Resolve confusing Kudu SHOW TABLE STATS output

This is an automated email from the ASF dual-hosted git repository.

joemcdonnell pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/impala.git


The following commit(s) were added to refs/heads/master by this push:
     new 7dd13f7  IMPALA-5308: Resolve confusing Kudu SHOW TABLE STATS output
7dd13f7 is described below

commit 7dd13f72784514a59f82c9a7a5e2250503dbfaf0
Author: Tamas Mate <tm...@cloudera.com>
AuthorDate: Tue Mar 3 13:09:50 2020 +0100

    IMPALA-5308: Resolve confusing Kudu SHOW TABLE STATS output
    
    This change modifies the output of the SHOW TABLE STATS and SHOW
    PARTITIONS for Kudu tables.
     - PARTITIONS: the #Row column has been removed
     - TABLE STATS: instead of showing partition informations it returns a
     resultset similar to HDFS table stats, #Rows, #Partitions, Size, Format
     and Location
    
    Example outputs can be seen in the doc changes.
    
    Testing:
    * kudu_stats.test is modified to verify the new result set
    * kudu_partition_ddl.test is modified to verify the new partitions style
    * Updated unit test with the new error message
    
    Change-Id: Ice4b8df65f0a53fe14b8fbe35d82c9887ab9a041
    Reviewed-on: http://gerrit.cloudera.org:8080/15199
    Reviewed-by: Thomas Tauber-Marshall <tm...@cloudera.com>
    Tested-by: Impala Public Jenkins <im...@cloudera.com>
---
 docs/topics/impala_compute_stats.xml               |   7 +-
 docs/topics/impala_show.xml                        |  76 ++++-----
 .../org/apache/impala/analysis/ShowStatsStmt.java  |   4 +-
 .../org/apache/impala/catalog/FeKuduTable.java     |  43 +++++-
 .../java/org/apache/impala/service/Frontend.java   |   3 +
 .../org/apache/impala/analysis/AnalyzeDDLTest.java |   2 +-
 .../queries/QueryTest/kudu_partition_ddl.test      | 170 ++++++++++-----------
 .../queries/QueryTest/kudu_stats.test              |  34 ++++-
 8 files changed, 190 insertions(+), 149 deletions(-)

diff --git a/docs/topics/impala_compute_stats.xml b/docs/topics/impala_compute_stats.xml
index 95343f4..487691c 100644
--- a/docs/topics/impala_compute_stats.xml
+++ b/docs/topics/impala_compute_stats.xml
@@ -485,11 +485,8 @@ Returned 2 row(s) in 0.01s</codeblock>
 
     <p rev="IMPALA-2830">
       The <codeph>COMPUTE STATS</codeph> statement applies to Kudu tables.
-      Impala does not compute the number of rows for each partition for
-      Kudu tables. Therefore, you do not need to re-run the operation when
-      you see -1 in the <codeph># Rows</codeph> column of the output from
-      <codeph>SHOW TABLE STATS</codeph>. That column always shows -1 for
-      all Kudu tables.
+      Impala only computes the number of rows for the whole Kudu table,
+      partition level row counts are not available.
     </p>
 
     <p conref="../shared/impala_common.xml#common/related_info"/>
diff --git a/docs/topics/impala_show.xml b/docs/topics/impala_show.xml
index 414a7d4..0e40717 100644
--- a/docs/topics/impala_show.xml
+++ b/docs/topics/impala_show.xml
@@ -969,35 +969,20 @@ show create table show_create_table_demo;
 
       <p rev="kudu IMPALA-2830">
         Because Kudu tables do not have characteristics derived from HDFS, such
-        as number of files, file format, and HDFS cache status, the output of
+        as number of files and HDFS cache status, the output of
         <codeph>SHOW TABLE STATS</codeph> reflects different characteristics
-        that apply to Kudu tables. If the Kudu table is created with the
-        clause <codeph>PARTITIONS 20</codeph>, then the result set of
-        <codeph>SHOW TABLE STATS</codeph> consists of 20 rows, each representing
-        one of the numbered partitions. For example:
+        that apply to Kudu tables.
       </p>
 
 <codeblock rev="kudu IMPALA-2830">
 show table stats kudu_table;
-+--------+-----------+----------+-----------------------+------------+
-| # Rows | Start Key | Stop Key | Leader Replica        | # Replicas |
-+--------+-----------+----------+-----------------------+------------+
-| -1     |           | 00000001 | host.example.com:7050 | 3          |
-| -1     | 00000001  | 00000002 | host.example.com:7050 | 3          |
-| -1     | 00000002  | 00000003 | host.example.com:7050 | 3          |
-| -1     | 00000003  | 00000004 | host.example.com:7050 | 3          |
-| -1     | 00000004  | 00000005 | host.example.com:7050 | 3          |
-...
++-------+-------------+---------+--------+------------------------------+
+| #Rows | #Partitions | Size    | Format | Location                     |
++-------+-------------+---------+--------+------------------------------+
+| 100   | 7           | 56.16MB | KUDU   | kudu-master.example.com:7051 |
++-------+-------------+---------+--------+------------------------------+
 </codeblock>
 
-      <p rev="IMPALA-2830">
-        Impala does not compute the number of rows for each partition for
-        Kudu tables. Therefore, you do not need to re-run <codeph>COMPUTE STATS</codeph>
-        when you see -1 in the <codeph># Rows</codeph> column of the output from
-        <codeph>SHOW TABLE STATS</codeph>. That column always shows -1 for
-        all Kudu tables.
-      </p>
-
       <p conref="../shared/impala_common.xml#common/example_blurb"/>
 
       <p>
@@ -1337,34 +1322,39 @@ show column stats store_sales;
 </codeblock>
 
       <p rev="kudu IMPALA-4403">
-        The following example shows the output for a Kudu table
-        using the hash partitioning mechanism. The number of
-        rows in the result set corresponds to the values used
-        in the <codeph>PARTITIONS <varname>N</varname></codeph>
-        clause of <codeph>CREATE TABLE</codeph>.
+        The following example shows the output for a Kudu table using the hash
+        partitioning mechanism. The number of rows in the result set corresponds to the
+        values used in the <codeph>PARTITIONS <varname>N</varname></codeph> clause of
+        <codeph>CREATE TABLE</codeph>. If the Kudu table is created with the clause
+        <codeph>PARTITIONS 20</codeph>, then the result set of
+        <codeph>SHOW PARTITIONS</codeph> consists of 20 rows, each representing one of
+        the numbered partitions. For example:
       </p>
 
 <codeblock rev="kudu IMPALA-4403"><![CDATA[
 show partitions million_rows_hash;
-
-+--------+-----------+----------+-----------------------+--
-| # Rows | Start Key | Stop Key | Leader Replica        | # Replicas
-+--------+-----------+----------+-----------------------+--
-| -1     |           | 00000001 | n236.example.com:7050 | 3
-| -1     | 00000001  | 00000002 | n236.example.com:7050 | 3
-| -1     | 00000002  | 00000003 | n336.example.com:7050 | 3
-| -1     | 00000003  | 00000004 | n238.example.com:7050 | 3
-| -1     | 00000004  | 00000005 | n338.example.com:7050 | 3
-....
-| -1     | 0000002E  | 0000002F | n240.example.com:7050 | 3
-| -1     | 0000002F  | 00000030 | n336.example.com:7050 | 3
-| -1     | 00000030  | 00000031 | n240.example.com:7050 | 3
-| -1     | 00000031  |          | n334.example.com:7050 | 3
-+--------+-----------+----------+-----------------------+--
-Fetched 50 row(s) in 0.05s
++-----------+----------+-----------------------+------------+
+| Start Key | Stop Key | Leader Replica        | # Replicas |
++-----------+----------+-----------------------+------------+
+|           | 00000001 | n236.example.com:7050 | 3          |
+| 00000001  | 00000002 | n236.example.com:7050 | 3          |
+| 00000002  | 00000003 | n336.example.com:7050 | 3          |
+| 00000003  | 00000004 | n238.example.com:7050 | 3          |
+| 00000004  | 00000005 | n338.example.com:7050 | 3          |
+...                                                         |
+| 0000002E  | 0000002F | n240.example.com:7050 | 3          |
+| 0000002F  | 00000030 | n336.example.com:7050 | 3          |
+| 00000030  | 00000031 | n240.example.com:7050 | 3          |
+| 00000031  |          | n334.example.com:7050 | 3          |
++-----------+----------+-----------------------+------------+
 ]]>
 </codeblock>
 
+      <p rev="IMPALA-2830">
+        Impala only computes the number of rows for the whole Kudu table, partition level
+        row counts are not available.
+      </p>
+
       <p rev="kudu IMPALA-4403">
         The following example shows the output for a Kudu table
         using the range partitioning mechanism:
diff --git a/fe/src/main/java/org/apache/impala/analysis/ShowStatsStmt.java b/fe/src/main/java/org/apache/impala/analysis/ShowStatsStmt.java
index 49a43f4..14698eb 100644
--- a/fe/src/main/java/org/apache/impala/analysis/ShowStatsStmt.java
+++ b/fe/src/main/java/org/apache/impala/analysis/ShowStatsStmt.java
@@ -99,8 +99,8 @@ public class ShowStatsStmt extends StatementBase {
         throw new AnalysisException(getSqlPrefix() + " must target a Kudu table: " +
             table_.getFullName());
       } else if (op_ == TShowStatsOp.PARTITIONS) {
-        throw new AnalysisException(getSqlPrefix() + " must target an HDFS table: " +
-            table_.getFullName());
+        throw new AnalysisException(getSqlPrefix() +
+            " must target an HDFS or Kudu table: " + table_.getFullName());
       }
     }
   }
diff --git a/fe/src/main/java/org/apache/impala/catalog/FeKuduTable.java b/fe/src/main/java/org/apache/impala/catalog/FeKuduTable.java
index 9f04f25..f2ae9bc 100644
--- a/fe/src/main/java/org/apache/impala/catalog/FeKuduTable.java
+++ b/fe/src/main/java/org/apache/impala/catalog/FeKuduTable.java
@@ -132,11 +132,42 @@ public interface FeKuduTable extends FeTable {
       TResultSetMetadata resultSchema = new TResultSetMetadata();
       result.setSchema(resultSchema);
 
-      resultSchema.addToColumns(new TColumn("# Rows", Type.INT.toThrift()));
+      // Build column header
+      resultSchema.addToColumns(new TColumn("#Rows", Type.BIGINT.toThrift()));
+      resultSchema.addToColumns(new TColumn("#Partitions", Type.BIGINT.toThrift()));
+      resultSchema.addToColumns(new TColumn("Size", Type.STRING.toThrift()));
+      resultSchema.addToColumns(new TColumn("Format", Type.STRING.toThrift()));
+      resultSchema.addToColumns(new TColumn("Location", Type.STRING.toThrift()));
+
+      KuduClient client = KuduUtil.getKuduClient(table.getKuduMasterHosts());
+      try {
+        org.apache.kudu.client.KuduTable kuduTable =
+            client.openTable(table.getKuduTableName());
+        List<LocatedTablet> tablets = kuduTable.getTabletsLocations(
+            BackendConfig.INSTANCE.getKuduClientTimeoutMs());
+        TResultRowBuilder tResultRowBuilder = new TResultRowBuilder();
+        tResultRowBuilder.add(table.getNumRows());
+        tResultRowBuilder.add(tablets.size());
+        tResultRowBuilder.addBytes(kuduTable.getTableStatistics().getOnDiskSize());
+        tResultRowBuilder.add("KUDU");
+        tResultRowBuilder.add(table.getKuduMasterHosts());
+        result.addToRows(tResultRowBuilder.get());
+      } catch (Exception e) {
+        throw new ImpalaRuntimeException("Error accessing Kudu for table stats.", e);
+      }
+      return result;
+    }
+
+    public static TResultSet getPartitions(FeKuduTable table)
+        throws ImpalaRuntimeException {
+      TResultSet result = new TResultSet();
+      TResultSetMetadata resultSchema = new TResultSetMetadata();
+      result.setSchema(resultSchema);
+
       resultSchema.addToColumns(new TColumn("Start Key", Type.STRING.toThrift()));
       resultSchema.addToColumns(new TColumn("Stop Key", Type.STRING.toThrift()));
       resultSchema.addToColumns(new TColumn("Leader Replica", Type.STRING.toThrift()));
-      resultSchema.addToColumns(new TColumn("# Replicas", Type.INT.toThrift()));
+      resultSchema.addToColumns(new TColumn("#Replicas", Type.INT.toThrift()));
 
       KuduClient client = KuduUtil.getKuduClient(table.getKuduMasterHosts());
       try {
@@ -147,12 +178,11 @@ public interface FeKuduTable extends FeTable {
         if (tablets.isEmpty()) {
           TResultRowBuilder builder = new TResultRowBuilder();
           result.addToRows(
-              builder.add("-1").add("N/A").add("N/A").add("N/A").add("-1").get());
+              builder.add("N/A").add("N/A").add("N/A").add("-1").get());
           return result;
         }
         for (LocatedTablet tab: tablets) {
           TResultRowBuilder builder = new TResultRowBuilder();
-          builder.add("-1");   // The Kudu client API doesn't expose tablet row counts.
           builder.add(DatatypeConverter.printHexBinary(
               tab.getPartition().getPartitionKeyStart()));
           builder.add(DatatypeConverter.printHexBinary(
@@ -161,16 +191,15 @@ public interface FeKuduTable extends FeTable {
           if (leader == null) {
             // Leader might be null, if it is not yet available (e.g. during
             // leader election in Kudu)
-            builder.add("Leader n/a");
+            builder.add("Leader N/A");
           } else {
             builder.add(leader.getRpcHost() + ":" + leader.getRpcPort().toString());
           }
           builder.add(tab.getReplicas().size());
           result.addToRows(builder.get());
         }
-
       } catch (Exception e) {
-        throw new ImpalaRuntimeException("Error accessing Kudu for table stats.", e);
+        throw new ImpalaRuntimeException("Error accessing Kudu for table partitions.", e);
       }
       return result;
     }
diff --git a/fe/src/main/java/org/apache/impala/service/Frontend.java b/fe/src/main/java/org/apache/impala/service/Frontend.java
index 1144ca7..8189222 100644
--- a/fe/src/main/java/org/apache/impala/service/Frontend.java
+++ b/fe/src/main/java/org/apache/impala/service/Frontend.java
@@ -1182,7 +1182,10 @@ public class Frontend {
     } else if (table instanceof FeKuduTable) {
       if (op == TShowStatsOp.RANGE_PARTITIONS) {
         return FeKuduTable.Utils.getRangePartitions((FeKuduTable) table);
+      } else if (op == TShowStatsOp.PARTITIONS) {
+        return FeKuduTable.Utils.getPartitions((FeKuduTable) table);
       } else {
+        Preconditions.checkState(op == TShowStatsOp.TABLE_STATS);
         return FeKuduTable.Utils.getTableStats((FeKuduTable) table);
       }
     } else {
diff --git a/fe/src/test/java/org/apache/impala/analysis/AnalyzeDDLTest.java b/fe/src/test/java/org/apache/impala/analysis/AnalyzeDDLTest.java
index a64b98a..e91179f 100644
--- a/fe/src/test/java/org/apache/impala/analysis/AnalyzeDDLTest.java
+++ b/fe/src/test/java/org/apache/impala/analysis/AnalyzeDDLTest.java
@@ -3771,7 +3771,7 @@ public class AnalyzeDDLTest extends FrontendTestBase {
     AnalysisError("show partitions functional.view_view",
         "SHOW PARTITIONS not applicable to a view: functional.view_view");
     AnalysisError("show partitions functional_hbase.alltypes",
-        "SHOW PARTITIONS must target an HDFS table: functional_hbase.alltypes");
+        "SHOW PARTITIONS must target an HDFS or Kudu table: functional_hbase.alltypes");
   }
 
   @Test
diff --git a/testdata/workloads/functional-query/queries/QueryTest/kudu_partition_ddl.test b/testdata/workloads/functional-query/queries/QueryTest/kudu_partition_ddl.test
index 17d365b..d33ee14 100644
--- a/testdata/workloads/functional-query/queries/QueryTest/kudu_partition_ddl.test
+++ b/testdata/workloads/functional-query/queries/QueryTest/kudu_partition_ddl.test
@@ -8,20 +8,20 @@ create table simple_hash (id int, name string, valf float, vali bigint,
 'Table has been created.'
 ====
 ---- QUERY
-show table stats simple_hash
+show partitions simple_hash
 ---- LABELS
-# Rows,Start Key,Stop Key,Leader Replica,# Replicas
+Start Key,Stop Key,Leader Replica,#Replicas
 ---- RESULTS
--1,'','0000000000000001',regex:.*?:\d+,3
--1,'0000000000000001','00000001',regex:.*?:\d+,3
--1,'00000001','0000000100000001',regex:.*?:\d+,3
--1,'0000000100000001','00000002',regex:.*?:\d+,3
--1,'00000002','0000000200000001',regex:.*?:\d+,3
--1,'0000000200000001','00000003',regex:.*?:\d+,3
--1,'00000003','0000000300000001',regex:.*?:\d+,3
--1,'0000000300000001','',regex:.*?:\d+,3
+'','0000000000000001',regex:.*?:\d+,3
+'0000000000000001','00000001',regex:.*?:\d+,3
+'00000001','0000000100000001',regex:.*?:\d+,3
+'0000000100000001','00000002',regex:.*?:\d+,3
+'00000002','0000000200000001',regex:.*?:\d+,3
+'0000000200000001','00000003',regex:.*?:\d+,3
+'00000003','0000000300000001',regex:.*?:\d+,3
+'0000000300000001','',regex:.*?:\d+,3
 ---- TYPES
-INT,STRING,STRING,STRING,INT
+STRING,STRING,STRING,INT
 ====
 ---- QUERY
 -- Test single column range partitioning with bounded and unbounded partitions
@@ -33,15 +33,15 @@ create table range_part_bounds (id int, name string, valf float, vali bigint,
 'Table has been created.'
 ====
 ---- QUERY
-show table stats range_part_bounds
+show partitions range_part_bounds
 ---- LABELS
-# Rows,Start Key,Stop Key,Leader Replica,# Replicas
+Start Key,Stop Key,Leader Replica,#Replicas
 ---- RESULTS
--1,'','8000000B',regex:.*?:\d+,3
--1,'8000000B','80000015',regex:.*?:\d+,3
--1,'80000015','',regex:.*?:\d+,3
+'','8000000B',regex:.*?:\d+,3
+'8000000B','80000015',regex:.*?:\d+,3
+'80000015','',regex:.*?:\d+,3
 ---- TYPES
-INT,STRING,STRING,STRING,INT
+STRING,STRING,STRING,INT
 ====
 ---- QUERY
 -- Test single column range partitioning with single value partitions
@@ -53,15 +53,15 @@ create table range_part_single (id int, name string, valf float, vali bigint,
 'Table has been created.'
 ====
 ---- QUERY
-show table stats range_part_single
+show partitions range_part_single
 ---- LABELS
-# Rows,Start Key,Stop Key,Leader Replica,# Replicas
+Start Key,Stop Key,Leader Replica,#Replicas
 ---- RESULTS
--1,'80000001','80000002',regex:.*?:\d+,3
--1,'8000000A','8000000B',regex:.*?:\d+,3
--1,'80000064','80000065',regex:.*?:\d+,3
+'80000001','80000002',regex:.*?:\d+,3
+'8000000A','8000000B',regex:.*?:\d+,3
+'80000064','80000065',regex:.*?:\d+,3
 ---- TYPES
-INT,STRING,STRING,STRING,INT
+STRING,STRING,STRING,INT
 ====
 ---- QUERY
 -- Test single column range partitioning with bounded, unbounded and single
@@ -74,17 +74,17 @@ create table range_part_multiple_bounds (id int, name string, valf float,
 'Table has been created.'
 ====
 ---- QUERY
-show table stats range_part_multiple_bounds
+show partitions range_part_multiple_bounds
 ---- LABELS
-# Rows,Start Key,Stop Key,Leader Replica,# Replicas
+Start Key,Stop Key,Leader Replica,#Replicas
 ---- RESULTS
--1,'','8000000B',regex:.*?:\d+,3
--1,'8000000B','80000015',regex:.*?:\d+,3
--1,'80000015','8000001F',regex:.*?:\d+,3
--1,'80000028','80000029',regex:.*?:\d+,3
--1,'80000032','80000033',regex:.*?:\d+,3
+'','8000000B',regex:.*?:\d+,3
+'8000000B','80000015',regex:.*?:\d+,3
+'80000015','8000001F',regex:.*?:\d+,3
+'80000028','80000029',regex:.*?:\d+,3
+'80000032','80000033',regex:.*?:\d+,3
 ---- TYPES
-INT,STRING,STRING,STRING,INT
+STRING,STRING,STRING,INT
 ====
 ---- QUERY
 -- Test multiple column range partitioning
@@ -104,15 +104,15 @@ show range partitions range_part_multiple_cols
 'VALUE = (30, "matthew")'
 ====
 ---- QUERY
-show table stats range_part_multiple_cols
+show partitions range_part_multiple_cols
 ---- LABELS
-# Rows,Start Key,Stop Key,Leader Replica,# Replicas
+Start Key,Stop Key,Leader Replica,#Replicas
 ---- RESULTS
--1,'8000000A6D617274696E','8000000A6D617274696E00',regex:.*?:\d+,3
--1,'8000001464696D6974726973','8000001464696D697472697300',regex:.*?:\d+,3
--1,'8000001E6D617474686577','8000001E6D61747468657700',regex:.*?:\d+,3
+'8000000A6D617274696E','8000000A6D617274696E00',regex:.*?:\d+,3
+'8000001464696D6974726973','8000001464696D697472697300',regex:.*?:\d+,3
+'8000001E6D617474686577','8000001E6D61747468657700',regex:.*?:\d+,3
 ---- TYPES
-INT,STRING,STRING,STRING,INT
+STRING,STRING,STRING,INT
 ====
 ---- QUERY
 -- Test single column range partitioning with string partition column
@@ -124,16 +124,16 @@ create table range_part_single_string_col (id int, name string, valf float,
 'Table has been created.'
 ====
 ---- QUERY
-show table stats range_part_single_string_col
+show partitions range_part_single_string_col
 ---- LABELS
-# Rows,Start Key,Stop Key,Leader Replica,# Replicas
+Start Key,Stop Key,Leader Replica,#Replicas
 ---- RESULTS
--1,'','61616100',regex:.*?:\d+,3
--1,'61616100','62626200',regex:.*?:\d+,3
--1,'62626200','63636300',regex:.*?:\d+,3
--1,'646464','64646400',regex:.*?:\d+,3
+'','61616100',regex:.*?:\d+,3
+'61616100','62626200',regex:.*?:\d+,3
+'62626200','63636300',regex:.*?:\d+,3
+'646464','64646400',regex:.*?:\d+,3
 ---- TYPES
-INT,STRING,STRING,STRING,INT
+STRING,STRING,STRING,INT
 ====
 ---- QUERY
 -- Test hash and range partitioning
@@ -144,20 +144,20 @@ create table simple_hash_range (id int, name string, valf float, vali bigint,
 'Table has been created.'
 ====
 ---- QUERY
-show table stats simple_hash_range
+show partitions simple_hash_range
 ---- LABELS
-# Rows,Start Key,Stop Key,Leader Replica,# Replicas
+Start Key,Stop Key,Leader Replica,#Replicas
 ---- RESULTS
--1,'000000008000000A6D617274696E','000000008000000A6D617274696E00',regex:.*?:\d+,3
--1,'0000000080000014616C6578','0000000080000014616C657800',regex:.*?:\d+,3
--1,'000000018000000A6D617274696E','000000018000000A6D617274696E00',regex:.*?:\d+,3
--1,'0000000180000014616C6578','0000000180000014616C657800',regex:.*?:\d+,3
--1,'000000028000000A6D617274696E','000000028000000A6D617274696E00',regex:.*?:\d+,3
--1,'0000000280000014616C6578','0000000280000014616C657800',regex:.*?:\d+,3
--1,'000000038000000A6D617274696E','000000038000000A6D617274696E00',regex:.*?:\d+,3
--1,'0000000380000014616C6578','0000000380000014616C657800',regex:.*?:\d+,3
+'000000008000000A6D617274696E','000000008000000A6D617274696E00',regex:.*?:\d+,3
+'0000000080000014616C6578','0000000080000014616C657800',regex:.*?:\d+,3
+'000000018000000A6D617274696E','000000018000000A6D617274696E00',regex:.*?:\d+,3
+'0000000180000014616C6578','0000000180000014616C657800',regex:.*?:\d+,3
+'000000028000000A6D617274696E','000000028000000A6D617274696E00',regex:.*?:\d+,3
+'0000000280000014616C6578','0000000280000014616C657800',regex:.*?:\d+,3
+'000000038000000A6D617274696E','000000038000000A6D617274696E00',regex:.*?:\d+,3
+'0000000380000014616C6578','0000000380000014616C657800',regex:.*?:\d+,3
 ---- TYPES
-INT,STRING,STRING,STRING,INT
+STRING,STRING,STRING,INT
 ====
 ---- QUERY
 create table simple_hash_range_ctas
@@ -169,20 +169,20 @@ as select * from simple_hash
 'Inserted 0 row(s)'
 ====
 ---- QUERY
-show table stats simple_hash_range_ctas
+show partitions simple_hash_range_ctas
 ---- LABELS
-# Rows,Start Key,Stop Key,Leader Replica,# Replicas
+Start Key,Stop Key,Leader Replica,#Replicas
 ---- RESULTS
--1,'000000008000000A6361736579','000000008000000A636173657900',regex:.*?:\d+,3
--1,'00000000800000146D617263656C','00000000800000146D617263656C00',regex:.*?:\d+,3
--1,'000000018000000A6361736579','000000018000000A636173657900',regex:.*?:\d+,3
--1,'00000001800000146D617263656C','00000001800000146D617263656C00',regex:.*?:\d+,3
--1,'000000028000000A6361736579','000000028000000A636173657900',regex:.*?:\d+,3
--1,'00000002800000146D617263656C','00000002800000146D617263656C00',regex:.*?:\d+,3
--1,'000000038000000A6361736579','000000038000000A636173657900',regex:.*?:\d+,3
--1,'00000003800000146D617263656C','00000003800000146D617263656C00',regex:.*?:\d+,3
+'000000008000000A6361736579','000000008000000A636173657900',regex:.*?:\d+,3
+'00000000800000146D617263656C','00000000800000146D617263656C00',regex:.*?:\d+,3
+'000000018000000A6361736579','000000018000000A636173657900',regex:.*?:\d+,3
+'00000001800000146D617263656C','00000001800000146D617263656C00',regex:.*?:\d+,3
+'000000028000000A6361736579','000000028000000A636173657900',regex:.*?:\d+,3
+'00000002800000146D617263656C','00000002800000146D617263656C00',regex:.*?:\d+,3
+'000000038000000A6361736579','000000038000000A636173657900',regex:.*?:\d+,3
+'00000003800000146D617263656C','00000003800000146D617263656C00',regex:.*?:\d+,3
 ---- TYPES
-INT,STRING,STRING,STRING,INT
+STRING,STRING,STRING,INT
 ====
 ---- QUERY
 -- Test hash defaults to all columns
@@ -192,16 +192,16 @@ create table simple_hash_all_columns (id int, name string, valf float, vali bigi
 'Table has been created.'
 ====
 ---- QUERY
-show table stats simple_hash_all_columns
+show partitions simple_hash_all_columns
 ---- LABELS
-# Rows,Start Key,Stop Key,Leader Replica,# Replicas
+Start Key,Stop Key,Leader Replica,#Replicas
 ---- RESULTS
--1,'','00000001',regex:.*?:\d+,3
--1,'00000001','00000002',regex:.*?:\d+,3
--1,'00000002','00000003',regex:.*?:\d+,3
--1,'00000003','',regex:.*?:\d+,3
+'','00000001',regex:.*?:\d+,3
+'00000001','00000002',regex:.*?:\d+,3
+'00000002','00000003',regex:.*?:\d+,3
+'00000003','',regex:.*?:\d+,3
 ---- TYPES
-INT,STRING,STRING,STRING,INT
+STRING,STRING,STRING,INT
 ====
 ---- QUERY
 -- Test range defaults to all columns
@@ -213,14 +213,14 @@ create table simple_range_all_columns (id int, name string, valf float, vali big
 'Table has been created.'
 ====
 ---- QUERY
-show table stats simple_range_all_columns
+show partitions simple_range_all_columns
 ---- LABELS
-# Rows,Start Key,Stop Key,Leader Replica,# Replicas
+Start Key,Stop Key,Leader Replica,#Replicas
 ---- RESULTS
--1,'8000000161','800000016100',regex:.*?:\d+,3
--1,'8000000262','800000026200',regex:.*?:\d+,3
+'8000000161','800000016100',regex:.*?:\d+,3
+'8000000262','800000026200',regex:.*?:\d+,3
 ---- TYPES
-INT,STRING,STRING,STRING,INT
+STRING,STRING,STRING,INT
 ====
 ---- QUERY
 -- Test using non-literal constant values in range-partition bounds
@@ -232,16 +232,16 @@ create table range_complex_const_boundary_vals (x int, y int, primary key (x))
 'Table has been created.'
 ====
 ---- QUERY
-show table stats range_complex_const_boundary_vals
+show partitions range_complex_const_boundary_vals
 ---- LABELS
-# Rows,Start Key,Stop Key,Leader Replica,# Replicas
+Start Key,Stop Key,Leader Replica,#Replicas
 ---- RESULTS
--1,'','80000002',regex:.*?:\d+,3
--1,'80000007','8000000A',regex:.*?:\d+,3
--1,'80000019','80000078',regex:.*?:\d+,3
--1,'800002D0','800002D1',regex:.*?:\d+,3
+'','80000002',regex:.*?:\d+,3
+'80000007','8000000A',regex:.*?:\d+,3
+'80000019','80000078',regex:.*?:\d+,3
+'800002D0','800002D1',regex:.*?:\d+,3
 ---- TYPES
-INT,STRING,STRING,STRING,INT
+STRING,STRING,STRING,INT
 ====
 ---- QUERY
 -- Test range partitioning with overlapping partitions
diff --git a/testdata/workloads/functional-query/queries/QueryTest/kudu_stats.test b/testdata/workloads/functional-query/queries/QueryTest/kudu_stats.test
index 23a43b5..f3794a9 100644
--- a/testdata/workloads/functional-query/queries/QueryTest/kudu_stats.test
+++ b/testdata/workloads/functional-query/queries/QueryTest/kudu_stats.test
@@ -7,15 +7,37 @@ create table simple (id int primary key, name string, valf float, vali bigint)
 'Table has been created.'
 ====
 ---- QUERY
-show table stats simple
+# Tests the SHOW TABLE STATS output without stats computed
+show table stats simple;
 ---- RESULTS
--1,'','8000000A',regex:.*?:\d+,1
--1,'8000000A','8000001E',regex:.*?:\d+,1
--1,'8000001E','',regex:.*?:\d+,1
+-1,3,regex:.*,'KUDU',regex:.*
 ---- TYPES
-INT,STRING,STRING,STRING,INT
+BIGINT,BIGINT,STRING,STRING,STRING
 ---- LABELS
-# Rows,Start Key,Stop Key,Leader Replica,# Replicas
+#Rows,#Partitions,Size,Format,Location
+====
+---- QUERY
+# Tests the SHOW TABLE STATS output after stats computed
+compute stats simple;
+show table stats simple;
+---- RESULTS
+0,3,regex:.*,'KUDU',regex:.*
+---- TYPES
+BIGINT,BIGINT,STRING,STRING,STRING
+---- LABELS
+#Rows,#Partitions,Size,Format,Location
+====
+---- QUERY
+# Tests the SHOW PARTITIONS output
+show partitions simple;
+---- RESULTS
+'','8000000A',regex:.*?:\d+,1
+'8000000A','8000001E',regex:.*?:\d+,1
+'8000001E','',regex:.*?:\d+,1
+---- TYPES
+STRING,STRING,STRING,INT
+---- LABELS
+Start Key,Stop Key,Leader Replica,#Replicas
 ====
 ---- QUERY
 # IMPALA-3373: Computing stats on a Kudu table lead to duplicate columns shown for the