You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@impala.apache.org by ta...@apache.org on 2018/08/07 17:18:22 UTC

[3/5] impala git commit: IMPALA-7354: planner test resource estimates for more workloads

IMPALA-7354: planner test resource estimates for more workloads

Adds the resource estimates for key benchmark workloads:
TPC-H, TPC-DS, TPC-H Nested and TPC-H Kudu to the planner
test so that we can track changes in resource requirements
and estimates for these queries.

Also don't show decimal places for MB and KB estimates. The
estimates are not accurate to that level and displaying
extra precision has some disadvantages:
* It communicates to readers that the estimates have a high level of
  precision.
* It increases the odds of small variations in file sizes, etc
  causing test failures.

Also fixed a regex in the stress test that didn't escape the decimal
point correctly.

Testing:
Ran core tests.

Change-Id: I6a9f836699200ea87fb03bf36abad0e23949ac26
Reviewed-on: http://gerrit.cloudera.org:8080/11087
Reviewed-by: Impala Public Jenkins <im...@cloudera.com>
Tested-by: Impala Public Jenkins <im...@cloudera.com>


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

Branch: refs/heads/master
Commit: a8d7a50bd51f49c67291915164ad36154e83b8da
Parents: df3f165
Author: Tim Armstrong <ta...@cloudera.com>
Authored: Mon Jul 30 17:31:06 2018 -0700
Committer: Impala Public Jenkins <im...@cloudera.com>
Committed: Tue Aug 7 08:34:33 2018 +0000

----------------------------------------------------------------------
 .../org/apache/impala/common/PrintUtils.java    |  16 +
 .../java/org/apache/impala/planner/Planner.java |   2 +-
 .../org/apache/impala/planner/PlannerTest.java  |  16 +-
 .../apache/impala/planner/PlannerTestBase.java  |  54 ++-
 .../org/apache/impala/util/PrintUtilsTest.java  |  81 ++++
 .../queries/PlannerTest/disable-codegen.test    |  20 +-
 .../queries/PlannerTest/max-row-size.test       |  12 +-
 .../PlannerTest/resource-requirements.test      | 218 +++++------
 .../PlannerTest/spillable-buffer-sizing.test    |  28 +-
 .../queries/PlannerTest/tpcds-all.test          | 372 +++++++++++++------
 .../queries/PlannerTest/tpch-all.test           | 132 +++++++
 .../queries/PlannerTest/tpch-kudu.test          |  44 +++
 .../queries/PlannerTest/tpch-nested.test        | 308 +++++++++------
 .../admission-reject-mem-estimate.test          |   8 +-
 .../queries/QueryTest/stats-extrapolation.test  |   4 +-
 tests/stress/concurrent_select.py               |   2 +-
 16 files changed, 925 insertions(+), 392 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/impala/blob/a8d7a50b/fe/src/main/java/org/apache/impala/common/PrintUtils.java
----------------------------------------------------------------------
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 9f75134..264518f 100644
--- a/fe/src/main/java/org/apache/impala/common/PrintUtils.java
+++ b/fe/src/main/java/org/apache/impala/common/PrintUtils.java
@@ -49,6 +49,22 @@ public class PrintUtils {
     return bytes + "B";
   }
 
+  /**
+   * Same as printBytes() except 0 decimal points are shown for MB and KB.
+   */
+  public static String printBytesRoundedToMb(long bytes) {
+    double result = bytes;
+    // Avoid String.format() due to IMPALA-1572 which happens on JDK7 but not JDK6.
+    // IMPALA-6759: Please update tests/stress/concurrent_select.py MEM_ESTIMATE_PATTERN
+    // if you add additional unit prefixes.
+    if (bytes >= PETABYTE) return new DecimalFormat(".00PB").format(result / PETABYTE);
+    if (bytes >= TERABYTE) return new DecimalFormat(".00TB").format(result / TERABYTE);
+    if (bytes >= GIGABYTE) return new DecimalFormat(".00GB").format(result / GIGABYTE);
+    if (bytes >= MEGABYTE) return new DecimalFormat("0MB").format(result / MEGABYTE);
+    if (bytes >= KILOBYTE) return new DecimalFormat("0KB").format(result / KILOBYTE);
+    return bytes + "B";
+  }
+
   public static String printCardinality(String prefix, long cardinality) {
     return prefix + "cardinality=" +
         ((cardinality != -1) ? String.valueOf(cardinality) : "unavailable");

http://git-wip-us.apache.org/repos/asf/impala/blob/a8d7a50b/fe/src/main/java/org/apache/impala/planner/Planner.java
----------------------------------------------------------------------
diff --git a/fe/src/main/java/org/apache/impala/planner/Planner.java b/fe/src/main/java/org/apache/impala/planner/Planner.java
index 1496d9a..818c526 100644
--- a/fe/src/main/java/org/apache/impala/planner/Planner.java
+++ b/fe/src/main/java/org/apache/impala/planner/Planner.java
@@ -273,7 +273,7 @@ public class Planner {
           PrintUtils.printBytes(request.getMax_per_host_min_mem_reservation()),
           request.getMax_per_host_thread_reservation()));
       str.append(String.format("Per-Host Resource Estimates: Memory=%s\n",
-          PrintUtils.printBytes(request.getPer_host_mem_estimate())));
+          PrintUtils.printBytesRoundedToMb(request.getPer_host_mem_estimate())));
       hasHeader = true;
     }
     if (request.query_ctx.disable_codegen_hint) {

http://git-wip-us.apache.org/repos/asf/impala/blob/a8d7a50b/fe/src/test/java/org/apache/impala/planner/PlannerTest.java
----------------------------------------------------------------------
diff --git a/fe/src/test/java/org/apache/impala/planner/PlannerTest.java b/fe/src/test/java/org/apache/impala/planner/PlannerTest.java
index 063169d..87cd518 100644
--- a/fe/src/test/java/org/apache/impala/planner/PlannerTest.java
+++ b/fe/src/test/java/org/apache/impala/planner/PlannerTest.java
@@ -228,7 +228,9 @@ public class PlannerTest extends PlannerTestBase {
 
   @Test
   public void testTpch() {
-    runPlannerTestFile("tpch-all", "tpch");
+    runPlannerTestFile("tpch-all", "tpch",
+        ImmutableSet.of(PlannerTestOption.INCLUDE_RESOURCE_HEADER,
+            PlannerTestOption.VALIDATE_RESOURCES));
   }
 
   @Test
@@ -246,7 +248,9 @@ public class PlannerTest extends PlannerTestBase {
 
   @Test
   public void testTpchNested() {
-    runPlannerTestFile("tpch-nested", "tpch_nested_parquet");
+    runPlannerTestFile("tpch-nested", "tpch_nested_parquet",
+        ImmutableSet.of(PlannerTestOption.INCLUDE_RESOURCE_HEADER,
+            PlannerTestOption.VALIDATE_RESOURCES));
   }
 
   @Test
@@ -254,7 +258,9 @@ public class PlannerTest extends PlannerTestBase {
     // Uses ss_sold_date_sk as the partition key of store_sales to allow static partition
     // pruning. The original predicates were rephrased in terms of the ss_sold_date_sk
     // partition key, with the query semantics identical to the original queries.
-    runPlannerTestFile("tpcds-all", "tpcds");
+    runPlannerTestFile("tpcds-all", "tpcds",
+        ImmutableSet.of(PlannerTestOption.INCLUDE_RESOURCE_HEADER,
+            PlannerTestOption.VALIDATE_RESOURCES));
   }
 
   @Test
@@ -367,7 +373,9 @@ public class PlannerTest extends PlannerTestBase {
   @Test
   public void testKuduTpch() {
     Assume.assumeTrue(RuntimeEnv.INSTANCE.isKuduSupported());
-    runPlannerTestFile("tpch-kudu");
+    runPlannerTestFile("tpch-kudu", ImmutableSet.of(
+        PlannerTestOption.INCLUDE_RESOURCE_HEADER,
+        PlannerTestOption.VALIDATE_RESOURCES));
   }
 
   @Test

http://git-wip-us.apache.org/repos/asf/impala/blob/a8d7a50b/fe/src/test/java/org/apache/impala/planner/PlannerTestBase.java
----------------------------------------------------------------------
diff --git a/fe/src/test/java/org/apache/impala/planner/PlannerTestBase.java b/fe/src/test/java/org/apache/impala/planner/PlannerTestBase.java
index 0c51036..da5b1bd 100644
--- a/fe/src/test/java/org/apache/impala/planner/PlannerTestBase.java
+++ b/fe/src/test/java/org/apache/impala/planner/PlannerTestBase.java
@@ -23,7 +23,6 @@ import java.io.FileWriter;
 import java.io.IOException;
 import java.nio.file.Paths;
 import java.util.ArrayList;
-import java.util.Arrays;
 import java.util.Collections;
 import java.util.HashSet;
 import java.util.List;
@@ -58,7 +57,6 @@ import org.apache.impala.thrift.THdfsPartitionLocation;
 import org.apache.impala.thrift.THdfsScanNode;
 import org.apache.impala.thrift.THdfsTable;
 import org.apache.impala.thrift.TLineageGraph;
-import org.apache.impala.thrift.TNetworkAddress;
 import org.apache.impala.thrift.TPlanExecInfo;
 import org.apache.impala.thrift.TPlanFragment;
 import org.apache.impala.thrift.TPlanNode;
@@ -522,9 +520,8 @@ public class PlannerTestBase extends FrontendTestBase {
     if (execRequest == null) return null;
 
     String explainStr = explainBuilder.toString();
-    if (!testOptions.contains(PlannerTestOption.INCLUDE_EXPLAIN_HEADER)) {
-      explainStr = removeExplainHeader(explainStr);
-    }
+    explainStr = removeExplainHeader(explainBuilder.toString(), testOptions);
+
     actualOutput.append(explainStr);
     LOG.info(section.toString() + ":" + explainStr);
     if (expectedErrorMsg != null) {
@@ -569,8 +566,8 @@ public class PlannerTestBase extends FrontendTestBase {
       queryCtx.client_request.getQuery_options().setExplain_level(origExplainLevel);
     }
     Preconditions.checkNotNull(execRequest);
-    String explainStr = removeExplainHeader(explainBuilder.toString());
-    return explainStr;
+    return removeExplainHeader(
+        explainBuilder.toString(), Collections.<PlannerTestOption>emptySet());
   }
 
   private void checkScanRangeLocations(TestCase testCase, TExecRequest execRequest,
@@ -747,20 +744,28 @@ public class PlannerTestBase extends FrontendTestBase {
   }
 
   /**
-   * Strips out the header containing resource estimates and the warning about missing
-   * stats from the given explain plan, because the estimates can change easily with
-   * stats/cardinality.
+   * If required by 'testOptions', strip out all or part of the the header containing
+   * resource estimates and the warning about missing stats from the given explain plan.
    */
-  private String removeExplainHeader(String explain) {
-    String[] lines = explain.split("\n");
-    // Find the first empty line - the end of the header.
-    for (int i = 0; i < lines.length - 1; ++i) {
-      if (lines[i].isEmpty()) {
-        return Joiner.on("\n").join(Arrays.copyOfRange(lines, i + 1 , lines.length))
-            + "\n";
+  private String removeExplainHeader(String explain, Set<PlannerTestOption> testOptions) {
+    if (testOptions.contains(PlannerTestOption.INCLUDE_EXPLAIN_HEADER)) return explain;
+    boolean keepResources =
+        testOptions.contains(PlannerTestOption.INCLUDE_RESOURCE_HEADER);
+    StringBuilder builder = new StringBuilder();
+    boolean inHeader = true;
+    for (String line: explain.split("\n")) {
+      if (inHeader) {
+        // The first empty line indicates the end of the header.
+        if (line.isEmpty()) {
+          inHeader = false;
+        } else if (keepResources && line.contains("Resource")) {
+          builder.append(line).append("\n");
+        }
+      } else {
+        builder.append(line).append("\n");
       }
     }
-    return explain;
+    return builder.toString();
   }
 
   /**
@@ -772,8 +777,12 @@ public class PlannerTestBase extends FrontendTestBase {
     EXTENDED_EXPLAIN,
     // Include the header of the explain plan (default is to strip the explain header).
     INCLUDE_EXPLAIN_HEADER,
-    // Validate the values of resource requirements (default is to ignore differences
-    // in resource values).
+    // Include the part of the explain header that has top-level resource consumption.
+    // If INCLUDE_EXPLAIN_HEADER is enabled, these are already included.
+    INCLUDE_RESOURCE_HEADER,
+    // Validate the values of resource requirement values within the plan (default is to
+    // ignore differences in resource values). Operator- and fragment-level resource
+    // requirements are only included if EXTENDED_EXPLAIN is also enabled.
     VALIDATE_RESOURCES,
   }
 
@@ -792,6 +801,11 @@ public class PlannerTestBase extends FrontendTestBase {
     runPlannerTestFile(testFile, "default", defaultQueryOptions(), testOptions);
   }
 
+  protected void runPlannerTestFile(
+      String testFile, String dbName, Set<PlannerTestOption> testOptions) {
+    runPlannerTestFile(testFile, dbName, defaultQueryOptions(), testOptions);
+  }
+
   private void runPlannerTestFile(String testFile, String dbName, TQueryOptions options,
         Set<PlannerTestOption> testOptions) {
     String fileName = testDir_.resolve(testFile + ".test").toString();

http://git-wip-us.apache.org/repos/asf/impala/blob/a8d7a50b/fe/src/test/java/org/apache/impala/util/PrintUtilsTest.java
----------------------------------------------------------------------
diff --git a/fe/src/test/java/org/apache/impala/util/PrintUtilsTest.java b/fe/src/test/java/org/apache/impala/util/PrintUtilsTest.java
new file mode 100644
index 0000000..8f55c7d
--- /dev/null
+++ b/fe/src/test/java/org/apache/impala/util/PrintUtilsTest.java
@@ -0,0 +1,81 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+package org.apache.impala.util;
+
+import static org.junit.Assert.*;
+
+import org.apache.impala.common.PrintUtils;
+import org.junit.Test;
+
+/**
+ * Unit tests for PrintUtils functions.
+ */
+public class PrintUtilsTest {
+
+  @Test
+  public void testPrintBytes() {
+    assertEquals("0B", PrintUtils.printBytes(0L));
+    assertEquals("55B", PrintUtils.printBytes(55L));
+    assertEquals("1023B", PrintUtils.printBytes(1023L));
+    assertEquals("1.00KB", PrintUtils.printBytes(1024L));
+    assertEquals("54.16KB", PrintUtils.printBytes(55463L));
+    assertEquals("54.16KB", PrintUtils.printBytes(55463L));
+    // The exact threshold before rounding is used to choose the unit.
+    // TODO: this behaviour seems fairly harmless but wrong
+    assertEquals("1024.00KB", PrintUtils.printBytes(1024L * 1024L - 1L));
+    assertEquals("1.00MB", PrintUtils.printBytes(1024L * 1024L));
+    assertEquals("1.00MB", PrintUtils.printBytes(1024L * 1024L + 10L));
+    assertEquals("1.00GB", PrintUtils.printBytes(1024L * 1024L * 1024L));
+    assertEquals("1.50GB", PrintUtils.printBytes((long)(1024L * 1024L * 1024L * 1.5)));
+    assertEquals("4.00TB", PrintUtils.printBytes(1024L * 1024L * 1024L * 1024L * 4L));
+    assertEquals("8.42PB",
+        PrintUtils.printBytes((long)(1024L * 1024L * 1024L * 1024L * 1024L * 8.42)));
+
+    // Negative values always get bytes as unit.
+    // TODO: fix this behaviour if needed.
+    assertEquals("-10B", PrintUtils.printBytes(-10L));
+    assertEquals("-123456789B", PrintUtils.printBytes(-123456789L));
+  }
+
+  @Test
+  public void testPrintBytesRoundedToMb() {
+    assertEquals("0B", PrintUtils.printBytesRoundedToMb(0L));
+    assertEquals("55B", PrintUtils.printBytesRoundedToMb(55L));
+    assertEquals("1023B", PrintUtils.printBytesRoundedToMb(1023L));
+    assertEquals("1KB", PrintUtils.printBytesRoundedToMb(1024L));
+    assertEquals("54KB", PrintUtils.printBytesRoundedToMb(55463L));
+    assertEquals("54KB", PrintUtils.printBytesRoundedToMb(55463L));
+    // The exact threshold before rounding is used to choose the unit.
+    // TODO: this behaviour seems fairly harmless but wrong
+    assertEquals("1024KB", PrintUtils.printBytesRoundedToMb(1024L * 1024L - 1L));
+    assertEquals("1MB", PrintUtils.printBytesRoundedToMb(1024L * 1024L));
+    assertEquals("1MB", PrintUtils.printBytesRoundedToMb(1024L * 1024L + 10L));
+    assertEquals("1.00GB", PrintUtils.printBytesRoundedToMb(1024L * 1024L * 1024L));
+    assertEquals("1.50GB",
+        PrintUtils.printBytesRoundedToMb((long)(1024L * 1024L * 1024L * 1.5)));
+    assertEquals("4.00TB",
+        PrintUtils.printBytesRoundedToMb(1024L * 1024L * 1024L * 1024L * 4L));
+    assertEquals("8.42PB", PrintUtils.printBytesRoundedToMb(
+        (long)(1024L * 1024L * 1024L * 1024L * 1024L * 8.42)));
+
+    // Negative values always get bytes as unit.
+    // TODO: fix this behaviour if needed.
+    assertEquals("-10B", PrintUtils.printBytesRoundedToMb(-10L));
+    assertEquals("-123456789B", PrintUtils.printBytesRoundedToMb(-123456789L));
+  }
+}

http://git-wip-us.apache.org/repos/asf/impala/blob/a8d7a50b/testdata/workloads/functional-planner/queries/PlannerTest/disable-codegen.test
----------------------------------------------------------------------
diff --git a/testdata/workloads/functional-planner/queries/PlannerTest/disable-codegen.test b/testdata/workloads/functional-planner/queries/PlannerTest/disable-codegen.test
index 2adecc3..3987410 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/disable-codegen.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/disable-codegen.test
@@ -2,7 +2,7 @@
 select count(*) from functional.alltypes
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=32.00KB Threads=3
-Per-Host Resource Estimates: Memory=148.00MB
+Per-Host Resource Estimates: Memory=148MB
 Codegen disabled by planner
 
 PLAN-ROOT SINK
@@ -22,7 +22,7 @@ PLAN-ROOT SINK
 select count(*) from functional.alltypesagg
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=128.00KB Threads=3
-Per-Host Resource Estimates: Memory=100.00MB
+Per-Host Resource Estimates: Memory=100MB
 
 PLAN-ROOT SINK
 |
@@ -41,7 +41,7 @@ PLAN-ROOT SINK
 select count(*) from functional_parquet.alltypes
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=16.00KB Threads=3
-Per-Host Resource Estimates: Memory=21.00MB
+Per-Host Resource Estimates: Memory=21MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional_parquet.alltypes
 
@@ -62,7 +62,7 @@ PLAN-ROOT SINK
 select * from functional_parquet.alltypes
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=88.00KB Threads=3
-Per-Host Resource Estimates: Memory=128.00MB
+Per-Host Resource Estimates: Memory=128MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional_parquet.alltypes
 
@@ -79,7 +79,7 @@ from functional.alltypes t1
 join functional.alltypestiny t2 on t1.id = t2.id
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=2.98MB Threads=5
-Per-Host Resource Estimates: Memory=182.94MB
+Per-Host Resource Estimates: Memory=183MB
 Codegen disabled by planner
 
 PLAN-ROOT SINK
@@ -109,7 +109,7 @@ PLAN-ROOT SINK
 select count(*) from functional.alltypes t1, functional.alltypes t2
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=64.00KB Threads=5
-Per-Host Resource Estimates: Memory=276.00MB
+Per-Host Resource Estimates: Memory=276MB
 
 PLAN-ROOT SINK
 |
@@ -138,7 +138,7 @@ select count(*) from (
   select * from functional.alltypestiny) v
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=32.00KB Threads=3
-Per-Host Resource Estimates: Memory=148.00MB
+Per-Host Resource Estimates: Memory=148MB
 Codegen disabled by planner
 
 PLAN-ROOT SINK
@@ -167,7 +167,7 @@ select count(*) from (
   select * from functional.alltypes) v
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=32.00KB Threads=3
-Per-Host Resource Estimates: Memory=148.00MB
+Per-Host Resource Estimates: Memory=148MB
 
 PLAN-ROOT SINK
 |
@@ -194,7 +194,7 @@ select sum(l_discount)
 from (select * from tpch.lineitem limit 1000) v
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=3
-Per-Host Resource Estimates: Memory=274.00MB
+Per-Host Resource Estimates: Memory=274MB
 Codegen disabled by planner
 
 PLAN-ROOT SINK
@@ -215,7 +215,7 @@ select sum(l_discount)
 from (select * from tpch.lineitem where l_orderkey > 100 limit 1000) v
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=3
-Per-Host Resource Estimates: Memory=274.00MB
+Per-Host Resource Estimates: Memory=274MB
 
 PLAN-ROOT SINK
 |

http://git-wip-us.apache.org/repos/asf/impala/blob/a8d7a50b/testdata/workloads/functional-planner/queries/PlannerTest/max-row-size.test
----------------------------------------------------------------------
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 cec6162..c228c0b 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/max-row-size.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/max-row-size.test
@@ -5,7 +5,7 @@ from tpch_parquet.customer
     inner join tpch_parquet.nation on c_nationkey = n_nationkey
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=33.97MB Threads=5
-Per-Host Resource Estimates: Memory=57.94MB
+Per-Host Resource Estimates: Memory=58MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -57,7 +57,7 @@ from tpch_parquet.lineitem
     left join tpch_parquet.orders on l_orderkey = o_orderkey
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=110.00MB Threads=5
-Per-Host Resource Estimates: Memory=420.41MB
+Per-Host Resource Estimates: Memory=420MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -106,7 +106,7 @@ select * from tpch_parquet.lineitem
 where l_orderkey not in (select o_orderkey from tpch_parquet.orders)
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=78.00MB Threads=5
-Per-Host Resource Estimates: Memory=154.00MB
+Per-Host Resource Estimates: Memory=154MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -157,7 +157,7 @@ group by 1, 2
 having count(*) = 1
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=125.00MB Threads=7
-Per-Host Resource Estimates: Memory=253.12MB
+Per-Host Resource Estimates: Memory=253MB
 
 F04:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -278,7 +278,7 @@ from tpch_parquet.lineitem
 group by 1, 2
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=98.00MB Threads=4
-Per-Host Resource Estimates: Memory=482.91MB
+Per-Host Resource Estimates: Memory=483MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -324,7 +324,7 @@ select max(tinyint_col) over(partition by int_col)
 from functional.alltypes
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=40.03MB Threads=4
-Per-Host Resource Estimates: Memory=56.00MB
+Per-Host Resource Estimates: Memory=56MB
 Codegen disabled by planner
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1

http://git-wip-us.apache.org/repos/asf/impala/blob/a8d7a50b/testdata/workloads/functional-planner/queries/PlannerTest/resource-requirements.test
----------------------------------------------------------------------
diff --git a/testdata/workloads/functional-planner/queries/PlannerTest/resource-requirements.test b/testdata/workloads/functional-planner/queries/PlannerTest/resource-requirements.test
index ee9a790..5a2e1c5 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/resource-requirements.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/resource-requirements.test
@@ -2,7 +2,7 @@
 select * from tpch_parquet.lineitem
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=40.00MB Threads=2
-Per-Host Resource Estimates: Memory=80.00MB
+Per-Host Resource Estimates: Memory=80MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=80.00MB mem-reservation=40.00MB thread-reservation=2
@@ -19,7 +19,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=40.00MB Threads=3
-Per-Host Resource Estimates: Memory=80.00MB
+Per-Host Resource Estimates: Memory=80MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -42,7 +42,7 @@ Per-Host Resources: mem-estimate=80.00MB mem-reservation=40.00MB thread-reservat
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=80.00MB Threads=3
-Per-Host Resource Estimates: Memory=160.00MB
+Per-Host Resource Estimates: Memory=160MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -69,7 +69,7 @@ Per-Host Resources: mem-estimate=160.00MB mem-reservation=80.00MB thread-reserva
 select l_comment from tpch_parquet.lineitem
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=4.00MB Threads=2
-Per-Host Resource Estimates: Memory=80.00MB
+Per-Host Resource Estimates: Memory=80MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=80.00MB mem-reservation=4.00MB thread-reservation=2
@@ -86,7 +86,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=42B cardinality=6001215
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=4.00MB Threads=3
-Per-Host Resource Estimates: Memory=80.00MB
+Per-Host Resource Estimates: Memory=80MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -109,7 +109,7 @@ Per-Host Resources: mem-estimate=80.00MB mem-reservation=4.00MB thread-reservati
    tuple-ids=0 row-size=42B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=3
-Per-Host Resource Estimates: Memory=160.00MB
+Per-Host Resource Estimates: Memory=160MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -135,7 +135,7 @@ Per-Host Resources: mem-estimate=160.00MB mem-reservation=8.00MB thread-reservat
 select string_col from functional_parquet.alltypes;
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=16.00KB Threads=2
-Per-Host Resource Estimates: Memory=16.00MB
+Per-Host Resource Estimates: Memory=16MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional_parquet.alltypes
 
@@ -155,7 +155,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=16B cardinality=unavailable
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=16.00KB Threads=3
-Per-Host Resource Estimates: Memory=16.00MB
+Per-Host Resource Estimates: Memory=16MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional_parquet.alltypes
 
@@ -181,7 +181,7 @@ Per-Host Resources: mem-estimate=16.00MB mem-reservation=16.00KB thread-reservat
    tuple-ids=0 row-size=16B cardinality=unavailable
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=32.00KB Threads=3
-Per-Host Resource Estimates: Memory=32.00MB
+Per-Host Resource Estimates: Memory=32MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional_parquet.alltypes
 
@@ -211,7 +211,7 @@ Per-Host Resources: mem-estimate=32.00MB mem-reservation=32.00KB thread-reservat
 select int_col, float_col, string_col from functional_parquet.alltypes;
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=24.00KB Threads=2
-Per-Host Resource Estimates: Memory=16.00MB
+Per-Host Resource Estimates: Memory=16MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional_parquet.alltypes
 
@@ -231,7 +231,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=24B cardinality=unavailable
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=24.00KB Threads=3
-Per-Host Resource Estimates: Memory=16.00MB
+Per-Host Resource Estimates: Memory=16MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional_parquet.alltypes
 
@@ -257,7 +257,7 @@ Per-Host Resources: mem-estimate=16.00MB mem-reservation=24.00KB thread-reservat
    tuple-ids=0 row-size=24B cardinality=unavailable
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=48.00KB Threads=3
-Per-Host Resource Estimates: Memory=32.00MB
+Per-Host Resource Estimates: Memory=32MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional_parquet.alltypes
 
@@ -287,7 +287,7 @@ Per-Host Resources: mem-estimate=32.00MB mem-reservation=48.00KB thread-reservat
 select 'foo' from functional_parquet.alltypes
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=16.00KB Threads=2
-Per-Host Resource Estimates: Memory=10.00MB
+Per-Host Resource Estimates: Memory=10MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional_parquet.alltypes
 
@@ -307,7 +307,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=0B cardinality=unavailable
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=16.00KB Threads=3
-Per-Host Resource Estimates: Memory=10.00MB
+Per-Host Resource Estimates: Memory=10MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional_parquet.alltypes
 
@@ -333,7 +333,7 @@ Per-Host Resources: mem-estimate=1.00MB mem-reservation=16.00KB thread-reservati
    tuple-ids=0 row-size=0B cardinality=unavailable
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=32.00KB Threads=3
-Per-Host Resource Estimates: Memory=32.00MB
+Per-Host Resource Estimates: Memory=32MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional_parquet.alltypes
 
@@ -363,7 +363,7 @@ Per-Host Resources: mem-estimate=32.00MB mem-reservation=32.00KB thread-reservat
 select c_nationkey from tpch_parquet.customer
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=128.00KB Threads=2
-Per-Host Resource Estimates: Memory=24.00MB
+Per-Host Resource Estimates: Memory=24MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=24.00MB mem-reservation=128.00KB thread-reservation=2
@@ -384,7 +384,7 @@ PLAN-ROOT SINK
 select c_custkey from tpch_parquet.customer
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=2.00MB Threads=2
-Per-Host Resource Estimates: Memory=24.00MB
+Per-Host Resource Estimates: Memory=24MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=24.00MB mem-reservation=2.00MB thread-reservation=2
@@ -406,7 +406,7 @@ PLAN-ROOT SINK
 select c_mktsegment from functional_parquet.customer_multiblock
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=512.00KB Threads=2
-Per-Host Resource Estimates: Memory=16.00MB
+Per-Host Resource Estimates: Memory=16MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional_parquet.customer_multiblock
 
@@ -429,7 +429,7 @@ select o_orderkey, o_orderstatus, o_totalprice, o_orderdate, o_orderpriority, o_
 from tpch_nested_parquet.customer.c_orders
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=24.00MB Threads=2
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=88.00MB mem-reservation=24.00MB thread-reservation=2
@@ -451,7 +451,7 @@ select o_orderkey, pos
 from tpch_nested_parquet.customer.c_orders
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=4.00MB Threads=2
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=88.00MB mem-reservation=4.00MB thread-reservation=2
@@ -473,7 +473,7 @@ select pos
 from tpch_nested_parquet.customer.c_orders
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=4.00MB Threads=2
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=88.00MB mem-reservation=4.00MB thread-reservation=2
@@ -496,7 +496,7 @@ select c_custkey, o_orderkey, o_orderstatus, o_totalprice, o_orderdate, o_orderp
 from tpch_nested_parquet.customer c, c.c_orders
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=32.00MB Threads=2
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_nested_parquet.customer
 
@@ -539,7 +539,7 @@ select c_custkey, o_orderkey, pos
 from tpch_nested_parquet.customer c, c.c_orders
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=2
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_nested_parquet.customer
 
@@ -583,7 +583,7 @@ select c_custkey, pos
 from tpch_nested_parquet.customer c, c.c_orders
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=2
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_nested_parquet.customer
 
@@ -627,7 +627,7 @@ select c_custkey
 from tpch_nested_parquet.customer c, c.c_orders
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=2
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_nested_parquet.customer
 
@@ -670,7 +670,7 @@ select o_orderkey
 from tpch_nested_parquet.customer c, c.c_orders
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=4.00MB Threads=2
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_nested_parquet.customer
 
@@ -713,7 +713,7 @@ select c_custkey, o_orderkey, l_comment
 from tpch_nested_parquet.customer c, c.c_orders o, o.o_lineitems
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=16.00MB Threads=2
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_nested_parquet.customer
 
@@ -773,7 +773,7 @@ PLAN-ROOT SINK
 select * from tpch.lineitem
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=2
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=2
@@ -790,7 +790,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=3
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -813,7 +813,7 @@ Per-Host Resources: mem-estimate=88.00MB mem-reservation=8.00MB thread-reservati
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=16.00MB Threads=3
-Per-Host Resource Estimates: Memory=176.00MB
+Per-Host Resource Estimates: Memory=176MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -839,7 +839,7 @@ Per-Host Resources: mem-estimate=176.00MB mem-reservation=16.00MB thread-reserva
 select l_comment from tpch.lineitem
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=2
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=2
@@ -856,7 +856,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=42B cardinality=6001215
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=3
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -879,7 +879,7 @@ Per-Host Resources: mem-estimate=88.00MB mem-reservation=8.00MB thread-reservati
    tuple-ids=0 row-size=42B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=16.00MB Threads=3
-Per-Host Resource Estimates: Memory=176.00MB
+Per-Host Resource Estimates: Memory=176MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -905,7 +905,7 @@ Per-Host Resources: mem-estimate=176.00MB mem-reservation=16.00MB thread-reserva
 select * from functional.alltypes
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=32.00KB Threads=2
-Per-Host Resource Estimates: Memory=16.00MB
+Per-Host Resource Estimates: Memory=16MB
 Codegen disabled by planner
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
@@ -924,7 +924,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=97B cardinality=7300
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=32.00KB Threads=3
-Per-Host Resource Estimates: Memory=16.00MB
+Per-Host Resource Estimates: Memory=16MB
 Codegen disabled by planner
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
@@ -949,7 +949,7 @@ Per-Host Resources: mem-estimate=16.00MB mem-reservation=32.00KB thread-reservat
    tuple-ids=0 row-size=97B cardinality=7300
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=64.00KB Threads=3
-Per-Host Resource Estimates: Memory=32.00MB
+Per-Host Resource Estimates: Memory=32MB
 Codegen disabled by planner
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
@@ -977,7 +977,7 @@ Per-Host Resources: mem-estimate=32.00MB mem-reservation=64.00KB thread-reservat
 select * from tpch_avro.orders
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=2
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_avro.orders
 
@@ -996,7 +996,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=108B cardinality=unavailable
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=3
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_avro.orders
 
@@ -1021,7 +1021,7 @@ Per-Host Resources: mem-estimate=88.00MB mem-reservation=8.00MB thread-reservati
    tuple-ids=0 row-size=108B cardinality=unavailable
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=16.00MB Threads=5
-Per-Host Resource Estimates: Memory=176.00MB
+Per-Host Resource Estimates: Memory=176MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_avro.orders
 
@@ -1049,7 +1049,7 @@ Per-Host Resources: mem-estimate=176.00MB mem-reservation=16.00MB thread-reserva
 select * from tpch_rc.customer
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=2
-Per-Host Resource Estimates: Memory=32.00MB
+Per-Host Resource Estimates: Memory=32MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_rc.customer
 
@@ -1068,7 +1068,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=98B cardinality=unavailable
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=3
-Per-Host Resource Estimates: Memory=32.00MB
+Per-Host Resource Estimates: Memory=32MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_rc.customer
 
@@ -1093,7 +1093,7 @@ Per-Host Resources: mem-estimate=32.00MB mem-reservation=8.00MB thread-reservati
    tuple-ids=0 row-size=98B cardinality=unavailable
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=16.00MB Threads=5
-Per-Host Resource Estimates: Memory=64.00MB
+Per-Host Resource Estimates: Memory=64MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_rc.customer
 
@@ -1121,7 +1121,7 @@ Per-Host Resources: mem-estimate=64.00MB mem-reservation=16.00MB thread-reservat
 select * from tpcds_seq_snap.web_returns
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=2
-Per-Host Resource Estimates: Memory=16.00MB
+Per-Host Resource Estimates: Memory=16MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpcds_seq_snap.web_returns
 
@@ -1140,7 +1140,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=104B cardinality=unavailable
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=3
-Per-Host Resource Estimates: Memory=16.00MB
+Per-Host Resource Estimates: Memory=16MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpcds_seq_snap.web_returns
 
@@ -1165,7 +1165,7 @@ Per-Host Resources: mem-estimate=16.00MB mem-reservation=8.00MB thread-reservati
    tuple-ids=0 row-size=104B cardinality=unavailable
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=16.00MB Threads=5
-Per-Host Resource Estimates: Memory=32.00MB
+Per-Host Resource Estimates: Memory=32MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpcds_seq_snap.web_returns
 
@@ -1193,7 +1193,7 @@ Per-Host Resources: mem-estimate=32.00MB mem-reservation=16.00MB thread-reservat
 select * from tpch_orc_def.lineitem
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=2
-Per-Host Resource Estimates: Memory=40.00MB
+Per-Host Resource Estimates: Memory=40MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=40.00MB mem-reservation=8.00MB thread-reservation=2
@@ -1213,7 +1213,7 @@ PLAN-ROOT SINK
 select l_comment from tpch_orc_def.lineitem
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=2
-Per-Host Resource Estimates: Memory=40.00MB
+Per-Host Resource Estimates: Memory=40MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=40.00MB mem-reservation=8.00MB thread-reservation=2
@@ -1233,7 +1233,7 @@ PLAN-ROOT SINK
 select * from functional_orc_def.alltypes
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=88.00KB Threads=2
-Per-Host Resource Estimates: Memory=16.00MB
+Per-Host Resource Estimates: Memory=16MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional_orc_def.alltypes
 
@@ -1256,7 +1256,7 @@ PLAN-ROOT SINK
 select * from functional.alltypesmixedformat
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=88.00KB Threads=2
-Per-Host Resource Estimates: Memory=16.00MB
+Per-Host Resource Estimates: Memory=16MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional.alltypesmixedformat
 
@@ -1276,7 +1276,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=88B cardinality=unavailable
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=88.00KB Threads=3
-Per-Host Resource Estimates: Memory=16.00MB
+Per-Host Resource Estimates: Memory=16MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional.alltypesmixedformat
 
@@ -1302,7 +1302,7 @@ Per-Host Resources: mem-estimate=16.00MB mem-reservation=88.00KB thread-reservat
    tuple-ids=0 row-size=88B cardinality=unavailable
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=176.00KB Threads=5
-Per-Host Resource Estimates: Memory=32.00MB
+Per-Host Resource Estimates: Memory=32MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional.alltypesmixedformat
 
@@ -1456,7 +1456,7 @@ union all
 select * from tpch.lineitem
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=2
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=88.00MB mem-reservation=8.00MB thread-reservation=2
@@ -1487,7 +1487,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=8.00MB Threads=3
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 
 F03:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -1524,7 +1524,7 @@ Per-Host Resources: mem-estimate=88.00MB mem-reservation=8.00MB thread-reservati
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=16.00MB Threads=3
-Per-Host Resource Estimates: Memory=176.00MB
+Per-Host Resource Estimates: Memory=176MB
 
 F03:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -1566,7 +1566,7 @@ from tpch_parquet.lineitem
 group by l_orderkey
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=38.00MB Threads=2
-Per-Host Resource Estimates: Memory=114.00MB
+Per-Host Resource Estimates: Memory=114MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=114.00MB mem-reservation=38.00MB thread-reservation=2
@@ -1589,7 +1589,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=8B cardinality=6001215
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=46.50MB Threads=4
-Per-Host Resource Estimates: Memory=124.00MB
+Per-Host Resource Estimates: Memory=124MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -1630,7 +1630,7 @@ Per-Host Resources: mem-estimate=114.00MB mem-reservation=38.00MB thread-reserva
    tuple-ids=0 row-size=8B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=85.50MB Threads=5
-Per-Host Resource Estimates: Memory=248.00MB
+Per-Host Resource Estimates: Memory=248MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -1674,7 +1674,7 @@ Per-Host Resources: mem-estimate=228.00MB mem-reservation=76.00MB thread-reserva
 select count(*) from tpch_parquet.lineitem
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=128.00KB Threads=2
-Per-Host Resource Estimates: Memory=11.00MB
+Per-Host Resource Estimates: Memory=11MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=11.00MB mem-reservation=128.00KB thread-reservation=2
@@ -1696,7 +1696,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=8B cardinality=6001215
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=128.00KB Threads=3
-Per-Host Resource Estimates: Memory=21.00MB
+Per-Host Resource Estimates: Memory=21MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=10.00MB mem-reservation=0B thread-reservation=1
@@ -1729,7 +1729,7 @@ Per-Host Resources: mem-estimate=11.00MB mem-reservation=128.00KB thread-reserva
    tuple-ids=0 row-size=8B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=256.00KB Threads=3
-Per-Host Resource Estimates: Memory=190.00MB
+Per-Host Resource Estimates: Memory=190MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=10.00MB mem-reservation=0B thread-reservation=1
@@ -1767,7 +1767,7 @@ from tpch_parquet.lineitem
 order by l_comment
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=52.00MB Threads=2
-Per-Host Resource Estimates: Memory=120.00MB
+Per-Host Resource Estimates: Memory=120MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=120.00MB mem-reservation=52.00MB thread-reservation=2
@@ -1789,7 +1789,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=52.00MB Threads=3
-Per-Host Resource Estimates: Memory=120.00MB
+Per-Host Resource Estimates: Memory=120MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -1818,7 +1818,7 @@ Per-Host Resources: mem-estimate=120.00MB mem-reservation=52.00MB thread-reserva
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=104.00MB Threads=3
-Per-Host Resource Estimates: Memory=240.00MB
+Per-Host Resource Estimates: Memory=240MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -1853,7 +1853,7 @@ order by l_comment
 limit 100
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=40.00MB Threads=2
-Per-Host Resource Estimates: Memory=80.03MB
+Per-Host Resource Estimates: Memory=80MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=80.03MB mem-reservation=40.00MB thread-reservation=2
@@ -1875,7 +1875,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=40.00MB Threads=3
-Per-Host Resource Estimates: Memory=80.03MB
+Per-Host Resource Estimates: Memory=80MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -1905,7 +1905,7 @@ Per-Host Resources: mem-estimate=80.03MB mem-reservation=40.00MB thread-reservat
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=80.00MB Threads=3
-Per-Host Resource Estimates: Memory=160.05MB
+Per-Host Resource Estimates: Memory=160MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -1939,7 +1939,7 @@ select *
 from tpch.lineitem inner join tpch.orders on l_orderkey = o_orderkey
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=51.00MB Threads=3
-Per-Host Resource Estimates: Memory=477.41MB
+Per-Host Resource Estimates: Memory=477MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=477.41MB mem-reservation=51.00MB thread-reservation=3 runtime-filters-memory=1.00MB
@@ -1973,7 +1973,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=51.00MB Threads=5
-Per-Host Resource Estimates: Memory=477.41MB
+Per-Host Resource Estimates: Memory=477MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -2019,7 +2019,7 @@ Per-Host Resources: mem-estimate=389.41MB mem-reservation=43.00MB thread-reserva
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=102.00MB Threads=5
-Per-Host Resource Estimates: Memory=954.83MB
+Per-Host Resource Estimates: Memory=955MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -2077,7 +2077,7 @@ select *
 from tpch.lineitem inner join /* +shuffle */ tpch.orders on l_orderkey = o_orderkey
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=51.00MB Threads=3
-Per-Host Resource Estimates: Memory=477.41MB
+Per-Host Resource Estimates: Memory=477MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=477.41MB mem-reservation=51.00MB thread-reservation=3 runtime-filters-memory=1.00MB
@@ -2111,7 +2111,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=52.00MB Threads=6
-Per-Host Resource Estimates: Memory=278.14MB
+Per-Host Resource Estimates: Memory=278MB
 
 F03:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -2163,7 +2163,7 @@ Per-Host Resources: mem-estimate=89.00MB mem-reservation=9.00MB thread-reservati
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=104.00MB Threads=7
-Per-Host Resource Estimates: Memory=456.14MB
+Per-Host Resource Estimates: Memory=456MB
 
 F03:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -2227,7 +2227,7 @@ select *
 from tpch.lineitem, tpch.orders
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=16.00MB Threads=3
-Per-Host Resource Estimates: Memory=449.10MB
+Per-Host Resource Estimates: Memory=449MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=449.10MB mem-reservation=16.00MB thread-reservation=3
@@ -2257,7 +2257,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=16.00MB Threads=5
-Per-Host Resource Estimates: Memory=449.10MB
+Per-Host Resource Estimates: Memory=449MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -2299,7 +2299,7 @@ Per-Host Resources: mem-estimate=361.10MB mem-reservation=8.00MB thread-reservat
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=32.00MB Threads=5
-Per-Host Resource Estimates: Memory=898.21MB
+Per-Host Resource Estimates: Memory=898MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -2351,7 +2351,7 @@ Per-Host Resources: mem-estimate=722.21MB mem-reservation=16.00MB thread-reserva
 select * from functional.alltypes where 1 = 2
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=0B Threads=1
-Per-Host Resource Estimates: Memory=10.00MB
+Per-Host Resource Estimates: Memory=10MB
 Codegen disabled by planner
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
@@ -2364,7 +2364,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=0B cardinality=0
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=0B Threads=1
-Per-Host Resource Estimates: Memory=10.00MB
+Per-Host Resource Estimates: Memory=10MB
 Codegen disabled by planner
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
@@ -2377,7 +2377,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=0B cardinality=0
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=0B Threads=1
-Per-Host Resource Estimates: Memory=10.00MB
+Per-Host Resource Estimates: Memory=10MB
 Codegen disabled by planner
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
@@ -2394,7 +2394,7 @@ select max(tinyint_col) over(partition by int_col)
 from functional.alltypes
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=10.03MB Threads=2
-Per-Host Resource Estimates: Memory=26.00MB
+Per-Host Resource Estimates: Memory=26MB
 Codegen disabled by planner
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
@@ -2424,7 +2424,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=5B cardinality=7300
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=10.03MB Threads=4
-Per-Host Resource Estimates: Memory=26.00MB
+Per-Host Resource Estimates: Memory=26MB
 Codegen disabled by planner
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
@@ -2466,7 +2466,7 @@ Per-Host Resources: mem-estimate=16.00MB mem-reservation=32.00KB thread-reservat
    tuple-ids=0 row-size=5B cardinality=7300
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=20.06MB Threads=5
-Per-Host Resource Estimates: Memory=52.00MB
+Per-Host Resource Estimates: Memory=52MB
 Codegen disabled by planner
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
@@ -2515,7 +2515,7 @@ select *, row_number() over (order by o_totalprice) rnum_price,
 from tpch_parquet.orders
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=48.00MB Threads=2
-Per-Host Resource Estimates: Memory=70.00MB
+Per-Host Resource Estimates: Memory=70MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=70.00MB mem-reservation=48.00MB thread-reservation=2
@@ -2568,7 +2568,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=191B cardinality=1500000
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=68.00MB Threads=3
-Per-Host Resource Estimates: Memory=102.00MB
+Per-Host Resource Estimates: Memory=102MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=44.00MB mem-reservation=32.00MB thread-reservation=1
@@ -2628,7 +2628,7 @@ Per-Host Resources: mem-estimate=58.00MB mem-reservation=36.00MB thread-reservat
    tuple-ids=0 row-size=191B cardinality=1500000
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=104.00MB Threads=3
-Per-Host Resource Estimates: Memory=160.00MB
+Per-Host Resource Estimates: Memory=160MB
 
 F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=44.00MB mem-reservation=32.00MB thread-reservation=1
@@ -2701,7 +2701,7 @@ from tpch_parquet.lineitem join tpch_parquet.orders on l_orderkey = o_orderkey
 where l_shipmode = 'F'
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=78.00MB Threads=3
-Per-Host Resource Estimates: Memory=142.58MB
+Per-Host Resource Estimates: Memory=143MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=142.58MB mem-reservation=78.00MB thread-reservation=3 runtime-filters-memory=3.00MB
@@ -2806,7 +2806,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=78B cardinality=600122
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=123.75MB Threads=12
-Per-Host Resource Estimates: Memory=348.33MB
+Per-Host Resource Estimates: Memory=348MB
 
 F09:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -2952,7 +2952,7 @@ Per-Host Resources: mem-estimate=81.00MB mem-reservation=25.00MB thread-reservat
    tuple-ids=0 row-size=78B cardinality=600122
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=243.75MB Threads=13
-Per-Host Resource Estimates: Memory=692.91MB
+Per-Host Resource Estimates: Memory=693MB
 
 F09:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -3158,7 +3158,7 @@ order by
 limit 100
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=92.25MB Threads=5
-Per-Host Resource Estimates: Memory=396.18MB
+Per-Host Resource Estimates: Memory=396MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=396.18MB mem-reservation=92.25MB thread-reservation=5 runtime-filters-memory=3.00MB
@@ -3242,7 +3242,7 @@ PLAN-ROOT SINK
    tuple-ids=2 row-size=16B cardinality=6001215
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=191.12MB Threads=11
-Per-Host Resource Estimates: Memory=523.66MB
+Per-Host Resource Estimates: Memory=524MB
 
 F07:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -3376,7 +3376,7 @@ Per-Host Resources: mem-estimate=90.00MB mem-reservation=10.00MB thread-reservat
    tuple-ids=2 row-size=16B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=336.88MB Threads=13
-Per-Host Resource Estimates: Memory=991.44MB
+Per-Host Resource Estimates: Memory=991MB
 
 F07:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -3541,7 +3541,7 @@ from functional.alltypes
 where year=2009 and month=05
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=32.00KB Threads=2
-Per-Host Resource Estimates: Memory=16.03MB
+Per-Host Resource Estimates: Memory=16MB
 Codegen disabled by planner
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
@@ -3561,7 +3561,7 @@ WRITE TO HDFS [functional.alltypesnopart, OVERWRITE=false]
    tuple-ids=0 row-size=89B cardinality=310
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=32.00KB Threads=2
-Per-Host Resource Estimates: Memory=16.03MB
+Per-Host Resource Estimates: Memory=16MB
 Codegen disabled by planner
 
 F00:PLAN FRAGMENT [RANDOM] hosts=1 instances=1
@@ -3581,7 +3581,7 @@ WRITE TO HDFS [functional.alltypesnopart, OVERWRITE=false]
    tuple-ids=0 row-size=89B cardinality=310
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=64.00KB Threads=2
-Per-Host Resource Estimates: Memory=32.03MB
+Per-Host Resource Estimates: Memory=32MB
 Codegen disabled by planner
 
 F00:PLAN FRAGMENT [RANDOM] hosts=1 instances=2
@@ -3606,7 +3606,7 @@ partitioned by (l_partkey) as
 select l_comment, l_partkey from tpch.lineitem
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=20.00MB Threads=2
-Per-Host Resource Estimates: Memory=306.99MB
+Per-Host Resource Estimates: Memory=307MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=306.99MB mem-reservation=20.00MB thread-reservation=2
@@ -3629,7 +3629,7 @@ WRITE TO HDFS [default.dummy_insert, OVERWRITE=false, PARTITION-KEYS=(l_partkey)
    tuple-ids=0 row-size=50B cardinality=6001215
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=20.00MB Threads=3
-Per-Host Resource Estimates: Memory=202.33MB
+Per-Host Resource Estimates: Memory=202MB
 
 F01:PLAN FRAGMENT [HASH(l_partkey)] hosts=3 instances=3
 |  Per-Host Resources: mem-estimate=114.33MB mem-reservation=12.00MB thread-reservation=1
@@ -3658,7 +3658,7 @@ Per-Host Resources: mem-estimate=88.00MB mem-reservation=8.00MB thread-reservati
    tuple-ids=0 row-size=50B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=40.00MB Threads=4
-Per-Host Resource Estimates: Memory=308.33MB
+Per-Host Resource Estimates: Memory=308MB
 
 F01:PLAN FRAGMENT [HASH(l_partkey)] hosts=3 instances=6
 |  Per-Host Resources: mem-estimate=132.33MB mem-reservation=24.00MB thread-reservation=2
@@ -3696,7 +3696,7 @@ from tpch_nested_parquet.customer c,
    order by o1.o_orderkey limit 100) v
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=85.94MB Threads=2
-Per-Host Resource Estimates: Memory=345.94MB
+Per-Host Resource Estimates: Memory=346MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_nested_parquet.customer
 
@@ -3759,7 +3759,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=66B cardinality=150000
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=119.94MB Threads=4
-Per-Host Resource Estimates: Memory=473.94MB
+Per-Host Resource Estimates: Memory=474MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_nested_parquet.customer
 
@@ -3839,7 +3839,7 @@ Per-Host Resources: mem-estimate=345.94MB mem-reservation=85.94MB thread-reserva
    tuple-ids=0 row-size=66B cardinality=150000
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=239.88MB Threads=5
-Per-Host Resource Estimates: Memory=947.88MB
+Per-Host Resource Estimates: Memory=948MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_nested_parquet.customer
 
@@ -3928,7 +3928,7 @@ from tpch_nested_parquet.customer c,
    from c.c_orders) v;
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=104.00MB Threads=2
-Per-Host Resource Estimates: Memory=136.00MB
+Per-Host Resource Estimates: Memory=136MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_nested_parquet.customer
 
@@ -4001,7 +4001,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=254B cardinality=150000
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=104.00MB Threads=3
-Per-Host Resource Estimates: Memory=136.00MB
+Per-Host Resource Estimates: Memory=136MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_nested_parquet.customer
 
@@ -4080,7 +4080,7 @@ Per-Host Resources: mem-estimate=136.00MB mem-reservation=104.00MB thread-reserv
    tuple-ids=0 row-size=254B cardinality=150000
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=208.00MB Threads=3
-Per-Host Resource Estimates: Memory=272.00MB
+Per-Host Resource Estimates: Memory=272MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 tpch_nested_parquet.customer
 
@@ -4173,7 +4173,7 @@ join (
 ) v1 on v1.k3 = t1.o_orderkey
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=99.00MB Threads=5
-Per-Host Resource Estimates: Memory=180.00MB
+Per-Host Resource Estimates: Memory=180MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=180.00MB mem-reservation=99.00MB thread-reservation=5 runtime-filters-memory=3.00MB
@@ -4241,7 +4241,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=191B cardinality=1500000
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=100.50MB Threads=10
-Per-Host Resource Estimates: Memory=228.27MB
+Per-Host Resource Estimates: Memory=228MB
 
 F05:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -4339,7 +4339,7 @@ Per-Host Resources: mem-estimate=78.77MB mem-reservation=59.00MB thread-reservat
    tuple-ids=0 row-size=191B cardinality=1500000
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=176.50MB Threads=11
-Per-Host Resource Estimates: Memory=432.03MB
+Per-Host Resource Estimates: Memory=432MB
 
 F05:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -4477,7 +4477,7 @@ join (
 ) v1
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=176.00KB Threads=5
-Per-Host Resource Estimates: Memory=137.99MB
+Per-Host Resource Estimates: Memory=138MB
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=137.99MB mem-reservation=176.00KB thread-reservation=5
@@ -4533,7 +4533,7 @@ PLAN-ROOT SINK
    tuple-ids=0 row-size=117B cardinality=25
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=176.00KB Threads=9
-Per-Host Resource Estimates: Memory=137.99MB
+Per-Host Resource Estimates: Memory=138MB
 
 F04:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -4613,7 +4613,7 @@ Per-Host Resources: mem-estimate=87.53MB mem-reservation=32.00KB thread-reservat
    tuple-ids=0 row-size=117B cardinality=25
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=352.00KB Threads=9
-Per-Host Resource Estimates: Memory=275.97MB
+Per-Host Resource Estimates: Memory=276MB
 
 F04:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -4723,7 +4723,7 @@ sum(smallint_col) over (partition by tinyint_col order by smallint_col
                                                 from functional.alltypesagg
 ---- PLAN
 Max Per-Host Resource Reservation: Memory=18.12MB Threads=2
-Per-Host Resource Estimates: Memory=34.00MB
+Per-Host Resource Estimates: Memory=34MB
 Codegen disabled by planner
 
 F00:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1

http://git-wip-us.apache.org/repos/asf/impala/blob/a8d7a50b/testdata/workloads/functional-planner/queries/PlannerTest/spillable-buffer-sizing.test
----------------------------------------------------------------------
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 e849da6..5acc9ae 100644
--- a/testdata/workloads/functional-planner/queries/PlannerTest/spillable-buffer-sizing.test
+++ b/testdata/workloads/functional-planner/queries/PlannerTest/spillable-buffer-sizing.test
@@ -4,7 +4,7 @@ from tpch_parquet.customer
     inner join tpch_parquet.nation on c_nationkey = n_nationkey
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=18.97MB Threads=5
-Per-Host Resource Estimates: Memory=42.94MB
+Per-Host Resource Estimates: Memory=43MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -50,7 +50,7 @@ Per-Host Resources: mem-estimate=26.94MB mem-reservation=18.94MB thread-reservat
    tuple-ids=0 row-size=238B cardinality=150000
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=37.94MB Threads=5
-Per-Host Resource Estimates: Memory=85.88MB
+Per-Host Resource Estimates: Memory=86MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -109,7 +109,7 @@ from tpch_parquet.lineitem
     left join tpch_parquet.orders on l_orderkey = o_orderkey
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=98.00MB Threads=5
-Per-Host Resource Estimates: Memory=420.41MB
+Per-Host Resource Estimates: Memory=420MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -153,7 +153,7 @@ Per-Host Resources: mem-estimate=380.41MB mem-reservation=74.00MB thread-reserva
    tuple-ids=0 row-size=263B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=196.00MB Threads=5
-Per-Host Resource Estimates: Memory=840.83MB
+Per-Host Resource Estimates: Memory=841MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -210,7 +210,7 @@ from tpch_parquet.orders
     join /*+shuffle*/ tpch_parquet.customer on o_custkey = c_custkey
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=76.00MB Threads=6
-Per-Host Resource Estimates: Memory=100.00MB
+Per-Host Resource Estimates: Memory=100MB
 
 F03:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -262,7 +262,7 @@ Per-Host Resources: mem-estimate=41.00MB mem-reservation=25.00MB thread-reservat
    tuple-ids=0 row-size=191B cardinality=1500000
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=118.00MB Threads=7
-Per-Host Resource Estimates: Memory=166.00MB
+Per-Host Resource Estimates: Memory=166MB
 
 F03:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -327,7 +327,7 @@ from tpch_parquet.orders
     join /*+broadcast*/ tpch_parquet.customer on o_custkey = c_custkey
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=75.00MB Threads=5
-Per-Host Resource Estimates: Memory=102.38MB
+Per-Host Resource Estimates: Memory=102MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -373,7 +373,7 @@ Per-Host Resources: mem-estimate=78.38MB mem-reservation=59.00MB thread-reservat
    tuple-ids=0 row-size=191B cardinality=1500000
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=150.00MB Threads=5
-Per-Host Resource Estimates: Memory=204.76MB
+Per-Host Resource Estimates: Memory=205MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -541,7 +541,7 @@ from tpch_parquet.customer
 group by c_nationkey
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=5.94MB Threads=4
-Per-Host Resource Estimates: Memory=44.00MB
+Per-Host Resource Estimates: Memory=44MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -582,7 +582,7 @@ Per-Host Resources: mem-estimate=34.00MB mem-reservation=4.00MB thread-reservati
    tuple-ids=0 row-size=10B cardinality=150000
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=11.88MB Threads=5
-Per-Host Resource Estimates: Memory=88.00MB
+Per-Host Resource Estimates: Memory=88MB
 
 F02:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -630,7 +630,7 @@ group by 1, 2
 having count(*) = 1
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=99.00MB Threads=7
-Per-Host Resource Estimates: Memory=227.12MB
+Per-Host Resource Estimates: Memory=227MB
 
 F04:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -701,7 +701,7 @@ Per-Host Resources: mem-estimate=81.00MB mem-reservation=5.00MB thread-reservati
    tuple-ids=0 row-size=8B cardinality=6001215
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=147.00MB Threads=9
-Per-Host Resource Estimates: Memory=363.00MB
+Per-Host Resource Estimates: Memory=363MB
 
 F04:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
 |  Per-Host Resources: mem-estimate=0B mem-reservation=0B thread-reservation=1
@@ -867,7 +867,7 @@ from functional_parquet.alltypestiny
 group by string_col
 ---- DISTRIBUTEDPLAN
 Max Per-Host Resource Reservation: Memory=68.01MB Threads=4
-Per-Host Resource Estimates: Memory=272.00MB
+Per-Host Resource Estimates: Memory=272MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional_parquet.alltypestiny
 
@@ -911,7 +911,7 @@ Per-Host Resources: mem-estimate=144.00MB mem-reservation=34.01MB thread-reserva
    tuple-ids=0 row-size=16B cardinality=unavailable
 ---- PARALLELPLANS
 Max Per-Host Resource Reservation: Memory=136.02MB Threads=5
-Per-Host Resource Estimates: Memory=544.00MB
+Per-Host Resource Estimates: Memory=544MB
 WARNING: The following tables are missing relevant table and/or column statistics.
 functional_parquet.alltypestiny