ClickHouse/src/Interpreters/InterpreterSelectQuery.h

Ignoring revisions in .git-blame-ignore-revs. Click here to bypass and see the normal blame view.

265 lines
10 KiB
C++
Raw Normal View History

2011-08-28 05:13:24 +00:00
#pragma once
#include <memory>
#include <optional>
2022-08-18 07:47:29 +00:00
#include <Access/EnabledRowPolicies.h>
#include <Core/QueryProcessingStage.h>
#include <Interpreters/ExpressionActions.h>
2018-02-23 06:00:48 +00:00
#include <Interpreters/ExpressionAnalyzer.h>
2020-11-01 13:54:07 +00:00
#include <Interpreters/IInterpreterUnionOrSelectQuery.h>
2021-10-29 11:51:41 +00:00
#include <Interpreters/PreparedSets.h>
2020-11-01 13:54:07 +00:00
#include <Interpreters/StorageID.h>
#include <Parsers/ASTSelectQuery.h>
#include <Storages/ReadInOrderOptimizer.h>
#include <Storages/SelectQueryInfo.h>
2020-06-22 09:49:21 +00:00
#include <Storages/TableLockHolder.h>
2022-05-20 19:49:31 +00:00
#include <QueryPipeline/Pipe.h>
2011-08-28 05:13:24 +00:00
2019-10-27 18:12:40 +00:00
#include <Columns/FilterDescription.h>
namespace Poco
{
class Logger;
}
2011-08-28 05:13:24 +00:00
namespace DB
{
2022-08-18 07:47:29 +00:00
class SubqueryForSet;
class InterpreterSelectWithUnionQuery;
class Context;
class QueryPlan;
2015-05-06 23:35:37 +00:00
2022-05-10 16:12:03 +00:00
struct GroupingSetsParams;
using GroupingSetsParamsList = std::vector<GroupingSetsParams>;
struct TreeRewriterResult;
using TreeRewriterResultPtr = std::shared_ptr<const TreeRewriterResult>;
2015-05-06 23:35:37 +00:00
2022-08-18 07:47:29 +00:00
struct RowPolicy;
using RowPolicyPtr = std::shared_ptr<const RowPolicy>;
2017-06-02 21:37:28 +00:00
/** Interprets the SELECT query. Returns the stream of blocks with the results of the query before `to_stage` stage.
2011-08-28 05:13:24 +00:00
*/
2020-11-01 13:54:07 +00:00
class InterpreterSelectQuery : public IInterpreterUnionOrSelectQuery
2011-08-28 05:13:24 +00:00
{
public:
/**
* query_ptr
* - A query AST to interpret.
*
2018-02-27 19:00:55 +00:00
* required_result_column_names
2018-03-02 04:05:20 +00:00
* - don't calculate all columns except the specified ones from the query
* - it is used to remove calculation (and reading) of unnecessary columns from subqueries.
* empty means - use all columns.
*/
InterpreterSelectQuery(
2017-05-23 18:24:43 +00:00
const ASTPtr & query_ptr_,
const ContextPtr & context_,
const SelectQueryOptions &,
2019-08-03 11:02:40 +00:00
const Names & required_result_column_names_ = Names{});
InterpreterSelectQuery(
const ASTPtr & query_ptr_,
const ContextMutablePtr & context_,
const SelectQueryOptions &,
const Names & required_result_column_names_ = Names{});
/// Read data not from the table specified in the query, but from the prepared pipe `input`.
InterpreterSelectQuery(
const ASTPtr & query_ptr_,
const ContextPtr & context_,
Pipe input_pipe_,
const SelectQueryOptions & = {});
/// Read data not from the table specified in the query, but from the specified `storage_`.
InterpreterSelectQuery(
const ASTPtr & query_ptr_,
const ContextPtr & context_,
const StoragePtr & storage_,
2020-06-17 11:52:19 +00:00
const StorageMetadataPtr & metadata_snapshot_ = nullptr,
const SelectQueryOptions & = {});
2022-07-18 15:53:30 +00:00
/// Reuse existing prepared_sets for another pass of analysis. It's used for projection.
/// TODO: Find a general way of sharing sets among different interpreters, such as subqueries.
2021-10-29 11:51:41 +00:00
InterpreterSelectQuery(
const ASTPtr & query_ptr_,
const ContextPtr & context_,
2021-10-29 11:51:41 +00:00
const SelectQueryOptions &,
2022-07-18 15:53:30 +00:00
PreparedSetsPtr prepared_sets_);
2021-10-29 11:51:41 +00:00
~InterpreterSelectQuery() override;
2018-02-25 00:50:53 +00:00
/// Execute a query. Get the stream of blocks to read.
2015-06-18 02:11:05 +00:00
BlockIO execute() override;
/// Builds QueryPlan for current query.
void buildQueryPlan(QueryPlan & query_plan) override;
bool ignoreLimits() const override { return options.ignore_limits; }
bool ignoreQuota() const override { return options.ignore_quota; }
void ignoreWithTotals() override;
ASTPtr getQuery() const { return query_ptr; }
const SelectQueryInfo & getQueryInfo() const { return query_info; }
2021-10-29 11:51:41 +00:00
SelectQueryExpressionAnalyzer * getQueryAnalyzer() const { return query_analyzer.get(); }
2021-04-21 16:00:27 +00:00
const ExpressionAnalysisResult & getAnalysisResult() const { return analysis_result; }
const Names & getRequiredColumns() const { return required_columns; }
bool hasAggregation() const { return query_analyzer->hasAggregation(); }
static void addEmptySourceToQueryPlan(
QueryPlan & query_plan, const Block & source_header, const SelectQueryInfo & query_info, const ContextPtr & context_);
2020-09-25 13:19:26 +00:00
Names getRequiredColumns() { return required_columns; }
bool supportsTransactions() const override { return true; }
2022-07-28 09:40:09 +00:00
FilterDAGInfoPtr getAdditionalQueryInfo() const { return additional_filter_info; }
RowPolicyFilterPtr getRowPolicyFilter() const;
2022-08-13 13:03:16 +00:00
void extendQueryLogElemImpl(QueryLogElement & elem, const ASTPtr & ast, ContextPtr context) const override;
static SortDescription getSortDescription(const ASTSelectQuery & query, const ContextPtr & context);
static UInt64 getLimitForSorting(const ASTSelectQuery & query, const ContextPtr & context);
2021-11-02 15:40:37 +00:00
2023-05-25 12:41:04 +00:00
static bool isQueryWithFinal(const SelectQueryInfo & info);
static std::pair<UInt64, UInt64> getLimitLengthAndOffset(const ASTSelectQuery & query, const ContextPtr & context);
2023-07-11 11:57:19 +00:00
/// Adjust the parallel replicas settings (enabled, disabled) based on the query analysis
bool adjustParallelReplicasAfterAnalysis();
2011-08-28 05:13:24 +00:00
private:
InterpreterSelectQuery(
const ASTPtr & query_ptr_,
const ContextPtr & context_,
std::optional<Pipe> input_pipe,
const StoragePtr & storage_,
const SelectQueryOptions &,
2020-06-17 11:52:19 +00:00
const Names & required_result_column_names = {},
2021-10-29 11:51:41 +00:00
const StorageMetadataPtr & metadata_snapshot_ = nullptr,
2022-07-18 15:53:30 +00:00
PreparedSetsPtr prepared_sets_ = nullptr);
InterpreterSelectQuery(
const ASTPtr & query_ptr_,
const ContextMutablePtr & context_,
std::optional<Pipe> input_pipe,
const StoragePtr & storage_,
const SelectQueryOptions &,
const Names & required_result_column_names = {},
const StorageMetadataPtr & metadata_snapshot_ = nullptr,
2022-07-18 15:53:30 +00:00
PreparedSetsPtr prepared_sets_ = nullptr);
ASTSelectQuery & getSelectQuery() { return query_ptr->as<ASTSelectQuery &>(); }
2019-03-12 14:07:02 +00:00
void addPrewhereAliasActions();
2023-07-06 08:08:27 +00:00
void applyFiltersToPrewhereInAnalysis(ExpressionAnalysisResult & analysis) const;
bool shouldMoveToPrewhere() const;
Block getSampleBlockImpl();
2021-10-13 18:22:02 +00:00
void executeImpl(QueryPlan & query_plan, std::optional<Pipe> prepared_pipe);
2018-02-23 06:00:48 +00:00
2017-06-02 21:37:28 +00:00
/// Different stages of query execution.
2021-02-15 19:48:06 +00:00
void executeFetchColumns(QueryProcessingStage::Enum processing_stage, QueryPlan & query_plan);
2020-11-03 11:28:28 +00:00
void executeWhere(QueryPlan & query_plan, const ActionsDAGPtr & expression, bool remove_filter);
void executeAggregation(
QueryPlan & query_plan, const ActionsDAGPtr & expression, bool overflow_row, bool final, InputOrderInfoPtr group_by_info);
2022-05-05 13:56:16 +00:00
void executeMergeAggregated(QueryPlan & query_plan, bool overflow_row, bool final, bool has_grouping_sets);
void executeTotalsAndHaving(QueryPlan & query_plan, bool has_having, const ActionsDAGPtr & expression, bool remove_filter, bool overflow_row, bool final);
void executeHaving(QueryPlan & query_plan, const ActionsDAGPtr & expression, bool remove_filter);
2020-11-03 11:28:28 +00:00
static void executeExpression(QueryPlan & query_plan, const ActionsDAGPtr & expression, const std::string & description);
/// FIXME should go through ActionsDAG to behave as a proper function
void executeWindow(QueryPlan & query_plan);
void executeOrder(QueryPlan & query_plan, InputOrderInfoPtr sorting_info);
void executeOrderOptimized(QueryPlan & query_plan, InputOrderInfoPtr sorting_info, UInt64 limit, SortDescription & output_order_descr);
void executeWithFill(QueryPlan & query_plan);
void executeMergeSorted(QueryPlan & query_plan, const std::string & description);
void executePreLimit(QueryPlan & query_plan, bool do_not_skip_offset);
void executeLimitBy(QueryPlan & query_plan);
void executeLimit(QueryPlan & query_plan);
void executeOffset(QueryPlan & query_plan);
2020-11-03 11:28:28 +00:00
static void executeProjection(QueryPlan & query_plan, const ActionsDAGPtr & expression);
void executeDistinct(QueryPlan & query_plan, bool before_order, Names columns, bool pre_distinct);
void executeExtremes(QueryPlan & query_plan);
void executeSubqueriesInSetsAndJoins(QueryPlan & query_plan);
bool autoFinalOnQuery(ASTSelectQuery & select_query);
std::optional<UInt64> getTrivialCount(UInt64 max_parallel_replicas);
2023-07-05 14:05:40 +00:00
/// Check if we can limit block size to read based on LIMIT clause
UInt64 maxBlockSizeByLimit() const;
2019-03-26 18:28:37 +00:00
2018-09-20 17:51:42 +00:00
enum class Modificator
{
ROLLUP = 0,
CUBE = 1,
2018-09-20 17:51:42 +00:00
};
2018-09-20 20:57:06 +00:00
void executeRollupOrCube(QueryPlan & query_plan, Modificator modificator);
2019-03-26 18:28:37 +00:00
2017-06-02 21:37:28 +00:00
/** If there is a SETTINGS section in the SELECT query, then apply settings from it.
*
2017-06-02 21:37:28 +00:00
* Section SETTINGS - settings for a specific query.
* Normally, the settings can be passed in other ways, not inside the query.
* But the use of this section is justified if you need to set the settings for one subquery.
*/
void initSettings();
TreeRewriterResultPtr syntax_analyzer_result;
std::unique_ptr<SelectQueryExpressionAnalyzer> query_analyzer;
2019-07-19 10:14:27 +00:00
SelectQueryInfo query_info;
/// Is calculated in getSampleBlock. Is used later in readImpl.
ExpressionAnalysisResult analysis_result;
/// For row-level security.
RowPolicyFilterPtr row_policy_filter;
FilterDAGInfoPtr filter_info;
2022-06-21 11:24:46 +00:00
/// For additional_filter setting.
FilterDAGInfoPtr additional_filter_info;
2023-01-11 09:57:13 +00:00
2023-01-11 10:42:01 +00:00
/// For "per replica" filter when multiple replicas are used
FilterDAGInfoPtr parallel_replicas_custom_filter_info;
2022-06-21 11:24:46 +00:00
2019-10-03 15:47:42 +00:00
QueryProcessingStage::Enum from_stage = QueryProcessingStage::FetchColumns;
/// List of columns to read to execute the query.
Names required_columns;
/// Structure of query source (table, subquery, etc).
Block source_header;
/// Actions to calculate ALIAS if required.
ActionsDAGPtr alias_actions;
/// The subquery interpreter, if the subquery
std::unique_ptr<InterpreterSelectWithUnionQuery> interpreter_subquery;
2017-06-02 21:37:28 +00:00
/// Table from where to read data, if not subquery.
StoragePtr storage;
StorageID table_id = StorageID::createEmpty(); /// Will be initialized if storage is not nullptr
2020-06-18 16:10:47 +00:00
TableLockHolder table_lock;
2018-02-21 03:26:06 +00:00
/// Used when we read from prepared input, not table or subquery.
std::optional<Pipe> input_pipe;
2018-02-21 03:26:06 +00:00
Poco::Logger * log;
2020-06-17 11:52:19 +00:00
StorageMetadataPtr metadata_snapshot;
StorageSnapshotPtr storage_snapshot;
2021-10-29 11:51:41 +00:00
/// Reuse already built sets for multiple passes of analysis, possibly across interpreters.
2022-07-18 15:53:30 +00:00
PreparedSetsPtr prepared_sets;
2011-08-28 05:13:24 +00:00
};
}