Fixed tests

This commit is contained in:
Maksim Kita 2024-04-09 12:54:51 +03:00
parent 0111f30ed2
commit bd1189de70
26 changed files with 95 additions and 40 deletions

View File

@ -74,12 +74,12 @@ bool isStorageUsedInTree(const StoragePtr & storage, const IQueryTreeNode * root
const auto * subtree_node = nodes_to_process.back(); const auto * subtree_node = nodes_to_process.back();
nodes_to_process.pop_back(); nodes_to_process.pop_back();
auto * table_node = subtree_node->as<TableNode>(); const auto * table_node = subtree_node->as<TableNode>();
auto * table_function_node = subtree_node->as<TableFunctionNode>(); const auto * table_function_node = subtree_node->as<TableFunctionNode>();
if (table_node || table_function_node) if (table_node || table_function_node)
{ {
auto table_storage = table_node ? table_node->getStorage() : table_function_node->getStorage(); const auto & table_storage = table_node ? table_node->getStorage() : table_function_node->getStorage();
if (table_storage->getStorageID() == storage->getStorageID()) if (table_storage->getStorageID() == storage->getStorageID())
return true; return true;
} }

View File

@ -56,6 +56,14 @@ struct SelectQueryOptions
std::optional<UInt32> shard_num; std::optional<UInt32> shard_num;
std::optional<UInt32> shard_count; std::optional<UInt32> shard_count;
/** During read from MergeTree parts will be removed from snapshot after they are not needed.
* This optimization will break subsequent execution of the same query tree, because table node
* will no more have valid snapshot.
*
* TODO: Implement this functionality in safer way
*/
bool merge_tree_enable_remove_parts_from_snapshot_optimization = true;
SelectQueryOptions( /// NOLINT(google-explicit-constructor) SelectQueryOptions( /// NOLINT(google-explicit-constructor)
QueryProcessingStage::Enum stage = QueryProcessingStage::Complete, QueryProcessingStage::Enum stage = QueryProcessingStage::Complete,
size_t depth = 0, size_t depth = 0,

View File

@ -1382,6 +1382,7 @@ void Planner::buildPlanForQueryNode()
select_query_info.has_window = hasWindowFunctionNodes(query_tree); select_query_info.has_window = hasWindowFunctionNodes(query_tree);
select_query_info.has_aggregates = hasAggregateFunctionNodes(query_tree); select_query_info.has_aggregates = hasAggregateFunctionNodes(query_tree);
select_query_info.need_aggregate = query_node.hasGroupBy() || select_query_info.has_aggregates; select_query_info.need_aggregate = query_node.hasGroupBy() || select_query_info.has_aggregates;
select_query_info.merge_tree_enable_remove_parts_from_snapshot_optimization = select_query_options.merge_tree_enable_remove_parts_from_snapshot_optimization;
if (!select_query_info.has_window && query_node.hasQualify()) if (!select_query_info.has_window && query_node.hasQualify())
{ {

View File

@ -292,6 +292,7 @@ ReadFromMergeTree::ReadFromMergeTree(
, log(std::move(log_)) , log(std::move(log_))
, analyzed_result_ptr(analyzed_result_ptr_) , analyzed_result_ptr(analyzed_result_ptr_)
, is_parallel_reading_from_replicas(enable_parallel_reading) , is_parallel_reading_from_replicas(enable_parallel_reading)
, enable_remove_parts_from_snapshot_optimization(query_info_.merge_tree_enable_remove_parts_from_snapshot_optimization)
{ {
if (is_parallel_reading_from_replicas) if (is_parallel_reading_from_replicas)
{ {
@ -1970,6 +1971,13 @@ void ReadFromMergeTree::initializePipeline(QueryPipelineBuilder & pipeline, cons
{ {
auto result = getAnalysisResult(); auto result = getAnalysisResult();
if (enable_remove_parts_from_snapshot_optimization)
{
/// Do not keep data parts in snapshot.
/// They are stored separately, and some could be released after PK analysis.
storage_snapshot->data = std::make_unique<MergeTreeData::SnapshotData>();
}
result.checkLimits(context->getSettingsRef(), query_info); result.checkLimits(context->getSettingsRef(), query_info);
shared_virtual_fields.emplace("_sample_factor", result.sampling.used_sample_factor); shared_virtual_fields.emplace("_sample_factor", result.sampling.used_sample_factor);

View File

@ -280,6 +280,7 @@ private:
std::optional<MergeTreeAllRangesCallback> all_ranges_callback; std::optional<MergeTreeAllRangesCallback> all_ranges_callback;
std::optional<MergeTreeReadTaskCallback> read_task_callback; std::optional<MergeTreeReadTaskCallback> read_task_callback;
bool enable_vertical_final = false; bool enable_vertical_final = false;
bool enable_remove_parts_from_snapshot_optimization = true;
}; };
} }

View File

@ -171,7 +171,11 @@ private:
auto & query_to_execute = recursive_step > 0 ? recursive_query : non_recursive_query; auto & query_to_execute = recursive_step > 0 ? recursive_query : non_recursive_query;
++recursive_step; ++recursive_step;
auto interpreter = std::make_unique<InterpreterSelectQueryAnalyzer>(query_to_execute, recursive_query_context, SelectQueryOptions{});
SelectQueryOptions select_query_options;
select_query_options.merge_tree_enable_remove_parts_from_snapshot_optimization = false;
auto interpreter = std::make_unique<InterpreterSelectQueryAnalyzer>(query_to_execute, recursive_query_context, select_query_options);
auto pipeline_builder = interpreter->buildQueryPipeline(); auto pipeline_builder = interpreter->buildQueryPipeline();
pipeline_builder.addSimpleTransform([&](const Block & in_header) pipeline_builder.addSimpleTransform([&](const Block & in_header)

View File

@ -235,6 +235,9 @@ struct SelectQueryInfo
/// For IStorageSystemOneBlock /// For IStorageSystemOneBlock
std::vector<UInt8> columns_mask; std::vector<UInt8> columns_mask;
/// During read from MergeTree parts will be removed from snapshot after they are not needed
bool merge_tree_enable_remove_parts_from_snapshot_optimization = true;
bool isFinal() const; bool isFinal() const;
}; };
} }

View File

@ -1,5 +1,6 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
WITH RECURSIVE recursive_cte AS (SELECT 1 AS n UNION ALL SELECT n + 1 FROM recursive_cte WHERE n < 10) WITH RECURSIVE recursive_cte AS (SELECT 1 AS n UNION ALL SELECT n + 1 FROM recursive_cte WHERE n < 10)
SELECT n FROM recursive_cte; SELECT n FROM recursive_cte;
1 1

View File

@ -1,5 +1,7 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
WITH RECURSIVE recursive_cte AS (SELECT 1 AS n UNION ALL SELECT n + 1 FROM recursive_cte WHERE n < 10) WITH RECURSIVE recursive_cte AS (SELECT 1 AS n UNION ALL SELECT n + 1 FROM recursive_cte WHERE n < 10)
SELECT n FROM recursive_cte; SELECT n FROM recursive_cte;

View File

@ -1,3 +1,5 @@
SET allow_experimental_analyzer = 1;
DROP TABLE IF EXISTS tree; DROP TABLE IF EXISTS tree;
CREATE TABLE tree CREATE TABLE tree
( (

View File

@ -1,5 +1,6 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
DROP TABLE IF EXISTS tree; DROP TABLE IF EXISTS tree;
CREATE TABLE tree CREATE TABLE tree
( (

View File

@ -1,5 +1,7 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
DROP TABLE IF EXISTS tree; DROP TABLE IF EXISTS tree;
CREATE TABLE tree CREATE TABLE tree
( (

View File

@ -32,6 +32,7 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
-- WITH RECURSIVE -- WITH RECURSIVE
-- sum of 1..100 -- sum of 1..100

View File

@ -32,6 +32,8 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
-- WITH RECURSIVE -- WITH RECURSIVE
-- sum of 1..100 -- sum of 1..100

View File

@ -32,6 +32,7 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
-- --
-- Some examples with a tree -- Some examples with a tree
-- --

View File

@ -32,6 +32,8 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
-- --
-- Some examples with a tree -- Some examples with a tree
-- --

View File

@ -32,6 +32,7 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
-- --
-- different tree example -- different tree example
-- --

View File

@ -32,6 +32,8 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
-- --
-- different tree example -- different tree example
-- --

View File

@ -32,6 +32,7 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
-- --
-- test cycle detection -- test cycle detection
-- --

View File

@ -32,6 +32,8 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
-- --
-- test cycle detection -- test cycle detection
-- --

View File

@ -32,13 +32,14 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
-- --
-- test multiple WITH queries -- test multiple WITH queries
-- --
WITH RECURSIVE WITH RECURSIVE
y AS (SELECT 1 AS id), y AS (SELECT 1 AS id),
x AS (SELECT * FROM y UNION ALL SELECT id+1 FROM x WHERE id < 5) x AS (SELECT * FROM y UNION ALL SELECT id+1 FROM x WHERE id < 5)
SELECT * FROM x; SELECT * FROM x ORDER BY id;
1 1
2 2
3 3
@ -48,7 +49,7 @@ SELECT * FROM x;
WITH RECURSIVE WITH RECURSIVE
x AS (SELECT * FROM y UNION ALL SELECT id+1 FROM x WHERE id < 5), x AS (SELECT * FROM y UNION ALL SELECT id+1 FROM x WHERE id < 5),
y AS (SELECT 1 AS id) y AS (SELECT 1 AS id)
SELECT * FROM x; SELECT * FROM x ORDER BY id;
1 1
2 2
3 3
@ -59,7 +60,7 @@ WITH RECURSIVE
(SELECT 1 AS id UNION ALL SELECT id+1 FROM x WHERE id < 5), (SELECT 1 AS id UNION ALL SELECT id+1 FROM x WHERE id < 5),
y AS y AS
(SELECT 1 AS id UNION ALL SELECT id+1 FROM y WHERE id < 10) (SELECT 1 AS id UNION ALL SELECT id+1 FROM y WHERE id < 10)
SELECT y.*, x.* FROM y LEFT JOIN x USING (id); SELECT y.*, x.* FROM y LEFT JOIN x USING (id) ORDER BY y.id;
1 1 1 1
2 2 2 2
3 3 3 3
@ -75,7 +76,7 @@ WITH RECURSIVE
(SELECT 1 AS id UNION ALL SELECT id+1 FROM x WHERE id < 5), (SELECT 1 AS id UNION ALL SELECT id+1 FROM x WHERE id < 5),
y AS y AS
(SELECT 1 AS id UNION ALL SELECT id+1 FROM x WHERE id < 10) (SELECT 1 AS id UNION ALL SELECT id+1 FROM x WHERE id < 10)
SELECT y.*, x.* FROM y LEFT JOIN x USING (id); SELECT y.*, x.* FROM y LEFT JOIN x USING (id) ORDER BY y.id;
1 1 1 1
2 2 2 2
3 3 3 3
@ -89,32 +90,32 @@ WITH RECURSIVE
(SELECT * FROM x UNION ALL SELECT * FROM x), (SELECT * FROM x UNION ALL SELECT * FROM x),
z AS z AS
(SELECT * FROM x UNION ALL SELECT id+1 FROM z WHERE id < 10) (SELECT * FROM x UNION ALL SELECT id+1 FROM z WHERE id < 10)
SELECT * FROM z; SELECT * FROM z ORDER BY id;
1 1
2 2
3
2 2
3 3
4 3
3 3
4 4
5 4
4 4
5 5
6 5
5 5
6 6
7 6
6 6
7 7
8 7
7 7
8 8
9 8
8 8
9 9
10
9 9
9
10
10 10
10 10
WITH RECURSIVE WITH RECURSIVE
@ -124,58 +125,58 @@ WITH RECURSIVE
(SELECT * FROM x UNION ALL SELECT * FROM x), (SELECT * FROM x UNION ALL SELECT * FROM x),
z AS z AS
(SELECT * FROM y UNION ALL SELECT id+1 FROM z WHERE id < 10) (SELECT * FROM y UNION ALL SELECT id+1 FROM z WHERE id < 10)
SELECT * FROM z; SELECT * FROM z ORDER BY id;
1 1
1 1
2 2
2 2
3
3
2 2
2 2
3 3
3 3
4 3
4 3
3 3
3 3
4 4
4 4
5 4
5 4
4 4
4 4
5 5
5 5
6 5
6 5
5 5
5 5
6 6
6 6
7 6
7 6
6 6
6 6
7 7
7 7
8 7
8 7
7 7
7 7
8 8
8 8
9 8
9 8
8 8
8 8
9 9
9 9
10 9
10 9
9 9
9 9
10 10
10 10
10 10
10 10
10
10

View File

@ -1,3 +1,4 @@
/** /**
* Based on https://github.com/postgres/postgres/blob/master/src/test/regress/sql/with.sql, license: * Based on https://github.com/postgres/postgres/blob/master/src/test/regress/sql/with.sql, license:
* *
@ -32,33 +33,35 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
-- --
-- test multiple WITH queries -- test multiple WITH queries
-- --
WITH RECURSIVE WITH RECURSIVE
y AS (SELECT 1 AS id), y AS (SELECT 1 AS id),
x AS (SELECT * FROM y UNION ALL SELECT id+1 FROM x WHERE id < 5) x AS (SELECT * FROM y UNION ALL SELECT id+1 FROM x WHERE id < 5)
SELECT * FROM x; SELECT * FROM x ORDER BY id;
-- forward reference OK -- forward reference OK
WITH RECURSIVE WITH RECURSIVE
x AS (SELECT * FROM y UNION ALL SELECT id+1 FROM x WHERE id < 5), x AS (SELECT * FROM y UNION ALL SELECT id+1 FROM x WHERE id < 5),
y AS (SELECT 1 AS id) y AS (SELECT 1 AS id)
SELECT * FROM x; SELECT * FROM x ORDER BY id;
WITH RECURSIVE WITH RECURSIVE
x AS x AS
(SELECT 1 AS id UNION ALL SELECT id+1 FROM x WHERE id < 5), (SELECT 1 AS id UNION ALL SELECT id+1 FROM x WHERE id < 5),
y AS y AS
(SELECT 1 AS id UNION ALL SELECT id+1 FROM y WHERE id < 10) (SELECT 1 AS id UNION ALL SELECT id+1 FROM y WHERE id < 10)
SELECT y.*, x.* FROM y LEFT JOIN x USING (id); SELECT y.*, x.* FROM y LEFT JOIN x USING (id) ORDER BY y.id;
WITH RECURSIVE WITH RECURSIVE
x AS x AS
(SELECT 1 AS id UNION ALL SELECT id+1 FROM x WHERE id < 5), (SELECT 1 AS id UNION ALL SELECT id+1 FROM x WHERE id < 5),
y AS y AS
(SELECT 1 AS id UNION ALL SELECT id+1 FROM x WHERE id < 10) (SELECT 1 AS id UNION ALL SELECT id+1 FROM x WHERE id < 10)
SELECT y.*, x.* FROM y LEFT JOIN x USING (id); SELECT y.*, x.* FROM y LEFT JOIN x USING (id) ORDER BY y.id;
WITH RECURSIVE WITH RECURSIVE
x AS x AS
@ -67,7 +70,7 @@ WITH RECURSIVE
(SELECT * FROM x UNION ALL SELECT * FROM x), (SELECT * FROM x UNION ALL SELECT * FROM x),
z AS z AS
(SELECT * FROM x UNION ALL SELECT id+1 FROM z WHERE id < 10) (SELECT * FROM x UNION ALL SELECT id+1 FROM z WHERE id < 10)
SELECT * FROM z; SELECT * FROM z ORDER BY id;
WITH RECURSIVE WITH RECURSIVE
x AS x AS
@ -76,6 +79,6 @@ WITH RECURSIVE
(SELECT * FROM x UNION ALL SELECT * FROM x), (SELECT * FROM x UNION ALL SELECT * FROM x),
z AS z AS
(SELECT * FROM y UNION ALL SELECT id+1 FROM z WHERE id < 10) (SELECT * FROM y UNION ALL SELECT id+1 FROM z WHERE id < 10)
SELECT * FROM z; SELECT * FROM z ORDER BY id;
-- { echoOff } -- { echoOff }

View File

@ -32,6 +32,7 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
-- --
-- error cases -- error cases
-- --

View File

@ -32,6 +32,8 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
-- --
-- error cases -- error cases
-- --

View File

@ -32,6 +32,7 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
WITH RECURSIVE foo AS WITH RECURSIVE foo AS
(SELECT 1 AS i (SELECT 1 AS i
UNION ALL UNION ALL

View File

@ -32,6 +32,8 @@
-- { echoOn } -- { echoOn }
SET allow_experimental_analyzer = 1;
WITH RECURSIVE foo AS WITH RECURSIVE foo AS
(SELECT 1 AS i (SELECT 1 AS i
UNION ALL UNION ALL