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 2020/06/24 16:50:02 UTC

[impala] 02/02: IMPALA-8755: Unlock Z-ordering by default

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

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

commit 71a64591e35b5de818ca0cc1212b720d2e69c111
Author: norbert.luksa <no...@cloudera.com>
AuthorDate: Fri May 29 18:01:45 2020 +0200

    IMPALA-8755: Unlock Z-ordering by default
    
    Z-ordering has been around for a while behind a feature flag
    (unlock_zorder_sort). It's around time to turn this flag on by default.
    
    This commit beside setting the flag to true, merges the Z-order tests
    from custom cluster tests into the normal test files.
    
    Tests:
      - Run all related tests.
    
    Change-Id: I653e0e2db8f7bc2dd077943b3acf667514d45811
    Reviewed-on: http://gerrit.cloudera.org:8080/16003
    Reviewed-by: Impala Public Jenkins <im...@cloudera.com>
    Tested-by: Impala Public Jenkins <im...@cloudera.com>
---
 be/src/common/global-flags.cc                      |   4 +-
 fe/src/main/cup/sql-parser.cup                     |   6 +-
 .../org/apache/impala/analysis/AnalyzeDDLTest.java |  20 --
 .../apache/impala/analysis/AnalyzeKuduDDLTest.java |  11 +-
 .../org/apache/impala/analysis/ParserTest.java     |   9 -
 .../java/org/apache/impala/analysis/ToSqlTest.java |  11 -
 .../org/apache/impala/analysis/ToSqlUtilsTest.java |   6 -
 .../authorization/AuthorizationStmtTest.java       |   2 -
 .../org/apache/impala/planner/PlannerTest.java     |   5 -
 .../queries/QueryTest/alter-table-zorder.test      | 244 ---------------------
 .../queries/QueryTest/alter-table.test             | 243 ++++++++++++++++++++
 .../QueryTest/create-table-as-select-zorder.test   |  12 -
 .../queries/QueryTest/create-table-as-select.test  |  11 +
 .../QueryTest/create-table-like-file-zorder.test   |  13 --
 .../queries/QueryTest/create-table-like-file.test  |  12 +
 .../QueryTest/create-table-like-table-zorder.test  |  31 ---
 .../queries/QueryTest/create-table-like-table.test |  32 ++-
 .../queries/QueryTest/create-table-zorder.test     |  12 -
 .../queries/QueryTest/create-table.test            |  11 +
 tests/custom_cluster/test_zorder.py                |  85 -------
 20 files changed, 315 insertions(+), 465 deletions(-)

diff --git a/be/src/common/global-flags.cc b/be/src/common/global-flags.cc
index 17e29dd..357e56b 100644
--- a/be/src/common/global-flags.cc
+++ b/be/src/common/global-flags.cc
@@ -299,8 +299,8 @@ DEFINE_bool_hidden(mt_dop_auto_fallback, false,
 DEFINE_bool_hidden(recursively_list_partitions, true,
     "If true, recursively list the content of partition directories.");
 
-DEFINE_bool(unlock_zorder_sort, false,
-    "(Experimental) If true, enables using ZORDER option for SORT BY.");
+DEFINE_bool(unlock_zorder_sort, true,
+    "If true, enables using ZORDER option for SORT BY.");
 
 DEFINE_string(min_privilege_set_for_show_stmts, "any",
     "Comma separated list of privileges. Any one of them is required to show a database "
diff --git a/fe/src/main/cup/sql-parser.cup b/fe/src/main/cup/sql-parser.cup
index 0ea21d6..3679b97 100644
--- a/fe/src/main/cup/sql-parser.cup
+++ b/fe/src/main/cup/sql-parser.cup
@@ -1228,7 +1228,8 @@ alter_tbl_stmt ::=
     RPAREN
   {:
     if (!BackendConfig.INSTANCE.isZOrderSortUnlocked()) {
-      throw new NotImplementedException("Z-ordering is not yet implemented");
+      throw new NotImplementedException("Z-ordering is not enabled. You can use the " +
+          "unlock_zorder_sort flag to enable it.");
     }
     RESULT = new AlterTableSortByStmt(table, col_names, TSortingOrder.ZORDER);
   :}
@@ -1584,7 +1585,8 @@ opt_sort_cols ::=
   | KW_SORT KW_BY KW_ZORDER LPAREN opt_ident_list:col_names RPAREN
   {:
     if (!BackendConfig.INSTANCE.isZOrderSortUnlocked()) {
-      throw new NotImplementedException("Z-ordering is not yet implemented");
+      throw new NotImplementedException("Z-ordering is not enabled. You can use the " +
+          "unlock_zorder_sort flag to enable it.");
     }
     RESULT = new Pair<List<String>, TSortingOrder>(
       col_names, TSortingOrder.ZORDER);
diff --git a/fe/src/test/java/org/apache/impala/analysis/AnalyzeDDLTest.java b/fe/src/test/java/org/apache/impala/analysis/AnalyzeDDLTest.java
index 35b7403..964162b 100644
--- a/fe/src/test/java/org/apache/impala/analysis/AnalyzeDDLTest.java
+++ b/fe/src/test/java/org/apache/impala/analysis/AnalyzeDDLTest.java
@@ -1291,9 +1291,6 @@ public class AnalyzeDDLTest extends FrontendTestBase {
 
   @Test
   public void TestAlterTableSortByZOrder() {
-
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(true);
-
     AnalyzesOk("alter table functional.alltypes sort by zorder (int_col,id)");
     AnalyzesOk("alter table functional.alltypes sort by zorder (bool_col,int_col,id)");
     AnalyzesOk(
@@ -1308,8 +1305,6 @@ public class AnalyzeDDLTest extends FrontendTestBase {
         "not find SORT BY column 'foo' in table.");
     AnalysisError("alter table functional_hbase.alltypes sort by zorder (id, foo)",
         "ALTER TABLE SORT BY not supported on HBase tables.");
-
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(false);
   }
 
   @Test
@@ -1946,9 +1941,6 @@ public class AnalyzeDDLTest extends FrontendTestBase {
 
   @Test
   public void TestCreateTableLikeFile() throws AnalysisException {
-
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(true);
-
     // check that we analyze all of the CREATE TABLE options
     AnalyzesOk("create table if not exists newtbl_DNE like parquet "
         + "'/test-warehouse/schemas/alltypestiny.parquet'");
@@ -2013,10 +2005,6 @@ public class AnalyzeDDLTest extends FrontendTestBase {
     AnalysisError("create table newtbl_kudu like parquet " +
         "'/test-warehouse/schemas/alltypestiny.parquet' stored as kudu",
         "CREATE TABLE LIKE FILE statement is not supported for Kudu tables.");
-
-
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(false);
-
   }
 
   @Test
@@ -2346,8 +2334,6 @@ public class AnalyzeDDLTest extends FrontendTestBase {
         "Could not find SORT BY column 'foo' in table.");
 
     // Test sort by zorder columns.
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(true);
-
     AnalyzesOk("create table tbl sort by zorder (int_col,id) like functional.alltypes");
     AnalyzesOk("create table tbl sort by zorder (float_col,id) like functional.alltypes");
     AnalyzesOk("create table tbl sort by zorder (double_col,id) like " +
@@ -2356,8 +2342,6 @@ public class AnalyzeDDLTest extends FrontendTestBase {
         "functional.alltypes");
     AnalysisError("create table tbl sort by zorder (int_col,foo) like " +
         "functional.alltypes", "Could not find SORT BY column 'foo' in table.");
-
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(false);
   }
 
   @Test
@@ -2647,8 +2631,6 @@ public class AnalyzeDDLTest extends FrontendTestBase {
           type.name());
     }
 
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(true);
-
     // Tables with sort columns
     AnalyzesOk("create table functional.new_table (i int, j int) sort by (i)");
     AnalyzesOk("create table functional.new_table (i int, j int) sort by (i, j)");
@@ -2691,8 +2673,6 @@ public class AnalyzeDDLTest extends FrontendTestBase {
     AnalysisError("create table functional.new_table (i int) PARTITIONED BY (d decimal)" +
         "sort by zorder (i, d)", "SORT BY column list must not contain partition " +
         "column: 'd'");
-
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(false);
   }
 
   @Test
diff --git a/fe/src/test/java/org/apache/impala/analysis/AnalyzeKuduDDLTest.java b/fe/src/test/java/org/apache/impala/analysis/AnalyzeKuduDDLTest.java
index c00ccfc..1552ceb 100644
--- a/fe/src/test/java/org/apache/impala/analysis/AnalyzeKuduDDLTest.java
+++ b/fe/src/test/java/org/apache/impala/analysis/AnalyzeKuduDDLTest.java
@@ -19,7 +19,6 @@ package org.apache.impala.analysis;
 
 import org.apache.impala.catalog.KuduTable;
 import org.apache.impala.common.FrontendTestBase;
-import org.apache.impala.service.BackendConfig;
 import org.apache.impala.testutil.TestUtils;
 import org.apache.kudu.ColumnSchema.CompressionAlgorithm;
 import org.apache.kudu.ColumnSchema.Encoding;
@@ -431,15 +430,11 @@ public class AnalyzeKuduDDLTest extends FrontendTestBase {
         "partitions 8 sort by(i) stored as kudu", "SORT BY is not supported for Kudu " +
         "tables.", isExternalPurgeTbl);
 
-    // Z-Sort columns are not supported for Kudu tables.
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(true);
-
+    // Z-Order sorted columns are not supported for Kudu tables.
     AnalysisError("create table tab (i int, x int primary key) partition by hash(x) " +
         "partitions 8 sort by zorder(i) stored as kudu", "SORT BY is not " +
         "supported for Kudu tables.", isExternalPurgeTbl);
 
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(false);
-
     // Range partitions with TIMESTAMP
     AnalyzesOk("create table ts_ranges (ts timestamp primary key) " +
         "partition by range (partition cast('2009-01-01 00:00:00' as timestamp) " +
@@ -715,8 +710,6 @@ public class AnalyzeKuduDDLTest extends FrontendTestBase {
     AnalysisError("alter table functional_kudu.alltypes sort by (int_col)",
         "ALTER TABLE SORT BY not supported on Kudu tables.");
 
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(true);
-
     // ALTER TABLE SORT BY ZORDER
     AnalysisError("alter table functional_kudu.alltypes sort by zorder (int_col)",
         "ALTER TABLE SORT BY not supported on Kudu tables.");
@@ -730,7 +723,5 @@ public class AnalyzeKuduDDLTest extends FrontendTestBase {
     AnalysisError("alter table functional_kudu.alltypes set tblproperties("
         + "'sort.order'='true')",
         "'sort.*' table properties are not supported for Kudu tables.");
-
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(false);
   }
 }
diff --git a/fe/src/test/java/org/apache/impala/analysis/ParserTest.java b/fe/src/test/java/org/apache/impala/analysis/ParserTest.java
index a22a163..86d4c0b 100644
--- a/fe/src/test/java/org/apache/impala/analysis/ParserTest.java
+++ b/fe/src/test/java/org/apache/impala/analysis/ParserTest.java
@@ -29,7 +29,6 @@ import org.apache.impala.analysis.TimestampArithmeticExpr.TimeUnit;
 import org.apache.impala.common.AnalysisException;
 import org.apache.impala.common.FrontendTestBase;
 import org.apache.impala.compat.MetastoreShim;
-import org.apache.impala.service.BackendConfig;
 import org.junit.Test;
 
 import com.google.common.base.Preconditions;
@@ -2470,14 +2469,10 @@ public class ParserTest extends FrontendTestBase {
 
   @Test
   public void TestAlterTableZSortBy() {
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(true);
-
     ParsesOk("ALTER TABLE TEST SORT BY ZORDER (int_col, id)");
     ParsesOk("ALTER TABLE TEST SORT BY ZORDER ()");
     ParserError("ALTER TABLE TEST PARTITION (year=2009, month=4) SORT BY ZORDER " +
         "(int_col, id)");
-
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(false);
   }
 
   @Test
@@ -2609,8 +2604,6 @@ public class ParserTest extends FrontendTestBase {
     ParserError("CREATE TABLE Foo SORT BY (id) LIKE PARQUET '/user/foo'");
 
     // SORT BY ZORDER clause
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(true);
-
     ParsesOk("CREATE TABLE Foo (i int, j int) SORT BY ZORDER ()");
     ParsesOk("CREATE TABLE Foo (i int) SORT BY ZORDER (i)");
     ParsesOk("CREATE TABLE Foo (i int) SORT BY ZORDER (j)");
@@ -2651,8 +2644,6 @@ public class ParserTest extends FrontendTestBase {
     ParsesOk("CREATE TABLE Foo LIKE PARQUET '/user/foo' SORT BY ZORDER (id)");
     ParserError("CREATE TABLE Foo SORT BY ZORDER (id) LIKE PARQUET '/user/foo'");
 
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(false);
-
     // Column comments
     ParsesOk("CREATE TABLE Foo (i int COMMENT 'hello', s string)");
     ParsesOk("CREATE TABLE Foo (i int COMMENT 'hello', s string COMMENT 'hi')");
diff --git a/fe/src/test/java/org/apache/impala/analysis/ToSqlTest.java b/fe/src/test/java/org/apache/impala/analysis/ToSqlTest.java
index 079b0b3..473808a 100644
--- a/fe/src/test/java/org/apache/impala/analysis/ToSqlTest.java
+++ b/fe/src/test/java/org/apache/impala/analysis/ToSqlTest.java
@@ -23,7 +23,6 @@ import static org.junit.Assert.fail;
 import org.apache.impala.authorization.Privilege;
 import org.apache.impala.common.AnalysisException;
 import org.apache.impala.common.FrontendTestBase;
-import org.apache.impala.service.BackendConfig;
 import org.apache.impala.testutil.TestUtils;
 import org.junit.Test;
 
@@ -345,15 +344,11 @@ public class ToSqlTest extends FrontendTestBase {
         "SORT BY LEXICAL ( a, b ) STORED AS TEXTFILE" , true);
 
     // Table with SORT BY ZORDER clause.
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(true);
-
     testToSql("create table p (a int, b int) partitioned by (day string) sort by zorder" +
         "(a ,b) ", "default",
         "CREATE TABLE default.p ( a INT, b INT ) PARTITIONED BY ( day STRING ) " +
         "SORT BY ZORDER ( a, b ) STORED AS TEXTFILE" , true);
 
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(false);
-
     // Kudu table with a TIMESTAMP column default value
     String kuduMasters = catalog_.getDefaultKuduMasterHosts();
     testToSql(String.format("create table p (a bigint primary key, " +
@@ -412,14 +407,12 @@ public class ToSqlTest extends FrontendTestBase {
         "( string_col ) STORED AS TEXTFILE AS SELECT double_col, string_col, int_col " +
         "FROM functional.alltypes", true);
     // Table with SORT BY ZORDER clause.
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(true);
     testToSql("create table p partitioned by (string_col) sort by zorder (int_col, " +
         "bool_col) as select int_col, bool_col, string_col from functional.alltypes",
         "default",
         "CREATE TABLE default.p PARTITIONED BY ( string_col ) SORT BY ZORDER " +
         "( int_col, bool_col ) STORED AS TEXTFILE AS SELECT " +
         "int_col, bool_col, string_col FROM functional.alltypes", true);
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(false);
     // Kudu table with multiple partition params
     String kuduMasters = catalog_.getDefaultKuduMasterHosts();
     testToSql(String.format("create table p primary key (a,b) " +
@@ -445,11 +438,9 @@ public class ToSqlTest extends FrontendTestBase {
     testToSql("create table p sort by LEXICAL (id) like functional.alltypes", "default",
         "CREATE TABLE p SORT BY LEXICAL (id) LIKE functional.alltypes");
     // Table with ZORDER sort columns.
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(true);
     testToSql("create table p sort by zorder (bool_col, int_col) like " +
         "functional.alltypes",  "default",
         "CREATE TABLE p SORT BY ZORDER (bool_col,int_col) LIKE functional.alltypes");
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(false);
   }
 
   @Test
@@ -473,13 +464,11 @@ public class ToSqlTest extends FrontendTestBase {
         "'hdfs://localhost:20500/test-warehouse/schemas/alltypestiny.parquet' " +
         "SORT BY LEXICAL ( int_col, id ) STORED AS TEXTFILE", true);
     // Table with ZORDER sort columns.
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(true);
     testToSql("create table if not exists p like parquet " +
         "'/test-warehouse/schemas/alltypestiny.parquet' sort by zorder (int_col, id)",
         "default", "CREATE TABLE IF NOT EXISTS default.p LIKE PARQUET " +
         "'hdfs://localhost:20500/test-warehouse/schemas/alltypestiny.parquet' " +
         "SORT BY ZORDER ( int_col, id ) STORED AS TEXTFILE", true);
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(false);
   }
 
   @Test
diff --git a/fe/src/test/java/org/apache/impala/analysis/ToSqlUtilsTest.java b/fe/src/test/java/org/apache/impala/analysis/ToSqlUtilsTest.java
index 936eed7..093df28 100644
--- a/fe/src/test/java/org/apache/impala/analysis/ToSqlUtilsTest.java
+++ b/fe/src/test/java/org/apache/impala/analysis/ToSqlUtilsTest.java
@@ -36,9 +36,7 @@ import org.apache.impala.catalog.KuduTable;
 import org.apache.impala.catalog.ScalarFunction;
 import org.apache.impala.catalog.Type;
 import org.apache.impala.common.FrontendTestBase;
-import org.apache.impala.service.BackendConfig;
 import org.apache.impala.thrift.TFunctionBinaryType;
-import org.apache.impala.thrift.TSortingOrder;
 import org.junit.Test;
 
 import com.google.common.collect.Lists;
@@ -137,8 +135,6 @@ public class ToSqlUtilsTest extends FrontendTestBase {
 
   @Test
   public void testSortOrder() {
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(true);
-
     Map<String,String> props = new HashMap<>();
     props.put("foo", "foo-value");
     // Sorting order is LEXICAL by default
@@ -151,8 +147,6 @@ public class ToSqlUtilsTest extends FrontendTestBase {
     props.put(AlterTableSortByStmt.TBL_PROP_SORT_ORDER, "LEXICAL");
     // Returns false if zorder property set to LEXICAL
     assertEquals("LEXICAL", ToSqlUtils.getSortingOrder(props));
-
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(false);
   }
 
   private FeTable getTable(String dbName, String tableName) {
diff --git a/fe/src/test/java/org/apache/impala/authorization/AuthorizationStmtTest.java b/fe/src/test/java/org/apache/impala/authorization/AuthorizationStmtTest.java
index db9c34e..724ce09 100644
--- a/fe/src/test/java/org/apache/impala/authorization/AuthorizationStmtTest.java
+++ b/fe/src/test/java/org/apache/impala/authorization/AuthorizationStmtTest.java
@@ -2060,7 +2060,6 @@ public class AuthorizationStmtTest extends AuthorizationTestBase {
 
   @Test
   public void testAlterTable() throws ImpalaException {
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(true);
     for (AuthzTest test: new AuthzTest[]{
         authorize("alter table functional.alltypes add column c1 int"),
         authorize("alter table functional.alltypes add columns(c1 int)"),
@@ -2104,7 +2103,6 @@ public class AuthorizationStmtTest extends AuthorizationTestBase {
               allExcept(TPrivilegeLevel.ALL, TPrivilegeLevel.OWNER,
               TPrivilegeLevel.ALTER)));
     }
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(false);
 
     try {
       // We cannot set an owner to a role that doesn't exist
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 a89df60..d2b89eb 100644
--- a/fe/src/test/java/org/apache/impala/planner/PlannerTest.java
+++ b/fe/src/test/java/org/apache/impala/planner/PlannerTest.java
@@ -30,7 +30,6 @@ import org.apache.impala.catalog.Type;
 import org.apache.impala.common.ImpalaException;
 import org.apache.impala.common.RuntimeEnv;
 import org.apache.impala.datagenerator.HBaseTestDataRegionAssignment;
-import org.apache.impala.service.BackendConfig;
 import org.apache.impala.service.Frontend.PlanCtx;
 import org.apache.impala.testutil.TestUtils;
 import org.apache.impala.testutil.TestUtils.IgnoreValueFilter;
@@ -302,8 +301,6 @@ public class PlannerTest extends PlannerTestBase {
   public void testInsertSortByZorder() {
     // Add a test table with a SORT BY ZORDER clause to test that the corresponding sort
     // nodes are added by the insert statements in insert-sort-by.test.
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(true);
-
     addTestDb("test_sort_by_zorder", "Test DB for SORT BY ZORDER clause.");
     addTestTable("create table test_sort_by_zorder.t (id int, int_col int, " +
         "bool_col boolean) partitioned by (year int, month int) " +
@@ -311,8 +308,6 @@ public class PlannerTest extends PlannerTestBase {
     addTestTable("create table test_sort_by_zorder.t_nopart (id int, int_col int, " +
         "bool_col boolean) sort by zorder (int_col, bool_col) location '/'");
     runPlannerTestFile("insert-sort-by-zorder", "test_sort_by_zorder");
-
-    BackendConfig.INSTANCE.setZOrderSortUnlocked(false);
   }
 
   @Test
diff --git a/testdata/workloads/functional-query/queries/QueryTest/alter-table-zorder.test b/testdata/workloads/functional-query/queries/QueryTest/alter-table-zorder.test
deleted file mode 100644
index cdd18bb..0000000
--- a/testdata/workloads/functional-query/queries/QueryTest/alter-table-zorder.test
+++ /dev/null
@@ -1,244 +0,0 @@
-====
----- QUERY
-create table insert_data_z (i int, d int, f int, b boolean);
-insert into insert_data_z values (1, 2, 3, false), (4, 5, 6, true);
-====
----- QUERY
-create table insert_zsorted (i int, d int, f int, b boolean);
-====
----- QUERY
-# Test setting the sort.columns and sort.zorder property
-alter table insert_zsorted sort by zorder(i, d);
-describe formatted insert_zsorted;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','i,d                 '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-# Test inserting after alter table
-insert into table insert_zsorted select i, d, f, b from insert_data_z;
----- RUNTIME_PROFILE
-row_regex: .*order by: ZORDER: i, d
-====
----- QUERY
-# Test selection after alter table
-select count(*) from insert_zsorted;
----- RESULTS
-2
-====
----- QUERY
-# Test altering the sort.columns andd sort.order property
-alter table insert_zsorted sort by zorder(b, d, f);
-describe formatted insert_zsorted;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','b,d,f               '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-# Test inserting after alter table
-insert into table insert_zsorted select i, d, f, b from insert_data_z;
----- RUNTIME_PROFILE
-row_regex: .*order by: ZORDER: b, d, f
-====
----- QUERY
-# Test selection after alter table
-select count(*) from insert_zsorted;
----- RESULTS
-4
-====
----- QUERY
-# Test renaming a column in the sort by zorder list.
-alter table insert_zsorted change d e int;
-describe formatted insert_zsorted;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','b,e,f               '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-# Test inserting after alter table
-insert into table insert_zsorted select i, d, f, b from insert_data_z;
----- RUNTIME_PROFILE
-row_regex: .*order by: ZORDER: b, d, f
-====
----- QUERY
-# Test selection after alter table
-select count(*) from insert_zsorted;
----- RESULTS
-6
-====
----- QUERY
-# Test replacing the column list, including a column in the sort by zorder list.
-alter table insert_zsorted replace columns (i bigint, e decimal(12,2), f boolean);
-describe formatted insert_zsorted;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','e,f                 '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-# Test inserting after alter table
-insert into table insert_zsorted select i, cast(d as decimal(12,2)), b from insert_data_z;
----- RUNTIME_PROFILE
-row_regex: .*order by: ZORDER: CAST\(d AS DECIMAL\(12,2\)\), b
-====
----- QUERY
-# Test selection after alter table
-select count(*) from insert_zsorted;
----- RESULTS
-8
-====
----- QUERY
-# Test dropping a column in the sort by list
-alter table insert_zsorted drop column f;
-describe formatted insert_zsorted;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','e                   '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-# Test that a dropped column cannot be added as a sort column
-alter table insert_zsorted sort by zorder(f);
----- CATCH
-AnalysisException: Could not find SORT BY column 'f' in table.
-====
----- QUERY
-# Test that erroneous query didn't change sort columns
-describe formatted insert_zsorted;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','e                   '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-# Test removing the sort.columns property
-alter table insert_zsorted sort by zorder ();
-describe formatted insert_zsorted;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','                    '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-# Test inserting after alter table
-insert into table insert_zsorted select i, cast(d as decimal(12,2)) from insert_data_z;
----- RUNTIME_PROFILE
-aggregation(SUM, InitialRunsCreated): 0
-====
----- QUERY
-# Test selection after alter table
-select count(*) from insert_zsorted;
----- RESULTS
-10
-====
----- QUERY
-create table insert_zsorted_partitioned (i int, d int, f int, b boolean) partitioned by (p int) sort by zorder (i, d);
-====
----- QUERY
-# Test removing all sort columns.
-alter table insert_zsorted_partitioned sort by zorder();
-describe formatted insert_zsorted_partitioned;
----- RESULTS: VERIFY_IS_NOT_IN
-'','sort.columns        ','i,d                 '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-# Test inserting after alter table
-insert into table insert_zsorted_partitioned partition (p=1) select i, d, f, b from insert_data_z;
----- RUNTIME_PROFILE
-aggregation(SUM, InitialRunsCreated): 0
-====
----- QUERY
-# Test selection after alter table
-select count(*) from insert_zsorted_partitioned;
----- RESULTS
-2
-====
----- QUERY
-# Re-add a sort column.
-alter table insert_zsorted_partitioned sort by zorder(d, i);
-describe formatted insert_zsorted_partitioned;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','d,i                 '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-insert into table insert_zsorted_partitioned partition (p=1) select i, d, f, b from insert_data_z;
----- RUNTIME_PROFILE
-row_regex: .*order by: ZORDER: d, i
-====
----- QUERY
-# Test selection after alter table
-select count(*) from insert_zsorted_partitioned;
----- RESULTS
-4
-====
----- QUERY
-describe formatted insert_zsorted_partitioned;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','d,i                 '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-# Test dropping one sort column, so that one remains.
-alter table insert_zsorted_partitioned drop column i;
-describe formatted insert_zsorted_partitioned;
----- RESULTS: VERIFY_IS_NOT_IN
-'','sort.columns        ','d,i                 '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-# Creating table and swapping sorting orders
-create table swapping_table (i int, j int, k int) sort by zorder (i, j);
-describe formatted swapping_table;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','i,j                 '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-# Creating table and swapping sorting orders
-alter table swapping_table sort by lexical (i);
-describe formatted swapping_table;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','i                   '
-'','sort.order          ','LEXICAL             '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-# Creating table and swapping sorting orders
-alter table swapping_table sort by zorder (i, j, k);
-describe formatted swapping_table;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','i,j,k               '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-# Creating table and swapping sorting orders
-alter table swapping_table sort by ();
-describe formatted swapping_table;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','                    '
-'','sort.order          ','LEXICAL             '
----- TYPES
-STRING,STRING,STRING
-====
diff --git a/testdata/workloads/functional-query/queries/QueryTest/alter-table.test b/testdata/workloads/functional-query/queries/QueryTest/alter-table.test
index 7f97eca..1e29099 100644
--- a/testdata/workloads/functional-query/queries/QueryTest/alter-table.test
+++ b/testdata/workloads/functional-query/queries/QueryTest/alter-table.test
@@ -1535,3 +1535,246 @@ select * from del_table_part order by c0;
 ---- TYPES
 STRING,STRING,STRING,INT
 ====
+---- QUERY
+create table insert_data_z (i int, d int, f int, b boolean);
+insert into insert_data_z values (1, 2, 3, false), (4, 5, 6, true);
+====
+---- QUERY
+create table insert_zsorted (i int, d int, f int, b boolean);
+====
+---- QUERY
+# Test setting the sort.columns and sort.zorder property
+alter table insert_zsorted sort by zorder(i, d);
+describe formatted insert_zsorted;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','i,d                 '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+# Test inserting after alter table
+insert into table insert_zsorted select i, d, f, b from insert_data_z;
+---- RUNTIME_PROFILE
+row_regex: .*order by: ZORDER: i, d
+====
+---- QUERY
+# Test selection after alter table
+select count(*) from insert_zsorted;
+---- RESULTS
+2
+====
+---- QUERY
+# Test altering the sort.columns andd sort.order property
+alter table insert_zsorted sort by zorder(b, d, f);
+describe formatted insert_zsorted;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','b,d,f               '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+# Test inserting after alter table
+insert into table insert_zsorted select i, d, f, b from insert_data_z;
+---- RUNTIME_PROFILE
+row_regex: .*order by: ZORDER: b, d, f
+====
+---- QUERY
+# Test selection after alter table
+select count(*) from insert_zsorted;
+---- RESULTS
+4
+====
+---- QUERY
+# Test renaming a column in the sort by zorder list.
+alter table insert_zsorted change d e int;
+describe formatted insert_zsorted;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','b,e,f               '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+# Test inserting after alter table
+insert into table insert_zsorted select i, d, f, b from insert_data_z;
+---- RUNTIME_PROFILE
+row_regex: .*order by: ZORDER: b, d, f
+====
+---- QUERY
+# Test selection after alter table
+select count(*) from insert_zsorted;
+---- RESULTS
+6
+====
+---- QUERY
+# Test replacing the column list, including a column in the sort by zorder list.
+alter table insert_zsorted replace columns (i bigint, e decimal(12,2), f boolean);
+describe formatted insert_zsorted;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','e,f                 '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+# Test inserting after alter table
+insert into table insert_zsorted select i, cast(d as decimal(12,2)), b from insert_data_z;
+---- RUNTIME_PROFILE
+row_regex: .*order by: ZORDER: CAST\(d AS DECIMAL\(12,2\)\), b
+====
+---- QUERY
+# Test selection after alter table
+select count(*) from insert_zsorted;
+---- RESULTS
+8
+====
+---- QUERY
+# Test dropping a column in the sort by list
+alter table insert_zsorted drop column f;
+describe formatted insert_zsorted;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','e                   '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+# Test that a dropped column cannot be added as a sort column
+alter table insert_zsorted sort by zorder(f);
+---- CATCH
+AnalysisException: Could not find SORT BY column 'f' in table.
+====
+---- QUERY
+# Test that erroneous query didn't change sort columns
+describe formatted insert_zsorted;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','e                   '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+# Test removing the sort.columns property
+alter table insert_zsorted sort by zorder ();
+describe formatted insert_zsorted;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','                    '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+# Test inserting after alter table
+insert into table insert_zsorted select i, cast(d as decimal(12,2)) from insert_data_z;
+---- RUNTIME_PROFILE
+aggregation(SUM, InitialRunsCreated): 0
+====
+---- QUERY
+# Test selection after alter table
+select count(*) from insert_zsorted;
+---- RESULTS
+10
+====
+---- QUERY
+create table insert_zsorted_partitioned (i int, d int, f int, b boolean) partitioned by (p int) sort by zorder (i, d);
+====
+---- QUERY
+# Test removing all sort columns.
+alter table insert_zsorted_partitioned sort by zorder();
+describe formatted insert_zsorted_partitioned;
+---- RESULTS: VERIFY_IS_NOT_IN
+'','sort.columns        ','i,d                 '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+# Test inserting after alter table
+insert into table insert_zsorted_partitioned partition (p=1) select i, d, f, b from insert_data_z;
+---- RUNTIME_PROFILE
+aggregation(SUM, InitialRunsCreated): 0
+====
+---- QUERY
+# Test selection after alter table
+select count(*) from insert_zsorted_partitioned;
+---- RESULTS
+2
+====
+---- QUERY
+# Re-add a sort column.
+alter table insert_zsorted_partitioned sort by zorder(d, i);
+describe formatted insert_zsorted_partitioned;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','d,i                 '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+insert into table insert_zsorted_partitioned partition (p=1) select i, d, f, b from insert_data_z;
+---- RUNTIME_PROFILE
+row_regex: .*order by: ZORDER: d, i
+====
+---- QUERY
+# Test selection after alter table
+select count(*) from insert_zsorted_partitioned;
+---- RESULTS
+4
+====
+---- QUERY
+describe formatted insert_zsorted_partitioned;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','d,i                 '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+# Test dropping one sort column, so that one remains.
+alter table insert_zsorted_partitioned drop column i;
+describe formatted insert_zsorted_partitioned;
+---- RESULTS: VERIFY_IS_NOT_IN
+'','sort.columns        ','d,i                 '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+# Creating table and swapping sorting orders
+create table swapping_table (i int, j int, k int) sort by zorder (i, j);
+describe formatted swapping_table;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','i,j                 '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+# Creating table and swapping sorting orders
+alter table swapping_table sort by lexical (i);
+describe formatted swapping_table;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','i                   '
+'','sort.order          ','LEXICAL             '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+# Creating table and swapping sorting orders
+alter table swapping_table sort by zorder (i, j, k);
+describe formatted swapping_table;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','i,j,k               '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+# Creating table and swapping sorting orders
+alter table swapping_table sort by ();
+describe formatted swapping_table;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','                    '
+'','sort.order          ','LEXICAL             '
+---- TYPES
+STRING,STRING,STRING
+====
diff --git a/testdata/workloads/functional-query/queries/QueryTest/create-table-as-select-zorder.test b/testdata/workloads/functional-query/queries/QueryTest/create-table-as-select-zorder.test
deleted file mode 100644
index b3a429a..0000000
--- a/testdata/workloads/functional-query/queries/QueryTest/create-table-as-select-zorder.test
+++ /dev/null
@@ -1,12 +0,0 @@
-====
----- QUERY
-# Test adding sort.columns and sort.zorder when creating a table from a select query.
-create table zsortbytest sort by zorder (int_col, bool_col) as
-select * from functional.alltypessmall;
-describe formatted zsortbytest;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','int_col,bool_col    '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
\ No newline at end of file
diff --git a/testdata/workloads/functional-query/queries/QueryTest/create-table-as-select.test b/testdata/workloads/functional-query/queries/QueryTest/create-table-as-select.test
index f157681..f9123e6 100644
--- a/testdata/workloads/functional-query/queries/QueryTest/create-table-as-select.test
+++ b/testdata/workloads/functional-query/queries/QueryTest/create-table-as-select.test
@@ -199,3 +199,14 @@ describe formatted sortbytest;
 ---- TYPES
 STRING,STRING,STRING
 ====
+---- QUERY
+# Test adding sort.columns and sort.zorder when creating a table from a select query.
+create table zsortbytest sort by zorder (int_col, bool_col) as
+select * from functional.alltypessmall;
+describe formatted zsortbytest;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','int_col,bool_col    '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
\ No newline at end of file
diff --git a/testdata/workloads/functional-query/queries/QueryTest/create-table-like-file-zorder.test b/testdata/workloads/functional-query/queries/QueryTest/create-table-like-file-zorder.test
deleted file mode 100644
index d9d04c1..0000000
--- a/testdata/workloads/functional-query/queries/QueryTest/create-table-like-file-zorder.test
+++ /dev/null
@@ -1,13 +0,0 @@
-====
----- QUERY
-# Test adding sort.columns and sort.zorder when creating a table like a parquet file.
-create table $DATABASE.decimal_file like parquet
-'$FILESYSTEM_PREFIX/test-warehouse/schemas/decimal.parquet'
-sort by zorder (d32, d11) stored as textfile;
-describe formatted $DATABASE.decimal_file;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','d32,d11             '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
\ No newline at end of file
diff --git a/testdata/workloads/functional-query/queries/QueryTest/create-table-like-file.test b/testdata/workloads/functional-query/queries/QueryTest/create-table-like-file.test
index fd81aee..8237533 100644
--- a/testdata/workloads/functional-query/queries/QueryTest/create-table-like-file.test
+++ b/testdata/workloads/functional-query/queries/QueryTest/create-table-like-file.test
@@ -180,3 +180,15 @@ describe formatted $DATABASE.sorted_zipcodes_file;
 ---- TYPES
 STRING,STRING,STRING
 ====
+---- QUERY
+# Test adding sort.columns and sort.zorder when creating a table like a parquet file.
+create table $DATABASE.decimal_file like parquet
+'$FILESYSTEM_PREFIX/test-warehouse/schemas/decimal.parquet'
+sort by zorder (d32, d11) stored as textfile;
+describe formatted $DATABASE.decimal_file;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','d32,d11             '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
\ No newline at end of file
diff --git a/testdata/workloads/functional-query/queries/QueryTest/create-table-like-table-zorder.test b/testdata/workloads/functional-query/queries/QueryTest/create-table-like-table-zorder.test
deleted file mode 100644
index 4c2e014..0000000
--- a/testdata/workloads/functional-query/queries/QueryTest/create-table-like-table-zorder.test
+++ /dev/null
@@ -1,31 +0,0 @@
-====
----- QUERY
-# Test setting zsort.columns and sort.zorder when using create table like.
-create table zsortbytest sort by zorder (int_col, bool_col) like functional.alltypes;
-describe formatted zsortbytest;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','int_col,bool_col    '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-# Test that sort.columns and sort.ordering will be inherited from the source table.
-create table zsortbytest_clone like zsortbytest;
-describe formatted zsortbytest_clone;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','int_col,bool_col    '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
----- QUERY
-# Test that sort.columns and sort.zorder can be overridden in the query.
-create table zsortbytest_override sort by zorder (id, int_col) like zsortbytest;
-describe formatted zsortbytest_override;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','id,int_col          '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
diff --git a/testdata/workloads/functional-query/queries/QueryTest/create-table-like-table.test b/testdata/workloads/functional-query/queries/QueryTest/create-table-like-table.test
index 456f499..cd2f629 100644
--- a/testdata/workloads/functional-query/queries/QueryTest/create-table-like-table.test
+++ b/testdata/workloads/functional-query/queries/QueryTest/create-table-like-table.test
@@ -277,4 +277,34 @@ describe allcomplextypes_clone
 'month','int',''
 ---- TYPES
 STRING, STRING, STRING
-====
\ No newline at end of file
+====
+---- QUERY
+# Test setting zsort.columns and sort.zorder when using create table like.
+create table zsortbytest sort by zorder (int_col, bool_col) like functional.alltypes;
+describe formatted zsortbytest;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','int_col,bool_col    '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+# Test that sort.columns and sort.ordering will be inherited from the source table.
+create table zsortbytest_clone like zsortbytest;
+describe formatted zsortbytest_clone;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','int_col,bool_col    '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
+---- QUERY
+# Test that sort.columns and sort.zorder can be overridden in the query.
+create table zsortbytest_override sort by zorder (id, int_col) like zsortbytest;
+describe formatted zsortbytest_override;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','id,int_col          '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
diff --git a/testdata/workloads/functional-query/queries/QueryTest/create-table-zorder.test b/testdata/workloads/functional-query/queries/QueryTest/create-table-zorder.test
deleted file mode 100644
index b1218e5..0000000
--- a/testdata/workloads/functional-query/queries/QueryTest/create-table-zorder.test
+++ /dev/null
@@ -1,12 +0,0 @@
-====
----- QUERY
-# Make sure that specifying sort by zorder columns sets the 'sort.columns'
-# and 'sort.order' properties correctly.
-create table zsortbytest (i int, d int, b boolean) sort by zorder (d, i);
-describe formatted zsortbytest;
----- RESULTS: VERIFY_IS_SUBSET
-'','sort.columns        ','d,i                 '
-'','sort.order          ','ZORDER              '
----- TYPES
-STRING,STRING,STRING
-====
\ No newline at end of file
diff --git a/testdata/workloads/functional-query/queries/QueryTest/create-table.test b/testdata/workloads/functional-query/queries/QueryTest/create-table.test
index 7f91adc..0f3841b 100644
--- a/testdata/workloads/functional-query/queries/QueryTest/create-table.test
+++ b/testdata/workloads/functional-query/queries/QueryTest/create-table.test
@@ -298,3 +298,14 @@ describe formatted sortbytest;
 ---- TYPES
 STRING,STRING,STRING
 ====
+---- QUERY
+# Make sure that specifying sort by zorder columns sets the 'sort.columns'
+# and 'sort.order' properties correctly.
+create table zsortbytest (i int, d int, b boolean) sort by zorder (d, i);
+describe formatted zsortbytest;
+---- RESULTS: VERIFY_IS_SUBSET
+'','sort.columns        ','d,i                 '
+'','sort.order          ','ZORDER              '
+---- TYPES
+STRING,STRING,STRING
+====
\ No newline at end of file
diff --git a/tests/custom_cluster/test_zorder.py b/tests/custom_cluster/test_zorder.py
deleted file mode 100644
index 64fc710..0000000
--- a/tests/custom_cluster/test_zorder.py
+++ /dev/null
@@ -1,85 +0,0 @@
-# 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.
-
-import shlex
-import pprint
-import re
-
-from tests.common.custom_cluster_test_suite import CustomClusterTestSuite
-from tests.common.parametrize import UniqueDatabase
-from tests.common.skip import SkipIfLocal
-from tests.common.test_dimensions import create_uncompressed_text_dimension
-from tests.util.test_file_parser import QueryTestSectionReader, remove_comments
-
-# Temporary classes for testing Z-ordering frontend features.
-# These classes are based on tests/metadata/test_ddl.py
-# TODO: merge the testfiles and delete this test when Z-order tests do not
-# require custom cluster anymore.
-class TestZOrder(CustomClusterTestSuite):
-  VALID_SECTION_NAMES = ["CREATE_TABLE", "CREATE_VIEW", "QUERY", "RESULTS"]
-
-  @classmethod
-  def get_workload(cls):
-    return 'functional-query'
-
-  @classmethod
-  def add_test_dimensions(cls):
-    super(TestZOrder, cls).add_test_dimensions()
-
-  @CustomClusterTestSuite.with_args(impalad_args="--unlock_zorder_sort=true")
-  @SkipIfLocal.hdfs_client
-  @UniqueDatabase.parametrize(sync_ddl=True, num_dbs=2)
-  def test_alter_table(self, vector, unique_database):
-    vector.get_value('exec_option')['abort_on_error'] = False
-
-    # Create an unpartitioned table to get a filesystem directory that does not
-    # use the (key=value) format. The directory is automatically cleanup up
-    # by the unique_database fixture.
-    self.client.execute("create table {0}.part_data (i int)".format(unique_database))
-    assert self.filesystem_client.exists(
-        "test-warehouse/{0}.db/part_data".format(unique_database))
-    self.filesystem_client.create_file(
-        "test-warehouse/{0}.db/part_data/data.txt".format(unique_database),
-        file_data='1984')
-    self.run_test_case('QueryTest/alter-table-zorder', vector, use_db=unique_database)
-
-  @CustomClusterTestSuite.with_args(impalad_args="--unlock_zorder_sort=true")
-  @UniqueDatabase.parametrize(sync_ddl=True)
-  def test_create_table(self, vector, unique_database):
-    vector.get_value('exec_option')['abort_on_error'] = False
-    self.run_test_case('QueryTest/create-table-zorder', vector, use_db=unique_database)
-
-  @CustomClusterTestSuite.with_args(impalad_args="--unlock_zorder_sort=true")
-  @UniqueDatabase.parametrize(sync_ddl=True)
-  def test_create_table_like_table(self, vector, unique_database):
-    vector.get_value('exec_option')['abort_on_error'] = False
-    self.run_test_case('QueryTest/create-table-like-table-zorder', vector,
-        use_db=unique_database)
-
-  @CustomClusterTestSuite.with_args(impalad_args="--unlock_zorder_sort=true")
-  @UniqueDatabase.parametrize(sync_ddl=True)
-  def test_create_table_like_file(self, vector, unique_database):
-    vector.get_value('exec_option')['abort_on_error'] = False
-    self.run_test_case('QueryTest/create-table-like-file-zorder', vector,
-        use_db=unique_database)
-
-  @CustomClusterTestSuite.with_args(impalad_args="--unlock_zorder_sort=true")
-  @UniqueDatabase.parametrize(sync_ddl=True)
-  def test_create_table_as_select(self, vector, unique_database):
-    vector.get_value('exec_option')['abort_on_error'] = False
-    self.run_test_case('QueryTest/create-table-as-select-zorder', vector,
-        use_db=unique_database)