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 2019/02/26 23:54:22 UTC

[impala] branch master updated (e9936b0 -> 7ba6aa6)

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

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


    from e9936b0  IMPALA-8247: Fix tests missing from unified backend executable
     new 72fd7c3  IMPALA-8245: Add hostname to the error message in HdfsMonitoredOps
     new e4f46da  IMPALA-8069: Crash in impala::Sorter::Run::Run
     new 360f88e  IMPALA-8181: Abbreviate row counts in EXPLAIN
     new 7ba6aa6  IMPALA-8188: Fix DiskInfo::GetDeviceNames() for NVME disks

The 4 revisions listed above as "new" are entirely new to this
repository and will be described in separate emails.  The revisions
listed as "add" were already present in the repository and have only
been added to this reference.


Summary of changes:
 be/src/runtime/io/hdfs-monitored-ops.cc            |   3 +-
 be/src/util/disk-info.cc                           |  30 +-
 be/src/util/disk-info.h                            |   7 +
 be/src/util/sys-info-test.cc                       |  45 +-
 .../java/org/apache/impala/common/PrintUtils.java  |  20 +-
 .../org/apache/impala/planner/AnalyticPlanner.java |  13 +-
 .../org/apache/impala/planner/HdfsScanNode.java    |  33 +-
 .../java/org/apache/impala/planner/PlanNode.java   |   2 +-
 .../java/org/apache/impala/planner/ScanNode.java   |  18 +-
 .../java/org/apache/impala/testutil/TestUtils.java |   3 +-
 .../queries/PlannerTest/analytic-fns.test          |  42 +-
 .../queries/PlannerTest/constant-folding.test      |  46 +-
 .../queries/PlannerTest/fk-pk-join-detection.test  | 110 ++--
 .../queries/PlannerTest/max-row-size.test          |  40 +-
 .../queries/PlannerTest/mt-dop-validation.test     |  24 +-
 .../queries/PlannerTest/order.test                 |  17 +
 .../PlannerTest/parquet-filtering-disabled.test    |   6 +-
 .../queries/PlannerTest/parquet-filtering.test     |  16 +-
 .../queries/PlannerTest/resource-requirements.test | 558 ++++++++++-----------
 .../PlannerTest/sort-expr-materialization.test     |  32 +-
 .../PlannerTest/spillable-buffer-sizing.test       |  88 ++--
 .../queries/PlannerTest/tablesample.test           |  50 +-
 .../queries/PlannerTest/union.test                 |   8 +-
 .../queries/QueryTest/analytic-fns.test            |  16 +
 .../queries/QueryTest/explain-level2.test          |   4 +-
 .../queries/QueryTest/explain-level3.test          |   4 +-
 .../queries/QueryTest/stats-extrapolation.test     |  18 +-
 tests/custom_cluster/test_hdfs_timeout.py          |   3 +-
 tests/custom_cluster/test_stats_extrapolation.py   |   2 +-
 tests/metadata/test_stats_extrapolation.py         |   2 +-
 30 files changed, 706 insertions(+), 554 deletions(-)


[impala] 04/04: IMPALA-8188: Fix DiskInfo::GetDeviceNames() for NVME disks

Posted by jo...@apache.org.
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

commit 7ba6aa64a2455df30524ad42b10d7fd9227efacb
Author: Joe McDonnell <jo...@cloudera.com>
AuthorDate: Mon Feb 11 17:07:37 2019 -0800

    IMPALA-8188: Fix DiskInfo::GetDeviceNames() for NVME disks
    
    DiskInfo::GetDeviceNames() assumes a device name pattern where
    the last digits can be removed to get the main disk device
    (i.e. sda3 -> sda). NVME devices have a naming pattern of
    nvme{device_id}n{namespace_id}p{partition_id} where the main
    disk device removes the p{partition_id} part. This difference
    in naming prevents DiskInfo::GetDeviceNames() from looking
    up the associated information in /sys/block to find whether
    it is a rotational device (and thus use an appropriate
    number of DiskIo threads).
    
    This adds a condition to detect an NVME device name and
    handle it correctly.
    
    Testing:
     - Added unit test for the function used for detection
     - Tested this on an AWS m5-4xlarge, which is a Nitro instance
       type that exposes the main disk as an NVME device.
    
    Change-Id: I4d5bd93b4b09682a8c8248e7aa123d23d27cfeb4
    Reviewed-on: http://gerrit.cloudera.org:8080/12557
    Tested-by: Impala Public Jenkins <im...@cloudera.com>
    Reviewed-by: Philip Zeyliger <ph...@cloudera.com>
---
 be/src/util/disk-info.cc     | 30 +++++++++++++++++++++++++++--
 be/src/util/disk-info.h      |  7 +++++++
 be/src/util/sys-info-test.cc | 45 +++++++++++++++++++++++++++++++++++++++++++-
 3 files changed, 79 insertions(+), 3 deletions(-)

diff --git a/be/src/util/disk-info.cc b/be/src/util/disk-info.cc
index 0a383c8..25d9f02 100644
--- a/be/src/util/disk-info.cc
+++ b/be/src/util/disk-info.cc
@@ -17,6 +17,7 @@
 
 #include "util/disk-info.h"
 
+#include <regex>
 #ifdef __APPLE__
 #include <sys/mount.h>
 #else
@@ -50,6 +51,22 @@ vector<DiskInfo::Disk> DiskInfo::disks_;
 map<dev_t, int> DiskInfo::device_id_to_disk_id_;
 map<string, int> DiskInfo::disk_name_to_disk_id_;
 
+bool DiskInfo::TryNVMETrim(const std::string& name_in, std::string* basename_out) {
+  // NVME drives do not follow the typical device naming pattern. The pattern for NVME
+  // drives is nvme{device_id}n{namespace_id}p{partition_id}. The appropriate thing
+  // to do for this pattern is to trim the "p{partition_id}" part.
+  std::regex nvme_regex = std::regex("(nvme[0-9]+n[0-9]+)(p[0-9]+)*");
+  std::smatch nvme_match_result;
+  if (std::regex_match(name_in, nvme_match_result, nvme_regex)) {
+    DCHECK_GE(nvme_match_result.size(), 2);
+    // Match 0 contains the whole string.
+    // Match 1 contains the base nvme device without the partition.
+    *basename_out = nvme_match_result[1];
+    return true;
+  }
+  return false;
+}
+
 // Parses /proc/partitions to get the number of disks.  A bit of looking around
 // seems to indicate this as the best way to do this.
 // TODO: is there not something better than this?
@@ -70,8 +87,17 @@ void DiskInfo::GetDeviceNames() {
     string name = fields[3];
     if (name == "name") continue;
 
-    // Remove the partition# from the name.  e.g. sda2 --> sda
-    trim_right_if(name, is_any_of("0123456789"));
+    // NVME devices have a special format. Try to detect that, falling back to the normal
+    // method if this is not an NVME device.
+    std::string nvme_basename;
+    if (TryNVMETrim(name, &nvme_basename)) {
+      // This is an NVME device, use the returned basename
+      name = nvme_basename;
+    } else {
+      // Does not follow the NVME pattern, so use the logic for a normal disk device
+      // Remove the partition# from the name.  e.g. sda2 --> sda
+      trim_right_if(name, is_any_of("0123456789"));
+    }
 
     // Create a mapping of all device ids (one per partition) to the disk id.
     int major_dev_id = atoi(fields[0].c_str());
diff --git a/be/src/util/disk-info.h b/be/src/util/disk-info.h
index edfea21..cb43b15 100644
--- a/be/src/util/disk-info.h
+++ b/be/src/util/disk-info.h
@@ -67,6 +67,8 @@ class DiskInfo {
   static std::string DebugString();
 
  private:
+  friend class DiskInfoTest;
+
   static bool initialized_;
 
   struct Disk {
@@ -93,6 +95,11 @@ class DiskInfo {
   static std::map<std::string, int> disk_name_to_disk_id_;
 
   static void GetDeviceNames();
+
+  /// See if 'name_in' is an NVME device. If it is, set 'basename_out' to the base
+  /// NVME device name (i.e. nvme0n1p1 -> nvme0n1) and return true. Otherwise,
+  /// return false and leave 'basename_out' unmodified.
+  static bool TryNVMETrim(const std::string& name_in, std::string* basename_out);
 };
 }
 #endif
diff --git a/be/src/util/sys-info-test.cc b/be/src/util/sys-info-test.cc
index 7fe9da1..132c330 100644
--- a/be/src/util/sys-info-test.cc
+++ b/be/src/util/sys-info-test.cc
@@ -56,7 +56,22 @@ TEST(CpuInfoTest, InvalidSchedGetCpuValue) {
 
 }
 
-TEST(DiskInfoTest, Basic) {
+class DiskInfoTest : public ::testing::Test {
+ protected:
+  void TestTryNVMETrimPositive(const string& name, const string& expected_basename) {
+    string nvme_basename;
+    ASSERT_TRUE(DiskInfo::TryNVMETrim(name, &nvme_basename));
+    ASSERT_EQ(nvme_basename, expected_basename);
+  }
+
+  void TestTryNVMETrimNegative(const string& name) {
+    string nvme_basename;
+    ASSERT_FALSE(DiskInfo::TryNVMETrim(name, &nvme_basename));
+    ASSERT_EQ(nvme_basename, "");
+  }
+};
+
+TEST_F(DiskInfoTest, Basic) {
   cout << DiskInfo::DebugString();
   cout << "Device name for disk 0: " << DiskInfo::device_name(0) << endl;
 
@@ -69,5 +84,33 @@ TEST(DiskInfoTest, Basic) {
     cout << "No device name for /home";
   }
 }
+
+TEST_F(DiskInfoTest, NVMEDetection) {
+  // Positive case 1: NVME device with a partition specified
+  TestTryNVMETrimPositive("nvme0n1p2", "nvme0n1");
+
+  // Positive case 2: NVME device without partition specified
+  TestTryNVMETrimPositive("nvme0n1", "nvme0n1");
+
+  // Positive case 3: NVME multi-digit device id, namespace id, partition id
+  TestTryNVMETrimPositive("nvme15n13p24", "nvme15n13");
+
+  // Negative case 1: disk drive
+  TestTryNVMETrimNegative("sda3");
+  TestTryNVMETrimNegative("sda");
+
+  // Negative case 2: malformed NVME, missing namespace
+  TestTryNVMETrimNegative("nvme0p1");
+  TestTryNVMETrimNegative("nvme0");
+
+  // Negative case 3: malformed NVME, missing device id
+  TestTryNVMETrimNegative("nvmen1p3");
+  TestTryNVMETrimNegative("nvmen1");
+
+  // Negative case 4: extra garbage before / after
+  TestTryNVMETrimNegative("sdanvme0n1");
+  TestTryNVMETrimNegative("nvme0n1p0blah");
+}
+
 }
 


[impala] 02/04: IMPALA-8069: Crash in impala::Sorter::Run::Run

Posted by jo...@apache.org.
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

commit e4f46dad92f92c9a0e7730d7b393d42cf7ee8c7f
Author: Paul Rogers <pr...@cloudera.com>
AuthorDate: Fri Feb 22 19:05:29 2019 -0800

    IMPALA-8069: Crash in impala::Sorter::Run::Run
    
    Modifies the planner to omit a sorter node in the obscure case we have
    an analytic query with a constant sort. The analytic node requires a
    buffered tuple. When the sort is omitted, creates a buffered tuple on
    top of the node that would have been input to the sort.
    
    Testing:
    * Added a PlannerTest case.
    * Added an end-to-end test.
    * Reran the query that previously crashed: it now produces proper results.
    
    Change-Id: I19a57f3791ce9e41fe0ba79dc5834e762341c74e
    Reviewed-on: http://gerrit.cloudera.org:8080/12562
    Reviewed-by: Impala Public Jenkins <im...@cloudera.com>
    Tested-by: Impala Public Jenkins <im...@cloudera.com>
---
 .../org/apache/impala/planner/AnalyticPlanner.java | 13 +++++--
 .../queries/PlannerTest/analytic-fns.test          | 42 ++++++++++++++--------
 .../queries/PlannerTest/order.test                 | 17 +++++++++
 .../queries/QueryTest/analytic-fns.test            | 16 +++++++++
 4 files changed, 71 insertions(+), 17 deletions(-)

diff --git a/fe/src/main/java/org/apache/impala/planner/AnalyticPlanner.java b/fe/src/main/java/org/apache/impala/planner/AnalyticPlanner.java
index 92c44bb..3e58eae 100644
--- a/fe/src/main/java/org/apache/impala/planner/AnalyticPlanner.java
+++ b/fe/src/main/java/org/apache/impala/planner/AnalyticPlanner.java
@@ -324,8 +324,13 @@ public class AnalyticPlanner {
     ExprSubstitutionMap bufferedSmap = new ExprSubstitutionMap();
     boolean activePartition = activeExprs(partitionByExprs);
 
+    // IMPALA-8069: Ignore something like ORDER BY 0
+    boolean isConstSort = true;
+    for (OrderByElement elmt : orderByElements) {
+      isConstSort = isConstSort && elmt.getExpr().isConstant();
+    }
     // sort on partition by (pb) + order by (ob) exprs and create pb/ob predicates
-    if (activePartition || !orderByElements.isEmpty()) {
+    if (activePartition || !isConstSort) {
       // first sort on partitionExprs (direction doesn't matter)
       List<Expr> sortExprs = Lists.newArrayList(partitionByExprs);
       List<Boolean> isAsc =
@@ -365,10 +370,12 @@ public class AnalyticPlanner {
 
       root = sortNode;
       root.init(analyzer_);
-      sortSmap = sortNode.getOutputSmap();
+    }
+    if (activePartition || !orderByElements.isEmpty()) {
+      sortSmap = root.getOutputSmap();
 
       // create bufferedTupleDesc and bufferedSmap
-      sortTupleId = sortNode.tupleIds_.get(0);
+      sortTupleId = root.tupleIds_.get(0);
       bufferedTupleDesc =
           analyzer_.getDescTbl().copyTupleDescriptor(sortTupleId, "buffered-tuple");
       if (LOG.isTraceEnabled()) {
diff --git a/testdata/workloads/functional-planner/queries/PlannerTest/analytic-fns.test b/testdata/workloads/functional-planner/queries/PlannerTest/analytic-fns.test
index fef2844..a2038e4 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/analytic-fns.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/analytic-fns.test
@@ -2888,22 +2888,36 @@ PLAN-ROOT SINK
    constant-operands=3
    row-size=1B cardinality=3
 ====
-# IMPALA-6323 Order by a constant is equivalent to no ordering.
-select x, count() over(order by 1) from (VALUES((1 x), (2), (3))) T;
----- DISTRIBUTEDPLAN
+# Regression test for IMPALA-8069
+SELECT FIRST_VALUE(0) OVER (ORDER BY 0 ASC)
+FROM functional.alltypestiny
+---- PLAN
 PLAN-ROOT SINK
 |
-02:ANALYTIC
-|  functions: count()
-|  order by: 1 ASC
-|  window: RANGE BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW
-|  row-size=9B cardinality=3
+01:ANALYTIC
+|  functions: first_value(0)
+|  order by: 0 ASC
+|  window: ROWS BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW
+|  row-size=1B cardinality=8
 |
-01:SORT
-|  order by: 1 ASC
-|  row-size=1B cardinality=3
+00:SCAN HDFS [functional.alltypestiny]
+   partitions=4/4 files=4 size=460B
+   row-size=0B cardinality=8
+====
+# Regression test for IMPALA-8069
+# Same case as in end-to-end test analytic-fns.test
+SELECT FIRST_VALUE(0) OVER (ORDER BY 0 ASC), row_number() over (order by 0 ASC)
+FROM functional.alltypestiny
+---- PLAN
+PLAN-ROOT SINK
 |
-00:UNION
-   constant-operands=3
-   row-size=1B cardinality=3
+01:ANALYTIC
+|  functions: first_value(0), row_number()
+|  order by: 0 ASC
+|  window: ROWS BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW
+|  row-size=9B cardinality=8
+|
+00:SCAN HDFS [functional.alltypestiny]
+   partitions=4/4 files=4 size=460B
+   row-size=0B cardinality=8
 ====
diff --git a/testdata/workloads/functional-planner/queries/PlannerTest/order.test b/testdata/workloads/functional-planner/queries/PlannerTest/order.test
index fe02596..113b28c 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/order.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/order.test
@@ -1598,3 +1598,20 @@ PLAN-ROOT SINK
    limit: 1
    row-size=0B cardinality=1
 ====
+# Regression test for IMPALA-8069
+SELECT
+FIRST_VALUE(0) OVER (ORDER BY 0 ASC)
+FROM functional.alltypestiny
+---- PLAN
+PLAN-ROOT SINK
+|
+01:ANALYTIC
+|  functions: first_value(0)
+|  order by: 0 ASC
+|  window: ROWS BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW
+|  row-size=1B cardinality=8
+|
+00:SCAN HDFS [functional.alltypestiny]
+   partitions=4/4 files=4 size=460B
+   row-size=0B cardinality=8
+====
diff --git a/testdata/workloads/functional-query/queries/QueryTest/analytic-fns.test b/testdata/workloads/functional-query/queries/QueryTest/analytic-fns.test
index fbbd939..4188205 100644
--- a/testdata/workloads/functional-query/queries/QueryTest/analytic-fns.test
+++ b/testdata/workloads/functional-query/queries/QueryTest/analytic-fns.test
@@ -2140,3 +2140,19 @@ TINYINT, BIGINT
 2,3
 3,3
 ====
+---- QUERY
+# IMPALA-6323 Order by a constant is equivalent to no ordering.
+SELECT FIRST_VALUE(0) OVER (ORDER BY 0 ASC)
+FROM functional.alltypestiny;
+---- TYPES
+TINYINT
+---- RESULTS
+0
+0
+0
+0
+0
+0
+0
+0
+====


[impala] 01/04: IMPALA-8245: Add hostname to the error message in HdfsMonitoredOps

Posted by jo...@apache.org.
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

commit 72fd7c3d4d50f30479b5f6d50b327dff9ff013ac
Author: poojanilangekar <po...@cloudera.com>
AuthorDate: Tue Feb 26 09:20:57 2019 -0800

    IMPALA-8245: Add hostname to the error message in HdfsMonitoredOps
    
    When a task submitted via HdfsMonitoredOps fails, it raises
    TErrorCode::THREAD_POOL_TASK_TIMED_OUT or
    TErrorCode::THREAD_POOL_SUBMIT_FAILED errors. The error generated
    calls GetDescription() to provide information about the failed
    hdfs operation. This change appends the hostname to the description
    to enable the user to easily identify the host which has reached a
    bad connection state with the NameNode.
    
    Testing:
    Modified the test_hdfs_timeout.py to ensure that the hostname is
    logged in the error message.
    
    Change-Id: Ief1e21560b6fb54965f2fb2793c32c2ba5176ba2
    Reviewed-on: http://gerrit.cloudera.org:8080/12593
    Reviewed-by: Impala Public Jenkins <im...@cloudera.com>
    Tested-by: Impala Public Jenkins <im...@cloudera.com>
---
 be/src/runtime/io/hdfs-monitored-ops.cc   | 3 ++-
 tests/custom_cluster/test_hdfs_timeout.py | 3 ++-
 2 files changed, 4 insertions(+), 2 deletions(-)

diff --git a/be/src/runtime/io/hdfs-monitored-ops.cc b/be/src/runtime/io/hdfs-monitored-ops.cc
index f864b7c..10817fb 100644
--- a/be/src/runtime/io/hdfs-monitored-ops.cc
+++ b/be/src/runtime/io/hdfs-monitored-ops.cc
@@ -20,6 +20,7 @@
 #include "common/names.h"
 #include "common/status.h"
 #include "runtime/io/hdfs-monitored-ops.h"
+#include "util/debug-util.h"
 #include "util/hdfs-util.h"
 #include "util/time.h"
 
@@ -76,7 +77,7 @@ Status OpenHdfsFileOp::Execute() {
 }
 
 std::string OpenHdfsFileOp::GetDescription() {
-  return Substitute("hdfsOpenFile() for $0", fname_);
+  return Substitute("hdfsOpenFile() for $0 at backend $1", fname_, GetBackendString());
 }
 
 Status HdfsMonitor::OpenHdfsFileWithTimeout(const hdfsFS& fs, const std::string* fname,
diff --git a/tests/custom_cluster/test_hdfs_timeout.py b/tests/custom_cluster/test_hdfs_timeout.py
index 97e0aa3..0967427 100644
--- a/tests/custom_cluster/test_hdfs_timeout.py
+++ b/tests/custom_cluster/test_hdfs_timeout.py
@@ -81,7 +81,8 @@ class TestHdfsTimeouts(CustomClusterTestSuite):
       assert False, "Query should have failed, but instead returned {0}".format(result)
 
     # The exception should contain the appropriate error message
-    error_pattern = "hdfsOpenFile\(\) for.*failed to finish before the 5 second timeout"
+    error_pattern = "hdfsOpenFile\(\) for.*at backend.*"
+    "failed to finish before the 5 second timeout"
     assert len(re.findall(error_pattern, str(ex))) > 0
 
     # The timeout is 5 seconds and seems to be enforced within about 20 seconds, so it


[impala] 03/04: IMPALA-8181: Abbreviate row counts in EXPLAIN

Posted by jo...@apache.org.
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

commit 360f88e20709a4f1447c4997d1dc458e80874389
Author: Paul Rogers <pr...@cloudera.com>
AuthorDate: Sun Feb 10 17:37:23 2019 -0800

    IMPALA-8181: Abbreviate row counts in EXPLAIN
    
    A recent fix added node cardinality to the standard EXPLAIN output,
    displaying a large number like 123456780 as 123.46M. This patch applies
    the same fix to the remaining row count numbers: metadata, extrapolated
    rows, etc.
    
    Tests:
    * Rebased PlannerTest .test files as needed for the new row count
      format.
    * Reran all tests to check for dependencies on the old format.
    
    Change-Id: I08faaa9ad7b5ed42dcd7a15a333e8734bb45f10c
    Reviewed-on: http://gerrit.cloudera.org:8080/12438
    Reviewed-by: Impala Public Jenkins <im...@cloudera.com>
    Tested-by: Impala Public Jenkins <im...@cloudera.com>
---
 .../java/org/apache/impala/common/PrintUtils.java  |  20 +-
 .../org/apache/impala/planner/HdfsScanNode.java    |  33 +-
 .../java/org/apache/impala/planner/PlanNode.java   |   2 +-
 .../java/org/apache/impala/planner/ScanNode.java   |  18 +-
 .../java/org/apache/impala/testutil/TestUtils.java |   3 +-
 .../queries/PlannerTest/constant-folding.test      |  46 +-
 .../queries/PlannerTest/fk-pk-join-detection.test  | 110 ++--
 .../queries/PlannerTest/max-row-size.test          |  40 +-
 .../queries/PlannerTest/mt-dop-validation.test     |  24 +-
 .../PlannerTest/parquet-filtering-disabled.test    |   6 +-
 .../queries/PlannerTest/parquet-filtering.test     |  16 +-
 .../queries/PlannerTest/resource-requirements.test | 558 ++++++++++-----------
 .../PlannerTest/sort-expr-materialization.test     |  32 +-
 .../PlannerTest/spillable-buffer-sizing.test       |  88 ++--
 .../queries/PlannerTest/tablesample.test           |  50 +-
 .../queries/PlannerTest/union.test                 |   8 +-
 .../queries/QueryTest/explain-level2.test          |   4 +-
 .../queries/QueryTest/explain-level3.test          |   4 +-
 .../queries/QueryTest/stats-extrapolation.test     |  18 +-
 tests/custom_cluster/test_stats_extrapolation.py   |   2 +-
 tests/metadata/test_stats_extrapolation.py         |   2 +-
 21 files changed, 552 insertions(+), 532 deletions(-)

diff --git a/fe/src/main/java/org/apache/impala/common/PrintUtils.java b/fe/src/main/java/org/apache/impala/common/PrintUtils.java
index 1f347be..4941d8a 100644
--- a/fe/src/main/java/org/apache/impala/common/PrintUtils.java
+++ b/fe/src/main/java/org/apache/impala/common/PrintUtils.java
@@ -134,10 +134,28 @@ public class PrintUtils {
     return bytes + "B";
   }
 
-  public static String printCardinality(long cardinality) {
+  /**
+   * Print an estimated cardinality. No need to print the exact value
+   * because estimates are not super-precise.
+   */
+  public static String printEstCardinality(long cardinality) {
     return (cardinality != -1) ? printMetric(cardinality) : "unavailable";
   }
 
+  /**
+   * Print an exact cardinality (such as a row count) as one of three formats:
+   *
+   * * "unavailable" (if value < 0)
+   * * number (if value is small)
+   * * xx.xxU (dd,ddd) (if the value is large)
+   */
+  public static String printExactCardinality(long value) {
+    if (value == -1) return "unavailable";
+    String result = printMetric(value);
+    if (value < KILO) return result;
+    return String.format("%s (%,d)", result, value);
+  }
+
   public static String printNumHosts(String prefix, long numHosts) {
     return prefix + "hosts=" + ((numHosts != -1) ? numHosts : "unavailable");
   }
diff --git a/fe/src/main/java/org/apache/impala/planner/HdfsScanNode.java b/fe/src/main/java/org/apache/impala/planner/HdfsScanNode.java
index b1eefed..fb589b2 100644
--- a/fe/src/main/java/org/apache/impala/planner/HdfsScanNode.java
+++ b/fe/src/main/java/org/apache/impala/planner/HdfsScanNode.java
@@ -26,7 +26,6 @@ import java.util.List;
 import java.util.Map;
 import java.util.Set;
 
-import com.google.common.collect.Sets;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.FileSystem;
 import org.apache.impala.analysis.AggregateInfo;
@@ -92,6 +91,7 @@ import com.google.common.base.Objects;
 import com.google.common.base.Objects.ToStringHelper;
 import com.google.common.base.Preconditions;
 import com.google.common.collect.Lists;
+import com.google.common.collect.Sets;
 
 /**
  * Scan of a single table.
@@ -1284,19 +1284,19 @@ public class HdfsScanNode extends ScanNode {
       }
     }
     if (detailLevel.ordinal() >= TExplainLevel.EXTENDED.ordinal()) {
-      output.append(getStatsExplainString(detailPrefix));
-      output.append("\n");
-      String extrapRows = String.valueOf(extrapolatedNumRows_);
-      if (!FeFsTable.Utils.isStatsExtrapolationEnabled(tbl_)) {
+      output.append(getStatsExplainString(detailPrefix)).append("\n");
+      String extrapRows;
+      if (FeFsTable.Utils.isStatsExtrapolationEnabled(tbl_)) {
+        extrapRows = PrintUtils.printEstCardinality(extrapolatedNumRows_);
+      } else {
         extrapRows = "disabled";
-      } else if (extrapolatedNumRows_ == -1) {
-        extrapRows = "unavailable";
       }
       output.append(detailPrefix)
-        .append(String.format("extrapolated-rows=%s", extrapRows));
-      output.append(String.format(" max-scan-range-rows=%s",
-          maxScanRangeNumRows_ == -1 ? "unavailable" : maxScanRangeNumRows_));
-      output.append("\n");
+            .append("extrapolated-rows=")
+            .append(extrapRows)
+            .append(" max-scan-range-rows=")
+            .append(PrintUtils.printEstCardinality(maxScanRangeNumRows_))
+            .append("\n");
       if (numScanRangesNoDiskIds_ > 0) {
         output.append(detailPrefix)
           .append(String.format("missing disk ids: "
@@ -1382,18 +1382,17 @@ public class HdfsScanNode extends ScanNode {
   protected String getTableStatsExplainString(String prefix) {
     StringBuilder output = new StringBuilder();
     TTableStats tblStats = desc_.getTable().getTTableStats();
-    String numRows = String.valueOf(tblStats.num_rows);
-    if (tblStats.num_rows == -1) numRows = "unavailable";
     String totalBytes = PrintUtils.printBytes(tblStats.total_file_bytes);
     if (tblStats.total_file_bytes == -1) totalBytes = "unavailable";
     output.append(String.format("%stable: rows=%s size=%s",
-        prefix, numRows, totalBytes));
+        prefix,
+        PrintUtils.printEstCardinality(tblStats.num_rows),
+        totalBytes));
     if (tbl_.getNumClusteringCols() > 0) {
       output.append("\n");
-      String partNumRows = String.valueOf(partitionNumRows_);
-      if (partitionNumRows_ == -1) partNumRows = "unavailable";
       output.append(String.format("%spartitions: %s/%s rows=%s",
-          prefix, numPartitionsWithNumRows_, partitions_.size(), partNumRows));
+          prefix, numPartitionsWithNumRows_, partitions_.size(),
+          PrintUtils.printEstCardinality(partitionNumRows_)));
     }
     return output.toString();
   }
diff --git a/fe/src/main/java/org/apache/impala/planner/PlanNode.java b/fe/src/main/java/org/apache/impala/planner/PlanNode.java
index d264133..43642ab 100644
--- a/fe/src/main/java/org/apache/impala/planner/PlanNode.java
+++ b/fe/src/main/java/org/apache/impala/planner/PlanNode.java
@@ -345,7 +345,7 @@ abstract public class PlanNode extends TreeNode<PlanNode> {
       expBuilder.append("row-size=")
         .append(PrintUtils.printBytes(Math.round(avgRowSize_)))
         .append(" cardinality=")
-        .append(PrintUtils.printCardinality(cardinality_))
+        .append(PrintUtils.printEstCardinality(cardinality_))
         .append("\n");
     }
 
diff --git a/fe/src/main/java/org/apache/impala/planner/ScanNode.java b/fe/src/main/java/org/apache/impala/planner/ScanNode.java
index 99165d4..78c24af 100644
--- a/fe/src/main/java/org/apache/impala/planner/ScanNode.java
+++ b/fe/src/main/java/org/apache/impala/planner/ScanNode.java
@@ -26,6 +26,7 @@ import org.apache.impala.catalog.FeTable;
 import org.apache.impala.catalog.HdfsFileFormat;
 import org.apache.impala.catalog.Type;
 import org.apache.impala.common.NotImplementedException;
+import org.apache.impala.common.PrintUtils;
 import org.apache.impala.common.RuntimeEnv;
 import org.apache.impala.thrift.TNetworkAddress;
 import org.apache.impala.thrift.TQueryOptions;
@@ -111,12 +112,12 @@ abstract public class ScanNode extends PlanNode {
    * when the string returned by this method is embedded in a query's explain plan.
    */
   protected String getTableStatsExplainString(String prefix) {
-    StringBuilder output = new StringBuilder();
     TTableStats tblStats = desc_.getTable().getTTableStats();
-    String numRows = String.valueOf(tblStats.num_rows);
-    if (tblStats.num_rows == -1) numRows = "unavailable";
-    output.append(prefix + "table: rows=" + numRows);
-    return output.toString();
+    return new StringBuilder()
+      .append(prefix)
+      .append("table: rows=")
+      .append(PrintUtils.printEstCardinality(tblStats.num_rows))
+      .toString();
   }
 
   /**
@@ -132,12 +133,13 @@ abstract public class ScanNode extends PlanNode {
         columnsMissingStats.add(slot.getColumn().getName());
       }
     }
+    output.append(prefix);
     if (columnsMissingStats.isEmpty()) {
-      output.append(prefix + "columns: all");
+      output.append("columns: all");
     } else if (columnsMissingStats.size() == desc_.getSlots().size()) {
-      output.append(prefix + "columns: unavailable");
+      output.append("columns: unavailable");
     } else {
-      output.append(String.format("%scolumns missing stats: %s", prefix,
+      output.append(String.format("columns missing stats: %s",
           Joiner.on(", ").join(columnsMissingStats)));
     }
     return output.toString();
diff --git a/fe/src/test/java/org/apache/impala/testutil/TestUtils.java b/fe/src/test/java/org/apache/impala/testutil/TestUtils.java
index 26412fb..de6c27c 100644
--- a/fe/src/test/java/org/apache/impala/testutil/TestUtils.java
+++ b/fe/src/test/java/org/apache/impala/testutil/TestUtils.java
@@ -38,6 +38,7 @@ import javax.json.JsonWriterFactory;
 import javax.json.stream.JsonGenerator;
 
 import org.apache.impala.catalog.Catalog;
+import org.apache.impala.common.PrintUtils;
 import org.apache.impala.common.RuntimeEnv;
 import org.apache.impala.thrift.TClientRequest;
 import org.apache.impala.thrift.TNetworkAddress;
@@ -137,7 +138,7 @@ public class TestUtils {
 
   // Ignore the exact estimated row count, which depends on the file sizes.
   static IgnoreValueFilter SCAN_RANGE_ROW_COUNT_FILTER =
-      new IgnoreValueFilter("max-scan-range-rows", NUMBER_REGEX);
+      new IgnoreValueFilter("max-scan-range-rows", PrintUtils.METRIC_REGEX);
 
   // Filters that are always applied
   private static final List<ResultFilter> DEFAULT_FILTERS = Arrays.<ResultFilter>asList(
diff --git a/testdata/workloads/functional-planner/queries/PlannerTest/constant-folding.test b/testdata/workloads/functional-planner/queries/PlannerTest/constant-folding.test
index ad45570..c96efc6 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/constant-folding.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/constant-folding.test
@@ -63,9 +63,9 @@ PLAN-ROOT SINK
    predicates on o: !empty(o.o_lineitems), o_orderkey = CAST(4 AS BIGINT)
    predicates on o_lineitems: CAST(20 AS BIGINT) + CAST(l_linenumber AS BIGINT) < CAST(0 AS BIGINT)
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.23K
    parquet statistics predicates: c_custkey > CAST(10 AS BIGINT)
    parquet statistics predicates on o: o_orderkey = CAST(4 AS BIGINT)
    parquet dictionary predicates: c_custkey > CAST(10 AS BIGINT)
@@ -94,7 +94,7 @@ PLAN-ROOT SINK
    hbase filters: d:string_col EQUAL '4'
    predicates: tinyint_col = CAST(5 AS TINYINT), string_col = '4'
    stored statistics:
-     table: rows=10000
+     table: rows=10.00K
      columns: all
    mem-estimate=4.00KB mem-reservation=0B thread-reservation=0
    tuple-ids=0 row-size=107B cardinality=1
@@ -149,8 +149,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -188,8 +188,8 @@ PLAN-ROOT SINK
 |     partitions=24/24 files=24 size=478.45KB
 |     predicates: CAST(b.double_col AS DECIMAL(3,2)) > CAST(11.1 AS DECIMAL(6,1))
 |     stored statistics:
-|       table: rows=7300 size=478.45KB
-|       partitions: 24/24 rows=7300
+|       table: rows=7.30K size=478.45KB
+|       partitions: 24/24 rows=7.30K
 |       columns: all
 |     extrapolated-rows=disabled max-scan-range-rows=310
 |     mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -199,8 +199,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes a]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -237,8 +237,8 @@ PLAN-ROOT SINK
 |     partitions=24/24 files=24 size=478.45KB
 |     predicates: CAST(b.double_col AS DECIMAL(3,2)) > CAST(11.1 AS DECIMAL(6,1))
 |     stored statistics:
-|       table: rows=7300 size=478.45KB
-|       partitions: 24/24 rows=7300
+|       table: rows=7.30K size=478.45KB
+|       partitions: 24/24 rows=7.30K
 |       columns: all
 |     extrapolated-rows=disabled max-scan-range-rows=310
 |     mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -248,8 +248,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes a]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -289,8 +289,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -327,8 +327,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -368,8 +368,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -396,8 +396,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -461,8 +461,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    limit: 2
diff --git a/testdata/workloads/functional-planner/queries/PlannerTest/fk-pk-join-detection.test b/testdata/workloads/functional-planner/queries/PlannerTest/fk-pk-join-detection.test
index 5383858..bfe5e2e 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/fk-pk-join-detection.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/fk-pk-join-detection.test
@@ -21,9 +21,9 @@ PLAN-ROOT SINK
 |     partitions=1/1 files=1 size=12.60MB
 |     predicates: c_salutation = 'Mrs.'
 |     stored statistics:
-|       table: rows=100000 size=12.60MB
+|       table: rows=100.00K size=12.60MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=100000
+|     extrapolated-rows=disabled max-scan-range-rows=100.00K
 |     mem-estimate=48.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=219B cardinality=16.67K
 |     in pipelines: 01(GETNEXT)
@@ -32,10 +32,10 @@ PLAN-ROOT SINK
    partitions=1824/1824 files=1824 size=346.60MB
    runtime filters: RF000[bloom] -> ss_customer_sk
    stored statistics:
-     table: rows=2880404 size=346.60MB
-     partitions: 1824/1824 rows=2880404
+     table: rows=2.88M size=346.60MB
+     partitions: 1824/1824 rows=2.88M
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=130093
+   extrapolated-rows=disabled max-scan-range-rows=130.09K
    mem-estimate=128.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=100B cardinality=2.88M
    in pipelines: 00(GETNEXT)
@@ -64,9 +64,9 @@ PLAN-ROOT SINK
 |     partitions=1/1 files=1 size=12.60MB
 |     predicates: c_salutation = 'Mrs.'
 |     stored statistics:
-|       table: rows=100000 size=12.60MB
+|       table: rows=100.00K size=12.60MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=100000
+|     extrapolated-rows=disabled max-scan-range-rows=100.00K
 |     mem-estimate=48.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=219B cardinality=16.67K
 |     in pipelines: 01(GETNEXT)
@@ -74,10 +74,10 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpcds.store_sales]
    partitions=1824/1824 files=1824 size=346.60MB
    stored statistics:
-     table: rows=2880404 size=346.60MB
-     partitions: 1824/1824 rows=2880404
+     table: rows=2.88M size=346.60MB
+     partitions: 1824/1824 rows=2.88M
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=130093
+   extrapolated-rows=disabled max-scan-range-rows=130.09K
    mem-estimate=128.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=100B cardinality=2.88M
    in pipelines: 00(GETNEXT)
@@ -106,9 +106,9 @@ PLAN-ROOT SINK
 |     partitions=1/1 files=1 size=12.60MB
 |     predicates: c_salutation = 'Mrs.'
 |     stored statistics:
-|       table: rows=100000 size=12.60MB
+|       table: rows=100.00K size=12.60MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=100000
+|     extrapolated-rows=disabled max-scan-range-rows=100.00K
 |     mem-estimate=48.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=219B cardinality=16.67K
 |     in pipelines: 01(GETNEXT)
@@ -117,10 +117,10 @@ PLAN-ROOT SINK
    partitions=1824/1824 files=1824 size=346.60MB
    runtime filters: RF000[bloom] -> ss_customer_sk
    stored statistics:
-     table: rows=2880404 size=346.60MB
-     partitions: 1824/1824 rows=2880404
+     table: rows=2.88M size=346.60MB
+     partitions: 1824/1824 rows=2.88M
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=130093
+   extrapolated-rows=disabled max-scan-range-rows=130.09K
    mem-estimate=128.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=100B cardinality=2.88M
    in pipelines: 00(GETNEXT)
@@ -148,9 +148,9 @@ PLAN-ROOT SINK
 |     partitions=1/1 files=1 size=31.19MB
 |     predicates: sr_return_quantity < CAST(10 AS INT)
 |     stored statistics:
-|       table: rows=287514 size=31.19MB
+|       table: rows=287.51K size=31.19MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=287514
+|     extrapolated-rows=disabled max-scan-range-rows=287.51K
 |     mem-estimate=80.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=88B cardinality=28.75K
 |     in pipelines: 01(GETNEXT)
@@ -159,10 +159,10 @@ PLAN-ROOT SINK
    partitions=1824/1824 files=1824 size=346.60MB
    runtime filters: RF000[bloom] -> ss_item_sk, RF001[bloom] -> ss_ticket_number
    stored statistics:
-     table: rows=2880404 size=346.60MB
-     partitions: 1824/1824 rows=2880404
+     table: rows=2.88M size=346.60MB
+     partitions: 1824/1824 rows=2.88M
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=130093
+   extrapolated-rows=disabled max-scan-range-rows=130.09K
    mem-estimate=128.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=100B cardinality=2.88M
    in pipelines: 00(GETNEXT)
@@ -188,9 +188,9 @@ PLAN-ROOT SINK
 |--01:SCAN HDFS [tpcds.web_sales]
 |     partitions=1/1 files=1 size=140.07MB
 |     stored statistics:
-|       table: rows=719384 size=140.07MB
+|       table: rows=719.38K size=140.07MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=657377
+|     extrapolated-rows=disabled max-scan-range-rows=657.38K
 |     mem-estimate=160.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=144B cardinality=719.38K
 |     in pipelines: 01(GETNEXT)
@@ -199,10 +199,10 @@ PLAN-ROOT SINK
    partitions=1824/1824 files=1824 size=346.60MB
    runtime filters: RF000[bloom] -> ss_sold_time_sk
    stored statistics:
-     table: rows=2880404 size=346.60MB
-     partitions: 1824/1824 rows=2880404
+     table: rows=2.88M size=346.60MB
+     partitions: 1824/1824 rows=2.88M
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=130093
+   extrapolated-rows=disabled max-scan-range-rows=130.09K
    mem-estimate=128.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=100B cardinality=2.88M
    in pipelines: 00(GETNEXT)
@@ -230,9 +230,9 @@ PLAN-ROOT SINK
 |     partitions=1/1 files=1 size=9.84MB
 |     predicates: a.d_holiday = 'Y'
 |     stored statistics:
-|       table: rows=73049 size=9.84MB
+|       table: rows=73.05K size=9.84MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=73049
+|     extrapolated-rows=disabled max-scan-range-rows=73.05K
 |     mem-estimate=48.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=0 row-size=255B cardinality=36.52K
 |     in pipelines: 00(GETNEXT)
@@ -241,9 +241,9 @@ PLAN-ROOT SINK
    partitions=1/1 files=1 size=9.84MB
    runtime filters: RF000[bloom] -> b.d_date_sk
    stored statistics:
-     table: rows=73049 size=9.84MB
+     table: rows=73.05K size=9.84MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=73049
+   extrapolated-rows=disabled max-scan-range-rows=73.05K
    mem-estimate=48.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=1 row-size=255B cardinality=73.05K
    in pipelines: 01(GETNEXT)
@@ -274,9 +274,9 @@ PLAN-ROOT SINK
 |--02:SCAN HDFS [tpcds.customer]
 |     partitions=1/1 files=1 size=12.60MB
 |     stored statistics:
-|       table: rows=100000 size=12.60MB
+|       table: rows=100.00K size=12.60MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=100000
+|     extrapolated-rows=disabled max-scan-range-rows=100.00K
 |     mem-estimate=48.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=2 row-size=4B cardinality=100.00K
 |     in pipelines: 02(GETNEXT)
@@ -292,9 +292,9 @@ PLAN-ROOT SINK
 |--04:SCAN HDFS [tpcds.date_dim d2]
 |     partitions=1/1 files=1 size=9.84MB
 |     stored statistics:
-|       table: rows=73049 size=9.84MB
+|       table: rows=73.05K size=9.84MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=73049
+|     extrapolated-rows=disabled max-scan-range-rows=73.05K
 |     mem-estimate=48.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=4 row-size=4B cardinality=73.05K
 |     in pipelines: 04(GETNEXT)
@@ -319,9 +319,9 @@ PLAN-ROOT SINK
 |  |     partitions=1/1 files=1 size=9.84MB
 |  |     predicates: d1.d_fy_week_seq = CAST(1000 AS INT)
 |  |     stored statistics:
-|  |       table: rows=73049 size=9.84MB
+|  |       table: rows=73.05K size=9.84MB
 |  |       columns: all
-|  |     extrapolated-rows=disabled max-scan-range-rows=73049
+|  |     extrapolated-rows=disabled max-scan-range-rows=73.05K
 |  |     mem-estimate=48.00MB mem-reservation=8.00MB thread-reservation=1
 |  |     tuple-ids=3 row-size=8B cardinality=7
 |  |     in pipelines: 03(GETNEXT)
@@ -330,10 +330,10 @@ PLAN-ROOT SINK
 |     partitions=1824/1824 files=1824 size=346.60MB
 |     runtime filters: RF000[bloom] -> ss_addr_sk, RF008[bloom] -> ss_sold_date_sk
 |     stored statistics:
-|       table: rows=2880404 size=346.60MB
-|       partitions: 1824/1824 rows=2880404
+|       table: rows=2.88M size=346.60MB
+|       partitions: 1824/1824 rows=2.88M
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=130093
+|     extrapolated-rows=disabled max-scan-range-rows=130.09K
 |     mem-estimate=128.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=0 row-size=24B cardinality=2.88M
 |     in pipelines: 00(GETNEXT)
@@ -342,9 +342,9 @@ PLAN-ROOT SINK
    partitions=1/1 files=1 size=31.19MB
    runtime filters: RF002[bloom] -> sr_returned_date_sk, RF004[bloom] -> sr_item_sk, RF005[bloom] -> sr_ticket_number
    stored statistics:
-     table: rows=287514 size=31.19MB
+     table: rows=287.51K size=31.19MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=287514
+   extrapolated-rows=disabled max-scan-range-rows=287.51K
    mem-estimate=80.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=1 row-size=20B cardinality=287.51K
    in pipelines: 01(GETNEXT)
@@ -370,9 +370,9 @@ PLAN-ROOT SINK
 |--01:SCAN HDFS [tpcds.customer]
 |     partitions=1/1 files=1 size=12.60MB
 |     stored statistics:
-|       table: rows=100000 size=12.60MB
+|       table: rows=100.00K size=12.60MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=100000
+|     extrapolated-rows=disabled max-scan-range-rows=100.00K
 |     mem-estimate=48.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=219B cardinality=100.00K
 |     in pipelines: 01(GETNEXT)
@@ -381,10 +381,10 @@ PLAN-ROOT SINK
    partitions=1824/1824 files=1824 size=346.60MB
    runtime filters: RF000[bloom] -> ss_customer_sk % 10
    stored statistics:
-     table: rows=2880404 size=346.60MB
-     partitions: 1824/1824 rows=2880404
+     table: rows=2.88M size=346.60MB
+     partitions: 1824/1824 rows=2.88M
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=130093
+   extrapolated-rows=disabled max-scan-range-rows=130.09K
    mem-estimate=128.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=100B cardinality=2.88M
    in pipelines: 00(GETNEXT)
@@ -422,10 +422,10 @@ PLAN-ROOT SINK
    partitions=1824/1824 files=1824 size=346.60MB
    runtime filters: RF000[bloom] -> ss_customer_sk
    stored statistics:
-     table: rows=2880404 size=346.60MB
-     partitions: 1824/1824 rows=2880404
+     table: rows=2.88M size=346.60MB
+     partitions: 1824/1824 rows=2.88M
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=130093
+   extrapolated-rows=disabled max-scan-range-rows=130.09K
    mem-estimate=128.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=4B cardinality=2.88M
    in pipelines: 00(GETNEXT)
@@ -451,9 +451,9 @@ PLAN-ROOT SINK
 |--01:SCAN HDFS [tpcds.customer]
 |     partitions=1/1 files=1 size=12.60MB
 |     stored statistics:
-|       table: rows=100000 size=12.60MB
+|       table: rows=100.00K size=12.60MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=100000
+|     extrapolated-rows=disabled max-scan-range-rows=100.00K
 |     mem-estimate=48.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=4B cardinality=100.00K
 |     in pipelines: 01(GETNEXT)
@@ -499,9 +499,9 @@ PLAN-ROOT SINK
 |  01:SCAN HDFS [tpcds.web_sales]
 |     partitions=1/1 files=1 size=140.07MB
 |     stored statistics:
-|       table: rows=719384 size=140.07MB
+|       table: rows=719.38K size=140.07MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=657377
+|     extrapolated-rows=disabled max-scan-range-rows=657.38K
 |     mem-estimate=160.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=4B cardinality=719.38K
 |     in pipelines: 01(GETNEXT)
@@ -510,10 +510,10 @@ PLAN-ROOT SINK
    partitions=1824/1824 files=1824 size=346.60MB
    runtime filters: RF000[bloom] -> ss_sold_time_sk
    stored statistics:
-     table: rows=2880404 size=346.60MB
-     partitions: 1824/1824 rows=2880404
+     table: rows=2.88M size=346.60MB
+     partitions: 1824/1824 rows=2.88M
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=130093
+   extrapolated-rows=disabled max-scan-range-rows=130.09K
    mem-estimate=128.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=100B cardinality=2.88M
    in pipelines: 00(GETNEXT)
diff --git a/testdata/workloads/functional-planner/queries/PlannerTest/max-row-size.test b/testdata/workloads/functional-planner/queries/PlannerTest/max-row-size.test
index 793c231..867268b 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/max-row-size.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/max-row-size.test
@@ -50,9 +50,9 @@ Per-Host Resources: mem-estimate=41.95MB mem-reservation=33.94MB thread-reservat
    partitions=1/1 files=1 size=12.31MB
    runtime filters: RF000[bloom] -> c_nationkey
    stored statistics:
-     table: rows=150000 size=12.31MB
+     table: rows=150.00K size=12.31MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=150000
+   extrapolated-rows=disabled max-scan-range-rows=150.00K
    mem-estimate=24.00MB mem-reservation=16.00MB thread-reservation=1
    tuple-ids=0 row-size=218B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -97,9 +97,9 @@ Per-Host Resources: mem-estimate=359.29MB mem-reservation=86.00MB thread-reserva
 |  01:SCAN HDFS [tpch_parquet.orders, RANDOM]
 |     partitions=1/1 files=2 size=54.07MB
 |     stored statistics:
-|       table: rows=1500000 size=54.07MB
+|       table: rows=1.50M size=54.07MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1177136
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=40.00MB mem-reservation=24.00MB thread-reservation=1
 |     tuple-ids=1 row-size=171B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -107,9 +107,9 @@ Per-Host Resources: mem-estimate=359.29MB mem-reservation=86.00MB thread-reserva
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -153,9 +153,9 @@ Per-Host Resources: mem-estimate=124.02MB mem-reservation=74.00MB thread-reserva
 |  01:SCAN HDFS [tpch_parquet.orders, RANDOM]
 |     partitions=1/1 files=2 size=54.07MB
 |     stored statistics:
-|       table: rows=1500000 size=54.07MB
+|       table: rows=1.50M size=54.07MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1177136
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=1
 |     tuple-ids=1 row-size=8B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -163,9 +163,9 @@ Per-Host Resources: mem-estimate=124.02MB mem-reservation=74.00MB thread-reserva
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -237,9 +237,9 @@ Per-Host Resources: mem-estimate=99.65MB mem-reservation=66.00MB thread-reservat
 |  01:SCAN HDFS [tpch_parquet.orders, RANDOM]
 |     partitions=1/1 files=2 size=54.07MB
 |     stored statistics:
-|       table: rows=1500000 size=54.07MB
+|       table: rows=1.50M size=54.07MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1177136
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=40.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=21B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -255,9 +255,9 @@ Per-Host Resources: mem-estimate=81.00MB mem-reservation=5.00MB thread-reservati
    partitions=1/1 files=3 size=193.60MB
    runtime filters: RF000[bloom] -> l_orderkey
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=4.00MB thread-reservation=1
    tuple-ids=0 row-size=8B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -305,9 +305,9 @@ Per-Host Resources: mem-estimate=806.43MB mem-reservation=74.00MB thread-reserva
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -359,9 +359,9 @@ Per-Host Resources: mem-estimate=168.14MB mem-reservation=50.00MB thread-reserva
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=16.00MB thread-reservation=1
    tuple-ids=0 row-size=29B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -412,8 +412,8 @@ Per-Host Resources: mem-estimate=16.00MB mem-reservation=32.00KB thread-reservat
 00:SCAN HDFS [functional.alltypes, RANDOM]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=16.00MB mem-reservation=32.00KB thread-reservation=1
diff --git a/testdata/workloads/functional-planner/queries/PlannerTest/mt-dop-validation.test b/testdata/workloads/functional-planner/queries/PlannerTest/mt-dop-validation.test
index f749a4e..c25e99a 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/mt-dop-validation.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/mt-dop-validation.test
@@ -58,7 +58,7 @@ PLAN-ROOT SINK
 |  in pipelines: 01(GETNEXT), 00(OPEN)
 |
 00:SCAN HDFS [functional_parquet.alltypes]
-   partitions=24/24 files=24 size=189.28KB
+   partitions=24/24 files=24 size=190.12KB
    predicates: id < CAST(10 AS INT)
    stored statistics:
      table: rows=unavailable size=unavailable
@@ -113,7 +113,7 @@ Per-Host Resources: mem-estimate=432.00MB mem-reservation=102.07MB thread-reserv
 |  in pipelines: 00(GETNEXT)
 |
 00:SCAN HDFS [functional_parquet.alltypes, RANDOM]
-   partitions=24/24 files=24 size=189.28KB
+   partitions=24/24 files=24 size=190.12KB
    predicates: id < CAST(10 AS INT)
    stored statistics:
      table: rows=unavailable size=unavailable
@@ -152,7 +152,7 @@ PLAN-ROOT SINK
 |  in pipelines: 01(GETNEXT), 00(OPEN)
 |
 00:SCAN HDFS [functional_parquet.alltypes]
-   partitions=24/24 files=24 size=189.28KB
+   partitions=24/24 files=24 size=190.12KB
    predicates: id < CAST(10 AS INT)
    stored statistics:
      table: rows=unavailable size=unavailable
@@ -200,7 +200,7 @@ Per-Host Resources: mem-estimate=30.32MB mem-reservation=30.00MB thread-reservat
 F00:PLAN FRAGMENT [RANDOM] hosts=3 instances=9
 Per-Host Resources: mem-estimate=48.00MB mem-reservation=48.00KB thread-reservation=3
 00:SCAN HDFS [functional_parquet.alltypes, RANDOM]
-   partitions=24/24 files=24 size=189.28KB
+   partitions=24/24 files=24 size=190.12KB
    predicates: id < CAST(10 AS INT)
    stored statistics:
      table: rows=unavailable size=unavailable
@@ -273,9 +273,9 @@ PLAN-ROOT SINK
    predicates on o: !empty(o.o_lineitems), o_orderkey < CAST(5 AS BIGINT)
    predicates on o_lineitems: l_linenumber < CAST(3 AS INT)
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders
-   extrapolated-rows=disabled max-scan-range-rows=50116
+   extrapolated-rows=disabled max-scan-range-rows=50.11K
    parquet statistics predicates: c_custkey < CAST(10 AS BIGINT)
    parquet statistics predicates on o: o_orderkey < CAST(5 AS BIGINT)
    parquet statistics predicates on o_lineitems: l_linenumber < CAST(3 AS INT)
@@ -348,9 +348,9 @@ Per-Host Resources: mem-estimate=312.00MB mem-reservation=312.00MB thread-reserv
    predicates on o: !empty(o.o_lineitems), o_orderkey < CAST(5 AS BIGINT)
    predicates on o_lineitems: l_linenumber < CAST(3 AS INT)
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders
-   extrapolated-rows=disabled max-scan-range-rows=50116
+   extrapolated-rows=disabled max-scan-range-rows=50.11K
    parquet statistics predicates: c_custkey < CAST(10 AS BIGINT)
    parquet statistics predicates on o: o_orderkey < CAST(5 AS BIGINT)
    parquet statistics predicates on o_lineitems: l_linenumber < CAST(3 AS INT)
@@ -411,9 +411,9 @@ PLAN-ROOT SINK
    predicates: !empty(c.c_orders), !empty(c.c_orders)
    predicates on o1: o1.o_orderkey < CAST(5 AS BIGINT)
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders, c_orders
-   extrapolated-rows=disabled max-scan-range-rows=50116
+   extrapolated-rows=disabled max-scan-range-rows=50.11K
    parquet statistics predicates on o1: o1.o_orderkey < CAST(5 AS BIGINT)
    parquet dictionary predicates on o1: o1.o_orderkey < CAST(5 AS BIGINT)
    mem-estimate=88.00MB mem-reservation=16.00MB thread-reservation=0
@@ -472,9 +472,9 @@ Per-Host Resources: mem-estimate=269.81MB mem-reservation=53.81MB thread-reserva
    predicates: !empty(c.c_orders), !empty(c.c_orders)
    predicates on o1: o1.o_orderkey < CAST(5 AS BIGINT)
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders, c_orders
-   extrapolated-rows=disabled max-scan-range-rows=50116
+   extrapolated-rows=disabled max-scan-range-rows=50.11K
    parquet statistics predicates on o1: o1.o_orderkey < CAST(5 AS BIGINT)
    parquet dictionary predicates on o1: o1.o_orderkey < CAST(5 AS BIGINT)
    mem-estimate=88.00MB mem-reservation=16.00MB thread-reservation=0
diff --git a/testdata/workloads/functional-planner/queries/PlannerTest/parquet-filtering-disabled.test b/testdata/workloads/functional-planner/queries/PlannerTest/parquet-filtering-disabled.test
index 07e5ef2..3908f90 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/parquet-filtering-disabled.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/parquet-filtering-disabled.test
@@ -228,9 +228,9 @@ PLAN-ROOT SINK
    predicates on o: !empty(o.o_lineitems), o.o_orderkey > CAST(0 AS BIGINT)
    predicates on l: l.l_partkey > CAST(0 AS BIGINT)
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.22K
    mem-estimate=264.00MB mem-reservation=16.00MB thread-reservation=1
    tuple-ids=0 row-size=20B cardinality=15.00K
    in pipelines: 00(GETNEXT)
@@ -301,7 +301,7 @@ PLAN-ROOT SINK
    predicates on o: !empty(o.o_lineitems)
    predicates on l: l.l_shipdate = '1994-08-19', l.l_receiptdate = '1994-08-24', l.l_shipmode = 'RAIL', l.l_returnflag = 'R', l.l_comment IS NULL
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders
    extrapolated-rows=disabled max-scan-range-rows=44225
    mem-estimate=616.00MB mem-reservation=32.00MB thread-reservation=1
diff --git a/testdata/workloads/functional-planner/queries/PlannerTest/parquet-filtering.test b/testdata/workloads/functional-planner/queries/PlannerTest/parquet-filtering.test
index c520c6b..33edda9 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/parquet-filtering.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/parquet-filtering.test
@@ -52,8 +52,8 @@ PLAN-ROOT SINK
    partitions=24/24 files=24 size=478.45KB
    predicates: int_col IS NULL, int_col > CAST(1 AS INT), int_col > CAST(tinyint_col AS INT), CAST(int_col AS DOUBLE) * rand() > CAST(50 AS DOUBLE)
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -120,8 +120,8 @@ PLAN-ROOT SINK
    partitions=22/24 files=22 size=437.72KB
    predicates: id = CAST(1 AS INT), string_col IN ('aaaa', 'bbbb', 'cccc'), smallint_col IN (CAST(1 AS SMALLINT), CAST(2 AS SMALLINT), CAST(3 AS SMALLINT), CAST(4 AS SMALLINT), CAST(5 AS SMALLINT)), bool_col, bigint_col < CAST(5000 AS BIGINT), double_col > CAST(100.00 AS DOUBLE), float_col > CAST(50.00 AS FLOAT), tinyint_col < CAST(50 AS TINYINT), int_col % CAST(2 AS INT) = CAST(1 AS INT), timestamp_cmp(timestamp_col, TIMESTAMP '2016-11-20 00:00:00') = CAST(1 AS INT), date_string_col > ' [...]
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 22/22 rows=6680
+     table: rows=7.30K size=478.45KB
+     partitions: 22/22 rows=6.68K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=339
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -427,9 +427,9 @@ PLAN-ROOT SINK
    predicates on o: !empty(o.o_lineitems), o.o_orderkey > CAST(0 AS BIGINT)
    predicates on l: l.l_partkey > CAST(0 AS BIGINT)
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.23K
    parquet statistics predicates: c_custkey > CAST(0 AS BIGINT)
    parquet statistics predicates on o: o.o_orderkey > CAST(0 AS BIGINT)
    parquet statistics predicates on l: l.l_partkey > CAST(0 AS BIGINT)
@@ -552,9 +552,9 @@ PLAN-ROOT SINK
    predicates on o: !empty(o.o_lineitems)
    predicates on l: l.l_shipdate = '1994-08-19', l.l_receiptdate = '1994-08-24', l.l_shipmode = 'RAIL', l.l_returnflag = 'R', l.l_comment IS NULL
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.23K
    parquet statistics predicates on l: l.l_shipdate = '1994-08-19', l.l_receiptdate = '1994-08-24', l.l_shipmode = 'RAIL', l.l_returnflag = 'R'
    parquet dictionary predicates on l: l.l_shipdate = '1994-08-19', l.l_receiptdate = '1994-08-24', l.l_shipmode = 'RAIL', l.l_returnflag = 'R'
    mem-estimate=616.00MB mem-reservation=32.00MB thread-reservation=1
diff --git a/testdata/workloads/functional-planner/queries/PlannerTest/resource-requirements.test b/testdata/workloads/functional-planner/queries/PlannerTest/resource-requirements.test
index a90363f..d53a78b 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/resource-requirements.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/resource-requirements.test
@@ -13,9 +13,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_parquet.lineitem]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -39,9 +39,9 @@ Per-Host Resources: mem-estimate=80.00MB mem-reservation=40.00MB thread-reservat
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -65,9 +65,9 @@ Per-Host Resources: mem-estimate=160.00MB mem-reservation=80.00MB thread-reserva
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=0
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -88,9 +88,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_parquet.lineitem]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=4.00MB thread-reservation=1
    tuple-ids=0 row-size=38B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -114,9 +114,9 @@ Per-Host Resources: mem-estimate=80.00MB mem-reservation=4.00MB thread-reservati
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=4.00MB thread-reservation=1
    tuple-ids=0 row-size=38B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -140,9 +140,9 @@ Per-Host Resources: mem-estimate=160.00MB mem-reservation=8.00MB thread-reservat
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=4.00MB thread-reservation=0
    tuple-ids=0 row-size=38B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -417,9 +417,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_parquet.customer]
    partitions=1/1 files=1 size=12.31MB
    stored statistics:
-     table: rows=150000 size=12.31MB
+     table: rows=150.00K size=12.31MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=150000
+   extrapolated-rows=disabled max-scan-range-rows=150.00K
    mem-estimate=24.00MB mem-reservation=128.00KB thread-reservation=1
    tuple-ids=0 row-size=2B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -440,9 +440,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_parquet.customer]
    partitions=1/1 files=1 size=12.31MB
    stored statistics:
-     table: rows=150000 size=12.31MB
+     table: rows=150.00K size=12.31MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=150000
+   extrapolated-rows=disabled max-scan-range-rows=150.00K
    mem-estimate=24.00MB mem-reservation=2.00MB thread-reservation=1
    tuple-ids=0 row-size=8B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -490,9 +490,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_nested_parquet.customer.c_orders]
    partitions=1/1 files=4 size=288.99MB
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.22K
    mem-estimate=88.00MB mem-reservation=24.00MB thread-reservation=1
    tuple-ids=0 row-size=64B cardinality=1.50M
    in pipelines: 00(GETNEXT)
@@ -515,9 +515,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_nested_parquet.customer.c_orders]
    partitions=1/1 files=4 size=288.99MB
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.22K
    mem-estimate=88.00MB mem-reservation=4.00MB thread-reservation=1
    tuple-ids=0 row-size=16B cardinality=1.50M
    in pipelines: 00(GETNEXT)
@@ -539,9 +539,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_nested_parquet.customer.c_orders]
    partitions=1/1 files=4 size=288.99MB
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.22K
    mem-estimate=88.00MB mem-reservation=4.00MB thread-reservation=1
    tuple-ids=0 row-size=8B cardinality=1.50M
    in pipelines: 00(GETNEXT)
@@ -591,9 +591,9 @@ PLAN-ROOT SINK
    partitions=1/1 files=4 size=288.99MB
    predicates: !empty(c.c_orders)
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.23K
    mem-estimate=88.00MB mem-reservation=32.00MB thread-reservation=1
    tuple-ids=0 row-size=20B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -641,9 +641,9 @@ PLAN-ROOT SINK
    partitions=1/1 files=4 size=288.99MB
    predicates: !empty(c.c_orders)
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.23K
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=20B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -692,9 +692,9 @@ PLAN-ROOT SINK
    partitions=1/1 files=4 size=288.99MB
    predicates: !empty(c.c_orders)
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.23K
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=20B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -742,9 +742,9 @@ PLAN-ROOT SINK
    partitions=1/1 files=4 size=288.99MB
    predicates: !empty(c.c_orders)
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.23K
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=20B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -792,9 +792,9 @@ PLAN-ROOT SINK
    partitions=1/1 files=4 size=288.99MB
    predicates: !empty(c.c_orders)
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns: unavailable
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.23K
    mem-estimate=88.00MB mem-reservation=4.00MB thread-reservation=1
    tuple-ids=0 row-size=12B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -865,9 +865,9 @@ PLAN-ROOT SINK
    predicates: !empty(c.c_orders)
    predicates on o: !empty(o.o_lineitems)
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.23K
    mem-estimate=88.00MB mem-reservation=16.00MB thread-reservation=1
    tuple-ids=0 row-size=20B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -887,9 +887,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch.lineitem]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -913,9 +913,9 @@ Per-Host Resources: mem-estimate=88.00MB mem-reservation=8.00MB thread-reservati
 00:SCAN HDFS [tpch.lineitem, RANDOM]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -939,9 +939,9 @@ Per-Host Resources: mem-estimate=176.00MB mem-reservation=16.00MB thread-reserva
 00:SCAN HDFS [tpch.lineitem, RANDOM]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=0
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -961,9 +961,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch.lineitem]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=38B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -987,9 +987,9 @@ Per-Host Resources: mem-estimate=88.00MB mem-reservation=8.00MB thread-reservati
 00:SCAN HDFS [tpch.lineitem, RANDOM]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=38B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -1013,9 +1013,9 @@ Per-Host Resources: mem-estimate=176.00MB mem-reservation=16.00MB thread-reserva
 00:SCAN HDFS [tpch.lineitem, RANDOM]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=0
    tuple-ids=0 row-size=38B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -1036,8 +1036,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=16.00MB mem-reservation=32.00KB thread-reservation=1
@@ -1064,8 +1064,8 @@ Per-Host Resources: mem-estimate=16.00MB mem-reservation=32.00KB thread-reservat
 00:SCAN HDFS [functional.alltypes, RANDOM]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=16.00MB mem-reservation=32.00KB thread-reservation=1
@@ -1092,8 +1092,8 @@ Per-Host Resources: mem-estimate=32.00MB mem-reservation=64.00KB thread-reservat
 00:SCAN HDFS [functional.alltypes, RANDOM]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=16.00MB mem-reservation=32.00KB thread-reservation=0
@@ -1355,9 +1355,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_orc_def.lineitem]
    partitions=1/1 files=6 size=144.66MB
    stored statistics:
-     table: rows=6001215 size=144.66MB
+     table: rows=6.00M size=144.66MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1075682
+   extrapolated-rows=disabled max-scan-range-rows=1.08M
    mem-estimate=40.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -1377,9 +1377,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_orc_def.lineitem]
    partitions=1/1 files=6 size=144.66MB
    stored statistics:
-     table: rows=6001215 size=144.66MB
+     table: rows=6.00M size=144.66MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1075682
+   extrapolated-rows=disabled max-scan-range-rows=1.08M
    mem-estimate=40.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=38B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -1726,9 +1726,9 @@ PLAN-ROOT SINK
 |--02:SCAN HDFS [tpch.lineitem]
 |     partitions=1/1 files=1 size=718.94MB
 |     stored statistics:
-|       table: rows=6001215 size=718.94MB
+|       table: rows=6.00M size=718.94MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1068457
+|     extrapolated-rows=disabled max-scan-range-rows=1.07M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=231B cardinality=6.00M
 |     in pipelines: 02(GETNEXT)
@@ -1736,9 +1736,9 @@ PLAN-ROOT SINK
 01:SCAN HDFS [tpch.lineitem]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 01(GETNEXT)
@@ -1769,9 +1769,9 @@ Per-Host Resources: mem-estimate=88.00MB mem-reservation=8.00MB thread-reservati
 |--02:SCAN HDFS [tpch.lineitem, RANDOM]
 |     partitions=1/1 files=1 size=718.94MB
 |     stored statistics:
-|       table: rows=6001215 size=718.94MB
+|       table: rows=6.00M size=718.94MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1068457
+|     extrapolated-rows=disabled max-scan-range-rows=1.07M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=231B cardinality=6.00M
 |     in pipelines: 02(GETNEXT)
@@ -1779,9 +1779,9 @@ Per-Host Resources: mem-estimate=88.00MB mem-reservation=8.00MB thread-reservati
 01:SCAN HDFS [tpch.lineitem, RANDOM]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 01(GETNEXT)
@@ -1812,9 +1812,9 @@ Per-Host Resources: mem-estimate=176.00MB mem-reservation=16.00MB thread-reserva
 |--02:SCAN HDFS [tpch.lineitem, RANDOM]
 |     partitions=1/1 files=1 size=718.94MB
 |     stored statistics:
-|       table: rows=6001215 size=718.94MB
+|       table: rows=6.00M size=718.94MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1068457
+|     extrapolated-rows=disabled max-scan-range-rows=1.07M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=0
 |     tuple-ids=1 row-size=231B cardinality=6.00M
 |     in pipelines: 02(GETNEXT)
@@ -1822,9 +1822,9 @@ Per-Host Resources: mem-estimate=176.00MB mem-reservation=16.00MB thread-reserva
 01:SCAN HDFS [tpch.lineitem, RANDOM]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=0
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 01(GETNEXT)
@@ -1854,9 +1854,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_parquet.lineitem]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=4.00MB thread-reservation=1
    tuple-ids=0 row-size=8B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -1902,9 +1902,9 @@ Per-Host Resources: mem-estimate=114.00MB mem-reservation=38.00MB thread-reserva
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=4.00MB thread-reservation=1
    tuple-ids=0 row-size=8B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -1948,11 +1948,11 @@ Per-Host Resources: mem-estimate=228.00MB mem-reservation=76.00MB thread-reserva
 |  in pipelines: 00(GETNEXT)
 |
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
-   partitions=1/1 files=3 size=193.60MB
+   partitions=1/1 files=3 size=193.59MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=4.00MB thread-reservation=0
    tuple-ids=0 row-size=8B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -1978,9 +1978,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_parquet.lineitem]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=1.00MB mem-reservation=128.00KB thread-reservation=1
    tuple-ids=0 row-size=8B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2016,9 +2016,9 @@ Per-Host Resources: mem-estimate=11.00MB mem-reservation=128.00KB thread-reserva
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=1.00MB mem-reservation=128.00KB thread-reservation=1
    tuple-ids=0 row-size=8B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2054,9 +2054,9 @@ Per-Host Resources: mem-estimate=180.00MB mem-reservation=256.00KB thread-reserv
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=128.00KB thread-reservation=0
    tuple-ids=0 row-size=8B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2084,9 +2084,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_parquet.lineitem]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2117,9 +2117,9 @@ Per-Host Resources: mem-estimate=118.00MB mem-reservation=52.00MB thread-reserva
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2150,9 +2150,9 @@ Per-Host Resources: mem-estimate=236.00MB mem-reservation=104.00MB thread-reserv
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=0
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2182,9 +2182,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_parquet.lineitem]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2217,9 +2217,9 @@ Per-Host Resources: mem-estimate=80.02MB mem-reservation=40.00MB thread-reservat
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2252,9 +2252,9 @@ Per-Host Resources: mem-estimate=160.04MB mem-reservation=80.00MB thread-reserva
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=0
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2284,9 +2284,9 @@ PLAN-ROOT SINK
 |--01:SCAN HDFS [tpch.orders]
 |     partitions=1/1 files=1 size=162.56MB
 |     stored statistics:
-|       table: rows=1500000 size=162.56MB
+|       table: rows=1.50M size=162.56MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1181132
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=171B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -2295,9 +2295,9 @@ PLAN-ROOT SINK
    partitions=1/1 files=1 size=718.94MB
    runtime filters: RF000[bloom] -> l_orderkey
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2337,9 +2337,9 @@ Per-Host Resources: mem-estimate=368.29MB mem-reservation=43.00MB thread-reserva
 |  01:SCAN HDFS [tpch.orders, RANDOM]
 |     partitions=1/1 files=1 size=162.56MB
 |     stored statistics:
-|       table: rows=1500000 size=162.56MB
+|       table: rows=1.50M size=162.56MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1181132
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=171B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -2348,9 +2348,9 @@ Per-Host Resources: mem-estimate=368.29MB mem-reservation=43.00MB thread-reserva
    partitions=1/1 files=1 size=718.94MB
    runtime filters: RF000[bloom] -> l_orderkey
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2398,9 +2398,9 @@ Per-Host Resources: mem-estimate=715.89MB mem-reservation=86.00MB thread-reserva
 |  01:SCAN HDFS [tpch.orders, RANDOM]
 |     partitions=1/1 files=1 size=162.56MB
 |     stored statistics:
-|       table: rows=1500000 size=162.56MB
+|       table: rows=1.50M size=162.56MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1181132
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=0
 |     tuple-ids=1 row-size=171B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -2409,9 +2409,9 @@ Per-Host Resources: mem-estimate=715.89MB mem-reservation=86.00MB thread-reserva
    partitions=1/1 files=1 size=718.94MB
    runtime filters: RF000[bloom] -> l_orderkey
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=0
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2441,9 +2441,9 @@ PLAN-ROOT SINK
 |--01:SCAN HDFS [tpch.orders]
 |     partitions=1/1 files=1 size=162.56MB
 |     stored statistics:
-|       table: rows=1500000 size=162.56MB
+|       table: rows=1.50M size=162.56MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1181132
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=171B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -2452,9 +2452,9 @@ PLAN-ROOT SINK
    partitions=1/1 files=1 size=718.94MB
    runtime filters: RF000[bloom] -> l_orderkey
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2494,9 +2494,9 @@ Per-Host Resources: mem-estimate=111.68MB mem-reservation=35.00MB thread-reserva
 |  01:SCAN HDFS [tpch.orders, RANDOM]
 |     partitions=1/1 files=1 size=162.56MB
 |     stored statistics:
-|       table: rows=1500000 size=162.56MB
+|       table: rows=1.50M size=162.56MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1181132
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=171B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -2512,9 +2512,9 @@ Per-Host Resources: mem-estimate=89.00MB mem-reservation=9.00MB thread-reservati
    partitions=1/1 files=1 size=718.94MB
    runtime filters: RF000[bloom] -> l_orderkey
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2562,9 +2562,9 @@ Per-Host Resources: mem-estimate=114.40MB mem-reservation=70.00MB thread-reserva
 |  01:SCAN HDFS [tpch.orders, RANDOM]
 |     partitions=1/1 files=1 size=162.56MB
 |     stored statistics:
-|       table: rows=1500000 size=162.56MB
+|       table: rows=1.50M size=162.56MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1181132
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=0
 |     tuple-ids=1 row-size=171B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -2580,9 +2580,9 @@ Per-Host Resources: mem-estimate=178.00MB mem-reservation=18.00MB thread-reserva
    partitions=1/1 files=1 size=718.94MB
    runtime filters: RF000[bloom] -> l_orderkey
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=0
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2608,9 +2608,9 @@ PLAN-ROOT SINK
 |--01:SCAN HDFS [tpch.orders]
 |     partitions=1/1 files=1 size=162.56MB
 |     stored statistics:
-|       table: rows=1500000 size=162.56MB
+|       table: rows=1.50M size=162.56MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1181132
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=171B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -2618,9 +2618,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch.lineitem]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2656,9 +2656,9 @@ Per-Host Resources: mem-estimate=342.84MB mem-reservation=8.00MB thread-reservat
 |  01:SCAN HDFS [tpch.orders, RANDOM]
 |     partitions=1/1 files=1 size=162.56MB
 |     stored statistics:
-|       table: rows=1500000 size=162.56MB
+|       table: rows=1.50M size=162.56MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1181132
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=171B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -2666,9 +2666,9 @@ Per-Host Resources: mem-estimate=342.84MB mem-reservation=8.00MB thread-reservat
 00:SCAN HDFS [tpch.lineitem, RANDOM]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2711,9 +2711,9 @@ Per-Host Resources: mem-estimate=686.35MB mem-reservation=16.00MB thread-reserva
 |  01:SCAN HDFS [tpch.orders, RANDOM]
 |     partitions=1/1 files=1 size=162.56MB
 |     stored statistics:
-|       table: rows=1500000 size=162.56MB
+|       table: rows=1.50M size=162.56MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1181132
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=0
 |     tuple-ids=1 row-size=171B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -2721,9 +2721,9 @@ Per-Host Resources: mem-estimate=686.35MB mem-reservation=16.00MB thread-reserva
 00:SCAN HDFS [tpch.lineitem, RANDOM]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=0
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -2807,8 +2807,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=16.00MB mem-reservation=32.00KB thread-reservation=1
@@ -2856,8 +2856,8 @@ Per-Host Resources: mem-estimate=16.00MB mem-reservation=32.00KB thread-reservat
 00:SCAN HDFS [functional.alltypes, RANDOM]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=16.00MB mem-reservation=32.00KB thread-reservation=1
@@ -2905,8 +2905,8 @@ Per-Host Resources: mem-estimate=32.00MB mem-reservation=64.00KB thread-reservat
 00:SCAN HDFS [functional.alltypes, RANDOM]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=16.00MB mem-reservation=32.00KB thread-reservation=0
@@ -2976,9 +2976,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_parquet.orders]
    partitions=1/1 files=2 size=54.07MB
    stored statistics:
-     table: rows=1500000 size=54.07MB
+     table: rows=1.50M size=54.07MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1177136
+   extrapolated-rows=disabled max-scan-range-rows=1.18M
    mem-estimate=40.00MB mem-reservation=24.00MB thread-reservation=1
    tuple-ids=0 row-size=171B cardinality=1.50M
    in pipelines: 00(GETNEXT)
@@ -3047,9 +3047,9 @@ Per-Host Resources: mem-estimate=56.00MB mem-reservation=36.00MB thread-reservat
 00:SCAN HDFS [tpch_parquet.orders, RANDOM]
    partitions=1/1 files=2 size=54.07MB
    stored statistics:
-     table: rows=1500000 size=54.07MB
+     table: rows=1.50M size=54.07MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1177136
+   extrapolated-rows=disabled max-scan-range-rows=1.18M
    mem-estimate=40.00MB mem-reservation=24.00MB thread-reservation=1
    tuple-ids=0 row-size=171B cardinality=1.50M
    in pipelines: 00(GETNEXT)
@@ -3118,9 +3118,9 @@ Per-Host Resources: mem-estimate=112.00MB mem-reservation=72.00MB thread-reserva
 00:SCAN HDFS [tpch_parquet.orders, RANDOM]
    partitions=1/1 files=2 size=54.07MB
    stored statistics:
-     table: rows=1500000 size=54.07MB
+     table: rows=1.50M size=54.07MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1177136
+   extrapolated-rows=disabled max-scan-range-rows=1.18M
    mem-estimate=40.00MB mem-reservation=24.00MB thread-reservation=0
    tuple-ids=0 row-size=171B cardinality=1.50M
    in pipelines: 00(GETNEXT)
@@ -3171,9 +3171,9 @@ PLAN-ROOT SINK
 |  |--09:SCAN HDFS [tpch_parquet.orders]
 |  |     partitions=1/1 files=2 size=54.07MB
 |  |     stored statistics:
-|  |       table: rows=1500000 size=54.07MB
+|  |       table: rows=1.50M size=54.07MB
 |  |       columns: all
-|  |     extrapolated-rows=disabled max-scan-range-rows=1177136
+|  |     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |  |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=1
 |  |     tuple-ids=6 row-size=8B cardinality=1.50M
 |  |     in pipelines: 09(GETNEXT)
@@ -3183,9 +3183,9 @@ PLAN-ROOT SINK
 |     predicates: l_shipmode = 'F'
 |     runtime filters: RF004[bloom] -> l_orderkey
 |     stored statistics:
-|       table: rows=6001215 size=193.60MB
+|       table: rows=6.00M size=193.59MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=2141702
+|     extrapolated-rows=disabled max-scan-range-rows=2.14M
 |     parquet statistics predicates: l_shipmode = 'F'
 |     parquet dictionary predicates: l_shipmode = 'F'
 |     mem-estimate=80.00MB mem-reservation=24.00MB thread-reservation=1
@@ -3204,9 +3204,9 @@ PLAN-ROOT SINK
 |  |     partitions=1/1 files=2 size=54.07MB
 |  |     predicates: o_orderpriority = '2-HIGH'
 |  |     stored statistics:
-|  |       table: rows=1500000 size=54.07MB
+|  |       table: rows=1.50M size=54.07MB
 |  |       columns: all
-|  |     extrapolated-rows=disabled max-scan-range-rows=1177136
+|  |     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |  |     parquet statistics predicates: o_orderpriority = '2-HIGH'
 |  |     parquet dictionary predicates: o_orderpriority = '2-HIGH'
 |  |     mem-estimate=40.00MB mem-reservation=8.00MB thread-reservation=1
@@ -3217,9 +3217,9 @@ PLAN-ROOT SINK
 |     partitions=1/1 files=3 size=193.60MB
 |     runtime filters: RF002[bloom] -> l_orderkey
 |     stored statistics:
-|       table: rows=6001215 size=193.60MB
+|       table: rows=6.00M size=193.59MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=2141702
+|     extrapolated-rows=disabled max-scan-range-rows=2.14M
 |     mem-estimate=80.00MB mem-reservation=24.00MB thread-reservation=1
 |     tuple-ids=3 row-size=66B cardinality=6.00M
 |     in pipelines: 05(GETNEXT)
@@ -3241,9 +3241,9 @@ PLAN-ROOT SINK
 |--02:SCAN HDFS [tpch_parquet.orders]
 |     partitions=1/1 files=2 size=54.07MB
 |     stored statistics:
-|       table: rows=1500000 size=54.07MB
+|       table: rows=1.50M size=54.07MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1177136
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=1
 |     tuple-ids=1 row-size=8B cardinality=1.50M
 |     in pipelines: 02(GETNEXT)
@@ -3253,9 +3253,9 @@ PLAN-ROOT SINK
    predicates: l_tax > CAST(10 AS DECIMAL(3,0))
    runtime filters: RF000[bloom] -> l_orderkey
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    parquet statistics predicates: l_tax > CAST(10 AS DECIMAL(3,0))
    parquet dictionary predicates: l_tax > CAST(10 AS DECIMAL(3,0))
    mem-estimate=80.00MB mem-reservation=24.00MB thread-reservation=1
@@ -3309,9 +3309,9 @@ Per-Host Resources: mem-estimate=109.02MB mem-reservation=43.00MB thread-reserva
 |  |  09:SCAN HDFS [tpch_parquet.orders, RANDOM]
 |  |     partitions=1/1 files=2 size=54.07MB
 |  |     stored statistics:
-|  |       table: rows=1500000 size=54.07MB
+|  |       table: rows=1.50M size=54.07MB
 |  |       columns: all
-|  |     extrapolated-rows=disabled max-scan-range-rows=1177136
+|  |     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |  |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=1
 |  |     tuple-ids=6 row-size=8B cardinality=1.50M
 |  |     in pipelines: 09(GETNEXT)
@@ -3321,9 +3321,9 @@ Per-Host Resources: mem-estimate=109.02MB mem-reservation=43.00MB thread-reserva
 |     predicates: l_shipmode = 'F'
 |     runtime filters: RF004[bloom] -> l_orderkey
 |     stored statistics:
-|       table: rows=6001215 size=193.60MB
+|       table: rows=6.00M size=193.59MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=2141702
+|     extrapolated-rows=disabled max-scan-range-rows=2.14M
 |     parquet statistics predicates: l_shipmode = 'F'
 |     parquet dictionary predicates: l_shipmode = 'F'
 |     mem-estimate=80.00MB mem-reservation=24.00MB thread-reservation=1
@@ -3349,9 +3349,9 @@ Per-Host Resources: mem-estimate=109.02MB mem-reservation=43.00MB thread-reserva
 |  |     partitions=1/1 files=2 size=54.07MB
 |  |     predicates: o_orderpriority = '2-HIGH'
 |  |     stored statistics:
-|  |       table: rows=1500000 size=54.07MB
+|  |       table: rows=1.50M size=54.07MB
 |  |       columns: all
-|  |     extrapolated-rows=disabled max-scan-range-rows=1177136
+|  |     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |  |     parquet statistics predicates: o_orderpriority = '2-HIGH'
 |  |     parquet dictionary predicates: o_orderpriority = '2-HIGH'
 |  |     mem-estimate=40.00MB mem-reservation=8.00MB thread-reservation=1
@@ -3362,9 +3362,9 @@ Per-Host Resources: mem-estimate=109.02MB mem-reservation=43.00MB thread-reserva
 |     partitions=1/1 files=3 size=193.60MB
 |     runtime filters: RF002[bloom] -> l_orderkey
 |     stored statistics:
-|       table: rows=6001215 size=193.60MB
+|       table: rows=6.00M size=193.59MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=2141702
+|     extrapolated-rows=disabled max-scan-range-rows=2.14M
 |     mem-estimate=80.00MB mem-reservation=24.00MB thread-reservation=1
 |     tuple-ids=3 row-size=66B cardinality=6.00M
 |     in pipelines: 05(GETNEXT)
@@ -3406,9 +3406,9 @@ Per-Host Resources: mem-estimate=55.73MB mem-reservation=39.75MB thread-reservat
 |  02:SCAN HDFS [tpch_parquet.orders, RANDOM]
 |     partitions=1/1 files=2 size=54.07MB
 |     stored statistics:
-|       table: rows=1500000 size=54.07MB
+|       table: rows=1.50M size=54.07MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1177136
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=1
 |     tuple-ids=1 row-size=8B cardinality=1.50M
 |     in pipelines: 02(GETNEXT)
@@ -3425,9 +3425,9 @@ Per-Host Resources: mem-estimate=81.00MB mem-reservation=25.00MB thread-reservat
    predicates: l_tax > CAST(10 AS DECIMAL(3,0))
    runtime filters: RF000[bloom] -> l_orderkey
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    parquet statistics predicates: l_tax > CAST(10 AS DECIMAL(3,0))
    parquet dictionary predicates: l_tax > CAST(10 AS DECIMAL(3,0))
    mem-estimate=80.00MB mem-reservation=24.00MB thread-reservation=1
@@ -3489,9 +3489,9 @@ Per-Host Resources: mem-estimate=198.00MB mem-reservation=86.00MB thread-reserva
 |  |  09:SCAN HDFS [tpch_parquet.orders, RANDOM]
 |  |     partitions=1/1 files=2 size=54.07MB
 |  |     stored statistics:
-|  |       table: rows=1500000 size=54.07MB
+|  |       table: rows=1.50M size=54.07MB
 |  |       columns: all
-|  |     extrapolated-rows=disabled max-scan-range-rows=1177136
+|  |     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |  |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=0
 |  |     tuple-ids=6 row-size=8B cardinality=1.50M
 |  |     in pipelines: 09(GETNEXT)
@@ -3501,9 +3501,9 @@ Per-Host Resources: mem-estimate=198.00MB mem-reservation=86.00MB thread-reserva
 |     predicates: l_shipmode = 'F'
 |     runtime filters: RF004[bloom] -> l_orderkey
 |     stored statistics:
-|       table: rows=6001215 size=193.60MB
+|       table: rows=6.00M size=193.59MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=2141702
+|     extrapolated-rows=disabled max-scan-range-rows=2.14M
 |     parquet statistics predicates: l_shipmode = 'F'
 |     parquet dictionary predicates: l_shipmode = 'F'
 |     mem-estimate=80.00MB mem-reservation=24.00MB thread-reservation=0
@@ -3537,9 +3537,9 @@ Per-Host Resources: mem-estimate=198.00MB mem-reservation=86.00MB thread-reserva
 |  |     partitions=1/1 files=2 size=54.07MB
 |  |     predicates: o_orderpriority = '2-HIGH'
 |  |     stored statistics:
-|  |       table: rows=1500000 size=54.07MB
+|  |       table: rows=1.50M size=54.07MB
 |  |       columns: all
-|  |     extrapolated-rows=disabled max-scan-range-rows=1177136
+|  |     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |  |     parquet statistics predicates: o_orderpriority = '2-HIGH'
 |  |     parquet dictionary predicates: o_orderpriority = '2-HIGH'
 |  |     mem-estimate=40.00MB mem-reservation=8.00MB thread-reservation=0
@@ -3550,9 +3550,9 @@ Per-Host Resources: mem-estimate=198.00MB mem-reservation=86.00MB thread-reserva
 |     partitions=1/1 files=3 size=193.60MB
 |     runtime filters: RF002[bloom] -> l_orderkey
 |     stored statistics:
-|       table: rows=6001215 size=193.60MB
+|       table: rows=6.00M size=193.59MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=2141702
+|     extrapolated-rows=disabled max-scan-range-rows=2.14M
 |     mem-estimate=80.00MB mem-reservation=24.00MB thread-reservation=0
 |     tuple-ids=3 row-size=66B cardinality=6.00M
 |     in pipelines: 05(GETNEXT)
@@ -3602,9 +3602,9 @@ Per-Host Resources: mem-estimate=62.67MB mem-reservation=41.75MB thread-reservat
 |  02:SCAN HDFS [tpch_parquet.orders, RANDOM]
 |     partitions=1/1 files=2 size=54.07MB
 |     stored statistics:
-|       table: rows=1500000 size=54.07MB
+|       table: rows=1.50M size=54.07MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1177136
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=0
 |     tuple-ids=1 row-size=8B cardinality=1.50M
 |     in pipelines: 02(GETNEXT)
@@ -3621,9 +3621,9 @@ Per-Host Resources: mem-estimate=162.00MB mem-reservation=50.00MB thread-reserva
    predicates: l_tax > CAST(10 AS DECIMAL(3,0))
    runtime filters: RF000[bloom] -> l_orderkey
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    parquet statistics predicates: l_tax > CAST(10 AS DECIMAL(3,0))
    parquet dictionary predicates: l_tax > CAST(10 AS DECIMAL(3,0))
    mem-estimate=80.00MB mem-reservation=24.00MB thread-reservation=0
@@ -3712,9 +3712,9 @@ PLAN-ROOT SINK
 |  03:SCAN HDFS [tpch.lineitem]
 |     partitions=1/1 files=1 size=718.94MB
 |     stored statistics:
-|       table: rows=6001215 size=718.94MB
+|       table: rows=6.00M size=718.94MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1068457
+|     extrapolated-rows=disabled max-scan-range-rows=1.07M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=3 row-size=16B cardinality=6.00M
 |     in pipelines: 03(GETNEXT)
@@ -3730,9 +3730,9 @@ PLAN-ROOT SINK
 |--00:SCAN HDFS [tpch.customer]
 |     partitions=1/1 files=1 size=23.08MB
 |     stored statistics:
-|       table: rows=150000 size=23.08MB
+|       table: rows=150.00K size=23.08MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=150000
+|     extrapolated-rows=disabled max-scan-range-rows=150.00K
 |     mem-estimate=32.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=0 row-size=38B cardinality=150.00K
 |     in pipelines: 00(GETNEXT)
@@ -3749,9 +3749,9 @@ PLAN-ROOT SINK
 |     partitions=1/1 files=1 size=162.56MB
 |     runtime filters: RF000[bloom] -> o_orderkey, RF002[bloom] -> o_custkey
 |     stored statistics:
-|       table: rows=1500000 size=162.56MB
+|       table: rows=1.50M size=162.56MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1181132
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=46B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -3760,9 +3760,9 @@ PLAN-ROOT SINK
    partitions=1/1 files=1 size=718.94MB
    runtime filters: RF000[bloom] -> tpch.lineitem.l_orderkey, RF004[bloom] -> l_orderkey
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=2 row-size=16B cardinality=6.00M
    in pipelines: 02(GETNEXT)
@@ -3850,9 +3850,9 @@ Per-Host Resources: mem-estimate=149.22MB mem-reservation=116.38MB thread-reserv
 |  03:SCAN HDFS [tpch.lineitem, RANDOM]
 |     partitions=1/1 files=1 size=718.94MB
 |     stored statistics:
-|       table: rows=6001215 size=718.94MB
+|       table: rows=6.00M size=718.94MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1068457
+|     extrapolated-rows=disabled max-scan-range-rows=1.07M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=3 row-size=16B cardinality=6.00M
 |     in pipelines: 03(GETNEXT)
@@ -3875,9 +3875,9 @@ Per-Host Resources: mem-estimate=149.22MB mem-reservation=116.38MB thread-reserv
 |  00:SCAN HDFS [tpch.customer, RANDOM]
 |     partitions=1/1 files=1 size=23.08MB
 |     stored statistics:
-|       table: rows=150000 size=23.08MB
+|       table: rows=150.00K size=23.08MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=150000
+|     extrapolated-rows=disabled max-scan-range-rows=150.00K
 |     mem-estimate=32.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=0 row-size=38B cardinality=150.00K
 |     in pipelines: 00(GETNEXT)
@@ -3901,9 +3901,9 @@ Per-Host Resources: mem-estimate=149.22MB mem-reservation=116.38MB thread-reserv
 |     partitions=1/1 files=1 size=162.56MB
 |     runtime filters: RF000[bloom] -> o_orderkey, RF002[bloom] -> o_custkey
 |     stored statistics:
-|       table: rows=1500000 size=162.56MB
+|       table: rows=1.50M size=162.56MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1181132
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=46B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -3919,9 +3919,9 @@ Per-Host Resources: mem-estimate=90.00MB mem-reservation=10.00MB thread-reservat
    partitions=1/1 files=1 size=718.94MB
    runtime filters: RF000[bloom] -> tpch.lineitem.l_orderkey, RF004[bloom] -> l_orderkey
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=2 row-size=16B cardinality=6.00M
    in pipelines: 02(GETNEXT)
@@ -4017,9 +4017,9 @@ Per-Host Resources: mem-estimate=115.27MB mem-reservation=94.88MB thread-reserva
 |  03:SCAN HDFS [tpch.lineitem, RANDOM]
 |     partitions=1/1 files=1 size=718.94MB
 |     stored statistics:
-|       table: rows=6001215 size=718.94MB
+|       table: rows=6.00M size=718.94MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1068457
+|     extrapolated-rows=disabled max-scan-range-rows=1.07M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=0
 |     tuple-ids=3 row-size=16B cardinality=6.00M
 |     in pipelines: 03(GETNEXT)
@@ -4050,9 +4050,9 @@ Per-Host Resources: mem-estimate=115.27MB mem-reservation=94.88MB thread-reserva
 |  00:SCAN HDFS [tpch.customer, RANDOM]
 |     partitions=1/1 files=1 size=23.08MB
 |     stored statistics:
-|       table: rows=150000 size=23.08MB
+|       table: rows=150.00K size=23.08MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=150000
+|     extrapolated-rows=disabled max-scan-range-rows=150.00K
 |     mem-estimate=32.00MB mem-reservation=8.00MB thread-reservation=0
 |     tuple-ids=0 row-size=38B cardinality=150.00K
 |     in pipelines: 00(GETNEXT)
@@ -4084,9 +4084,9 @@ Per-Host Resources: mem-estimate=115.27MB mem-reservation=94.88MB thread-reserva
 |     partitions=1/1 files=1 size=162.56MB
 |     runtime filters: RF000[bloom] -> o_orderkey, RF002[bloom] -> o_custkey
 |     stored statistics:
-|       table: rows=1500000 size=162.56MB
+|       table: rows=1.50M size=162.56MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1181132
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=0
 |     tuple-ids=1 row-size=46B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -4102,9 +4102,9 @@ Per-Host Resources: mem-estimate=180.00MB mem-reservation=20.00MB thread-reserva
    partitions=1/1 files=1 size=718.94MB
    runtime filters: RF000[bloom] -> tpch.lineitem.l_orderkey, RF004[bloom] -> l_orderkey
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=0
    tuple-ids=2 row-size=16B cardinality=6.00M
    in pipelines: 02(GETNEXT)
@@ -4134,10 +4134,10 @@ WRITE TO HDFS [functional.alltypesnopart, OVERWRITE=false]
    partition predicates: `year` = CAST(2009 AS INT), `month` = CAST(5 AS INT)
    partitions=1/24 files=1 size=20.36KB
    stored statistics:
-     table: rows=7300 size=478.45KB
+     table: rows=7.30K size=478.45KB
      partitions: 1/1 rows=310
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=7300
+   extrapolated-rows=disabled max-scan-range-rows=7.30K
    mem-estimate=16.00MB mem-reservation=32.00KB thread-reservation=1
    tuple-ids=0 row-size=81B cardinality=310
    in pipelines: 00(GETNEXT)
@@ -4160,10 +4160,10 @@ WRITE TO HDFS [functional.alltypesnopart, OVERWRITE=false]
    partition predicates: `year` = CAST(2009 AS INT), `month` = CAST(5 AS INT)
    partitions=1/24 files=1 size=20.36KB
    stored statistics:
-     table: rows=7300 size=478.45KB
+     table: rows=7.30K size=478.45KB
      partitions: 1/1 rows=310
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=7300
+   extrapolated-rows=disabled max-scan-range-rows=7.30K
    mem-estimate=16.00MB mem-reservation=32.00KB thread-reservation=1
    tuple-ids=0 row-size=81B cardinality=310
    in pipelines: 00(GETNEXT)
@@ -4186,10 +4186,10 @@ WRITE TO HDFS [functional.alltypesnopart, OVERWRITE=false]
    partition predicates: `year` = CAST(2009 AS INT), `month` = CAST(5 AS INT)
    partitions=1/24 files=1 size=20.36KB
    stored statistics:
-     table: rows=7300 size=478.45KB
+     table: rows=7.30K size=478.45KB
      partitions: 1/1 rows=310
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=7300
+   extrapolated-rows=disabled max-scan-range-rows=7.30K
    mem-estimate=16.00MB mem-reservation=32.00KB thread-reservation=0
    tuple-ids=0 row-size=81B cardinality=310
    in pipelines: 00(GETNEXT)
@@ -4218,9 +4218,9 @@ WRITE TO HDFS [default.dummy_insert, OVERWRITE=false, PARTITION-KEYS=(l_partkey)
 00:SCAN HDFS [tpch.lineitem]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=46B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -4251,9 +4251,9 @@ Per-Host Resources: mem-estimate=88.00MB mem-reservation=8.00MB thread-reservati
 00:SCAN HDFS [tpch.lineitem, RANDOM]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=46B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -4284,9 +4284,9 @@ Per-Host Resources: mem-estimate=176.00MB mem-reservation=16.00MB thread-reserva
 00:SCAN HDFS [tpch.lineitem, RANDOM]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=0
    tuple-ids=0 row-size=46B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -4309,9 +4309,9 @@ WRITE TO HDFS [default.dummy_insert, OVERWRITE=false, PARTITION-KEYS=(l_partkey)
 00:SCAN HDFS [tpch.lineitem]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=46B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -4336,9 +4336,9 @@ Per-Host Resources: mem-estimate=88.00MB mem-reservation=8.00MB thread-reservati
 00:SCAN HDFS [tpch.lineitem, RANDOM]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=1
    tuple-ids=0 row-size=46B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -4363,9 +4363,9 @@ Per-Host Resources: mem-estimate=176.00MB mem-reservation=16.00MB thread-reserva
 00:SCAN HDFS [tpch.lineitem, RANDOM]
    partitions=1/1 files=1 size=718.94MB
    stored statistics:
-     table: rows=6001215 size=718.94MB
+     table: rows=6.00M size=718.94MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1068457
+   extrapolated-rows=disabled max-scan-range-rows=1.07M
    mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=0
    tuple-ids=0 row-size=46B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -4449,9 +4449,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_nested_parquet.customer c]
    partitions=1/1 files=4 size=288.99MB
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders, c_orders
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.23K
    mem-estimate=88.00MB mem-reservation=16.00MB thread-reservation=1
    tuple-ids=0 row-size=54B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -4546,9 +4546,9 @@ Per-Host Resources: mem-estimate=345.94MB mem-reservation=85.94MB thread-reserva
 00:SCAN HDFS [tpch_nested_parquet.customer c, RANDOM]
    partitions=1/1 files=4 size=288.99MB
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders, c_orders
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.23K
    mem-estimate=88.00MB mem-reservation=16.00MB thread-reservation=1
    tuple-ids=0 row-size=54B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -4643,9 +4643,9 @@ Per-Host Resources: mem-estimate=691.88MB mem-reservation=171.88MB thread-reserv
 00:SCAN HDFS [tpch_nested_parquet.customer c, RANDOM]
    partitions=1/1 files=4 size=288.99MB
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders, c_orders
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.23K
    mem-estimate=88.00MB mem-reservation=16.00MB thread-reservation=0
    tuple-ids=0 row-size=54B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -4740,9 +4740,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_nested_parquet.customer c]
    partitions=1/1 files=4 size=288.99MB
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.23K
    mem-estimate=88.00MB mem-reservation=56.00MB thread-reservation=1
    tuple-ids=0 row-size=230B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -4835,9 +4835,9 @@ Per-Host Resources: mem-estimate=136.00MB mem-reservation=104.00MB thread-reserv
 00:SCAN HDFS [tpch_nested_parquet.customer c, RANDOM]
    partitions=1/1 files=4 size=288.99MB
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.23K
    mem-estimate=88.00MB mem-reservation=56.00MB thread-reservation=1
    tuple-ids=0 row-size=230B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -4930,9 +4930,9 @@ Per-Host Resources: mem-estimate=272.00MB mem-reservation=208.00MB thread-reserv
 00:SCAN HDFS [tpch_nested_parquet.customer c, RANDOM]
    partitions=1/1 files=4 size=288.99MB
    stored statistics:
-     table: rows=150000 size=288.99MB
+     table: rows=150.00K size=288.98MB
      columns missing stats: c_orders
-   extrapolated-rows=disabled max-scan-range-rows=44225
+   extrapolated-rows=disabled max-scan-range-rows=44.23K
    mem-estimate=88.00MB mem-reservation=56.00MB thread-reservation=0
    tuple-ids=0 row-size=230B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -4992,9 +4992,9 @@ PLAN-ROOT SINK
 |  |  |--03:SCAN HDFS [tpch_parquet.orders t4]
 |  |  |     partitions=1/1 files=2 size=54.07MB
 |  |  |     stored statistics:
-|  |  |       table: rows=1500000 size=54.07MB
+|  |  |       table: rows=1.50M size=54.07MB
 |  |  |       columns: all
-|  |  |     extrapolated-rows=disabled max-scan-range-rows=1177136
+|  |  |     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |  |  |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=1
 |  |  |     tuple-ids=3 row-size=8B cardinality=1.50M
 |  |  |     in pipelines: 03(GETNEXT)
@@ -5003,9 +5003,9 @@ PLAN-ROOT SINK
 |  |     partitions=1/1 files=2 size=54.07MB
 |  |     runtime filters: RF004[bloom] -> t3.o_orderkey
 |  |     stored statistics:
-|  |       table: rows=1500000 size=54.07MB
+|  |       table: rows=1.50M size=54.07MB
 |  |       columns: all
-|  |     extrapolated-rows=disabled max-scan-range-rows=1177136
+|  |     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |  |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=1
 |  |     tuple-ids=2 row-size=8B cardinality=1.50M
 |  |     in pipelines: 02(GETNEXT)
@@ -5014,9 +5014,9 @@ PLAN-ROOT SINK
 |     partitions=1/1 files=2 size=54.07MB
 |     runtime filters: RF002[bloom] -> t2.o_orderkey
 |     stored statistics:
-|       table: rows=1500000 size=54.07MB
+|       table: rows=1.50M size=54.07MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1177136
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=1
 |     tuple-ids=1 row-size=8B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -5025,9 +5025,9 @@ PLAN-ROOT SINK
    partitions=1/1 files=2 size=54.07MB
    runtime filters: RF000[bloom] -> t1.o_orderkey
    stored statistics:
-     table: rows=1500000 size=54.07MB
+     table: rows=1.50M size=54.07MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1177136
+   extrapolated-rows=disabled max-scan-range-rows=1.18M
    mem-estimate=40.00MB mem-reservation=24.00MB thread-reservation=1
    tuple-ids=0 row-size=171B cardinality=1.50M
    in pipelines: 00(GETNEXT)
@@ -5094,9 +5094,9 @@ Per-Host Resources: mem-estimate=88.84MB mem-reservation=59.00MB thread-reservat
 |  |  |  03:SCAN HDFS [tpch_parquet.orders t4, RANDOM]
 |  |  |     partitions=1/1 files=2 size=54.07MB
 |  |  |     stored statistics:
-|  |  |       table: rows=1500000 size=54.07MB
+|  |  |       table: rows=1.50M size=54.07MB
 |  |  |       columns: all
-|  |  |     extrapolated-rows=disabled max-scan-range-rows=1177136
+|  |  |     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |  |  |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=1
 |  |  |     tuple-ids=3 row-size=8B cardinality=1.50M
 |  |  |     in pipelines: 03(GETNEXT)
@@ -5112,9 +5112,9 @@ Per-Host Resources: mem-estimate=88.84MB mem-reservation=59.00MB thread-reservat
 |  |     partitions=1/1 files=2 size=54.07MB
 |  |     runtime filters: RF004[bloom] -> t3.o_orderkey
 |  |     stored statistics:
-|  |       table: rows=1500000 size=54.07MB
+|  |       table: rows=1.50M size=54.07MB
 |  |       columns: all
-|  |     extrapolated-rows=disabled max-scan-range-rows=1177136
+|  |     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |  |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=1
 |  |     tuple-ids=2 row-size=8B cardinality=1.50M
 |  |     in pipelines: 02(GETNEXT)
@@ -5130,9 +5130,9 @@ Per-Host Resources: mem-estimate=88.84MB mem-reservation=59.00MB thread-reservat
 |     partitions=1/1 files=2 size=54.07MB
 |     runtime filters: RF002[bloom] -> t2.o_orderkey
 |     stored statistics:
-|       table: rows=1500000 size=54.07MB
+|       table: rows=1.50M size=54.07MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1177136
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=1
 |     tuple-ids=1 row-size=8B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -5141,9 +5141,9 @@ Per-Host Resources: mem-estimate=88.84MB mem-reservation=59.00MB thread-reservat
    partitions=1/1 files=2 size=54.07MB
    runtime filters: RF000[bloom] -> t1.o_orderkey
    stored statistics:
-     table: rows=1500000 size=54.07MB
+     table: rows=1.50M size=54.07MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1177136
+   extrapolated-rows=disabled max-scan-range-rows=1.18M
    mem-estimate=40.00MB mem-reservation=24.00MB thread-reservation=1
    tuple-ids=0 row-size=171B cardinality=1.50M
    in pipelines: 00(GETNEXT)
@@ -5234,9 +5234,9 @@ Per-Host Resources: mem-estimate=157.53MB mem-reservation=118.00MB thread-reserv
 |  |  |  03:SCAN HDFS [tpch_parquet.orders t4, RANDOM]
 |  |  |     partitions=1/1 files=2 size=54.07MB
 |  |  |     stored statistics:
-|  |  |       table: rows=1500000 size=54.07MB
+|  |  |       table: rows=1.50M size=54.07MB
 |  |  |       columns: all
-|  |  |     extrapolated-rows=disabled max-scan-range-rows=1177136
+|  |  |     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |  |  |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=0
 |  |  |     tuple-ids=3 row-size=8B cardinality=1.50M
 |  |  |     in pipelines: 03(GETNEXT)
@@ -5252,9 +5252,9 @@ Per-Host Resources: mem-estimate=157.53MB mem-reservation=118.00MB thread-reserv
 |  |     partitions=1/1 files=2 size=54.07MB
 |  |     runtime filters: RF004[bloom] -> t3.o_orderkey
 |  |     stored statistics:
-|  |       table: rows=1500000 size=54.07MB
+|  |       table: rows=1.50M size=54.07MB
 |  |       columns: all
-|  |     extrapolated-rows=disabled max-scan-range-rows=1177136
+|  |     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |  |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=0
 |  |     tuple-ids=2 row-size=8B cardinality=1.50M
 |  |     in pipelines: 02(GETNEXT)
@@ -5270,9 +5270,9 @@ Per-Host Resources: mem-estimate=157.53MB mem-reservation=118.00MB thread-reserv
 |     partitions=1/1 files=2 size=54.07MB
 |     runtime filters: RF002[bloom] -> t2.o_orderkey
 |     stored statistics:
-|       table: rows=1500000 size=54.07MB
+|       table: rows=1.50M size=54.07MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1177136
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=40.00MB mem-reservation=4.00MB thread-reservation=0
 |     tuple-ids=1 row-size=8B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -5281,9 +5281,9 @@ Per-Host Resources: mem-estimate=157.53MB mem-reservation=118.00MB thread-reserv
    partitions=1/1 files=2 size=54.07MB
    runtime filters: RF000[bloom] -> t1.o_orderkey
    stored statistics:
-     table: rows=1500000 size=54.07MB
+     table: rows=1.50M size=54.07MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1177136
+   extrapolated-rows=disabled max-scan-range-rows=1.18M
    mem-estimate=40.00MB mem-reservation=24.00MB thread-reservation=0
    tuple-ids=0 row-size=171B cardinality=1.50M
    in pipelines: 00(GETNEXT)
@@ -5335,9 +5335,9 @@ PLAN-ROOT SINK
 |  |  |--03:SCAN HDFS [tpch_parquet.supplier t4]
 |  |  |     partitions=1/1 files=1 size=882.47KB
 |  |  |     stored statistics:
-|  |  |       table: rows=10000 size=882.47KB
+|  |  |       table: rows=10.00K size=882.49KB
 |  |  |       columns: all
-|  |  |     extrapolated-rows=disabled max-scan-range-rows=10000
+|  |  |     extrapolated-rows=disabled max-scan-range-rows=10.00K
 |  |  |     mem-estimate=16.00MB mem-reservation=128.00KB thread-reservation=1
 |  |  |     tuple-ids=3 row-size=8B cardinality=10.00K
 |  |  |     in pipelines: 03(GETNEXT)
@@ -5345,7 +5345,7 @@ PLAN-ROOT SINK
 |  |  02:SCAN HDFS [tpch_parquet.nation t3]
 |  |     partitions=1/1 files=1 size=2.75KB
 |  |     stored statistics:
-|  |       table: rows=25 size=2.75KB
+|  |       table: rows=25 size=2.76KB
 |  |       columns: all
 |  |     extrapolated-rows=disabled max-scan-range-rows=25
 |  |     mem-estimate=16.00MB mem-reservation=8.00KB thread-reservation=1
@@ -5355,7 +5355,7 @@ PLAN-ROOT SINK
 |  01:SCAN HDFS [tpch_parquet.nation t2]
 |     partitions=1/1 files=1 size=2.75KB
 |     stored statistics:
-|       table: rows=25 size=2.75KB
+|       table: rows=25 size=2.76KB
 |       columns: all
 |     extrapolated-rows=disabled max-scan-range-rows=25
 |     mem-estimate=16.00MB mem-reservation=8.00KB thread-reservation=1
@@ -5365,7 +5365,7 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_parquet.nation t1]
    partitions=1/1 files=1 size=2.75KB
    stored statistics:
-     table: rows=25 size=2.75KB
+     table: rows=25 size=2.76KB
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=25
    mem-estimate=16.00MB mem-reservation=32.00KB thread-reservation=1
@@ -5431,9 +5431,9 @@ Per-Host Resources: mem-estimate=97.55MB mem-reservation=32.00KB thread-reservat
 |  |  |  03:SCAN HDFS [tpch_parquet.supplier t4, RANDOM]
 |  |  |     partitions=1/1 files=1 size=882.47KB
 |  |  |     stored statistics:
-|  |  |       table: rows=10000 size=882.47KB
+|  |  |       table: rows=10.00K size=882.49KB
 |  |  |       columns: all
-|  |  |     extrapolated-rows=disabled max-scan-range-rows=10000
+|  |  |     extrapolated-rows=disabled max-scan-range-rows=10.00K
 |  |  |     mem-estimate=16.00MB mem-reservation=128.00KB thread-reservation=1
 |  |  |     tuple-ids=3 row-size=8B cardinality=10.00K
 |  |  |     in pipelines: 03(GETNEXT)
@@ -5441,7 +5441,7 @@ Per-Host Resources: mem-estimate=97.55MB mem-reservation=32.00KB thread-reservat
 |  |  02:SCAN HDFS [tpch_parquet.nation t3, RANDOM]
 |  |     partitions=1/1 files=1 size=2.75KB
 |  |     stored statistics:
-|  |       table: rows=25 size=2.75KB
+|  |       table: rows=25 size=2.76KB
 |  |       columns: all
 |  |     extrapolated-rows=disabled max-scan-range-rows=25
 |  |     mem-estimate=16.00MB mem-reservation=8.00KB thread-reservation=1
@@ -5449,9 +5449,9 @@ Per-Host Resources: mem-estimate=97.55MB mem-reservation=32.00KB thread-reservat
 |  |     in pipelines: 02(GETNEXT)
 |  |
 |  01:SCAN HDFS [tpch_parquet.nation t2, RANDOM]
-|     partitions=1/1 files=1 size=2.75KB
+|     partitions=1/1 files=1 size=2.76KB
 |     stored statistics:
-|       table: rows=25 size=2.75KB
+|       table: rows=25 size=2.76KB
 |       columns: all
 |     extrapolated-rows=disabled max-scan-range-rows=25
 |     mem-estimate=16.00MB mem-reservation=8.00KB thread-reservation=1
@@ -5461,7 +5461,7 @@ Per-Host Resources: mem-estimate=97.55MB mem-reservation=32.00KB thread-reservat
 00:SCAN HDFS [tpch_parquet.nation t1, RANDOM]
    partitions=1/1 files=1 size=2.75KB
    stored statistics:
-     table: rows=25 size=2.75KB
+     table: rows=25 size=2.76KB
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=25
    mem-estimate=16.00MB mem-reservation=32.00KB thread-reservation=1
@@ -5548,9 +5548,9 @@ Per-Host Resources: mem-estimate=195.14MB mem-reservation=64.00KB thread-reserva
 |  |  |  03:SCAN HDFS [tpch_parquet.supplier t4, RANDOM]
 |  |  |     partitions=1/1 files=1 size=882.47KB
 |  |  |     stored statistics:
-|  |  |       table: rows=10000 size=882.47KB
+|  |  |       table: rows=10.00K size=882.49KB
 |  |  |       columns: all
-|  |  |     extrapolated-rows=disabled max-scan-range-rows=10000
+|  |  |     extrapolated-rows=disabled max-scan-range-rows=10.00K
 |  |  |     mem-estimate=16.00MB mem-reservation=128.00KB thread-reservation=0
 |  |  |     tuple-ids=3 row-size=8B cardinality=10.00K
 |  |  |     in pipelines: 03(GETNEXT)
@@ -5558,7 +5558,7 @@ Per-Host Resources: mem-estimate=195.14MB mem-reservation=64.00KB thread-reserva
 |  |  02:SCAN HDFS [tpch_parquet.nation t3, RANDOM]
 |  |     partitions=1/1 files=1 size=2.75KB
 |  |     stored statistics:
-|  |       table: rows=25 size=2.75KB
+|  |       table: rows=25 size=2.76KB
 |  |       columns: all
 |  |     extrapolated-rows=disabled max-scan-range-rows=25
 |  |     mem-estimate=16.00MB mem-reservation=8.00KB thread-reservation=0
@@ -5568,7 +5568,7 @@ Per-Host Resources: mem-estimate=195.14MB mem-reservation=64.00KB thread-reserva
 |  01:SCAN HDFS [tpch_parquet.nation t2, RANDOM]
 |     partitions=1/1 files=1 size=2.75KB
 |     stored statistics:
-|       table: rows=25 size=2.75KB
+|       table: rows=25 size=2.76KB
 |       columns: all
 |     extrapolated-rows=disabled max-scan-range-rows=25
 |     mem-estimate=16.00MB mem-reservation=8.00KB thread-reservation=0
@@ -5576,9 +5576,9 @@ Per-Host Resources: mem-estimate=195.14MB mem-reservation=64.00KB thread-reserva
 |     in pipelines: 01(GETNEXT)
 |
 00:SCAN HDFS [tpch_parquet.nation t1, RANDOM]
-   partitions=1/1 files=1 size=2.75KB
+   partitions=1/1 files=1 size=2.76KB
    stored statistics:
-     table: rows=25 size=2.75KB
+     table: rows=25 size=2.76KB
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=25
    mem-estimate=16.00MB mem-reservation=32.00KB thread-reservation=0
@@ -5645,10 +5645,10 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypesagg]
    partitions=11/11 files=11 size=814.73KB
    stored statistics:
-     table: rows=11000 size=814.73KB
-     partitions: 11/11 rows=11000
+     table: rows=11.00K size=814.73KB
+     partitions: 11/11 rows=11.00K
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1005
+   extrapolated-rows=disabled max-scan-range-rows=1.00K
    mem-estimate=16.00MB mem-reservation=128.00KB thread-reservation=1
    tuple-ids=0 row-size=3B cardinality=11.00K
    in pipelines: 00(GETNEXT)
@@ -5757,9 +5757,9 @@ PLAN-ROOT SINK
 00:SCAN HDFS [tpch_parquet.lineitem]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -5805,9 +5805,9 @@ Per-Host Resources: mem-estimate=806.43MB mem-reservation=74.00MB thread-reserva
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
diff --git a/testdata/workloads/functional-planner/queries/PlannerTest/sort-expr-materialization.test b/testdata/workloads/functional-planner/queries/PlannerTest/sort-expr-materialization.test
index 433ea07..a85b1a9 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/sort-expr-materialization.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/sort-expr-materialization.test
@@ -16,8 +16,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -42,8 +42,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -67,8 +67,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -94,8 +94,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -129,8 +129,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -156,8 +156,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -182,8 +182,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -208,8 +208,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
diff --git a/testdata/workloads/functional-planner/queries/PlannerTest/spillable-buffer-sizing.test b/testdata/workloads/functional-planner/queries/PlannerTest/spillable-buffer-sizing.test
index 68e5110..3283ddd 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/spillable-buffer-sizing.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/spillable-buffer-sizing.test
@@ -49,9 +49,9 @@ Per-Host Resources: mem-estimate=26.95MB mem-reservation=18.94MB thread-reservat
    partitions=1/1 files=1 size=12.31MB
    runtime filters: RF000[bloom] -> c_nationkey
    stored statistics:
-     table: rows=150000 size=12.31MB
+     table: rows=150.00K size=12.31MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=150000
+   extrapolated-rows=disabled max-scan-range-rows=150.00K
    mem-estimate=24.00MB mem-reservation=16.00MB thread-reservation=1
    tuple-ids=0 row-size=218B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -110,9 +110,9 @@ Per-Host Resources: mem-estimate=53.88MB mem-reservation=37.88MB thread-reservat
    partitions=1/1 files=1 size=12.31MB
    runtime filters: RF000[bloom] -> c_nationkey
    stored statistics:
-     table: rows=150000 size=12.31MB
+     table: rows=150.00K size=12.31MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=150000
+   extrapolated-rows=disabled max-scan-range-rows=150.00K
    mem-estimate=24.00MB mem-reservation=16.00MB thread-reservation=0
    tuple-ids=0 row-size=218B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -156,9 +156,9 @@ Per-Host Resources: mem-estimate=359.29MB mem-reservation=74.00MB thread-reserva
 |  01:SCAN HDFS [tpch_parquet.orders, RANDOM]
 |     partitions=1/1 files=2 size=54.07MB
 |     stored statistics:
-|       table: rows=1500000 size=54.07MB
+|       table: rows=1.50M size=54.07MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1177136
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=40.00MB mem-reservation=24.00MB thread-reservation=1
 |     tuple-ids=1 row-size=171B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -166,9 +166,9 @@ Per-Host Resources: mem-estimate=359.29MB mem-reservation=74.00MB thread-reserva
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -215,9 +215,9 @@ Per-Host Resources: mem-estimate=697.89MB mem-reservation=148.00MB thread-reserv
 |  01:SCAN HDFS [tpch_parquet.orders, RANDOM]
 |     partitions=1/1 files=2 size=54.07MB
 |     stored statistics:
-|       table: rows=1500000 size=54.07MB
+|       table: rows=1.50M size=54.07MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1177136
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=40.00MB mem-reservation=24.00MB thread-reservation=0
 |     tuple-ids=1 row-size=171B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -225,9 +225,9 @@ Per-Host Resources: mem-estimate=697.89MB mem-reservation=148.00MB thread-reserv
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=0
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -272,9 +272,9 @@ Per-Host Resources: mem-estimate=38.68MB mem-reservation=18.00MB thread-reservat
 |  01:SCAN HDFS [tpch_parquet.customer, RANDOM]
 |     partitions=1/1 files=1 size=12.31MB
 |     stored statistics:
-|       table: rows=150000 size=12.31MB
+|       table: rows=150.00K size=12.31MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=150000
+|     extrapolated-rows=disabled max-scan-range-rows=150.00K
 |     mem-estimate=24.00MB mem-reservation=16.00MB thread-reservation=1
 |     tuple-ids=1 row-size=218B cardinality=150.00K
 |     in pipelines: 01(GETNEXT)
@@ -290,9 +290,9 @@ Per-Host Resources: mem-estimate=41.00MB mem-reservation=25.00MB thread-reservat
    partitions=1/1 files=2 size=54.07MB
    runtime filters: RF000[bloom] -> o_custkey
    stored statistics:
-     table: rows=1500000 size=54.07MB
+     table: rows=1.50M size=54.07MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1177136
+   extrapolated-rows=disabled max-scan-range-rows=1.18M
    mem-estimate=40.00MB mem-reservation=24.00MB thread-reservation=1
    tuple-ids=0 row-size=171B cardinality=1.50M
    in pipelines: 00(GETNEXT)
@@ -340,9 +340,9 @@ Per-Host Resources: mem-estimate=40.48MB mem-reservation=19.00MB thread-reservat
 |  01:SCAN HDFS [tpch_parquet.customer, RANDOM]
 |     partitions=1/1 files=1 size=12.31MB
 |     stored statistics:
-|       table: rows=150000 size=12.31MB
+|       table: rows=150.00K size=12.31MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=150000
+|     extrapolated-rows=disabled max-scan-range-rows=150.00K
 |     mem-estimate=24.00MB mem-reservation=16.00MB thread-reservation=0
 |     tuple-ids=1 row-size=218B cardinality=150.00K
 |     in pipelines: 01(GETNEXT)
@@ -358,9 +358,9 @@ Per-Host Resources: mem-estimate=82.00MB mem-reservation=50.00MB thread-reservat
    partitions=1/1 files=2 size=54.07MB
    runtime filters: RF000[bloom] -> o_custkey
    stored statistics:
-     table: rows=1500000 size=54.07MB
+     table: rows=1.50M size=54.07MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1177136
+   extrapolated-rows=disabled max-scan-range-rows=1.18M
    mem-estimate=40.00MB mem-reservation=24.00MB thread-reservation=0
    tuple-ids=0 row-size=171B cardinality=1.50M
    in pipelines: 00(GETNEXT)
@@ -405,9 +405,9 @@ Per-Host Resources: mem-estimate=85.45MB mem-reservation=59.00MB thread-reservat
 |  01:SCAN HDFS [tpch_parquet.customer, RANDOM]
 |     partitions=1/1 files=1 size=12.31MB
 |     stored statistics:
-|       table: rows=150000 size=12.31MB
+|       table: rows=150.00K size=12.31MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=150000
+|     extrapolated-rows=disabled max-scan-range-rows=150.00K
 |     mem-estimate=24.00MB mem-reservation=16.00MB thread-reservation=1
 |     tuple-ids=1 row-size=218B cardinality=150.00K
 |     in pipelines: 01(GETNEXT)
@@ -416,9 +416,9 @@ Per-Host Resources: mem-estimate=85.45MB mem-reservation=59.00MB thread-reservat
    partitions=1/1 files=2 size=54.07MB
    runtime filters: RF000[bloom] -> o_custkey
    stored statistics:
-     table: rows=1500000 size=54.07MB
+     table: rows=1.50M size=54.07MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1177136
+   extrapolated-rows=disabled max-scan-range-rows=1.18M
    mem-estimate=40.00MB mem-reservation=24.00MB thread-reservation=1
    tuple-ids=0 row-size=171B cardinality=1.50M
    in pipelines: 00(GETNEXT)
@@ -466,9 +466,9 @@ Per-Host Resources: mem-estimate=150.47MB mem-reservation=118.00MB thread-reserv
 |  01:SCAN HDFS [tpch_parquet.customer, RANDOM]
 |     partitions=1/1 files=1 size=12.31MB
 |     stored statistics:
-|       table: rows=150000 size=12.31MB
+|       table: rows=150.00K size=12.31MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=150000
+|     extrapolated-rows=disabled max-scan-range-rows=150.00K
 |     mem-estimate=24.00MB mem-reservation=16.00MB thread-reservation=0
 |     tuple-ids=1 row-size=218B cardinality=150.00K
 |     in pipelines: 01(GETNEXT)
@@ -477,9 +477,9 @@ Per-Host Resources: mem-estimate=150.47MB mem-reservation=118.00MB thread-reserv
    partitions=1/1 files=2 size=54.07MB
    runtime filters: RF000[bloom] -> o_custkey
    stored statistics:
-     table: rows=1500000 size=54.07MB
+     table: rows=1.50M size=54.07MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1177136
+   extrapolated-rows=disabled max-scan-range-rows=1.18M
    mem-estimate=40.00MB mem-reservation=24.00MB thread-reservation=0
    tuple-ids=0 row-size=171B cardinality=1.50M
    in pipelines: 00(GETNEXT)
@@ -653,9 +653,9 @@ Per-Host Resources: mem-estimate=34.00MB mem-reservation=4.00MB thread-reservati
 00:SCAN HDFS [tpch_parquet.customer, RANDOM]
    partitions=1/1 files=1 size=12.31MB
    stored statistics:
-     table: rows=150000 size=12.31MB
+     table: rows=150.00K size=12.31MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=150000
+   extrapolated-rows=disabled max-scan-range-rows=150.00K
    mem-estimate=24.00MB mem-reservation=2.00MB thread-reservation=1
    tuple-ids=0 row-size=10B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -701,9 +701,9 @@ Per-Host Resources: mem-estimate=68.00MB mem-reservation=8.00MB thread-reservati
 00:SCAN HDFS [tpch_parquet.customer, RANDOM]
    partitions=1/1 files=1 size=12.31MB
    stored statistics:
-     table: rows=150000 size=12.31MB
+     table: rows=150.00K size=12.31MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=150000
+   extrapolated-rows=disabled max-scan-range-rows=150.00K
    mem-estimate=24.00MB mem-reservation=2.00MB thread-reservation=0
    tuple-ids=0 row-size=10B cardinality=150.00K
    in pipelines: 00(GETNEXT)
@@ -774,9 +774,9 @@ Per-Host Resources: mem-estimate=85.65MB mem-reservation=52.00MB thread-reservat
 |  01:SCAN HDFS [tpch_parquet.orders, RANDOM]
 |     partitions=1/1 files=2 size=54.07MB
 |     stored statistics:
-|       table: rows=1500000 size=54.07MB
+|       table: rows=1.50M size=54.07MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1177136
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=40.00MB mem-reservation=8.00MB thread-reservation=1
 |     tuple-ids=1 row-size=21B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -792,9 +792,9 @@ Per-Host Resources: mem-estimate=81.00MB mem-reservation=5.00MB thread-reservati
    partitions=1/1 files=3 size=193.60MB
    runtime filters: RF000[bloom] -> l_orderkey
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=4.00MB thread-reservation=1
    tuple-ids=0 row-size=8B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -866,9 +866,9 @@ Per-Host Resources: mem-estimate=107.20MB mem-reservation=87.00MB thread-reserva
 |  01:SCAN HDFS [tpch_parquet.orders, RANDOM]
 |     partitions=1/1 files=2 size=54.07MB
 |     stored statistics:
-|       table: rows=1500000 size=54.07MB
+|       table: rows=1.50M size=54.07MB
 |       columns: all
-|     extrapolated-rows=disabled max-scan-range-rows=1177136
+|     extrapolated-rows=disabled max-scan-range-rows=1.18M
 |     mem-estimate=40.00MB mem-reservation=8.00MB thread-reservation=0
 |     tuple-ids=1 row-size=21B cardinality=1.50M
 |     in pipelines: 01(GETNEXT)
@@ -884,9 +884,9 @@ Per-Host Resources: mem-estimate=162.00MB mem-reservation=10.00MB thread-reserva
    partitions=1/1 files=3 size=193.60MB
    runtime filters: RF000[bloom] -> l_orderkey
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=4.00MB thread-reservation=0
    tuple-ids=0 row-size=8B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -933,9 +933,9 @@ Per-Host Resources: mem-estimate=806.43MB mem-reservation=74.00MB thread-reserva
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=1
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
@@ -978,9 +978,9 @@ Per-Host Resources: mem-estimate=886.43MB mem-reservation=148.00MB thread-reserv
 00:SCAN HDFS [tpch_parquet.lineitem, RANDOM]
    partitions=1/1 files=3 size=193.60MB
    stored statistics:
-     table: rows=6001215 size=193.60MB
+     table: rows=6.00M size=193.59MB
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2141702
+   extrapolated-rows=disabled max-scan-range-rows=2.14M
    mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=0
    tuple-ids=0 row-size=231B cardinality=6.00M
    in pipelines: 00(GETNEXT)
diff --git a/testdata/workloads/functional-planner/queries/PlannerTest/tablesample.test b/testdata/workloads/functional-planner/queries/PlannerTest/tablesample.test
index 5758fe8..e358e34 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/tablesample.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/tablesample.test
@@ -9,10 +9,10 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=3/24 files=3 size=60.68KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2449
+   extrapolated-rows=disabled max-scan-range-rows=2.45K
    mem-estimate=32.00MB mem-reservation=32.00KB thread-reservation=1
    tuple-ids=0 row-size=89B cardinality=730
    in pipelines: 00(GETNEXT)
@@ -28,8 +28,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=12/24 files=12 size=239.26KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=621
    mem-estimate=80.00MB mem-reservation=32.00KB thread-reservation=1
@@ -50,8 +50,8 @@ PLAN-ROOT SINK
    partitions=12/24 files=12 size=239.26KB
    predicates: id < CAST(10 AS INT)
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=621
    mem-estimate=80.00MB mem-reservation=32.00KB thread-reservation=1
@@ -71,10 +71,10 @@ PLAN-ROOT SINK
    partition predicates: `year` = CAST(2009 AS INT)
    partitions=6/24 files=6 size=119.70KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 12/12 rows=3650
+     table: rows=7.30K size=478.45KB
+     partitions: 12/12 rows=3.65K
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=1241
+   extrapolated-rows=disabled max-scan-range-rows=1.24K
    mem-estimate=48.00MB mem-reservation=32.00KB thread-reservation=1
    tuple-ids=0 row-size=89B cardinality=1.82K
    in pipelines: 00(GETNEXT)
@@ -90,8 +90,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=0/24 files=0 size=0B
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=0
    mem-estimate=0B mem-reservation=0B thread-reservation=0
@@ -109,10 +109,10 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=1/24 files=1 size=19.95KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=7300
+   extrapolated-rows=disabled max-scan-range-rows=7.30K
    mem-estimate=32.00MB mem-reservation=32.00KB thread-reservation=1
    tuple-ids=0 row-size=89B cardinality=73
    in pipelines: 00(GETNEXT)
@@ -130,10 +130,10 @@ PLAN-ROOT SINK
    partition predicates: `year` = CAST(2010 AS INT)
    partitions=1/24 files=1 size=20.36KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 12/12 rows=3650
+     table: rows=7.30K size=478.45KB
+     partitions: 12/12 rows=3.65K
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=7300
+   extrapolated-rows=disabled max-scan-range-rows=7.30K
    mem-estimate=32.00MB mem-reservation=32.00KB thread-reservation=1
    tuple-ids=0 row-size=89B cardinality=37
    in pipelines: 00(GETNEXT)
@@ -149,8 +149,8 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -208,8 +208,8 @@ PLAN-ROOT SINK
    partitions=24/24 files=24 size=478.45KB
    runtime filters: RF000[bloom] -> t1.id
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -228,10 +228,10 @@ PLAN-ROOT SINK
 00:SCAN HDFS [functional.alltypes]
    partitions=3/24 files=3 size=60.68KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
-   extrapolated-rows=disabled max-scan-range-rows=2449
+   extrapolated-rows=disabled max-scan-range-rows=2.45K
    mem-estimate=32.00MB mem-reservation=32.00KB thread-reservation=1
    tuple-ids=0 row-size=4B cardinality=730
    in pipelines: 00(GETNEXT)
diff --git a/testdata/workloads/functional-planner/queries/PlannerTest/union.test b/testdata/workloads/functional-planner/queries/PlannerTest/union.test
index e481793..098e1a4 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/union.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/union.test
@@ -4135,8 +4135,8 @@ Per-Host Resources: mem-estimate=128.00MB mem-reservation=32.00KB thread-reserva
 |--02:SCAN HDFS [functional.alltypes, RANDOM]
 |     partitions=24/24 files=24 size=478.45KB
 |     stored statistics:
-|       table: rows=7300 size=478.45KB
-|       partitions: 24/24 rows=7300
+|       table: rows=7.30K size=478.45KB
+|       partitions: 24/24 rows=7.30K
 |       columns: all
 |     extrapolated-rows=disabled max-scan-range-rows=310
 |     mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
@@ -4194,8 +4194,8 @@ Per-Host Resources: mem-estimate=128.00MB mem-reservation=32.00KB thread-reserva
 01:SCAN HDFS [functional.alltypes, RANDOM]
    partitions=24/24 files=24 size=478.45KB
    stored statistics:
-     table: rows=7300 size=478.45KB
-     partitions: 24/24 rows=7300
+     table: rows=7.30K size=478.45KB
+     partitions: 24/24 rows=7.30K
      columns: all
    extrapolated-rows=disabled max-scan-range-rows=310
    mem-estimate=128.00MB mem-reservation=32.00KB thread-reservation=1
diff --git a/testdata/workloads/functional-query/queries/QueryTest/explain-level2.test b/testdata/workloads/functional-query/queries/QueryTest/explain-level2.test
index 6c07f77..9fc57ea 100644
--- a/testdata/workloads/functional-query/queries/QueryTest/explain-level2.test
+++ b/testdata/workloads/functional-query/queries/QueryTest/explain-level2.test
@@ -40,7 +40,7 @@ row_regex:.*Per-Host Resources: mem-estimate=[0-9.]*[A-Z]*B mem-reservation=[0-9
 '|  01:SCAN HDFS [tpch.orders, RANDOM]'
 row_regex:.*partitions=1/1 files=1 size=.*
 '|     stored statistics:'
-row_regex:.*table: rows=1500000 size=.*
+row_regex:.*table: rows=[0-9.]*[A-Z]* size=.*
 '|       columns: all'
 row_regex:.*extrapolated-rows=disabled max-scan-range-rows=[0-9]*.*
 row_regex:.*mem-estimate=[0-9.]*[A-Z]*B mem-reservation=[0-9.]*[A-Z]*B thread-reservation=1.*
@@ -51,7 +51,7 @@ row_regex:.*mem-estimate=[0-9.]*[A-Z]*B mem-reservation=[0-9.]*[A-Z]*B thread-re
 row_regex:.*partitions=1/1 files=1 size=.*
 '   runtime filters: RF000[bloom] -> l_orderkey'
 '   stored statistics:'
-row_regex:.*table: rows=6001215 size=.*
+row_regex:.*table: rows=[0-9.]*[A-Z]* size=.*
 '     columns: all'
 row_regex:.*extrapolated-rows=disabled max-scan-range-rows=[0-9]*.*
 row_regex:.*mem-estimate=[0-9.]*[A-Z]*B mem-reservation=[0-9.]*[A-Z]*B thread-reservation=1.*
diff --git a/testdata/workloads/functional-query/queries/QueryTest/explain-level3.test b/testdata/workloads/functional-query/queries/QueryTest/explain-level3.test
index d095028..a4bffe4 100644
--- a/testdata/workloads/functional-query/queries/QueryTest/explain-level3.test
+++ b/testdata/workloads/functional-query/queries/QueryTest/explain-level3.test
@@ -41,7 +41,7 @@ row_regex:.*mem-estimate=[0-9.]*[A-Z]*B mem-reservation=[0-9.]*[A-Z]*B thread-re
 row_regex:.*partitions=1/1 files=1 size=.*
 '     runtime filters: RF000[bloom] -> l_orderkey'
 '     stored statistics:'
-row_regex:.*table: rows=6001215 size=.*
+row_regex:.*table: rows=[0-9.]*[A-Z]* size=.*
 '       columns: all'
 row_regex:.*extrapolated-rows=disabled max-scan-range-rows=[0-9]*.*
 row_regex:.*mem-estimate=[0-9.]*[A-Z]*B mem-reservation=[0-9.]*[A-Z]*B thread-reservation=1.*
@@ -55,7 +55,7 @@ row_regex:.*  |  mem-estimate=[0-9.]*[A-Z]*B mem-reservation=[0-9.]*[A-Z]*B thre
 '  01:SCAN HDFS [tpch.orders, RANDOM]'
 row_regex:.*partitions=1/1 files=1 size=.*
 '     stored statistics:'
-row_regex:.*table: rows=1500000 size=.*
+row_regex:.*table: rows=[0-9.]*[A-Z]* size=.*
 '       columns: all'
 row_regex:.*   extrapolated-rows=disabled max-scan-range-rows=[0-9]*.*
 row_regex:.*     mem-estimate=[0-9.]*[A-Z]*B mem-reservation=[0-9.]*[A-Z]*B thread-reservation=.*
diff --git a/testdata/workloads/functional-query/queries/QueryTest/stats-extrapolation.test b/testdata/workloads/functional-query/queries/QueryTest/stats-extrapolation.test
index c7e2781..3c1e31f 100644
--- a/testdata/workloads/functional-query/queries/QueryTest/stats-extrapolation.test
+++ b/testdata/workloads/functional-query/queries/QueryTest/stats-extrapolation.test
@@ -65,10 +65,10 @@ row_regex:.*mem-estimate=.* mem-reservation=.*
 '00:SCAN HDFS [$DATABASE.alltypes]'
 row_regex:.*partitions=12/12 files=12 size=.*
 '   stored statistics:'
-row_regex:.*table: rows=3650 size=.*
+row_regex:.*table: rows=3.65K size=.*
 '     partitions: 0/12 rows=unavailable'
 '     columns: all'
-row_regex:.* extrapolated-rows=3650.*
+row_regex:.* extrapolated-rows=3.65K .*
 row_regex:.*mem-estimate=.* mem-reservation=.*
 '   tuple-ids=0 row-size=4B cardinality=3.65K'
 '   in pipelines: 00(GETNEXT)'
@@ -94,7 +94,7 @@ row_regex:.*mem-estimate=.* mem-reservation=.*
 '   partition predicates: `month` IN (CAST(1 AS INT), CAST(2 AS INT), CAST(3 AS INT))'
 row_regex:.*partitions=3/12 files=3 size=.*
 '   stored statistics:'
-row_regex:.*table: rows=3650 size=.*
+row_regex:.*table: rows=3.65K size=.*
 '     partitions: 0/3 rows=unavailable'
 '     columns: all'
 row_regex:.* extrapolated-rows=904.*
@@ -122,10 +122,10 @@ row_regex:.*mem-estimate=.* mem-reservation=.*
 '00:SCAN HDFS [$DATABASE.alltypes]'
 row_regex:.*partitions=12/12 files=24 size=.*
 '   stored statistics:'
-row_regex:.*table: rows=3650 size=.*
+row_regex:.*table: rows=3.65K size=.*
 '     partitions: 0/12 rows=unavailable'
 '     columns: all'
-row_regex:.* extrapolated-rows=7300.*
+row_regex:.* extrapolated-rows=7.30K .*
 row_regex:.*mem-estimate=.* mem-reservation=.*
 '   tuple-ids=0 row-size=4B cardinality=7.30K'
 '   in pipelines: 00(GETNEXT)'
@@ -153,10 +153,10 @@ row_regex:.*mem-estimate=.* mem-reservation=.*
 '   partition predicates: `year` = CAST(2010 AS INT)'
 row_regex:.*partitions=12/24 files=12 size=.*
 '   stored statistics:'
-row_regex:.*table: rows=3650 size=.*
+row_regex:.*table: rows=3.65K size=.*
 '     partitions: 0/12 rows=unavailable'
 '     columns: all'
-row_regex:.* extrapolated-rows=3651.*
+row_regex:.* extrapolated-rows=3.65K .*
 row_regex:.*mem-estimate=.* mem-reservation=.*
 '   tuple-ids=0 row-size=4B cardinality=3.65K'
 '   in pipelines: 00(GETNEXT)'
@@ -183,10 +183,10 @@ row_regex:.*mem-estimate=.* mem-reservation=.*
 '   partition predicates: `year` = CAST(2010 AS INT)'
 row_regex:.*partitions=12/24 files=12 size=.*
 '   stored statistics:'
-row_regex:.*table: rows=10950 size=.*
+row_regex:.*table: rows=10.95K size=.*
 '     partitions: 0/12 rows=unavailable'
 '     columns: all'
-row_regex:.* extrapolated-rows=3651
+row_regex:.* extrapolated-rows=3.65K .*
 row_regex:.*mem-estimate=.* mem-reservation=.*
 '   tuple-ids=0 row-size=4B cardinality=3.65K'
 '   in pipelines: 00(GETNEXT)'
diff --git a/tests/custom_cluster/test_stats_extrapolation.py b/tests/custom_cluster/test_stats_extrapolation.py
index 2910da2..cd1accd 100644
--- a/tests/custom_cluster/test_stats_extrapolation.py
+++ b/tests/custom_cluster/test_stats_extrapolation.py
@@ -44,7 +44,7 @@ class TestStatsExtrapolation(CustomClusterTestSuite):
     # Test row count extrapolation
     self.client.execute("set explain_level=2")
     explain_result = self.client.execute("explain select * from functional.alltypes")
-    assert "extrapolated-rows=7300" in " ".join(explain_result.data)
+    assert "extrapolated-rows=7.30K" in " ".join(explain_result.data)
     # Test COMPUTE STATS TABLESAMPLE
     part_test_tbl = unique_database + ".alltypes"
     self.clone_table("functional.alltypes", part_test_tbl, True, vector)
diff --git a/tests/metadata/test_stats_extrapolation.py b/tests/metadata/test_stats_extrapolation.py
index 61bdb39..c6bd8c4 100644
--- a/tests/metadata/test_stats_extrapolation.py
+++ b/tests/metadata/test_stats_extrapolation.py
@@ -124,7 +124,7 @@ class TestStatsExtrapolation(ImpalaTestSuite):
   def __run_sampling_test(self, tbl, cols, expected_tbl, perc, seed):
     """Drops stats on 'tbl' and then runs COMPUTE STATS TABLESAMPLE on 'tbl' with the
     given column restriction clause, sampling percent and random seed. Checks that
-    the resulting table and column stats are reasoanbly close to those of
+    the resulting table and column stats are reasonably close to those of
     'expected_tbl'."""
     self.client.execute("drop stats {0}".format(tbl))
     self.client.execute("compute stats {0}{1} tablesample system ({2}) repeatable ({3})"\