You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@quickstep.apache.org by zu...@apache.org on 2016/08/16 20:47:02 UTC

[1/4] incubator-quickstep git commit: Fix IWYU for ExecutionDAGVisualizer [Forced Update!]

Repository: incubator-quickstep
Updated Branches:
  refs/heads/q-opt-dist-exe-gen-test 5ed4a1c4b -> cdc1e053b (forced update)


Fix IWYU for ExecutionDAGVisualizer


Project: http://git-wip-us.apache.org/repos/asf/incubator-quickstep/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-quickstep/commit/8ec99ed8
Tree: http://git-wip-us.apache.org/repos/asf/incubator-quickstep/tree/8ec99ed8
Diff: http://git-wip-us.apache.org/repos/asf/incubator-quickstep/diff/8ec99ed8

Branch: refs/heads/q-opt-dist-exe-gen-test
Commit: 8ec99ed8b95a909d68de9c3f8b87d2a3b74372a6
Parents: 203d3ea
Author: Jianqiao Zhu <ji...@cs.wisc.edu>
Authored: Tue Aug 16 12:37:30 2016 -0500
Committer: Zuyu Zhang <zu...@twitter.com>
Committed: Tue Aug 16 11:17:37 2016 -0700

----------------------------------------------------------------------
 query_execution/ForemanSingleNode.cpp      | 2 --
 query_execution/ForemanSingleNode.hpp      | 2 +-
 query_execution/PolicyEnforcerBase.hpp     | 3 +--
 query_execution/QueryExecutionTypedefs.hpp | 1 +
 utility/CMakeLists.txt                     | 2 ++
 utility/ExecutionDAGVisualizer.cpp         | 3 ++-
 6 files changed, 7 insertions(+), 6 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8ec99ed8/query_execution/ForemanSingleNode.cpp
----------------------------------------------------------------------
diff --git a/query_execution/ForemanSingleNode.cpp b/query_execution/ForemanSingleNode.cpp
index 7596b00..4661c37 100644
--- a/query_execution/ForemanSingleNode.cpp
+++ b/query_execution/ForemanSingleNode.cpp
@@ -22,7 +22,6 @@
 #include <cstddef>
 #include <cstdio>
 #include <memory>
-#include <tuple>
 #include <utility>
 #include <vector>
 
@@ -245,7 +244,6 @@ void ForemanSingleNode::printWorkOrderProfilingResults(const std::size_t query_i
       policy_enforcer_->getProfilingResults(query_id);
   fputs("Query ID,Worker ID,NUMA Socket,Operator ID,Time (microseconds)\n", out);
   for (auto workorder_entry : recorded_times) {
-    // Note: Index of the "worker thread index" in the tuple is 0.
     const std::size_t worker_id = workorder_entry.worker_id;
     fprintf(out,
             "%lu,%lu,%d,%lu,%lu\n",

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8ec99ed8/query_execution/ForemanSingleNode.hpp
----------------------------------------------------------------------
diff --git a/query_execution/ForemanSingleNode.hpp b/query_execution/ForemanSingleNode.hpp
index 71ce99d..5a368aa 100644
--- a/query_execution/ForemanSingleNode.hpp
+++ b/query_execution/ForemanSingleNode.hpp
@@ -84,7 +84,7 @@ class ForemanSingleNode final : public ForemanBase {
    *        query.
    *
    * @param query_id The ID of the query for which the results are to be printed.
-   * @return A vector of tuples, each being a single profiling entry.
+   * @return A vector of records, each being a single profiling entry.
    **/
   const std::vector<WorkOrderTimeEntry>& getWorkOrderProfilingResults(
       const std::size_t query_id) const;

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8ec99ed8/query_execution/PolicyEnforcerBase.hpp
----------------------------------------------------------------------
diff --git a/query_execution/PolicyEnforcerBase.hpp b/query_execution/PolicyEnforcerBase.hpp
index e95799e..62906e9 100644
--- a/query_execution/PolicyEnforcerBase.hpp
+++ b/query_execution/PolicyEnforcerBase.hpp
@@ -23,7 +23,6 @@
 #include <cstddef>
 #include <memory>
 #include <queue>
-#include <tuple>
 #include <unordered_map>
 #include <vector>
 
@@ -126,7 +125,7 @@ class PolicyEnforcerBase {
    * @param query_id The ID of the query for which the profiling results are
    *        requested.
    *
-   * @return A vector of tuples, each being a single profiling entry.
+   * @return A vector of records, each being a single profiling entry.
    **/
   inline const std::vector<WorkOrderTimeEntry>& getProfilingResults(
       const std::size_t query_id) const {

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8ec99ed8/query_execution/QueryExecutionTypedefs.hpp
----------------------------------------------------------------------
diff --git a/query_execution/QueryExecutionTypedefs.hpp b/query_execution/QueryExecutionTypedefs.hpp
index bba67e3..22c0ae1 100644
--- a/query_execution/QueryExecutionTypedefs.hpp
+++ b/query_execution/QueryExecutionTypedefs.hpp
@@ -20,6 +20,7 @@
 #ifndef QUICKSTEP_QUERY_EXECUTION_QUERY_EXECUTION_TYPEDEFS_HPP_
 #define QUICKSTEP_QUERY_EXECUTION_QUERY_EXECUTION_TYPEDEFS_HPP_
 
+#include <cstddef>
 #include <unordered_map>
 #include <vector>
 

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8ec99ed8/utility/CMakeLists.txt
----------------------------------------------------------------------
diff --git a/utility/CMakeLists.txt b/utility/CMakeLists.txt
index ae1179d..ddaae45 100644
--- a/utility/CMakeLists.txt
+++ b/utility/CMakeLists.txt
@@ -238,6 +238,8 @@ target_link_libraries(quickstep_utility_ExecutionDAGVisualizer
                       quickstep_relationaloperators_BuildHashOperator
                       quickstep_relationaloperators_HashJoinOperator
                       quickstep_relationaloperators_SelectOperator
+                      quickstep_relationaloperators_RelationalOperator
+                      quickstep_utility_DAG
                       quickstep_utility_Macros
                       quickstep_utility_StringUtil)
 target_link_libraries(quickstep_utility_Glob

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8ec99ed8/utility/ExecutionDAGVisualizer.cpp
----------------------------------------------------------------------
diff --git a/utility/ExecutionDAGVisualizer.cpp b/utility/ExecutionDAGVisualizer.cpp
index 37d00cd..2938808 100644
--- a/utility/ExecutionDAGVisualizer.cpp
+++ b/utility/ExecutionDAGVisualizer.cpp
@@ -20,7 +20,6 @@
 #include "utility/ExecutionDAGVisualizer.hpp"
 
 #include <algorithm>
-#include <cmath>
 #include <cstddef>
 #include <iomanip>
 #include <limits>
@@ -36,7 +35,9 @@
 #include "relational_operators/AggregationOperator.hpp"
 #include "relational_operators/BuildHashOperator.hpp"
 #include "relational_operators/HashJoinOperator.hpp"
+#include "relational_operators/RelationalOperator.hpp"
 #include "relational_operators/SelectOperator.hpp"
+#include "utility/DAG.hpp"
 #include "utility/StringUtil.hpp"
 
 #include "glog/logging.h"


[3/4] incubator-quickstep git commit: Minor fixes to the distributed query execution engine.

Posted by zu...@apache.org.
Minor fixes to the distributed query execution engine.


Project: http://git-wip-us.apache.org/repos/asf/incubator-quickstep/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-quickstep/commit/59f4dab2
Tree: http://git-wip-us.apache.org/repos/asf/incubator-quickstep/tree/59f4dab2
Diff: http://git-wip-us.apache.org/repos/asf/incubator-quickstep/diff/59f4dab2

Branch: refs/heads/q-opt-dist-exe-gen-test
Commit: 59f4dab26aa31a7e2aa9dfdc60524e56060f935b
Parents: 8c811c8
Author: Zuyu Zhang <zu...@twitter.com>
Authored: Tue Aug 16 13:17:01 2016 -0700
Committer: Zuyu Zhang <zu...@twitter.com>
Committed: Tue Aug 16 13:30:41 2016 -0700

----------------------------------------------------------------------
 query_execution/ForemanDistributed.cpp      | 2 +-
 query_execution/ForemanDistributed.hpp      | 2 +-
 query_execution/QueryManagerDistributed.cpp | 2 +-
 query_execution/QueryManagerDistributed.hpp | 4 ++--
 4 files changed, 5 insertions(+), 5 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/59f4dab2/query_execution/ForemanDistributed.cpp
----------------------------------------------------------------------
diff --git a/query_execution/ForemanDistributed.cpp b/query_execution/ForemanDistributed.cpp
index 29f5b9b..9c20465 100644
--- a/query_execution/ForemanDistributed.cpp
+++ b/query_execution/ForemanDistributed.cpp
@@ -279,7 +279,7 @@ void ForemanDistributed::printWorkOrderProfilingResults(const std::size_t query_
 }
 
 void ForemanDistributed::processShiftbossRegistrationMessage(const client_id shiftboss_client_id,
-                                                              const std::size_t work_order_capacity) {
+                                                             const std::size_t work_order_capacity) {
   S::ShiftbossRegistrationResponseMessage proto;
   proto.set_shiftboss_index(shiftboss_directory_.size());
 

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/59f4dab2/query_execution/ForemanDistributed.hpp
----------------------------------------------------------------------
diff --git a/query_execution/ForemanDistributed.hpp b/query_execution/ForemanDistributed.hpp
index f9a326a..fc1ede5 100644
--- a/query_execution/ForemanDistributed.hpp
+++ b/query_execution/ForemanDistributed.hpp
@@ -102,7 +102,7 @@ class ForemanDistributed final : public ForemanBase {
                             const serialization::WorkOrderMessage &proto);
 
   void processShiftbossRegistrationMessage(const tmb::client_id shiftboss_client_id,
-                                            const std::size_t work_order_capacity);
+                                           const std::size_t work_order_capacity);
 
   void processSaveQueryResultResponseMessage(const tmb::client_id cli_id,
                                              const relation_id result_relation_id);

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/59f4dab2/query_execution/QueryManagerDistributed.cpp
----------------------------------------------------------------------
diff --git a/query_execution/QueryManagerDistributed.cpp b/query_execution/QueryManagerDistributed.cpp
index e300ce5..7d45933 100644
--- a/query_execution/QueryManagerDistributed.cpp
+++ b/query_execution/QueryManagerDistributed.cpp
@@ -47,7 +47,7 @@ using std::unique_ptr;
 namespace quickstep {
 
 QueryManagerDistributed::QueryManagerDistributed(QueryHandle *query_handle,
-                                                 ShiftbossDirectory *shiftboss_directory,
+                                                 const ShiftbossDirectory *shiftboss_directory,
                                                  const tmb::client_id foreman_client_id,
                                                  tmb::MessageBus *bus)
     : QueryManagerBase(query_handle),

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/59f4dab2/query_execution/QueryManagerDistributed.hpp
----------------------------------------------------------------------
diff --git a/query_execution/QueryManagerDistributed.hpp b/query_execution/QueryManagerDistributed.hpp
index b448528..e609ab8 100644
--- a/query_execution/QueryManagerDistributed.hpp
+++ b/query_execution/QueryManagerDistributed.hpp
@@ -58,7 +58,7 @@ class QueryManagerDistributed final : public QueryManagerBase {
    * @param bus The TMB used for communication.
    **/
   QueryManagerDistributed(QueryHandle *query_handle,
-                          ShiftbossDirectory *shiftboss_directory,
+                          const ShiftbossDirectory *shiftboss_directory,
                           const tmb::client_id foreman_client_id,
                           tmb::MessageBus *bus);
 
@@ -105,7 +105,7 @@ class QueryManagerDistributed final : public QueryManagerBase {
            (query_exec_state_->getNumRebuildWorkOrders(index) == 0);
   }
 
-  ShiftbossDirectory *shiftboss_directory_;
+  const ShiftbossDirectory *shiftboss_directory_;
 
   const tmb::client_id foreman_client_id_;
   tmb::MessageBus *bus_;


[4/4] incubator-quickstep git commit: Added unit tests for the distributed version.

Posted by zu...@apache.org.
Added unit tests for the distributed version.


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

Branch: refs/heads/q-opt-dist-exe-gen-test
Commit: cdc1e053b34aff46104397405642cd9c64b7d5f1
Parents: 59f4dab
Author: Zuyu Zhang <zu...@twitter.com>
Authored: Sun Aug 14 00:02:20 2016 -0700
Committer: Zuyu Zhang <zu...@twitter.com>
Committed: Tue Aug 16 13:33:54 2016 -0700

----------------------------------------------------------------------
 query_optimizer/tests/CMakeLists.txt            |  35 ++++
 .../tests/DistributedExecutionGeneratorTest.cpp |  62 +++++++
 .../DistributedExecutionGeneratorTestRunner.cpp | 162 +++++++++++++++++++
 .../DistributedExecutionGeneratorTestRunner.hpp | 118 ++++++++++++++
 .../tests/execution_generator/CMakeLists.txt    |  70 ++++++++
 5 files changed, 447 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/cdc1e053/query_optimizer/tests/CMakeLists.txt
----------------------------------------------------------------------
diff --git a/query_optimizer/tests/CMakeLists.txt b/query_optimizer/tests/CMakeLists.txt
index 4969ada..597dbe0 100644
--- a/query_optimizer/tests/CMakeLists.txt
+++ b/query_optimizer/tests/CMakeLists.txt
@@ -80,6 +80,14 @@ target_link_libraries(quickstep_queryoptimizer_tests_TestDatabaseLoader
                       quickstep_utility_Macros
                       tmb)
 
+if (ENABLE_DISTRIBUTED)
+  add_executable(quickstep_queryoptimizer_tests_DistributedExecutionGeneratorTest
+                 DistributedExecutionGeneratorTest.cpp
+                 DistributedExecutionGeneratorTestRunner.cpp
+                 DistributedExecutionGeneratorTestRunner.hpp
+                 "${PROJECT_SOURCE_DIR}/utility/textbased_test/TextBasedTest.cpp"
+                 "${PROJECT_SOURCE_DIR}/utility/textbased_test/TextBasedTest.hpp")
+endif(ENABLE_DISTRIBUTED)
 add_executable(quickstep_queryoptimizer_tests_ExecutionGeneratorTest
                ExecutionGeneratorTest.cpp
                ExecutionGeneratorTestRunner.cpp
@@ -109,6 +117,33 @@ add_executable(quickstep_queryoptimizer_tests_OptimizerTextTest
                "${PROJECT_SOURCE_DIR}/utility/textbased_test/TextBasedTest.cpp"
                "${PROJECT_SOURCE_DIR}/utility/textbased_test/TextBasedTest.hpp")
 
+if (ENABLE_DISTRIBUTED)
+  target_link_libraries(quickstep_queryoptimizer_tests_DistributedExecutionGeneratorTest
+                        glog
+                        gtest
+                        quickstep_catalog_CatalogTypedefs
+                        quickstep_cli_DropRelation
+                        quickstep_cli_PrintToScreen
+                        quickstep_parser_ParseStatement
+                        quickstep_parser_SqlParserWrapper
+                        quickstep_queryexecution_ForemanDistributed
+                        quickstep_queryexecution_QueryExecutionTypedefs
+                        quickstep_queryexecution_QueryExecutionUtil
+                        quickstep_queryexecution_Shiftboss
+                        quickstep_queryexecution_Worker
+                        quickstep_queryexecution_WorkerDirectory
+                        quickstep_queryoptimizer_Optimizer
+                        quickstep_queryoptimizer_OptimizerContext
+                        quickstep_queryoptimizer_QueryHandle
+                        quickstep_queryoptimizer_tests_TestDatabaseLoader
+                        quickstep_utility_Macros
+                        quickstep_utility_MemStream
+                        quickstep_utility_SqlError
+                        quickstep_utility_TextBasedTestDriver
+                        tmb
+                        ${GFLAGS_LIB_NAME}
+                        ${LIBS})
+endif(ENABLE_DISTRIBUTED)
 target_link_libraries(quickstep_queryoptimizer_tests_ExecutionGeneratorTest
                       glog
                       gtest

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/cdc1e053/query_optimizer/tests/DistributedExecutionGeneratorTest.cpp
----------------------------------------------------------------------
diff --git a/query_optimizer/tests/DistributedExecutionGeneratorTest.cpp b/query_optimizer/tests/DistributedExecutionGeneratorTest.cpp
new file mode 100644
index 0000000..af310bc
--- /dev/null
+++ b/query_optimizer/tests/DistributedExecutionGeneratorTest.cpp
@@ -0,0 +1,62 @@
+/**
+ * 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.
+ **/
+
+#include <fstream>
+#include <memory>
+
+#include "query_optimizer/tests/DistributedExecutionGeneratorTestRunner.hpp"
+#include "utility/textbased_test/TextBasedTestDriver.hpp"
+#include "utility/textbased_test/TextBasedTest.hpp"
+
+#include "gflags/gflags.h"
+#include "glog/logging.h"
+#include "gtest/gtest.h"
+
+using quickstep::TextBasedTest;
+
+QUICKSTEP_GENERATE_TEXT_TEST(DISTRIBUTED_EXECUTION_GENERATOR_TEST);
+
+int main(int argc, char** argv) {
+  google::InitGoogleLogging(argv[0]);
+  // Honor FLAGS_buffer_pool_slots in StorageManager.
+  gflags::ParseCommandLineFlags(&argc, &argv, true);
+
+  if (argc < 4) {
+    LOG(ERROR) << "Must have at least 3 arguments, but " << argc - 1
+               << " are provided";
+  }
+
+  std::ifstream input_file(argv[1]);
+  CHECK(input_file.is_open()) << argv[1];
+  std::unique_ptr<quickstep::optimizer::DistributedExecutionGeneratorTestRunner>
+      test_runner(
+          new quickstep::optimizer::DistributedExecutionGeneratorTestRunner(argv[3]));
+  test_driver.reset(
+      new quickstep::TextBasedTestDriver(&input_file, test_runner.get()));
+  test_driver->registerOption(
+      quickstep::optimizer::DistributedExecutionGeneratorTestRunner::kResetOption);
+
+  ::testing::InitGoogleTest(&argc, argv);
+  const int success = RUN_ALL_TESTS();
+  if (success != 0) {
+    test_driver->writeActualOutputToFile(argv[2]);
+  }
+
+  return success;
+}

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/cdc1e053/query_optimizer/tests/DistributedExecutionGeneratorTestRunner.cpp
----------------------------------------------------------------------
diff --git a/query_optimizer/tests/DistributedExecutionGeneratorTestRunner.cpp b/query_optimizer/tests/DistributedExecutionGeneratorTestRunner.cpp
new file mode 100644
index 0000000..5cccc21
--- /dev/null
+++ b/query_optimizer/tests/DistributedExecutionGeneratorTestRunner.cpp
@@ -0,0 +1,162 @@
+/**
+ * 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.
+ **/
+
+#include "query_optimizer/tests/DistributedExecutionGeneratorTestRunner.hpp"
+
+#include <cstdio>
+#include <set>
+#include <string>
+#include <vector>
+
+#include "catalog/CatalogTypedefs.hpp"
+#include "cli/DropRelation.hpp"
+#include "cli/PrintToScreen.hpp"
+#include "parser/ParseStatement.hpp"
+#include "query_execution/ForemanDistributed.hpp"
+#include "query_execution/QueryExecutionTypedefs.hpp"
+#include "query_optimizer/OptimizerContext.hpp"
+#include "query_optimizer/QueryHandle.hpp"
+#include "utility/MemStream.hpp"
+#include "utility/SqlError.hpp"
+
+#include "glog/logging.h"
+
+#include "tmb/id_typedefs.h"
+#include "tmb/message_bus.h"
+#include "tmb/tagged_message.h"
+
+using std::string;
+using std::make_unique;
+using std::vector;
+
+namespace quickstep {
+
+class CatalogRelation;
+
+namespace optimizer {
+
+const char *DistributedExecutionGeneratorTestRunner::kResetOption =
+    "reset_before_execution";
+
+DistributedExecutionGeneratorTestRunner::DistributedExecutionGeneratorTestRunner(const string &storage_path)
+    : query_id_(0),
+      test_database_loader_(storage_path) {
+  test_database_loader_.createTestRelation(false /* allow_vchar */);
+  test_database_loader_.loadTestRelation();
+
+  bus_.Initialize();
+
+  cli_id_ = bus_.Connect();
+  bus_.RegisterClientAsSender(cli_id_, kAdmitRequestMessage);
+  bus_.RegisterClientAsSender(cli_id_, kPoisonMessage);
+  bus_.RegisterClientAsReceiver(cli_id_, kQueryExecutionSuccessMessage);
+
+  // NOTE(zuyu): Foreman should initialize before Shiftboss so that the former
+  // could receive a registration message from the latter.
+  foreman_ = make_unique<ForemanDistributed>(&bus_, test_database_loader_.catalog_database());
+
+  // We don't use the NUMA aware version of worker code.
+  const vector<numa_node_id> numa_nodes(1 /* Number of worker threads per instance */,
+                                        kAnyNUMANodeID);
+
+  for (int i = 0; i < kNumInstances; ++i) {
+    workers_.push_back(make_unique<Worker>(0 /* worker_thread_index */, &bus_));
+
+    const vector<tmb::client_id> worker_client_ids(1, workers_[i]->getBusClientID());
+    worker_directories_.push_back(
+        make_unique<WorkerDirectory>(worker_client_ids.size(), worker_client_ids, numa_nodes));
+
+    shiftbosses_.push_back(
+        make_unique<Shiftboss>(&bus_, test_database_loader_.storage_manager(), worker_directories_[i].get()));
+  }
+
+  foreman_->start();
+
+  for (int i = 0; i < kNumInstances; ++i) {
+    shiftbosses_[i]->start();
+    workers_[i]->start();
+  }
+}
+
+void DistributedExecutionGeneratorTestRunner::runTestCase(
+    const string &input, const std::set<string> &options, string *output) {
+  // TODO(qzeng): Test multi-threaded query execution when we have a Sort operator.
+
+  VLOG(4) << "Test SQL(s): " << input;
+
+  if (options.find(kResetOption) != options.end()) {
+    test_database_loader_.clear();
+    test_database_loader_.createTestRelation(false /* allow_vchar */);
+    test_database_loader_.loadTestRelation();
+  }
+
+  MemStream output_stream;
+  sql_parser_.feedNextBuffer(new string(input));
+
+  while (true) {
+    ParseResult result = sql_parser_.getNextStatement();
+    if (result.condition != ParseResult::kSuccess) {
+      if (result.condition == ParseResult::kError) {
+        *output = result.error_message;
+      }
+      break;
+    }
+
+    const ParseStatement &parse_statement = *result.parsed_statement;
+    std::printf("%s\n", parse_statement.toString().c_str());
+    try {
+      OptimizerContext optimizer_context;
+      QueryHandle query_handle(query_id_++, cli_id_);
+
+      optimizer_.generateQueryHandle(parse_statement,
+                                     test_database_loader_.catalog_database(),
+                                     &optimizer_context,
+                                     &query_handle);
+
+      QueryExecutionUtil::ConstructAndSendAdmitRequestMessage(
+          cli_id_,
+          foreman_->getBusClientID(),
+          &query_handle,
+          &bus_);
+
+      const tmb::AnnotatedMessage annotated_message = bus_.Receive(cli_id_, 0, true);
+      DCHECK_EQ(kQueryExecutionSuccessMessage, annotated_message.tagged_message.message_type());
+
+      const CatalogRelation *query_result_relation = query_handle.getQueryResultRelation();
+      if (query_result_relation) {
+          PrintToScreen::PrintRelation(*query_result_relation,
+                                       test_database_loader_.storage_manager(),
+                                       output_stream.file());
+          DropRelation::Drop(*query_result_relation,
+                             test_database_loader_.catalog_database(),
+                             test_database_loader_.storage_manager());
+      }
+    } catch (const SqlError &error) {
+      *output = error.formatMessage(input);
+      break;
+    }
+  }
+
+  if (output->empty()) {
+    *output = output_stream.str();
+  }
+}
+
+}  // namespace optimizer
+}  // namespace quickstep

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/cdc1e053/query_optimizer/tests/DistributedExecutionGeneratorTestRunner.hpp
----------------------------------------------------------------------
diff --git a/query_optimizer/tests/DistributedExecutionGeneratorTestRunner.hpp b/query_optimizer/tests/DistributedExecutionGeneratorTestRunner.hpp
new file mode 100644
index 0000000..e4d0765
--- /dev/null
+++ b/query_optimizer/tests/DistributedExecutionGeneratorTestRunner.hpp
@@ -0,0 +1,118 @@
+/**
+ * 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.
+ **/
+
+#ifndef QUICKSTEP_QUERY_OPTIMIZER_TESTS_DISTRIBUTED_EXECUTION_GENERATOR_TEST_RUNNER_HPP_
+#define QUICKSTEP_QUERY_OPTIMIZER_TESTS_DISTRIBUTED_EXECUTION_GENERATOR_TEST_RUNNER_HPP_
+
+#include <cstddef>
+#include <memory>
+#include <set>
+#include <string>
+#include <utility>
+#include <vector>
+
+#include "parser/SqlParserWrapper.hpp"
+#include "query_execution/ForemanDistributed.hpp"
+#include "query_execution/QueryExecutionTypedefs.hpp"
+#include "query_execution/QueryExecutionUtil.hpp"
+#include "query_execution/Shiftboss.hpp"
+#include "query_execution/Worker.hpp"
+#include "query_execution/WorkerDirectory.hpp"
+#include "query_optimizer/Optimizer.hpp"
+#include "query_optimizer/tests/TestDatabaseLoader.hpp"
+#include "utility/Macros.hpp"
+#include "utility/textbased_test/TextBasedTestRunner.hpp"
+
+#include "glog/logging.h"
+
+#include "tmb/id_typedefs.h"
+#include "tmb/message_bus.h"
+#include "tmb/tagged_message.h"
+
+namespace quickstep {
+namespace optimizer {
+
+namespace {
+constexpr int kNumInstances = 1;
+}  // namespace
+
+/**
+ * @brief TextBasedTestRunner for testing the ExecutionGenerator in the
+ *        distributed version.
+ */
+class DistributedExecutionGeneratorTestRunner : public TextBasedTestRunner {
+ public:
+  /**
+   * @brief If this option is enabled, recreate the entire database and
+   *        repopulate the data before every test.
+   */
+  static const char *kResetOption;
+
+  /**
+   * @brief Constructor.
+   */
+  explicit DistributedExecutionGeneratorTestRunner(const std::string &storage_path);
+
+  ~DistributedExecutionGeneratorTestRunner() {
+    tmb::TaggedMessage poison_tagged_message(quickstep::kPoisonMessage);
+
+    const tmb::MessageBus::SendStatus send_status =
+        QueryExecutionUtil::SendTMBMessage(
+            &bus_,
+            cli_id_,
+            foreman_->getBusClientID(),
+            std::move(poison_tagged_message));
+    CHECK(send_status == tmb::MessageBus::SendStatus::kOK);
+
+    for (int i = 0; i < kNumInstances; ++i) {
+      workers_[i]->join();
+      shiftbosses_[i]->join();
+    }
+
+    foreman_->join();
+  }
+
+  void runTestCase(const std::string &input,
+                   const std::set<std::string> &options,
+                   std::string *output) override;
+
+ private:
+  std::size_t query_id_;
+
+  SqlParserWrapper sql_parser_;
+  TestDatabaseLoader test_database_loader_;
+  Optimizer optimizer_;
+
+  MessageBusImpl bus_;
+
+  tmb::client_id cli_id_;
+
+  std::unique_ptr<ForemanDistributed> foreman_;
+
+  std::vector<std::unique_ptr<Worker>> workers_;
+  std::vector<std::unique_ptr<WorkerDirectory>> worker_directories_;
+  std::vector<std::unique_ptr<Shiftboss>> shiftbosses_;
+
+  DISALLOW_COPY_AND_ASSIGN(DistributedExecutionGeneratorTestRunner);
+};
+
+}  // namespace optimizer
+}  // namespace quickstep
+
+#endif  // QUICKSTEP_QUERY_OPTIMIZER_TESTS_DISTRIBUTED_EXECUTION_GENERATOR_TEST_RUNNER_HPP_

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/cdc1e053/query_optimizer/tests/execution_generator/CMakeLists.txt
----------------------------------------------------------------------
diff --git a/query_optimizer/tests/execution_generator/CMakeLists.txt b/query_optimizer/tests/execution_generator/CMakeLists.txt
index 0c00ff6..1ea6a17 100644
--- a/query_optimizer/tests/execution_generator/CMakeLists.txt
+++ b/query_optimizer/tests/execution_generator/CMakeLists.txt
@@ -15,6 +15,63 @@
 # specific language governing permissions and limitations
 # under the License.
 
+if (ENABLE_DISTRIBUTED)
+  add_test(quickstep_queryoptimizer_tests_distributed_executiongenerator_create
+           "../quickstep_queryoptimizer_tests_DistributedExecutionGeneratorTest"
+           "${CMAKE_CURRENT_SOURCE_DIR}/Create.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedCreate.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedCreate/")
+  add_test(quickstep_queryoptimizer_tests_distributed_executiongenerator_delete
+           "../quickstep_queryoptimizer_tests_DistributedExecutionGeneratorTest"
+           "${CMAKE_CURRENT_SOURCE_DIR}/Delete.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedDelete.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedDelete/")
+  add_test(quickstep_queryoptimizer_tests_distributed_executiongenerator_distinct
+           "../quickstep_queryoptimizer_tests_DistributedExecutionGeneratorTest"
+           "${CMAKE_CURRENT_SOURCE_DIR}/Distinct.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedDistinct.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedDistinct/")
+  add_test(quickstep_queryoptimizer_tests_distributed_executiongenerator_drop
+           "../quickstep_queryoptimizer_tests_DistributedExecutionGeneratorTest"
+           "${CMAKE_CURRENT_SOURCE_DIR}/Drop.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedDrop.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedDrop/")
+  add_test(quickstep_queryoptimizer_tests_distributed_executiongenerator_index
+           "../quickstep_queryoptimizer_tests_DistributedExecutionGeneratorTest"
+           "${CMAKE_CURRENT_SOURCE_DIR}/Index.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedIndex.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedIndex/")
+  add_test(quickstep_queryoptimizer_tests_distributed_executiongenerator_insert
+           "../quickstep_queryoptimizer_tests_DistributedExecutionGeneratorTest"
+           "${CMAKE_CURRENT_SOURCE_DIR}/Insert.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedInsert.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedInsert/")
+  add_test(quickstep_queryoptimizer_tests_distributed_executiongenerator_join
+           "../quickstep_queryoptimizer_tests_DistributedExecutionGeneratorTest"
+           "${CMAKE_CURRENT_SOURCE_DIR}/Join.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedJoin.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedJoin/")
+  add_test(quickstep_queryoptimizer_tests_distributed_executiongenerator_select
+           "../quickstep_queryoptimizer_tests_DistributedExecutionGeneratorTest"
+           "${CMAKE_CURRENT_SOURCE_DIR}/Select.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedSelect.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedSelect/")
+  add_test(quickstep_queryoptimizer_tests_distributed_executiongenerator_stringpatternmatching
+           "../quickstep_queryoptimizer_tests_DistributedExecutionGeneratorTest"
+           "${CMAKE_CURRENT_SOURCE_DIR}/StringPatternMatching.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedStringPatternMatching.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedStringPatternMatching/")
+  add_test(quickstep_queryoptimizer_tests_distributed_executiongenerator_tablegenerator
+           "../quickstep_queryoptimizer_tests_DistributedExecutionGeneratorTest"
+           "${CMAKE_CURRENT_SOURCE_DIR}/TableGenerator.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedTableGenerator.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedTableGenerator/")
+  add_test(quickstep_queryoptimizer_tests_distributed_executiongenerator_update
+           "../quickstep_queryoptimizer_tests_DistributedExecutionGeneratorTest"
+           "${CMAKE_CURRENT_SOURCE_DIR}/Update.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedUpdate.test"
+           "${CMAKE_CURRENT_BINARY_DIR}/DistributedUpdate/")
+endif(ENABLE_DISTRIBUTED)
 add_test(quickstep_queryoptimizer_tests_executiongenerator_create
          "../quickstep_queryoptimizer_tests_ExecutionGeneratorTest"
          "${CMAKE_CURRENT_SOURCE_DIR}/Create.test"
@@ -76,6 +133,19 @@ add_test(quickstep_queryoptimizer_tests_executiongenerator_update
 file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/Create)
 file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/Delete)
 file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/Distinct)
+if (ENABLE_DISTRIBUTED)
+  file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/DistributedCreate)
+  file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/DistributedDelete)
+  file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/DistributedDistinct)
+  file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/DistributedDrop)
+  file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/DistributedIndex)
+  file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/DistributedInsert)
+  file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/DistributedJoin)
+  file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/DistributedSelect)
+  file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/DistributedStringPatternMatching)
+  file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/DistributedTableGenerator)
+  file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/DistributedUpdate)
+endif(ENABLE_DISTRIBUTED)
 file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/Drop)
 file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/Index)
 file(MAKE_DIRECTORY ${CMAKE_CURRENT_BINARY_DIR}/Insert)


[2/4] incubator-quickstep git commit: Refactored OptimizerContext and Optimizer.

Posted by zu...@apache.org.
Refactored OptimizerContext and Optimizer.

  * Construct physical generator in query_optimizer once for all queries.


Project: http://git-wip-us.apache.org/repos/asf/incubator-quickstep/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-quickstep/commit/8c811c85
Tree: http://git-wip-us.apache.org/repos/asf/incubator-quickstep/tree/8c811c85
Diff: http://git-wip-us.apache.org/repos/asf/incubator-quickstep/diff/8c811c85

Branch: refs/heads/q-opt-dist-exe-gen-test
Commit: 8c811c85e64ddb9889780ad3467dfc5abbfc9f28
Parents: 8ec99ed
Author: Zuyu Zhang <zu...@twitter.com>
Authored: Sat Aug 13 01:49:03 2016 -0700
Committer: Zuyu Zhang <zu...@twitter.com>
Committed: Tue Aug 16 13:26:38 2016 -0700

----------------------------------------------------------------------
 cli/tests/CMakeLists.txt                        |  8 +---
 cli/tests/CommandExecutorTestRunner.cpp         | 35 ++++++------------
 cli/tests/CommandExecutorTestRunner.hpp         |  2 +
 query_optimizer/CMakeLists.txt                  |  2 +-
 query_optimizer/ExecutionGenerator.cpp          | 35 +++++++++---------
 query_optimizer/ExecutionGenerator.hpp          | 12 +++---
 query_optimizer/LogicalGenerator.cpp            |  3 +-
 query_optimizer/LogicalGenerator.hpp            |  5 ++-
 query_optimizer/Optimizer.cpp                   | 12 +++---
 query_optimizer/Optimizer.hpp                   | 26 +++++--------
 query_optimizer/OptimizerContext.hpp            | 39 +-------------------
 query_optimizer/QueryProcessor.cpp              |  7 ++--
 query_optimizer/QueryProcessor.hpp              |  3 ++
 query_optimizer/resolver/Resolver.cpp           |  4 +-
 query_optimizer/resolver/Resolver.hpp           |  8 +++-
 query_optimizer/tests/CMakeLists.txt            | 12 +-----
 .../tests/ExecutionGeneratorTestRunner.cpp      | 35 +++++-------------
 .../tests/ExecutionGeneratorTestRunner.hpp      |  3 +-
 query_optimizer/tests/OptimizerTest.cpp         |  3 +-
 .../tests/OptimizerTextTestRunner.cpp           |  9 ++---
 20 files changed, 93 insertions(+), 170 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/cli/tests/CMakeLists.txt
----------------------------------------------------------------------
diff --git a/cli/tests/CMakeLists.txt b/cli/tests/CMakeLists.txt
index 3bceba8..99fa3a3 100644
--- a/cli/tests/CMakeLists.txt
+++ b/cli/tests/CMakeLists.txt
@@ -36,19 +36,13 @@ target_link_libraries(quickstep_cli_tests_CommandExecutorTest
                       quickstep_parser_SqlParserWrapper
                       quickstep_queryexecution_AdmitRequestMessage
                       quickstep_queryexecution_ForemanSingleNode
-                      quickstep_queryexecution_QueryContext
                       quickstep_queryexecution_QueryExecutionTypedefs
                       quickstep_queryexecution_QueryExecutionUtil
                       quickstep_queryexecution_Worker
                       quickstep_queryexecution_WorkerDirectory
-                      quickstep_queryexecution_WorkerMessage
-                      quickstep_queryoptimizer_ExecutionGenerator
-                      quickstep_queryoptimizer_LogicalGenerator
+                      quickstep_queryoptimizer_Optimizer
                       quickstep_queryoptimizer_OptimizerContext
-                      quickstep_queryoptimizer_PhysicalGenerator
                       quickstep_queryoptimizer_QueryHandle
-                      quickstep_queryoptimizer_QueryPlan
-                      quickstep_queryoptimizer_physical_Physical
                       quickstep_queryoptimizer_tests_TestDatabaseLoader
                       quickstep_utility_Macros
                       quickstep_utility_MemStream

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/cli/tests/CommandExecutorTestRunner.cpp
----------------------------------------------------------------------
diff --git a/cli/tests/CommandExecutorTestRunner.cpp b/cli/tests/CommandExecutorTestRunner.cpp
index 9c701cd..41cc9da 100644
--- a/cli/tests/CommandExecutorTestRunner.cpp
+++ b/cli/tests/CommandExecutorTestRunner.cpp
@@ -31,17 +31,10 @@
 #include "query_execution/AdmitRequestMessage.hpp"
 #include "query_execution/ForemanSingleNode.hpp"
 #include "query_execution/QueryExecutionTypedefs.hpp"
-#include "query_execution/Worker.hpp"
-#include "query_optimizer/ExecutionGenerator.hpp"
-#include "query_optimizer/LogicalGenerator.hpp"
+#include "query_optimizer/Optimizer.hpp"
 #include "query_optimizer/OptimizerContext.hpp"
-#include "query_optimizer/PhysicalGenerator.hpp"
 #include "query_optimizer/QueryHandle.hpp"
-#include "query_optimizer/QueryPlan.hpp"
-#include "query_optimizer/physical/Physical.hpp"
-#include "utility/Macros.hpp"
 #include "utility/MemStream.hpp"
-#include "utility/PtrList.hpp"
 #include "utility/SqlError.hpp"
 
 #include "glog/logging.h"
@@ -53,8 +46,6 @@ namespace quickstep {
 class CatalogRelation;
 
 namespace O = ::quickstep::optimizer;
-namespace P = ::quickstep::optimizer::physical;
-namespace L = ::quickstep::optimizer::logical;
 
 const char CommandExecutorTestRunner::kResetOption[] =
     "reset_before_execution";
@@ -77,19 +68,16 @@ void CommandExecutorTestRunner::runTestCase(
 
   while (true) {
     ParseResult result = sql_parser_.getNextStatement();
-
-    O::OptimizerContext optimizer_context(test_database_loader_.catalog_database(),
-                                          test_database_loader_.storage_manager());
-
     if (result.condition != ParseResult::kSuccess) {
       if (result.condition == ParseResult::kError) {
         *output = result.error_message;
       }
       break;
     } else {
-      std::printf("%s\n", result.parsed_statement->toString().c_str());
+      const ParseStatement &parse_statement = *result.parsed_statement;
+      std::printf("%s\n", parse_statement.toString().c_str());
       try {
-        if (result.parsed_statement->getStatementType() == ParseStatement::kCommand) {
+        if (parse_statement.getStatementType() == ParseStatement::kCommand) {
           quickstep::cli::executeCommand(
               *result.parsed_statement,
               *(test_database_loader_.catalog_database()),
@@ -99,15 +87,14 @@ void CommandExecutorTestRunner::runTestCase(
               test_database_loader_.storage_manager(),
               nullptr,
               output_stream.file());
-        } else  {
+        } else {
           QueryHandle query_handle(0 /* query_id */, main_thread_client_id_);
-          O::LogicalGenerator logical_generator(&optimizer_context);
-          O::PhysicalGenerator physical_generator;
-          O::ExecutionGenerator execution_generator(&optimizer_context, &query_handle);
-          const P::PhysicalPtr physical_plan =
-              physical_generator.generatePlan(
-                  logical_generator.generatePlan(*result.parsed_statement));
-          execution_generator.generatePlan(physical_plan);
+          O::OptimizerContext optimizer_context;
+
+          optimizer_.generateQueryHandle(parse_statement,
+                                         test_database_loader_.catalog_database(),
+                                         &optimizer_context,
+                                         &query_handle);
 
           AdmitRequestMessage request_message(&query_handle);
           TaggedMessage admit_tagged_message(

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/cli/tests/CommandExecutorTestRunner.hpp
----------------------------------------------------------------------
diff --git a/cli/tests/CommandExecutorTestRunner.hpp b/cli/tests/CommandExecutorTestRunner.hpp
index 682dc81..83c5a9a 100644
--- a/cli/tests/CommandExecutorTestRunner.hpp
+++ b/cli/tests/CommandExecutorTestRunner.hpp
@@ -33,6 +33,7 @@
 #include "query_execution/Worker.hpp"
 #include "query_execution/WorkerDirectory.hpp"
 #include "query_execution/WorkerMessage.hpp"
+#include "query_optimizer/Optimizer.hpp"
 #include "query_optimizer/tests/TestDatabaseLoader.hpp"
 #include "utility/Macros.hpp"
 #include "utility/textbased_test/TextBasedTestDriver.hpp"
@@ -103,6 +104,7 @@ class CommandExecutorTestRunner : public TextBasedTestRunner {
  private:
   SqlParserWrapper sql_parser_;
   optimizer::TestDatabaseLoader test_database_loader_;
+  optimizer::Optimizer optimizer_;
 
   tmb::client_id main_thread_client_id_;
 

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/CMakeLists.txt
----------------------------------------------------------------------
diff --git a/query_optimizer/CMakeLists.txt b/query_optimizer/CMakeLists.txt
index c24ee89..56ae52f 100644
--- a/query_optimizer/CMakeLists.txt
+++ b/query_optimizer/CMakeLists.txt
@@ -181,7 +181,6 @@ target_link_libraries(quickstep_queryoptimizer_LogicalToPhysicalMapper
 target_link_libraries(quickstep_queryoptimizer_Optimizer
                       quickstep_queryoptimizer_ExecutionGenerator
                       quickstep_queryoptimizer_LogicalGenerator
-                      quickstep_queryoptimizer_OptimizerContext
                       quickstep_queryoptimizer_PhysicalGenerator
                       quickstep_utility_Macros)
 target_link_libraries(quickstep_queryoptimizer_OptimizerContext
@@ -221,6 +220,7 @@ target_link_libraries(quickstep_queryoptimizer_QueryProcessor
                       quickstep_catalog_Catalog
                       quickstep_catalog_Catalog_proto
                       quickstep_queryoptimizer_Optimizer
+                      quickstep_queryoptimizer_OptimizerContext
                       quickstep_storage_StorageManager
                       quickstep_utility_Macros)
 target_link_libraries(quickstep_queryoptimizer_Validator

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/ExecutionGenerator.cpp
----------------------------------------------------------------------
diff --git a/query_optimizer/ExecutionGenerator.cpp b/query_optimizer/ExecutionGenerator.cpp
index a4b538e..2e03e09 100644
--- a/query_optimizer/ExecutionGenerator.cpp
+++ b/query_optimizer/ExecutionGenerator.cpp
@@ -195,14 +195,14 @@ void ExecutionGenerator::generatePlan(const P::PhysicalPtr &physical_plan) {
     const CatalogRelation *temporary_relation = temporary_relation_info.relation;
     if (temporary_relation == result_relation) {
       query_handle_->setQueryResultRelation(
-          optimizer_context_->catalog_database()->getRelationByIdMutable(result_relation->getID()));
+          catalog_database_->getRelationByIdMutable(result_relation->getID()));
       continue;
     }
     const QueryPlan::DAGNodeIndex drop_table_index =
         execution_plan_->addRelationalOperator(
             new DropTableOperator(query_handle_->query_id(),
                                   *temporary_relation,
-                                  optimizer_context_->catalog_database(),
+                                  catalog_database_,
                                   false /* only_drop_blocks */));
     DCHECK(!temporary_relation_info.isStoredRelation());
     execution_plan_->addDependenciesForDropOperator(
@@ -216,12 +216,12 @@ void ExecutionGenerator::generatePlan(const P::PhysicalPtr &physical_plan) {
   }
 
 #ifdef QUICKSTEP_DISTRIBUTED
-  catalog_database_cache_proto_->set_name(optimizer_context_->catalog_database()->getName());
+  catalog_database_cache_proto_->set_name(catalog_database_->getName());
 
   LOG(INFO) << "CatalogDatabaseCache proto has " << referenced_relation_ids_.size() << " relation(s)";
   for (const relation_id rel_id : referenced_relation_ids_) {
     const CatalogRelationSchema &relation =
-        optimizer_context_->catalog_database()->getRelationSchemaById(rel_id);
+        catalog_database_->getRelationSchemaById(rel_id);
     LOG(INFO) << "RelationSchema " << rel_id
               << ", name: " << relation.getName()
               << ", " << relation.size()  << " attribute(s)";
@@ -311,7 +311,7 @@ void ExecutionGenerator::createTemporaryCatalogRelation(
     const CatalogRelation **catalog_relation_output,
     S::InsertDestination *insert_destination_proto) {
   std::unique_ptr<CatalogRelation> catalog_relation(
-      new CatalogRelation(optimizer_context_->catalog_database(),
+      new CatalogRelation(catalog_database_,
                           getNewRelationName(),
                           -1 /* id */,
                           true /* is_temporary*/));
@@ -332,7 +332,7 @@ void ExecutionGenerator::createTemporaryCatalogRelation(
   }
 
   *catalog_relation_output = catalog_relation.get();
-  const relation_id output_rel_id = optimizer_context_->catalog_database()->addRelation(
+  const relation_id output_rel_id = catalog_database_->addRelation(
       catalog_relation.release());
 
 #ifdef QUICKSTEP_DISTRIBUTED
@@ -344,11 +344,10 @@ void ExecutionGenerator::createTemporaryCatalogRelation(
 }
 
 void ExecutionGenerator::dropAllTemporaryRelations() {
-  CatalogDatabase *catalog_database = optimizer_context_->catalog_database();
   for (const CatalogRelationInfo &temporary_relation_info :
        temporary_relation_info_vec_) {
     DCHECK_EQ(temporary_relation_info.relation->size_blocks(), 0u);
-    catalog_database->dropRelationById(temporary_relation_info.relation->getID());
+    catalog_database_->dropRelationById(temporary_relation_info.relation->getID());
   }
 }
 
@@ -618,7 +617,7 @@ void ExecutionGenerator::convertHashJoin(const P::HashJoinPtr &physical_plan) {
   for (const E::AttributeReferencePtr &left_join_attribute : left_join_attributes) {
     // Try to determine the original stored relation referenced in the Hash Join.
     referenced_stored_probe_relation =
-        optimizer_context_->catalog_database()->getRelationByName(left_join_attribute->relation_name());
+        catalog_database_->getRelationByName(left_join_attribute->relation_name());
     if (referenced_stored_probe_relation == nullptr) {
       // Hash Join optimizations are not possible, if the referenced relation cannot be determined.
       skip_hash_join_optimization = true;
@@ -642,7 +641,7 @@ void ExecutionGenerator::convertHashJoin(const P::HashJoinPtr &physical_plan) {
   for (const E::AttributeReferencePtr &right_join_attribute : right_join_attributes) {
     // Try to determine the original stored relation referenced in the Hash Join.
     referenced_stored_build_relation =
-        optimizer_context_->catalog_database()->getRelationByName(right_join_attribute->relation_name());
+        catalog_database_->getRelationByName(right_join_attribute->relation_name());
     if (referenced_stored_build_relation == nullptr) {
       // Hash Join optimizations are not possible, if the referenced relation cannot be determined.
       skip_hash_join_optimization = true;
@@ -958,7 +957,7 @@ void ExecutionGenerator::convertCreateIndex(
   const CatalogRelationInfo *input_relation_info =
       findRelationInfoOutputByPhysical(physical_plan->input());
   CatalogRelation *input_relation =
-      optimizer_context_->catalog_database()->getRelationByIdMutable(
+      catalog_database_->getRelationByIdMutable(
             input_relation_info->relation->getID());
 
   // Check if any index with the specified name already exists.
@@ -1004,7 +1003,7 @@ void ExecutionGenerator::convertCreateTable(
   // CreateTable is converted to a CreateTable operator.
 
   std::unique_ptr<CatalogRelation> catalog_relation(new CatalogRelation(
-      optimizer_context_->catalog_database(),
+      catalog_database_,
       physical_plan->relation_name(),
       -1 /* id */,
       false /* is_temporary*/));
@@ -1038,7 +1037,7 @@ void ExecutionGenerator::convertCreateTable(
   execution_plan_->addRelationalOperator(
       new CreateTableOperator(query_handle_->query_id(),
                               catalog_relation.release(),
-                              optimizer_context_->catalog_database()));
+                              catalog_database_));
 }
 
 void ExecutionGenerator::convertDeleteTuples(
@@ -1065,7 +1064,7 @@ void ExecutionGenerator::convertDeleteTuples(
         execution_plan_->addRelationalOperator(
             new DropTableOperator(query_handle_->query_id(),
                                   *input_relation_info->relation,
-                                  optimizer_context_->catalog_database(),
+                                  catalog_database_,
                                   true /* only_drop_blocks */));
     if (!input_relation_info->isStoredRelation()) {
       execution_plan_->addDirectDependency(drop_table_index,
@@ -1110,7 +1109,7 @@ void ExecutionGenerator::convertDropTable(
   execution_plan_->addRelationalOperator(
       new DropTableOperator(query_handle_->query_id(),
                             catalog_relation,
-                            optimizer_context_->catalog_database()));
+                            catalog_database_));
 }
 
 void ExecutionGenerator::convertInsertTuple(
@@ -1120,7 +1119,7 @@ void ExecutionGenerator::convertInsertTuple(
   const CatalogRelationInfo *input_relation_info =
       findRelationInfoOutputByPhysical(physical_plan->input());
   const CatalogRelation &input_relation =
-      *optimizer_context_->catalog_database()->getRelationById(
+      *catalog_database_->getRelationById(
           input_relation_info->relation->getID());
 
   // Construct the tuple proto to be inserted.
@@ -1320,7 +1319,7 @@ void ExecutionGenerator::convertUpdateTable(
   const QueryPlan::DAGNodeIndex update_operator_index =
       execution_plan_->addRelationalOperator(new UpdateOperator(
           query_handle_->query_id(),
-          *optimizer_context_->catalog_database()->getRelationById(
+          *catalog_database_->getRelationById(
               input_rel_id),
           relocation_destination_index,
           execution_predicate_index,
@@ -1578,7 +1577,7 @@ void ExecutionGenerator::convertSort(const P::SortPtr &physical_sort) {
           new DropTableOperator(
               query_handle_->query_id(),
               *merged_runs_relation,
-              optimizer_context_->catalog_database(),
+              catalog_database_,
               false /* only_drop_blocks */));
   execution_plan_->addDirectDependency(
       drop_merged_runs_index,

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/ExecutionGenerator.hpp
----------------------------------------------------------------------
diff --git a/query_optimizer/ExecutionGenerator.hpp b/query_optimizer/ExecutionGenerator.hpp
index d5c7b06..6017aa5 100644
--- a/query_optimizer/ExecutionGenerator.hpp
+++ b/query_optimizer/ExecutionGenerator.hpp
@@ -68,6 +68,7 @@
 namespace quickstep {
 
 class CatalogAttribute;
+class CatalogDatabase;
 class CatalogRelation;
 class Predicate;
 
@@ -82,8 +83,6 @@ class InsertDestination;
 
 namespace optimizer {
 
-class OptimizerContext;
-
 /** \addtogroup QueryOptimizer
  *  @{
  */
@@ -96,12 +95,12 @@ class ExecutionGenerator {
   /**
    * @brief Constructor. Does not take ownership of \p query_handle.
    *
-   * @param optimizer_context Optimizer context.
+   * @param catalog_database The catalog database where this query is executed.
    * @param query_handle The pointer to the output query handle.
    */
-  ExecutionGenerator(OptimizerContext *optimizer_context,
+  ExecutionGenerator(CatalogDatabase *catalog_database,
                      QueryHandle *query_handle)
-      : optimizer_context_(DCHECK_NOTNULL(optimizer_context)),
+      : catalog_database_(DCHECK_NOTNULL(catalog_database)),
         query_handle_(DCHECK_NOTNULL(query_handle)),
         execution_plan_(DCHECK_NOTNULL(query_handle->getQueryPlanMutable())),
         query_context_proto_(DCHECK_NOTNULL(query_handle->getQueryContextProtoMutable())),
@@ -383,7 +382,8 @@ class ExecutionGenerator {
    */
   void dropAllTemporaryRelations();
 
-  OptimizerContext *optimizer_context_;
+  CatalogDatabase *catalog_database_;
+
   QueryHandle *query_handle_;
   QueryPlan *execution_plan_;  // A part of QueryHandle.
   serialization::QueryContext *query_context_proto_;  // A part of QueryHandle.

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/LogicalGenerator.cpp
----------------------------------------------------------------------
diff --git a/query_optimizer/LogicalGenerator.cpp b/query_optimizer/LogicalGenerator.cpp
index 450c36a..abeca53 100644
--- a/query_optimizer/LogicalGenerator.cpp
+++ b/query_optimizer/LogicalGenerator.cpp
@@ -48,8 +48,9 @@ LogicalGenerator::LogicalGenerator(OptimizerContext *optimizer_context)
 LogicalGenerator::~LogicalGenerator() {}
 
 L::LogicalPtr LogicalGenerator::generatePlan(
+    const CatalogDatabase &catalog_database,
     const ParseStatement &parse_statement) {
-  resolver::Resolver resolver(optimizer_context_);
+  resolver::Resolver resolver(catalog_database, optimizer_context_);
   DVLOG(4) << "Parse tree:\n" << parse_statement.toString();
   logical_plan_ = resolver.resolve(parse_statement);
   DVLOG(4) << "Initial logical plan:\n" << logical_plan_->toString();

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/LogicalGenerator.hpp
----------------------------------------------------------------------
diff --git a/query_optimizer/LogicalGenerator.hpp b/query_optimizer/LogicalGenerator.hpp
index d4384c8..2898e6f 100644
--- a/query_optimizer/LogicalGenerator.hpp
+++ b/query_optimizer/LogicalGenerator.hpp
@@ -25,6 +25,7 @@
 
 namespace quickstep {
 
+class CatalogDatabase;
 class ParseStatement;
 
 namespace optimizer {
@@ -61,10 +62,12 @@ class LogicalGenerator {
    * @brief Converts the parse tree to a logical plan and optimizes the logical
    *        plan by using rules.
    *
+   * @param catalog_database The catalog database where this query is executed.
    * @param parse_statement The parse tree to be converted.
    * @return An optimized logical plan.
    */
-  logical::LogicalPtr generatePlan(const ParseStatement &parse_statement);
+  logical::LogicalPtr generatePlan(const CatalogDatabase &catalog_database,
+                                   const ParseStatement &parse_statement);
 
  private:
   /**

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/Optimizer.cpp
----------------------------------------------------------------------
diff --git a/query_optimizer/Optimizer.cpp b/query_optimizer/Optimizer.cpp
index b77e430..b14c938 100644
--- a/query_optimizer/Optimizer.cpp
+++ b/query_optimizer/Optimizer.cpp
@@ -21,20 +21,20 @@
 
 #include "query_optimizer/ExecutionGenerator.hpp"
 #include "query_optimizer/LogicalGenerator.hpp"
-#include "query_optimizer/PhysicalGenerator.hpp"
 
 namespace quickstep {
 namespace optimizer {
 
 void Optimizer::generateQueryHandle(const ParseStatement &parse_statement,
+                                    CatalogDatabase *catalog_database,
+                                    OptimizerContext *optimizer_context,
                                     QueryHandle *query_handle) {
-  LogicalGenerator logical_generator(&optimizer_context_);
-  PhysicalGenerator physical_generator;
-  ExecutionGenerator execution_generator(&optimizer_context_, query_handle);
+  LogicalGenerator logical_generator(optimizer_context);
+  ExecutionGenerator execution_generator(catalog_database, query_handle);
 
   execution_generator.generatePlan(
-      physical_generator.generatePlan(
-          logical_generator.generatePlan(parse_statement)));
+      physical_generator_.generatePlan(
+          logical_generator.generatePlan(*catalog_database, parse_statement)));
 }
 
 }  // namespace optimizer

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/Optimizer.hpp
----------------------------------------------------------------------
diff --git a/query_optimizer/Optimizer.hpp b/query_optimizer/Optimizer.hpp
index 132b0b8..36f956a 100644
--- a/query_optimizer/Optimizer.hpp
+++ b/query_optimizer/Optimizer.hpp
@@ -20,7 +20,7 @@
 #ifndef QUICKSTEP_QUERY_OPTIMIZER_OPTIMIZER_HPP_
 #define QUICKSTEP_QUERY_OPTIMIZER_OPTIMIZER_HPP_
 
-#include "query_optimizer/OptimizerContext.hpp"
+#include "query_optimizer/PhysicalGenerator.hpp"
 #include "utility/Macros.hpp"
 
 namespace quickstep {
@@ -28,10 +28,11 @@ namespace quickstep {
 class CatalogDatabase;
 class ParseStatement;
 class QueryHandle;
-class StorageManager;
 
 namespace optimizer {
 
+class OptimizerContext;
+
 /** \addtogroup QueryOptimizer
  *  @{
  */
@@ -44,13 +45,8 @@ class Optimizer {
  public:
   /**
    * @brief Constructor.
-   *
-   * @param database The database that the query is executed on.
-   * @param storage_manager The storage manager for the database.
    */
-  Optimizer(CatalogDatabase *database,
-            StorageManager *storage_manager)
-      : optimizer_context_(database, storage_manager) {}
+  Optimizer() {}
 
   /**
    * @brief Destructor.
@@ -64,21 +60,17 @@ class Optimizer {
    * @note Does not take ownership of \p query_handle.
    *
    * @param parse_statement The parse tree of the input query.
+   * @param catalog_database The database that the query is executed on.
+   * @param optimizer_context The optimizer context of the input query.
    * @param query_handle The generated query handle to output.
    */
   void generateQueryHandle(const ParseStatement &parse_statement,
+                           CatalogDatabase *catalog_database,
+                           OptimizerContext *optimizer_context,
                            QueryHandle *query_handle);
 
-  /**
-   * @return True if the catalog will be permanently changed after executing the
-   *         query and needs to be saved.
-   */
-  bool isCatalogChanged() const {
-    return optimizer_context_.is_catalog_changed();
-  }
-
  private:
-  OptimizerContext optimizer_context_;
+  PhysicalGenerator physical_generator_;
 
   DISALLOW_COPY_AND_ASSIGN(Optimizer);
 };

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/OptimizerContext.hpp
----------------------------------------------------------------------
diff --git a/query_optimizer/OptimizerContext.hpp b/query_optimizer/OptimizerContext.hpp
index 5be2a98..d52c96c 100644
--- a/query_optimizer/OptimizerContext.hpp
+++ b/query_optimizer/OptimizerContext.hpp
@@ -25,9 +25,6 @@
 
 namespace quickstep {
 
-class CatalogDatabase;
-class StorageManager;
-
 namespace optimizer {
 
 /** \addtogroup QueryOptimizer
@@ -44,16 +41,9 @@ class OptimizerContext {
 
   /**
    * @brief Constructor.
-   *
-   * @param catalog_database The catalog database where this query is executed.
-   * @param storage_manager The storage manager to use for allocating storage
-   *        blocks.
    */
-  OptimizerContext(CatalogDatabase *catalog_database,
-                   StorageManager *storage_manager)
+  OptimizerContext()
       : current_expr_id_(-1),
-        catalog_database_(catalog_database),
-        storage_manager_(storage_manager),
         has_nested_queries_(false),
         is_catalog_changed_(false) {}
 
@@ -63,30 +53,6 @@ class OptimizerContext {
   ~OptimizerContext() {}
 
   /**
-   * @return Const pointer to the catalog database
-   *         where the query is executed.
-   */
-  const CatalogDatabase* catalog_database() const { return catalog_database_; }
-
-  /**
-   * @return Mutable pointer to the catalog database
-   *         where this query is executed.
-   */
-  CatalogDatabase* catalog_database() { return catalog_database_; }
-
-  /**
-   * @return Const pointer to the storage manager to use for
-   *         allocating storage blocks.
-   */
-  const StorageManager* storage_manager() const { return storage_manager_; }
-
-  /**
-   * @return Mutable pointer to the storage manager to use for
-   *         allocating storage blocks.
-   */
-  StorageManager* storage_manager() { return storage_manager_; }
-
-  /**
    * @brief Gets the next ExprId.
    *
    * @return A new ExprId.
@@ -123,9 +89,6 @@ class OptimizerContext {
  private:
   expressions::ExprId current_expr_id_;
 
-  CatalogDatabase *catalog_database_;
-  StorageManager *storage_manager_;
-
   bool has_nested_queries_;
 
   bool is_catalog_changed_;

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/QueryProcessor.cpp
----------------------------------------------------------------------
diff --git a/query_optimizer/QueryProcessor.cpp b/query_optimizer/QueryProcessor.cpp
index ea9f126..d74eeba 100644
--- a/query_optimizer/QueryProcessor.cpp
+++ b/query_optimizer/QueryProcessor.cpp
@@ -26,6 +26,7 @@
 #include "catalog/Catalog.hpp"
 #include "catalog/Catalog.pb.h"
 #include "query_optimizer/Optimizer.hpp"
+#include "query_optimizer/OptimizerContext.hpp"
 
 using std::ifstream;
 using std::ofstream;
@@ -34,11 +35,11 @@ namespace quickstep {
 
 void QueryProcessor::generateQueryHandle(const ParseStatement &statement,
                                          QueryHandle *query_handle) {
-  optimizer::Optimizer optimizer(getDefaultDatabase(), storage_manager_.get());
+  optimizer::OptimizerContext optimizer_context;
 
-  optimizer.generateQueryHandle(statement, query_handle);
+  optimizer_.generateQueryHandle(statement, getDefaultDatabase(), &optimizer_context, query_handle);
 
-  if (optimizer.isCatalogChanged() && !catalog_altered_) {
+  if (optimizer_context.is_catalog_changed() && !catalog_altered_) {
     catalog_altered_ = true;
   }
 

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/QueryProcessor.hpp
----------------------------------------------------------------------
diff --git a/query_optimizer/QueryProcessor.hpp b/query_optimizer/QueryProcessor.hpp
index 2a7e942..5d3818e 100644
--- a/query_optimizer/QueryProcessor.hpp
+++ b/query_optimizer/QueryProcessor.hpp
@@ -26,6 +26,7 @@
 #include <string>
 
 #include "catalog/Catalog.hpp"
+#include "query_optimizer/Optimizer.hpp"
 #include "storage/StorageManager.hpp"
 #include "utility/Macros.hpp"
 
@@ -194,6 +195,8 @@ class QueryProcessor {
  private:
   void loadCatalog();  // If it exists, free catalog_ before calling this
 
+  optimizer::Optimizer optimizer_;
+
   std::string catalog_filename_;
 
   std::unique_ptr<Catalog> catalog_;

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/resolver/Resolver.cpp
----------------------------------------------------------------------
diff --git a/query_optimizer/resolver/Resolver.cpp b/query_optimizer/resolver/Resolver.cpp
index 4931629..38ec24a 100644
--- a/query_optimizer/resolver/Resolver.cpp
+++ b/query_optimizer/resolver/Resolver.cpp
@@ -439,7 +439,7 @@ L::LogicalPtr Resolver::resolveCreateTable(
   // Resolve relation name.
   const std::string relation_name =
       create_table_statement.relation_name()->value();
-  if (context_->catalog_database()->hasRelationWithName(relation_name)) {
+  if (catalog_database_.hasRelationWithName(relation_name)) {
     THROW_SQL_ERROR_AT(create_table_statement.relation_name())
         << "Relation " << create_table_statement.relation_name()->value()
         << " already exists";
@@ -1719,7 +1719,7 @@ E::WindowInfo Resolver::resolveWindow(const ParseWindow &parse_window,
 const CatalogRelation* Resolver::resolveRelationName(
     const ParseString *relation_name) {
   const CatalogRelation *relation =
-      context_->catalog_database()->getRelationByName(
+      catalog_database_.getRelationByName(
           ToLower(relation_name->value()));
   if (relation == nullptr) {
     THROW_SQL_ERROR_AT(relation_name) << "Unrecognized relation "

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/resolver/Resolver.hpp
----------------------------------------------------------------------
diff --git a/query_optimizer/resolver/Resolver.hpp b/query_optimizer/resolver/Resolver.hpp
index 7bfeed2..855e6ba 100644
--- a/query_optimizer/resolver/Resolver.hpp
+++ b/query_optimizer/resolver/Resolver.hpp
@@ -98,10 +98,12 @@ class Resolver {
   /**
    * @brief Constructor. Does not take ownership of \p context.
    *
+   * @param catalog_database The database that the query is executed on.
    * @param context The context of this query.
    */
-  explicit Resolver(OptimizerContext *context)
-      : context_(context) {}
+  Resolver(const CatalogDatabase &catalog_database, OptimizerContext *context)
+      : catalog_database_(catalog_database),
+        context_(context) {}
 
   /**
    * @brief Validates the query is semantically correct and converts the parse
@@ -673,6 +675,8 @@ class Resolver {
       std::vector<expressions::NamedExpressionPtr> *select_list_expressions,
       logical::LogicalPtr *logical_plan);
 
+  const CatalogDatabase &catalog_database_;
+
   OptimizerContext *context_;
   WithQueriesInfo with_queries_info_;
 

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/tests/CMakeLists.txt
----------------------------------------------------------------------
diff --git a/query_optimizer/tests/CMakeLists.txt b/query_optimizer/tests/CMakeLists.txt
index 1453291..4969ada 100644
--- a/query_optimizer/tests/CMakeLists.txt
+++ b/query_optimizer/tests/CMakeLists.txt
@@ -110,7 +110,6 @@ add_executable(quickstep_queryoptimizer_tests_OptimizerTextTest
                "${PROJECT_SOURCE_DIR}/utility/textbased_test/TextBasedTest.hpp")
 
 target_link_libraries(quickstep_queryoptimizer_tests_ExecutionGeneratorTest
-                      ${GFLAGS_LIB_NAME}
                       glog
                       gtest
                       quickstep_catalog_CatalogDatabase
@@ -118,21 +117,14 @@ target_link_libraries(quickstep_queryoptimizer_tests_ExecutionGeneratorTest
                       quickstep_cli_PrintToScreen
                       quickstep_parser_ParseStatement
                       quickstep_parser_SqlParserWrapper
-                      quickstep_queryexecution_AdmitRequestMessage
                       quickstep_queryexecution_ForemanSingleNode
-                      quickstep_queryexecution_QueryContext
                       quickstep_queryexecution_QueryExecutionTypedefs
                       quickstep_queryexecution_QueryExecutionUtil
                       quickstep_queryexecution_Worker
                       quickstep_queryexecution_WorkerDirectory
-                      quickstep_queryexecution_WorkerMessage
-                      quickstep_queryoptimizer_ExecutionGenerator
-                      quickstep_queryoptimizer_LogicalGenerator
+                      quickstep_queryoptimizer_Optimizer
                       quickstep_queryoptimizer_OptimizerContext
-                      quickstep_queryoptimizer_PhysicalGenerator
                       quickstep_queryoptimizer_QueryHandle
-                      quickstep_queryoptimizer_QueryPlan
-                      quickstep_queryoptimizer_physical_Physical
                       quickstep_queryoptimizer_tests_TestDatabaseLoader
                       quickstep_threading_ThreadIDBasedMap
                       quickstep_utility_Macros
@@ -140,13 +132,13 @@ target_link_libraries(quickstep_queryoptimizer_tests_ExecutionGeneratorTest
                       quickstep_utility_SqlError
                       quickstep_utility_TextBasedTestDriver
                       tmb
+                      ${GFLAGS_LIB_NAME}
                       ${LIBS})
 target_link_libraries(quickstep_queryoptimizer_tests_OptimizerTextTest
                       ${GFLAGS_LIB_NAME}
                       glog
                       gtest
                       gtest_main
-                      quickstep_parser_ParseStatement
                       quickstep_parser_SqlParserWrapper
                       quickstep_queryoptimizer_LogicalGenerator
                       quickstep_queryoptimizer_OptimizerContext

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/tests/ExecutionGeneratorTestRunner.cpp
----------------------------------------------------------------------
diff --git a/query_optimizer/tests/ExecutionGeneratorTestRunner.cpp b/query_optimizer/tests/ExecutionGeneratorTestRunner.cpp
index 2e01c7b..06397d4 100644
--- a/query_optimizer/tests/ExecutionGeneratorTestRunner.cpp
+++ b/query_optimizer/tests/ExecutionGeneratorTestRunner.cpp
@@ -26,27 +26,16 @@
 #include "cli/DropRelation.hpp"
 #include "cli/PrintToScreen.hpp"
 #include "parser/ParseStatement.hpp"
-#include "query_execution/AdmitRequestMessage.hpp"
 #include "query_execution/ForemanSingleNode.hpp"
 #include "query_execution/QueryExecutionUtil.hpp"
-#include "query_execution/Worker.hpp"
-#include "query_optimizer/ExecutionGenerator.hpp"
-#include "query_optimizer/LogicalGenerator.hpp"
+#include "query_optimizer/Optimizer.hpp"
 #include "query_optimizer/OptimizerContext.hpp"
-#include "query_optimizer/PhysicalGenerator.hpp"
 #include "query_optimizer/QueryHandle.hpp"
-#include "query_optimizer/QueryPlan.hpp"
-#include "query_optimizer/physical/Physical.hpp"
-#include "utility/Macros.hpp"
 #include "utility/MemStream.hpp"
-#include "utility/PtrList.hpp"
 #include "utility/SqlError.hpp"
 
 #include "glog/logging.h"
 
-#include "tmb/id_typedefs.h"
-#include "tmb/message_bus.h"
-
 namespace quickstep {
 
 class CatalogRelation;
@@ -74,28 +63,22 @@ void ExecutionGeneratorTestRunner::runTestCase(
 
   while (true) {
     ParseResult result = sql_parser_.getNextStatement();
-
-    OptimizerContext optimizer_context(test_database_loader_.catalog_database(),
-                                       test_database_loader_.storage_manager());
-
     if (result.condition != ParseResult::kSuccess) {
       if (result.condition == ParseResult::kError) {
         *output = result.error_message;
       }
       break;
     } else {
-      std::printf("%s\n", result.parsed_statement->toString().c_str());
+      const ParseStatement &parse_statement = *result.parsed_statement;
+      std::printf("%s\n", parse_statement.toString().c_str());
       try {
         QueryHandle query_handle(0 /* query_id */, main_thread_client_id_);
-        LogicalGenerator logical_generator(&optimizer_context);
-        PhysicalGenerator physical_generator;
-        ExecutionGenerator execution_generator(&optimizer_context,
-                                               &query_handle);
-
-        const physical::PhysicalPtr physical_plan =
-            physical_generator.generatePlan(
-                logical_generator.generatePlan(*result.parsed_statement));
-        execution_generator.generatePlan(physical_plan);
+        OptimizerContext optimizer_context;
+
+        optimizer_.generateQueryHandle(parse_statement,
+                                       test_database_loader_.catalog_database(),
+                                       &optimizer_context,
+                                       &query_handle);
 
         QueryExecutionUtil::ConstructAndSendAdmitRequestMessage(
             main_thread_client_id_,

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/tests/ExecutionGeneratorTestRunner.hpp
----------------------------------------------------------------------
diff --git a/query_optimizer/tests/ExecutionGeneratorTestRunner.hpp b/query_optimizer/tests/ExecutionGeneratorTestRunner.hpp
index b8cd02a..f4e11ea 100644
--- a/query_optimizer/tests/ExecutionGeneratorTestRunner.hpp
+++ b/query_optimizer/tests/ExecutionGeneratorTestRunner.hpp
@@ -32,6 +32,7 @@
 #include "query_execution/Worker.hpp"
 #include "query_execution/WorkerDirectory.hpp"
 #include "query_execution/WorkerMessage.hpp"
+#include "query_optimizer/Optimizer.hpp"
 #include "query_optimizer/tests/TestDatabaseLoader.hpp"
 #include "threading/ThreadIDBasedMap.hpp"
 #include "utility/Macros.hpp"
@@ -41,7 +42,6 @@
 #include "tmb/message_bus.h"
 
 namespace quickstep {
-
 namespace optimizer {
 
 /**
@@ -106,6 +106,7 @@ class ExecutionGeneratorTestRunner : public TextBasedTestRunner {
  private:
   SqlParserWrapper sql_parser_;
   TestDatabaseLoader test_database_loader_;
+  Optimizer optimizer_;
 
   MessageBusImpl bus_;
   std::unique_ptr<ForemanSingleNode> foreman_;

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/tests/OptimizerTest.cpp
----------------------------------------------------------------------
diff --git a/query_optimizer/tests/OptimizerTest.cpp b/query_optimizer/tests/OptimizerTest.cpp
index a93db3e..3838638 100644
--- a/query_optimizer/tests/OptimizerTest.cpp
+++ b/query_optimizer/tests/OptimizerTest.cpp
@@ -61,8 +61,7 @@ OptimizerTest::OptimizerTest()
     : catalog_(new Catalog),
       catalog_database_(
           new CatalogDatabase(catalog_.get(), "TestDatabase" /* name */, 0)),
-      optimizer_context_(new OptimizerContext(catalog_database_.get(),
-                                              nullptr /* storage_manager */)),
+      optimizer_context_(new OptimizerContext),
       physical_generator_(new PhysicalGenerator()) {}
 
 E::AliasPtr OptimizerTest::createAlias(const E::ExpressionPtr &expression,

http://git-wip-us.apache.org/repos/asf/incubator-quickstep/blob/8c811c85/query_optimizer/tests/OptimizerTextTestRunner.cpp
----------------------------------------------------------------------
diff --git a/query_optimizer/tests/OptimizerTextTestRunner.cpp b/query_optimizer/tests/OptimizerTextTestRunner.cpp
index 93f3123..b9238c9 100644
--- a/query_optimizer/tests/OptimizerTextTestRunner.cpp
+++ b/query_optimizer/tests/OptimizerTextTestRunner.cpp
@@ -23,7 +23,6 @@
 #include <string>
 #include <vector>
 
-#include "parser/ParseStatement.hpp"
 #include "query_optimizer/LogicalGenerator.hpp"
 #include "query_optimizer/OptimizerContext.hpp"
 #include "query_optimizer/PhysicalGenerator.hpp"
@@ -49,8 +48,7 @@ void OptimizerTextTestRunner::runTestCase(const std::string &input,
   sql_parser_.feedNextBuffer(new std::string(input));
   ParseResult result = sql_parser_.getNextStatement();
 
-  OptimizerContext optimizer_context(test_database_loader_.catalog_database(),
-                                     nullptr /* storage_manager */);
+  OptimizerContext optimizer_context;
   if (result.condition != ParseResult::kSuccess) {
     *output = result.error_message;
   } else {
@@ -115,7 +113,7 @@ void OptimizerTextTestRunner::runTestCase(const std::string &input,
 logical::LogicalPtr OptimizerTextTestRunner::resolveParseTree(
     const ParseStatement &parse_statement,
     OptimizerContext *optimizer_context) {
-  resolver::Resolver resolver(optimizer_context);
+  resolver::Resolver resolver(*test_database_loader_.catalog_database(), optimizer_context);
   return resolver.resolve(parse_statement);
 }
 
@@ -123,7 +121,8 @@ logical::LogicalPtr OptimizerTextTestRunner::generateLogicalPlan(
     const ParseStatement &parse_statement,
     OptimizerContext *optimizer_context) {
   LogicalGenerator logical_generator(optimizer_context);
-  return logical_generator.generatePlan(parse_statement);
+  return logical_generator.generatePlan(*test_database_loader_.catalog_database(),
+                                        parse_statement);
 }
 
 physical::PhysicalPtr OptimizerTextTestRunner::generatePhysicalPlan(