Make metadata single structure

This commit is contained in:
alesapin 2020-06-05 20:29:40 +03:00
parent 2b23d1aa33
commit abaf47f0cd
24 changed files with 576 additions and 389 deletions

View File

@ -253,9 +253,9 @@ void DatabaseOrdinary::alterTable(
ast_create_query.columns_list->setOrReplace(ast_create_query.columns_list->indices, new_indices);
ast_create_query.columns_list->setOrReplace(ast_create_query.columns_list->constraints, new_constraints);
if (metadata.select)
if (metadata.select.select_query)
{
ast->replace(ast_create_query.select, metadata.select);
ast->replace(ast_create_query.select, metadata.select.select_query);
}
/// MaterializedView is one type of CREATE query without storage.
@ -263,17 +263,17 @@ void DatabaseOrdinary::alterTable(
{
ASTStorage & storage_ast = *ast_create_query.storage;
/// ORDER BY may change, but cannot appear, it's required construction
if (metadata.order_by_ast && storage_ast.order_by)
storage_ast.set(storage_ast.order_by, metadata.order_by_ast);
if (metadata.sorting_key.definition_ast && storage_ast.order_by)
storage_ast.set(storage_ast.order_by, metadata.sorting_key.definition_ast);
if (metadata.primary_key_ast)
storage_ast.set(storage_ast.primary_key, metadata.primary_key_ast);
if (metadata.primary_key.definition_ast)
storage_ast.set(storage_ast.primary_key, metadata.primary_key.definition_ast);
if (metadata.ttl_for_table_ast)
storage_ast.set(storage_ast.ttl_table, metadata.ttl_for_table_ast);
if (metadata.table_ttl.definition_ast)
storage_ast.set(storage_ast.ttl_table, metadata.table_ttl.definition_ast);
if (metadata.settings_ast)
storage_ast.set(storage_ast.settings, metadata.settings_ast);
if (metadata.settings_changes)
storage_ast.set(storage_ast.settings, metadata.settings_changes);
}
statement = getObjectDefinitionFromCreateQuery(ast);

View File

@ -316,14 +316,14 @@ void AlterCommand::apply(StorageInMemoryMetadata & metadata, const Context & con
}
else if (type == MODIFY_ORDER_BY)
{
if (!metadata.primary_key_ast && metadata.order_by_ast)
if (metadata.primary_key.definition_ast == nullptr && metadata.sorting_key.definition_ast != nullptr)
{
/// Primary and sorting key become independent after this ALTER so we have to
/// save the old ORDER BY expression as the new primary key.
metadata.primary_key_ast = metadata.order_by_ast->clone();
metadata.primary_key = metadata.sorting_key;
}
metadata.order_by_ast = order_by;
metadata.sorting_key = KeyDescription::getKeyFromAST(order_by, metadata.columns, context);
}
else if (type == COMMENT_COLUMN)
{
@ -430,15 +430,15 @@ void AlterCommand::apply(StorageInMemoryMetadata & metadata, const Context & con
}
else if (type == MODIFY_TTL)
{
metadata.ttl_for_table_ast = ttl;
metadata.table_ttl = TTLTableDescription::getTTLForTableFromAST(ttl, metadata.columns, context, metadata.primary_key);
}
else if (type == MODIFY_QUERY)
{
metadata.select = select;
metadata.select = SelectQueryDescription::getSelectQueryFromASTForMatView(select, context);
}
else if (type == MODIFY_SETTING)
{
auto & settings_from_storage = metadata.settings_ast->as<ASTSetQuery &>().changes;
auto & settings_from_storage = metadata.settings_changes->as<ASTSetQuery &>().changes;
for (const auto & change : settings_changes)
{
auto finder = [&change](const SettingChange & c) { return c.name == change.name; };
@ -465,8 +465,11 @@ void AlterCommand::apply(StorageInMemoryMetadata & metadata, const Context & con
rename_visitor.visit(column_to_modify.ttl);
});
}
if (metadata.ttl_for_table_ast)
rename_visitor.visit(metadata.ttl_for_table_ast);
if (metadata.table_ttl.definition_ast)
rename_visitor.visit(metadata.table_ttl.definition_ast);
metadata.table_ttl = TTLTableDescription::getTTLForTableFromAST(
metadata.table_ttl.definition_ast, metadata.columns, context, metadata.primary_key);
for (auto & constraint : metadata.constraints.constraints)
rename_visitor.visit(constraint);
@ -832,7 +835,7 @@ void AlterCommands::validate(const StorageInMemoryMetadata & metadata, const Con
}
else if (command.type == AlterCommand::MODIFY_SETTING)
{
if (metadata.settings_ast == nullptr)
if (metadata.settings_changes == nullptr)
throw Exception{"Cannot alter settings, because table engine doesn't support settings changes", ErrorCodes::BAD_ARGUMENTS};
}
else if (command.type == AlterCommand::RENAME_COLUMN)

View File

@ -54,4 +54,19 @@ ConstraintsExpressions ConstraintsDescription::getExpressions(const DB::Context
return res;
}
ConstraintsDescription::ConstraintsDescription(const ConstraintsDescription & other)
{
constraints.reserve(other.constraints.size());
for (const auto & constraint : other.constraints)
constraints.emplace_back(constraint->clone());
}
ConstraintsDescription & ConstraintsDescription::operator=(const ConstraintsDescription & other)
{
constraints.resize(other.constraints.size());
for (size_t i = 0; i < constraints.size(); ++i)
constraints[i] = other.constraints[i]->clone();
return *this;
}
}

View File

@ -20,6 +20,9 @@ struct ConstraintsDescription
static ConstraintsDescription parse(const String & str);
ConstraintsExpressions getExpressions(const Context & context, const NamesAndTypesList & source_columns_) const;
ConstraintsDescription(const ConstraintsDescription & other);
ConstraintsDescription & operator=(const ConstraintsDescription & other);
};
}

View File

@ -34,23 +34,23 @@ namespace ErrorCodes
const ColumnsDescription & IStorage::getColumns() const
{
return columns;
return metadata.columns;
}
const IndicesDescription & IStorage::getSecondaryIndices() const
{
return secondary_indices;
return metadata.secondary_indices;
}
bool IStorage::hasSecondaryIndices() const
{
return !secondary_indices.empty();
return !metadata.secondary_indices.empty();
}
const ConstraintsDescription & IStorage::getConstraints() const
{
return constraints;
return metadata.constraints;
}
Block IStorage::getSampleBlock() const
@ -292,17 +292,17 @@ void IStorage::setColumns(ColumnsDescription columns_)
{
if (columns_.getOrdinary().empty())
throw Exception("Empty list of columns passed", ErrorCodes::EMPTY_LIST_OF_COLUMNS_PASSED);
columns = std::move(columns_);
metadata.columns = std::move(columns_);
}
void IStorage::setSecondaryIndices(IndicesDescription secondary_indices_)
{
secondary_indices = std::move(secondary_indices_);
metadata.secondary_indices = std::move(secondary_indices_);
}
void IStorage::setConstraints(ConstraintsDescription constraints_)
{
constraints = std::move(constraints_);
metadata.constraints = std::move(constraints_);
}
bool IStorage::isVirtualColumn(const String & column_name) const
@ -373,11 +373,6 @@ TableStructureWriteLockHolder IStorage::lockExclusively(const String & query_id,
return result;
}
StorageInMemoryMetadata IStorage::getInMemoryMetadata() const
{
return StorageInMemoryMetadata(getColumns(), getSecondaryIndices(), getConstraints());
}
void IStorage::alter(
const AlterCommands & params,
const Context & context,
@ -385,8 +380,8 @@ void IStorage::alter(
{
lockStructureExclusively(table_lock_holder, context.getCurrentQueryId(), context.getSettingsRef().lock_acquire_timeout);
auto table_id = getStorageID();
StorageInMemoryMetadata metadata = getInMemoryMetadata();
params.apply(metadata, context);
StorageInMemoryMetadata old_metadata = getInMemoryMetadata();
params.apply(old_metadata, context);
DatabaseCatalog::instance().getDatabase(table_id.database_name)->alterTable(context, table_id, metadata);
setColumns(std::move(metadata.columns));
}
@ -423,135 +418,135 @@ NamesAndTypesList IStorage::getVirtuals() const
const KeyDescription & IStorage::getPartitionKey() const
{
return partition_key;
return metadata.partition_key;
}
void IStorage::setPartitionKey(const KeyDescription & partition_key_)
{
partition_key = partition_key_;
metadata.partition_key = partition_key_;
}
bool IStorage::isPartitionKeyDefined() const
{
return partition_key.definition_ast != nullptr;
return metadata.partition_key.definition_ast != nullptr;
}
bool IStorage::hasPartitionKey() const
{
return !partition_key.column_names.empty();
return !metadata.partition_key.column_names.empty();
}
Names IStorage::getColumnsRequiredForPartitionKey() const
{
if (hasPartitionKey())
return partition_key.expression->getRequiredColumns();
return metadata.partition_key.expression->getRequiredColumns();
return {};
}
const KeyDescription & IStorage::getSortingKey() const
{
return sorting_key;
return metadata.sorting_key;
}
void IStorage::setSortingKey(const KeyDescription & sorting_key_)
{
sorting_key = sorting_key_;
metadata.sorting_key = sorting_key_;
}
bool IStorage::isSortingKeyDefined() const
{
return sorting_key.definition_ast != nullptr;
return metadata.sorting_key.definition_ast != nullptr;
}
bool IStorage::hasSortingKey() const
{
return !sorting_key.column_names.empty();
return !metadata.sorting_key.column_names.empty();
}
Names IStorage::getColumnsRequiredForSortingKey() const
{
if (hasSortingKey())
return sorting_key.expression->getRequiredColumns();
return metadata.sorting_key.expression->getRequiredColumns();
return {};
}
Names IStorage::getSortingKeyColumns() const
{
if (hasSortingKey())
return sorting_key.column_names;
return metadata.sorting_key.column_names;
return {};
}
const KeyDescription & IStorage::getPrimaryKey() const
{
return primary_key;
return metadata.primary_key;
}
void IStorage::setPrimaryKey(const KeyDescription & primary_key_)
{
primary_key = primary_key_;
metadata.primary_key = primary_key_;
}
bool IStorage::isPrimaryKeyDefined() const
{
return primary_key.definition_ast != nullptr;
return metadata.primary_key.definition_ast != nullptr;
}
bool IStorage::hasPrimaryKey() const
{
return !primary_key.column_names.empty();
return !metadata.primary_key.column_names.empty();
}
Names IStorage::getColumnsRequiredForPrimaryKey() const
{
if (hasPrimaryKey())
return primary_key.expression->getRequiredColumns();
return metadata.primary_key.expression->getRequiredColumns();
return {};
}
Names IStorage::getPrimaryKeyColumns() const
{
if (hasSortingKey())
return primary_key.column_names;
return metadata.primary_key.column_names;
return {};
}
const KeyDescription & IStorage::getSamplingKey() const
{
return sampling_key;
return metadata.sampling_key;
}
void IStorage::setSamplingKey(const KeyDescription & sampling_key_)
{
sampling_key = sampling_key_;
metadata.sampling_key = sampling_key_;
}
bool IStorage::isSamplingKeyDefined() const
{
return sampling_key.definition_ast != nullptr;
return metadata.sampling_key.definition_ast != nullptr;
}
bool IStorage::hasSamplingKey() const
{
return !sampling_key.column_names.empty();
return !metadata.sampling_key.column_names.empty();
}
Names IStorage::getColumnsRequiredForSampling() const
{
if (hasSamplingKey())
return sampling_key.expression->getRequiredColumns();
return metadata.sampling_key.expression->getRequiredColumns();
return {};
}
const TTLTableDescription & IStorage::getTableTTLs() const
{
return table_ttl;
return metadata.table_ttl;
}
void IStorage::setTableTTLs(const TTLTableDescription & table_ttl_)
{
table_ttl = table_ttl_;
metadata.table_ttl = table_ttl_;
}
bool IStorage::hasAnyTableTTL() const
@ -561,37 +556,37 @@ bool IStorage::hasAnyTableTTL() const
const TTLColumnsDescription & IStorage::getColumnTTLs() const
{
return column_ttls_by_name;
return metadata.column_ttls_by_name;
}
void IStorage::setColumnTTLs(const TTLColumnsDescription & column_ttls_by_name_)
{
column_ttls_by_name = column_ttls_by_name_;
metadata.column_ttls_by_name = column_ttls_by_name_;
}
bool IStorage::hasAnyColumnTTL() const
{
return !column_ttls_by_name.empty();
return !metadata.column_ttls_by_name.empty();
}
const TTLDescription & IStorage::getRowsTTL() const
{
return table_ttl.rows_ttl;
return metadata.table_ttl.rows_ttl;
}
bool IStorage::hasRowsTTL() const
{
return table_ttl.rows_ttl.expression != nullptr;
return metadata.table_ttl.rows_ttl.expression != nullptr;
}
const TTLDescriptions & IStorage::getMoveTTLs() const
{
return table_ttl.move_ttl;
return metadata.table_ttl.move_ttl;
}
bool IStorage::hasAnyMoveTTL() const
{
return !table_ttl.move_ttl.empty();
return !metadata.table_ttl.move_ttl.empty();
}
@ -656,30 +651,30 @@ ColumnDependencies IStorage::getColumnDependencies(const NameSet & updated_colum
const ASTPtr & IStorage::getSettingsChanges() const
{
return settings_changes;
return metadata.settings_changes;
}
void IStorage::setSettingsChanges(const ASTPtr & settings_changes_)
{
if (settings_changes_)
settings_changes = settings_changes_->clone();
metadata.settings_changes = settings_changes_->clone();
else
settings_changes = nullptr;
metadata.settings_changes = nullptr;
}
const SelectQueryDescription & IStorage::getSelectQuery() const
{
return select;
return metadata.select;
}
void IStorage::setSelectQuery(const SelectQueryDescription & select_)
{
select = select_;
metadata.select = select_;
}
bool IStorage::hasSelectQuery() const
{
return select.select_query != nullptr;
return metadata.select.select_query != nullptr;
}
}

View File

@ -10,12 +10,7 @@
#include <Storages/SelectQueryInfo.h>
#include <Storages/TableStructureLockHolder.h>
#include <Storages/CheckResults.h>
#include <Storages/ColumnsDescription.h>
#include <Storages/IndicesDescription.h>
#include <Storages/ConstraintsDescription.h>
#include <Storages/StorageInMemoryMetadata.h>
#include <Storages/TTLDescription.h>
#include <Storages/KeyDescription.h>
#include <Storages/ColumnDependency.h>
#include <Storages/SelectQueryDescription.h>
#include <Common/ActionLock.h>
@ -157,15 +152,13 @@ public: /// thread-unsafe part. lockStructure must be acquired
/// Storage settings
const ASTPtr & getSettingsChanges() const;
void setSettingsChanges(const ASTPtr & settings_changes_);
bool hasSettingsChanges() const { return settings_changes != nullptr; }
bool hasSettingsChanges() const { return metadata.settings_changes != nullptr; }
const SelectQueryDescription & getSelectQuery() const;
void setSelectQuery(const SelectQueryDescription & select_);
bool hasSelectQuery() const;
/// Returns storage metadata copy. Direct modification of
/// result structure doesn't affect storage.
virtual StorageInMemoryMetadata getInMemoryMetadata() const;
const StorageInMemoryMetadata & getInMemoryMetadata() const { return metadata; }
Block getSampleBlock() const; /// ordinary + materialized.
Block getSampleBlockWithVirtuals() const; /// ordinary + materialized + virtuals.
@ -210,21 +203,8 @@ private:
StorageID storage_id;
mutable std::mutex id_mutex;
ColumnsDescription columns;
IndicesDescription secondary_indices;
ConstraintsDescription constraints;
KeyDescription partition_key;
KeyDescription primary_key;
KeyDescription sorting_key;
KeyDescription sampling_key;
TTLColumnsDescription column_ttls_by_name;
TTLTableDescription table_ttl;
ASTPtr settings_changes;
SelectQueryDescription select;
StorageInMemoryMetadata metadata;
private:
RWLockImpl::LockHolder tryLockTimed(
const RWLock & rwlock, RWLockImpl::Type type, const String & query_id, const SettingSeconds & acquire_timeout) const;
@ -462,7 +442,7 @@ public:
/// struct).
void setPartitionKey(const KeyDescription & partition_key_);
/// Returns ASTExpressionList of partition key expression for storage or nullptr if there is none.
ASTPtr getPartitionKeyAST() const { return partition_key.definition_ast; }
ASTPtr getPartitionKeyAST() const { return metadata.partition_key.definition_ast; }
/// Storage has user-defined (in CREATE query) partition key.
bool isPartitionKeyDefined() const;
/// Storage has partition key.
@ -477,7 +457,7 @@ public:
/// struct).
void setSortingKey(const KeyDescription & sorting_key_);
/// Returns ASTExpressionList of sorting key expression for storage or nullptr if there is none.
ASTPtr getSortingKeyAST() const { return sorting_key.definition_ast; }
ASTPtr getSortingKeyAST() const { return metadata.sorting_key.definition_ast; }
/// Storage has user-defined (in CREATE query) sorting key.
bool isSortingKeyDefined() const;
/// Storage has sorting key. It means, that it contains at least one column.
@ -494,7 +474,7 @@ public:
/// struct).
void setPrimaryKey(const KeyDescription & primary_key_);
/// Returns ASTExpressionList of primary key expression for storage or nullptr if there is none.
ASTPtr getPrimaryKeyAST() const { return primary_key.definition_ast; }
ASTPtr getPrimaryKeyAST() const { return metadata.primary_key.definition_ast; }
/// Storage has user-defined (in CREATE query) sorting key.
bool isPrimaryKeyDefined() const;
/// Storage has primary key (maybe part of some other key). It means, that
@ -512,7 +492,7 @@ public:
/// struct).
void setSamplingKey(const KeyDescription & sampling_key_);
/// Returns sampling expression AST for storage or nullptr if there is none.
ASTPtr getSamplingKeyAST() const { return sampling_key.definition_ast; }
ASTPtr getSamplingKeyAST() const { return metadata.sampling_key.definition_ast; }
/// Storage has user-defined (in CREATE query) sampling key.
bool isSamplingKeyDefined() const;
/// Storage has sampling key.

View File

@ -19,6 +19,43 @@ namespace ErrorCodes
extern const int LOGICAL_ERROR;
};
IndexDescription::IndexDescription(const IndexDescription & other)
: definition_ast(other.definition_ast ? other.definition_ast->clone() : nullptr)
, expression_list_ast(other.expression_list_ast ? other.expression_list_ast->clone() : nullptr)
, name(other.name)
, type(other.type)
, expression(other.expression) /// actions never changed
, arguments(other.arguments)
, column_names(other.column_names)
, data_types(other.data_types)
, sample_block(other.sample_block)
, granularity(other.granularity)
{
}
IndexDescription & IndexDescription::operator=(const IndexDescription & other)
{
if (other.definition_ast)
definition_ast = other.definition_ast->clone();
else
definition_ast.reset();
if (other.expression_list_ast)
expression_list_ast = other.expression_list_ast->clone();
else
expression_list_ast.reset();
name = other.name;
type = other.type;
expression = other.expression;
arguments = other.arguments;
column_names = other.column_names;
data_types = other.data_types;
sample_block = other.sample_block;
granularity = other.granularity;
return *this;
}
IndexDescription IndexDescription::getIndexFromAST(const ASTPtr & definition_ast, const ColumnsDescription & columns, const Context & context)
{

View File

@ -48,6 +48,13 @@ struct IndexDescription
/// Parse index from definition AST
static IndexDescription getIndexFromAST(const ASTPtr & definition_ast, const ColumnsDescription & columns, const Context & context);
IndexDescription() = default;
/// We need custom copy constructors because we don't want
/// unintentionaly share AST variables and modify them.
IndexDescription(const IndexDescription & other);
IndexDescription & operator=(const IndexDescription & other);
};
/// All secondary indices in storage

View File

@ -0,0 +1,69 @@
#include <Storages/KeyDescription.h>
#include <Functions/IFunction.h>
#include <Interpreters/ExpressionActions.h>
#include <Interpreters/ExpressionAnalyzer.h>
#include <Interpreters/SyntaxAnalyzer.h>
#include <Storages/extractKeyExpressionList.h>
namespace DB
{
KeyDescription::KeyDescription(const KeyDescription & other)
: definition_ast(other.definition_ast ? other.definition_ast->clone() : nullptr)
, expression_list_ast(other.expression_list_ast ? other.expression_list_ast->clone() : nullptr)
, expression(other.expression)
, sample_block(other.sample_block)
, column_names(other.column_names)
, data_types(other.data_types)
{
}
KeyDescription & KeyDescription::operator=(const KeyDescription & other)
{
if (other.definition_ast)
definition_ast = other.definition_ast->clone();
else
definition_ast.reset();
if (other.expression_list_ast)
expression_list_ast = other.expression_list_ast->clone();
expression = other.expression;
sample_block = other.sample_block;
column_names = other.column_names;
data_types = other.data_types;
return *this;
}
KeyDescription KeyDescription::getKeyFromAST(const ASTPtr & definition_ast, const ColumnsDescription & columns, const Context & context, ASTPtr additional_key_expression)
{
KeyDescription result;
result.definition_ast = definition_ast;
result.expression_list_ast = extractKeyExpressionList(definition_ast);
if (additional_key_expression)
result.expression_list_ast->children.push_back(additional_key_expression);
if (result.expression_list_ast->children.empty())
return result;
const auto & children = result.expression_list_ast->children;
for (const auto & child : children)
result.column_names.emplace_back(child->getColumnName());
{
auto expr = result.expression_list_ast->clone();
auto syntax_result = SyntaxAnalyzer(context).analyze(expr, columns.getAllPhysical());
result.expression = ExpressionAnalyzer(expr, syntax_result, context).getActions(true);
result.sample_block = result.expression->getSampleBlock();
}
for (size_t i = 0; i < result.sample_block.columns(); ++i)
result.data_types.emplace_back(result.sample_block.getByPosition(i).type);
return result;
}
}

View File

@ -0,0 +1,45 @@
#pragma once
#include <Interpreters/ExpressionActions.h>
#include <Parsers/IAST_fwd.h>
#include <Storages/ColumnsDescription.h>
namespace DB
{
/// Common structure for primary, partition and other storage keys
struct KeyDescription
{
/// User defined AST in CREATE/ALTER query. This field may be empty, but key
/// can exists because some of them maybe set implicitly (for example,
/// primary key in merge tree can be part of sorting key)
ASTPtr definition_ast;
/// ASTExpressionList with key fields, example: (x, toStartOfMonth(date))).
ASTPtr expression_list_ast;
/// Expression from expression_list_ast created by ExpressionAnalyzer. Useful,
/// when you need to get required columns for key, example: a, date, b.
ExpressionActionsPtr expression;
/// Sample block with key columns (names, types, empty column)
Block sample_block;
/// Column names in key definition, example: x, toStartOfMonth(date), a * b.
Names column_names;
/// Types from sample block ordered in columns order.
DataTypes data_types;
/// Parse key structure from key definition. Requires all columns, available
/// in storage.
static KeyDescription getKeyFromAST(const ASTPtr & definition_ast, const ColumnsDescription & columns, const Context & context, ASTPtr additional_key_expression = nullptr);
KeyDescription() = default;
/// We need custom copy constructors because we don't want
/// unintentionaly share AST variables and modify them.
KeyDescription(const KeyDescription & other);
KeyDescription & operator=(const KeyDescription & other);
};
}

View File

@ -144,23 +144,21 @@ MergeTreeData::MergeTreeData(
if (relative_data_path.empty())
throw Exception("MergeTree storages require data path", ErrorCodes::INCORRECT_FILE_NAME);
setSettingsChanges(metadata.settings_ast);
setSettingsChanges(metadata.settings_changes);
const auto settings = getSettings();
setProperties(metadata, /*only_check*/ false, attach);
/// NOTE: using the same columns list as is read when performing actual merges.
merging_params.check(getColumns().getAllPhysical());
if (metadata.sample_by_ast != nullptr)
if (metadata.sampling_key.definition_ast != nullptr)
{
KeyDescription candidate_sampling_key = KeyDescription::getKeyFromAST(metadata.sample_by_ast, getColumns(), global_context);
const auto & pk_sample_block = getPrimaryKey().sample_block;
if (!pk_sample_block.has(candidate_sampling_key.column_names[0]) && !attach
if (!pk_sample_block.has(metadata.sampling_key.column_names[0]) && !attach
&& !settings->compatibility_allow_sampling_expression_not_in_primary_key) /// This is for backward compatibility.
throw Exception("Sampling expression must be present in the primary key", ErrorCodes::BAD_ARGUMENTS);
setSamplingKey(candidate_sampling_key);
setSamplingKey(metadata.sampling_key);
}
MergeTreeDataFormatVersion min_format_version(0);
@ -169,7 +167,8 @@ MergeTreeData::MergeTreeData(
try
{
auto partition_by_ast = makeASTFunction("toYYYYMM", std::make_shared<ASTIdentifier>(date_column_name));
initPartitionKey(partition_by_ast);
auto partition_key = KeyDescription::getKeyFromAST(partition_by_ast, getColumns(), global_context);
initPartitionKey(partition_key);
if (minmax_idx_date_column_pos == -1)
throw Exception("Could not find Date column", ErrorCodes::BAD_TYPE_OF_FIELD);
@ -184,11 +183,11 @@ MergeTreeData::MergeTreeData(
else
{
is_custom_partitioned = true;
initPartitionKey(metadata.partition_by_ast);
initPartitionKey(metadata.partition_key);
min_format_version = MERGE_TREE_DATA_MIN_FORMAT_VERSION_WITH_CUSTOM_PARTITIONING;
}
setTTLExpressions(metadata.columns, metadata.ttl_for_table_ast);
setTTLExpressions(metadata.columns, metadata.table_ttl);
/// format_file always contained on any data path
PathWithDisk version_file;
@ -245,32 +244,6 @@ MergeTreeData::MergeTreeData(
LOG_WARNING(log, "{} Settings 'min_bytes_for_wide_part' and 'min_bytes_for_wide_part' will be ignored.", reason);
}
StorageInMemoryMetadata MergeTreeData::getInMemoryMetadata() const
{
StorageInMemoryMetadata metadata(getColumns(), getSecondaryIndices(), getConstraints());
if (isPartitionKeyDefined())
metadata.partition_by_ast = getPartitionKeyAST()->clone();
if (isSortingKeyDefined())
metadata.order_by_ast = getSortingKeyAST()->clone();
if (isPrimaryKeyDefined())
metadata.primary_key_ast = getPrimaryKeyAST()->clone();
if (hasAnyTableTTL())
metadata.ttl_for_table_ast = getTableTTLs().definition_ast->clone();
if (isSamplingKeyDefined())
metadata.sample_by_ast = getSamplingKeyAST()->clone();
if (hasSettingsChanges())
metadata.settings_ast = getSettingsChanges();
return metadata;
}
StoragePolicyPtr MergeTreeData::getStoragePolicy() const
{
return global_context.getStoragePolicy(getSettings()->storage_policy);
@ -306,35 +279,43 @@ static void checkKeyExpression(const ExpressionActions & expr, const Block & sam
void MergeTreeData::setProperties(const StorageInMemoryMetadata & metadata, bool only_check, bool attach)
{
if (!metadata.order_by_ast)
KeyDescription new_sorting_key = metadata.sorting_key;
KeyDescription new_primary_key = metadata.primary_key;
if (!new_sorting_key.definition_ast)
throw Exception("ORDER BY cannot be empty", ErrorCodes::BAD_ARGUMENTS);
ASTPtr new_sorting_key_expr_list = extractKeyExpressionList(metadata.order_by_ast);
ASTPtr new_primary_key_expr_list = metadata.primary_key_ast
? extractKeyExpressionList(metadata.primary_key_ast) : new_sorting_key_expr_list->clone();
if (merging_params.mode == MergeTreeData::MergingParams::VersionedCollapsing)
new_sorting_key_expr_list->children.push_back(std::make_shared<ASTIdentifier>(merging_params.version_column));
new_sorting_key = KeyDescription::getKeyFromAST(
metadata.sorting_key.definition_ast,
metadata.columns,
global_context,
std::make_shared<ASTIdentifier>(merging_params.version_column));
size_t primary_key_size = new_primary_key_expr_list->children.size();
size_t sorting_key_size = new_sorting_key_expr_list->children.size();
/// Primary key not defined at all
if (new_primary_key.definition_ast == nullptr)
{
/// We copy sorting key, and restore definition_ast to empty value
new_primary_key = metadata.sorting_key;
new_primary_key.definition_ast = nullptr;
}
size_t sorting_key_size = new_sorting_key.column_names.size();
size_t primary_key_size = new_primary_key.column_names.size();
if (primary_key_size > sorting_key_size)
throw Exception("Primary key must be a prefix of the sorting key, but its length: "
+ toString(primary_key_size) + " is greater than the sorting key length: " + toString(sorting_key_size),
ErrorCodes::BAD_ARGUMENTS);
Names new_primary_key_columns;
Names new_sorting_key_columns;
NameSet primary_key_columns_set;
for (size_t i = 0; i < sorting_key_size; ++i)
{
String sorting_key_column = new_sorting_key_expr_list->children[i]->getColumnName();
new_sorting_key_columns.push_back(sorting_key_column);
const String & sorting_key_column = new_sorting_key.column_names[i];
if (i < primary_key_size)
{
String pk_column = new_primary_key_expr_list->children[i]->getColumnName();
const String & pk_column = new_primary_key.column_names[i];
if (pk_column != sorting_key_column)
throw Exception("Primary key must be a prefix of the sorting key, but in position "
+ toString(i) + " its column is " + pk_column + ", not " + sorting_key_column,
@ -343,7 +324,6 @@ void MergeTreeData::setProperties(const StorageInMemoryMetadata & metadata, bool
if (!primary_key_columns_set.emplace(pk_column).second)
throw Exception("Primary key contains duplicate columns", ErrorCodes::BAD_ARGUMENTS);
new_primary_key_columns.push_back(pk_column);
}
}
@ -355,6 +335,9 @@ void MergeTreeData::setProperties(const StorageInMemoryMetadata & metadata, bool
/// This is ALTER, not CREATE/ATTACH TABLE. Let us check that all new columns used in the sorting key
/// expression have just been added (so that the sorting order is guaranteed to be valid with the new key).
Names new_primary_key_columns = new_primary_key.column_names;
Names new_sorting_key_columns = new_sorting_key.column_names;
ASTPtr added_key_column_expr_list = std::make_shared<ASTExpressionList>();
const auto & old_sorting_key_columns = getSortingKeyColumns();
for (size_t new_i = 0, old_i = 0; new_i < sorting_key_size; ++new_i)
@ -362,12 +345,12 @@ void MergeTreeData::setProperties(const StorageInMemoryMetadata & metadata, bool
if (old_i < old_sorting_key_columns.size())
{
if (new_sorting_key_columns[new_i] != old_sorting_key_columns[old_i])
added_key_column_expr_list->children.push_back(new_sorting_key_expr_list->children[new_i]);
added_key_column_expr_list->children.push_back(new_sorting_key.expression_list_ast->children[new_i]);
else
++old_i;
}
else
added_key_column_expr_list->children.push_back(new_sorting_key_expr_list->children[new_i]);
added_key_column_expr_list->children.push_back(new_sorting_key.expression_list_ast->children[new_i]);
}
if (!added_key_column_expr_list->children.empty())
@ -394,37 +377,9 @@ void MergeTreeData::setProperties(const StorageInMemoryMetadata & metadata, bool
}
}
auto new_sorting_key_syntax = SyntaxAnalyzer(global_context).analyze(new_sorting_key_expr_list, all_columns);
auto new_sorting_key_expr = ExpressionAnalyzer(new_sorting_key_expr_list, new_sorting_key_syntax, global_context)
.getActions(false);
auto new_sorting_key_sample =
ExpressionAnalyzer(new_sorting_key_expr_list, new_sorting_key_syntax, global_context)
.getActions(true)->getSampleBlock();
checkKeyExpression(*new_sorting_key_expr, new_sorting_key_sample, "Sorting");
auto new_primary_key_syntax = SyntaxAnalyzer(global_context).analyze(new_primary_key_expr_list, all_columns);
auto new_primary_key_expr = ExpressionAnalyzer(new_primary_key_expr_list, new_primary_key_syntax, global_context)
.getActions(false);
Block new_primary_key_sample;
DataTypes new_primary_key_data_types;
for (size_t i = 0; i < primary_key_size; ++i)
{
const auto & elem = new_sorting_key_sample.getByPosition(i);
new_primary_key_sample.insert(elem);
new_primary_key_data_types.push_back(elem.type);
}
DataTypes new_sorting_key_data_types;
for (size_t i = 0; i < sorting_key_size; ++i)
{
new_sorting_key_data_types.push_back(new_sorting_key_sample.getByPosition(i).type);
}
if (!metadata.secondary_indices.empty())
{
std::set<String> indices_names;
std::unordered_set<String> indices_names;
for (const auto & index : metadata.secondary_indices)
{
@ -444,22 +399,8 @@ void MergeTreeData::setProperties(const StorageInMemoryMetadata & metadata, bool
{
setColumns(std::move(metadata.columns));
KeyDescription new_sorting_key;
new_sorting_key.definition_ast = metadata.order_by_ast;
new_sorting_key.column_names = std::move(new_sorting_key_columns);
new_sorting_key.expression_list_ast = std::move(new_sorting_key_expr_list);
new_sorting_key.expression = std::move(new_sorting_key_expr);
new_sorting_key.sample_block = std::move(new_sorting_key_sample);
new_sorting_key.data_types = std::move(new_sorting_key_data_types);
setSortingKey(new_sorting_key);
KeyDescription new_primary_key;
new_primary_key.definition_ast = metadata.primary_key_ast;
new_primary_key.column_names = std::move(new_primary_key_columns);
new_primary_key.expression_list_ast = std::move(new_primary_key_expr_list);
new_primary_key.expression = std::move(new_primary_key_expr);
new_primary_key.sample_block = std::move(new_primary_key_sample);
new_primary_key.data_types = std::move(new_primary_key_data_types);
setPrimaryKey(new_primary_key);
setSecondaryIndices(metadata.secondary_indices);
@ -521,10 +462,8 @@ ASTPtr MergeTreeData::extractKeyExpressionList(const ASTPtr & node)
}
void MergeTreeData::initPartitionKey(ASTPtr partition_by_ast)
void MergeTreeData::initPartitionKey(const KeyDescription & new_partition_key)
{
KeyDescription new_partition_key = KeyDescription::getKeyFromAST(partition_by_ast, getColumns(), global_context);
if (new_partition_key.expression_list_ast->children.empty())
return;
@ -580,10 +519,10 @@ void MergeTreeData::initPartitionKey(ASTPtr partition_by_ast)
}
/// Todo replace columns with TTL for columns
void MergeTreeData::setTTLExpressions(const ColumnsDescription & new_columns,
const ASTPtr & new_ttl_table_ast, bool only_check)
const TTLTableDescription & new_table_ttl, bool only_check)
{
auto new_column_ttls_asts = new_columns.getColumnTTLs();
TTLColumnsDescription new_column_ttl_by_name = getColumnTTLs();
@ -614,56 +553,24 @@ void MergeTreeData::setTTLExpressions(const ColumnsDescription & new_columns,
setColumnTTLs(new_column_ttl_by_name);
}
if (new_ttl_table_ast)
if (new_table_ttl.definition_ast)
{
TTLDescriptions update_move_ttl_entries;
TTLDescription update_rows_ttl_entry;
bool seen_delete_ttl = false;
for (const auto & ttl_element_ptr : new_ttl_table_ast->children)
for (const auto & move_ttl : new_table_ttl.move_ttl)
{
const auto * ttl_element = ttl_element_ptr->as<ASTTTLElement>();
if (!ttl_element)
throw Exception("Unexpected AST element in TTL expression", ErrorCodes::UNEXPECTED_AST_STRUCTURE);
if (ttl_element->destination_type == DataDestinationType::DELETE)
if (!getDestinationForTTL(move_ttl))
{
if (seen_delete_ttl)
{
throw Exception("More than one DELETE TTL expression is not allowed", ErrorCodes::BAD_TTL_EXPRESSION);
}
update_rows_ttl_entry = TTLDescription::getTTLFromAST(ttl_element_ptr, new_columns, global_context, getPrimaryKey());
seen_delete_ttl = true;
}
else
{
auto new_ttl_entry = TTLDescription::getTTLFromAST(ttl_element_ptr, new_columns, global_context, getPrimaryKey());
if (!getDestinationForTTL(new_ttl_entry))
{
String message;
if (new_ttl_entry.destination_type == DataDestinationType::DISK)
message = "No such disk " + backQuote(new_ttl_entry.destination_name) + " for given storage policy.";
else
message = "No such volume " + backQuote(new_ttl_entry.destination_name) + " for given storage policy.";
throw Exception(message, ErrorCodes::BAD_TTL_EXPRESSION);
}
update_move_ttl_entries.emplace_back(std::move(new_ttl_entry));
String message;
if (move_ttl.destination_type == DataDestinationType::DISK)
message = "No such disk " + backQuote(move_ttl.destination_name) + " for given storage policy.";
else
message = "No such volume " + backQuote(move_ttl.destination_name) + " for given storage policy.";
throw Exception(message, ErrorCodes::BAD_TTL_EXPRESSION);
}
}
if (!only_check)
{
TTLTableDescription new_table_ttl
{
.definition_ast = new_ttl_table_ast,
.rows_ttl = update_rows_ttl_entry,
.move_ttl = update_move_ttl_entries,
};
auto move_ttl_entries_lock = std::lock_guard<std::mutex>(move_ttl_entries_mutex);
setTableTTLs(new_table_ttl);
}
@ -1458,13 +1365,13 @@ void MergeTreeData::checkAlterIsPossible(const AlterCommands & commands, const S
setProperties(metadata, /* only_check = */ true);
setTTLExpressions(metadata.columns, metadata.ttl_for_table_ast, /* only_check = */ true);
setTTLExpressions(metadata.columns, metadata.table_ttl, /* only_check = */ true);
if (hasSettingsChanges())
{
const auto & current_changes = getSettingsChanges()->as<const ASTSetQuery &>().changes;
const auto & new_changes = metadata.settings_ast->as<const ASTSetQuery &>().changes;
const auto & new_changes = metadata.settings_changes->as<const ASTSetQuery &>().changes;
for (const auto & changed_setting : new_changes)
{
if (MergeTreeSettings::findIndex(changed_setting.name) == MergeTreeSettings::npos)

View File

@ -333,9 +333,6 @@ public:
BrokenPartCallback broken_part_callback_ = [](const String &){});
/// See comments about methods below in IStorage interface
StorageInMemoryMetadata getInMemoryMetadata() const override;
StoragePolicyPtr getStoragePolicy() const override;
bool supportsPrewhere() const override { return true; }
@ -785,10 +782,10 @@ protected:
void setProperties(const StorageInMemoryMetadata & metadata, bool only_check = false, bool attach = false);
void initPartitionKey(ASTPtr partition_by_ast);
void initPartitionKey(const KeyDescription & new_partition_key);
void setTTLExpressions(const ColumnsDescription & columns,
const ASTPtr & new_ttl_table_ast, bool only_check = false);
const TTLTableDescription & new_table_ttl, bool only_check = false);
void checkStoragePolicy(const StoragePolicyPtr & new_storage_policy) const;

View File

@ -39,11 +39,6 @@ public:
return part->storage.mayBenefitFromIndexForIn(left_in_operand, query_context);
}
StorageInMemoryMetadata getInMemoryMetadata() const override
{
return part->storage.getInMemoryMetadata();
}
NamesAndTypesList getVirtuals() const override
{
return part->storage.getVirtuals();

View File

@ -572,47 +572,46 @@ static StoragePtr create(const StorageFactory::Arguments & args)
}
String date_column_name;
ASTPtr partition_by_ast;
ASTPtr order_by_ast;
ASTPtr primary_key_ast;
ASTPtr sample_by_ast;
ASTPtr ttl_table_ast;
ASTPtr settings_ast;
IndicesDescription indices_description;
ConstraintsDescription constraints_description;
StorageInMemoryMetadata metadata;
metadata.columns = args.columns;
std::unique_ptr<MergeTreeSettings> storage_settings = std::make_unique<MergeTreeSettings>(args.context.getMergeTreeSettings());
if (is_extended_storage_def)
{
if (args.storage_def->partition_by)
partition_by_ast = args.storage_def->partition_by->ptr();
metadata.partition_key = KeyDescription::getKeyFromAST(
args.storage_def->partition_by->ptr(), metadata.columns, args.context);
if (!args.storage_def->order_by)
throw Exception("You must provide an ORDER BY expression in the table definition. "
"If you don't want this table to be sorted, use ORDER BY tuple()",
ErrorCodes::BAD_ARGUMENTS);
order_by_ast = args.storage_def->order_by->ptr();
metadata.sorting_key = KeyDescription::getKeyFromAST(args.storage_def->order_by->ptr(), metadata.columns, args.context);
if (args.storage_def->primary_key)
primary_key_ast = args.storage_def->primary_key->ptr();
metadata.primary_key = KeyDescription::getKeyFromAST(args.storage_def->primary_key->ptr(), metadata.columns, args.context);
if (args.storage_def->sample_by)
sample_by_ast = args.storage_def->sample_by->ptr();
metadata.sampling_key = KeyDescription::getKeyFromAST(args.storage_def->sample_by->ptr(), metadata.columns, args.context);
if (args.storage_def->ttl_table)
ttl_table_ast = args.storage_def->ttl_table->ptr();
metadata.table_ttl = TTLTableDescription::getTTLForTableFromAST(
args.storage_def->ttl_table->ptr(),
metadata.columns,
args.context,
metadata.primary_key);
if (args.query.columns_list && args.query.columns_list->indices)
for (auto & index : args.query.columns_list->indices->children)
indices_description.push_back(IndexDescription::getIndexFromAST(index, args.columns, args.context));
metadata.secondary_indices.push_back(IndexDescription::getIndexFromAST(index, args.columns, args.context));
storage_settings->loadFromQuery(*args.storage_def);
if (args.storage_def->settings)
settings_ast = args.storage_def->settings->ptr();
metadata.settings_changes = args.storage_def->settings->ptr();
}
else
{
@ -627,12 +626,12 @@ static StoragePtr create(const StorageFactory::Arguments & args)
/// If there is an expression for sampling
if (arg_cnt - arg_num == 3)
{
sample_by_ast = engine_args[arg_num];
metadata.sampling_key = KeyDescription::getKeyFromAST(engine_args[arg_num], metadata.columns, args.context);
++arg_num;
}
/// Now only two parameters remain - primary_key, index_granularity.
order_by_ast = engine_args[arg_num];
metadata.sorting_key = KeyDescription::getKeyFromAST(engine_args[arg_num], metadata.columns, args.context);
++arg_num;
const auto * ast = engine_args[arg_num]->as<ASTLiteral>();
@ -648,18 +647,10 @@ static StoragePtr create(const StorageFactory::Arguments & args)
if (arg_num != arg_cnt)
throw Exception("Wrong number of engine arguments.", ErrorCodes::BAD_ARGUMENTS);
if (!args.attach && !indices_description.empty() && !args.local_context.getSettingsRef().allow_experimental_data_skipping_indices)
if (!args.attach && !metadata.secondary_indices.empty() && !args.local_context.getSettingsRef().allow_experimental_data_skipping_indices)
throw Exception("You must set the setting `allow_experimental_data_skipping_indices` to 1 " \
"before using data skipping indices.", ErrorCodes::BAD_ARGUMENTS);
StorageInMemoryMetadata metadata(args.columns, indices_description, args.constraints);
metadata.partition_by_ast = partition_by_ast;
metadata.order_by_ast = order_by_ast;
metadata.primary_key_ast = primary_key_ast;
metadata.ttl_for_table_ast = ttl_table_ast;
metadata.sample_by_ast = sample_by_ast;
metadata.settings_ast = settings_ast;
if (replicated)
return StorageReplicatedMergeTree::create(
zookeeper_path, replica_name, args.attach, args.table_id, args.relative_data_path,

View File

@ -0,0 +1,117 @@
#include <Storages/SelectQueryDescription.h>
#include <Parsers/ASTSelectWithUnionQuery.h>
#include <Parsers/ASTSelectQuery.h>
#include <Interpreters/getTableExpressions.h>
#include <Interpreters/AddDefaultDatabaseVisitor.h>
#include <Interpreters/Context.h>
namespace DB
{
namespace ErrorCodes
{
extern const int QUERY_IS_NOT_SUPPORTED_IN_MATERIALIZED_VIEW;
extern const int LOGICAL_ERROR;
}
SelectQueryDescription::SelectQueryDescription(const SelectQueryDescription & other)
: select_table_id(other.select_table_id)
, select_query(other.select_query ? other.select_query->clone() : nullptr)
, inner_query(other.inner_query ? other.inner_query->clone() : nullptr)
{
}
SelectQueryDescription & SelectQueryDescription::SelectQueryDescription::operator=(const SelectQueryDescription & other)
{
select_table_id = other.select_table_id;
if (other.select_query)
select_query = other.select_query->clone();
else
select_query.reset();
if (other.inner_query)
inner_query = other.inner_query->clone();
else
inner_query.reset();
return *this;
}
namespace
{
StorageID extractDependentTableFromSelectQuery(ASTSelectQuery & query, const Context & context, bool add_default_db = true)
{
if (add_default_db)
{
AddDefaultDatabaseVisitor visitor(context.getCurrentDatabase(), nullptr);
visitor.visit(query);
}
if (auto db_and_table = getDatabaseAndTable(query, 0))
{
return StorageID(db_and_table->database, db_and_table->table/*, db_and_table->uuid*/);
}
else if (auto subquery = extractTableExpression(query, 0))
{
auto * ast_select = subquery->as<ASTSelectWithUnionQuery>();
if (!ast_select)
throw Exception("Logical error while creating StorageMaterializedView. "
"Could not retrieve table name from select query.",
DB::ErrorCodes::LOGICAL_ERROR);
if (ast_select->list_of_selects->children.size() != 1)
throw Exception("UNION is not supported for MATERIALIZED VIEW",
ErrorCodes::QUERY_IS_NOT_SUPPORTED_IN_MATERIALIZED_VIEW);
auto & inner_query = ast_select->list_of_selects->children.at(0);
return extractDependentTableFromSelectQuery(inner_query->as<ASTSelectQuery &>(), context, false);
}
else
return StorageID::createEmpty();
}
void checkAllowedQueries(const ASTSelectQuery & query)
{
if (query.prewhere() || query.final() || query.sampleSize())
throw Exception("MATERIALIZED VIEW cannot have PREWHERE, SAMPLE or FINAL.", DB::ErrorCodes::QUERY_IS_NOT_SUPPORTED_IN_MATERIALIZED_VIEW);
ASTPtr subquery = extractTableExpression(query, 0);
if (!subquery)
return;
if (const auto * ast_select = subquery->as<ASTSelectWithUnionQuery>())
{
if (ast_select->list_of_selects->children.size() != 1)
throw Exception("UNION is not supported for MATERIALIZED VIEW", ErrorCodes::QUERY_IS_NOT_SUPPORTED_IN_MATERIALIZED_VIEW);
const auto & inner_query = ast_select->list_of_selects->children.at(0);
checkAllowedQueries(inner_query->as<ASTSelectQuery &>());
}
}
}
SelectQueryDescription SelectQueryDescription::getSelectQueryFromASTForMatView(const ASTPtr & select, const Context & context)
{
auto & new_select = select->as<ASTSelectWithUnionQuery &>();
if (new_select.list_of_selects->children.size() != 1)
throw Exception("UNION is not supported for MATERIALIZED VIEW", ErrorCodes::QUERY_IS_NOT_SUPPORTED_IN_MATERIALIZED_VIEW);
SelectQueryDescription result;
result.inner_query = new_select.list_of_selects->children.at(0)->clone();
auto & select_query = result.inner_query->as<ASTSelectQuery &>();
checkAllowedQueries(select_query);
result.select_table_id = extractDependentTableFromSelectQuery(select_query, context);
result.select_query = select->clone();
return result;
}
}

View File

@ -0,0 +1,27 @@
#pragma once
#include <Interpreters/StorageID.h>
namespace DB
{
/// Select query for different view in storages
struct SelectQueryDescription
{
/// Table id for select query
StorageID select_table_id = StorageID::createEmpty();
/// Select query itself (ASTSelectWithUnionQuery)
ASTPtr select_query;
/// First query from select_query list
ASTPtr inner_query;
/// Parse description from select query for materialized view. Also
/// validates query.
static SelectQueryDescription getSelectQueryFromASTForMatView(const ASTPtr & select, const Context & context);
SelectQueryDescription() = default;
SelectQueryDescription(const SelectQueryDescription & other);
SelectQueryDescription & operator=(const SelectQueryDescription & other);
};
}

View File

@ -12,73 +12,4 @@ StorageInMemoryMetadata::StorageInMemoryMetadata(
, constraints(constraints_)
{
}
StorageInMemoryMetadata::StorageInMemoryMetadata(const StorageInMemoryMetadata & other)
: columns(other.columns)
, secondary_indices(other.secondary_indices)
, constraints(other.constraints)
{
if (other.partition_by_ast)
partition_by_ast = other.partition_by_ast->clone();
if (other.order_by_ast)
order_by_ast = other.order_by_ast->clone();
if (other.primary_key_ast)
primary_key_ast = other.primary_key_ast->clone();
if (other.ttl_for_table_ast)
ttl_for_table_ast = other.ttl_for_table_ast->clone();
if (other.sample_by_ast)
sample_by_ast = other.sample_by_ast->clone();
if (other.settings_ast)
settings_ast = other.settings_ast->clone();
if (other.select)
select = other.select->clone();
}
StorageInMemoryMetadata & StorageInMemoryMetadata::operator=(const StorageInMemoryMetadata & other)
{
if (this == &other)
return *this;
columns = other.columns;
secondary_indices = other.secondary_indices;
constraints = other.constraints;
if (other.partition_by_ast)
partition_by_ast = other.partition_by_ast->clone();
else
partition_by_ast.reset();
if (other.order_by_ast)
order_by_ast = other.order_by_ast->clone();
else
order_by_ast.reset();
if (other.primary_key_ast)
primary_key_ast = other.primary_key_ast->clone();
else
primary_key_ast.reset();
if (other.ttl_for_table_ast)
ttl_for_table_ast = other.ttl_for_table_ast->clone();
else
ttl_for_table_ast.reset();
if (other.sample_by_ast)
sample_by_ast = other.sample_by_ast->clone();
else
sample_by_ast.reset();
if (other.settings_ast)
settings_ast = other.settings_ast->clone();
else
settings_ast.reset();
if (other.select)
select = other.select->clone();
else
select.reset();
return *this;
}
}

View File

@ -1,9 +1,13 @@
#pragma once
#include <Storages/ColumnsDescription.h>
#include <Storages/IndicesDescription.h>
#include <Storages/ConstraintsDescription.h>
#include <Parsers/IAST_fwd.h>
#include <Storages/ColumnsDescription.h>
#include <Storages/ConstraintsDescription.h>
#include <Storages/IndicesDescription.h>
#include <Storages/KeyDescription.h>
#include <Storages/TTLDescription.h>
#include <Storages/SelectQueryDescription.h>
namespace DB
{
@ -21,26 +25,25 @@ struct StorageInMemoryMetadata
/// Table constraints. Currently supported for MergeTree only.
ConstraintsDescription constraints;
/// PARTITION BY expression. Currently supported for MergeTree only.
ASTPtr partition_by_ast = nullptr;
KeyDescription partition_key;
/// PRIMARY KEY expression. If absent, than equal to order_by_ast.
KeyDescription primary_key;
/// ORDER BY expression. Required field for all MergeTree tables
/// even in old syntax MergeTree(partition_key, order_by, ...)
ASTPtr order_by_ast = nullptr;
/// PRIMARY KEY expression. If absent, than equal to order_by_ast.
ASTPtr primary_key_ast = nullptr;
/// TTL expression for whole table. Supported for MergeTree only.
ASTPtr ttl_for_table_ast = nullptr;
KeyDescription sorting_key;
/// SAMPLE BY expression. Supported for MergeTree only.
ASTPtr sample_by_ast = nullptr;
KeyDescription sampling_key;
/// Separate ttl expressions for columns
TTLColumnsDescription column_ttls_by_name;
/// TTL expressions for table (Move and Rows)
TTLTableDescription table_ttl;
/// SETTINGS expression. Supported for MergeTree, Buffer and Kafka.
ASTPtr settings_ast = nullptr;
/// SELECT QUERY. Supported for MaterializedView only.
ASTPtr select = nullptr;
ASTPtr settings_changes;
/// SELECT QUERY. Supported for MaterializedView and View (have to support LiveView).
SelectQueryDescription select;
StorageInMemoryMetadata(const StorageInMemoryMetadata & other);
StorageInMemoryMetadata() = default;
StorageInMemoryMetadata(const ColumnsDescription & columns_, const IndicesDescription & secondary_indices_, const ConstraintsDescription & constraints_);
StorageInMemoryMetadata & operator=(const StorageInMemoryMetadata & other);
};
}

View File

@ -100,13 +100,6 @@ StorageMaterializedView::StorageMaterializedView(
DatabaseCatalog::instance().addDependency(select.select_table_id, getStorageID());
}
StorageInMemoryMetadata StorageMaterializedView::getInMemoryMetadata() const
{
StorageInMemoryMetadata result(getColumns(), getSecondaryIndices(), getConstraints());
result.select = getSelectQuery().select_query;
return result;
}
QueryProcessingStage::Enum StorageMaterializedView::getQueryProcessingStage(const Context & context, QueryProcessingStage::Enum to_stage, const ASTPtr & query_ptr) const
{
return getTargetTable()->getQueryProcessingStage(context, to_stage, query_ptr);
@ -207,7 +200,7 @@ void StorageMaterializedView::alter(
/// start modify query
if (context.getSettingsRef().allow_experimental_alter_materialized_view_structure)
{
auto new_select = SelectQueryDescription::getSelectQueryFromASTForMatView(metadata.select, context);
const auto & new_select = metadata.select;
const auto & old_select = getSelectQuery();
DatabaseCatalog::instance().updateDependency(old_select.select_table_id, table_id, new_select.select_table_id, table_id);

View File

@ -21,8 +21,6 @@ public:
bool hasInnerTable() const { return has_inner_table; }
StorageInMemoryMetadata getInMemoryMetadata() const override;
bool supportsSampling() const override { return getTargetTable()->supportsSampling(); }
bool supportsPrewhere() const override { return getTargetTable()->supportsPrewhere(); }
bool supportsFinal() const override { return getTargetTable()->supportsFinal(); }

View File

@ -260,7 +260,7 @@ void StorageMergeTree::alter(
{
lockStructureExclusively(table_lock_holder, context.getCurrentQueryId(), context.getSettingsRef().lock_acquire_timeout);
changeSettings(metadata.settings_ast, table_lock_holder);
changeSettings(metadata.settings_changes, table_lock_holder);
DatabaseCatalog::instance().getDatabase(table_id.database_name)->alterTable(context, table_id, metadata);
}
@ -271,11 +271,11 @@ void StorageMergeTree::alter(
auto merges_block = getActionLock(ActionLocks::PartsMerge);
lockStructureExclusively(table_lock_holder, context.getCurrentQueryId(), context.getSettingsRef().lock_acquire_timeout);
changeSettings(metadata.settings_ast, table_lock_holder);
changeSettings(metadata.settings_changes, table_lock_holder);
/// Reinitialize primary key because primary key column types might have changed.
setProperties(metadata);
setTTLExpressions(metadata.columns, metadata.ttl_for_table_ast);
setTTLExpressions(metadata.columns, metadata.table_ttl);
DatabaseCatalog::instance().getDatabase(table_id.database_name)->alterTable(context, table_id, metadata);

View File

@ -483,20 +483,22 @@ void StorageReplicatedMergeTree::setTableStructure(ColumnsDescription new_column
ParserNotEmptyExpressionList parser(false);
auto new_sorting_key_expr_list = parseQuery(parser, metadata_diff.new_sorting_key, 0, DBMS_DEFAULT_MAX_PARSER_DEPTH);
ASTPtr order_by_ast;
if (new_sorting_key_expr_list->children.size() == 1)
metadata.order_by_ast = new_sorting_key_expr_list->children[0];
order_by_ast = new_sorting_key_expr_list->children[0];
else
{
auto tuple = makeASTFunction("tuple");
tuple->arguments->children = new_sorting_key_expr_list->children;
metadata.order_by_ast = tuple;
order_by_ast = tuple;
}
metadata.sorting_key = KeyDescription::getKeyFromAST(order_by_ast, metadata.columns, global_context);
if (!isPrimaryKeyDefined())
{
/// Primary and sorting key become independent after this ALTER so we have to
/// save the old ORDER BY expression as the new primary key.
metadata.primary_key_ast = getSortingKeyAST()->clone();
metadata.primary_key = getSortingKey();
}
}
@ -509,7 +511,8 @@ void StorageReplicatedMergeTree::setTableStructure(ColumnsDescription new_column
if (metadata_diff.ttl_table_changed)
{
ParserTTLExpressionList parser;
metadata.ttl_for_table_ast = parseQuery(parser, metadata_diff.new_ttl_table, 0, DBMS_DEFAULT_MAX_PARSER_DEPTH);
auto ttl_for_table_ast = parseQuery(parser, metadata_diff.new_ttl_table, 0, DBMS_DEFAULT_MAX_PARSER_DEPTH);
metadata.table_ttl = TTLTableDescription::getTTLForTableFromAST(ttl_for_table_ast, metadata.columns, global_context, metadata.primary_key);
}
}
@ -519,7 +522,7 @@ void StorageReplicatedMergeTree::setTableStructure(ColumnsDescription new_column
/// Even if the primary/sorting keys didn't change we must reinitialize it
/// because primary key column types might have changed.
setProperties(metadata);
setTTLExpressions(new_columns, metadata.ttl_for_table_ast);
setTTLExpressions(new_columns, metadata.table_ttl);
}
@ -3295,7 +3298,7 @@ void StorageReplicatedMergeTree::alter(
params.apply(metadata, query_context);
changeSettings(metadata.settings_ast, table_lock_holder);
changeSettings(metadata.settings_changes, table_lock_holder);
DatabaseCatalog::instance().getDatabase(table_id.database_name)->alterTable(query_context, table_id, metadata);
return;
@ -3329,11 +3332,11 @@ void StorageReplicatedMergeTree::alter(
params.apply(future_metadata, query_context);
ReplicatedMergeTreeTableMetadata future_metadata_in_zk(*this);
if (ast_to_str(future_metadata.order_by_ast) != ast_to_str(current_metadata.order_by_ast))
future_metadata_in_zk.sorting_key = serializeAST(*extractKeyExpressionList(future_metadata.order_by_ast));
if (ast_to_str(future_metadata.sorting_key.definition_ast) != ast_to_str(current_metadata.sorting_key.definition_ast))
future_metadata_in_zk.sorting_key = serializeAST(*future_metadata.sorting_key.expression_list_ast);
if (ast_to_str(future_metadata.ttl_for_table_ast) != ast_to_str(current_metadata.ttl_for_table_ast))
future_metadata_in_zk.ttl_table = serializeAST(*future_metadata.ttl_for_table_ast);
if (ast_to_str(future_metadata.table_ttl.definition_ast) != ast_to_str(current_metadata.table_ttl.definition_ast))
future_metadata_in_zk.ttl_table = serializeAST(*future_metadata.table_ttl.definition_ast);
String new_indices_str = future_metadata.secondary_indices.toString();
if (new_indices_str != current_metadata.secondary_indices.toString())
@ -3352,13 +3355,13 @@ void StorageReplicatedMergeTree::alter(
ops.emplace_back(zkutil::makeSetRequest(zookeeper_path + "/columns", new_columns_str, -1));
if (ast_to_str(current_metadata.settings_ast) != ast_to_str(future_metadata.settings_ast))
if (ast_to_str(current_metadata.settings_changes) != ast_to_str(future_metadata.settings_changes))
{
lockStructureExclusively(
table_lock_holder, query_context.getCurrentQueryId(), query_context.getSettingsRef().lock_acquire_timeout);
/// Just change settings
current_metadata.settings_ast = future_metadata.settings_ast;
changeSettings(current_metadata.settings_ast, table_lock_holder);
current_metadata.settings_changes = future_metadata.settings_changes;
changeSettings(current_metadata.settings_changes, table_lock_holder);
DatabaseCatalog::instance().getDatabase(table_id.database_name)->alterTable(query_context, table_id, current_metadata);
}

View File

@ -55,6 +55,41 @@ void checkTTLExpression(const ExpressionActionsPtr & ttl_expression, const Strin
}
TTLDescription::TTLDescription(const TTLDescription & other)
: mode(other.mode)
, expression_ast(other.expression_ast ? other.expression_ast->clone() : nullptr)
, expression(other.expression)
, result_column(other.result_column)
, where_expression(other.where_expression)
, where_result_column(other.where_result_column)
, group_by_keys(other.group_by_keys)
, set_parts(other.set_parts)
, aggregate_descriptions(other.aggregate_descriptions)
, destination_type(other.destination_type)
, destination_name(other.destination_name)
{
}
TTLDescription & TTLDescription::operator=(const TTLDescription & other)
{
mode = other.mode;
if (other.expression_ast)
expression_ast = other.expression_ast->clone();
else
expression_ast.reset();
expression = other.expression;
result_column = other.result_column;
where_expression = other.where_expression;
where_result_column = other.where_result_column;
group_by_keys = other.group_by_keys;
set_parts = other.set_parts;
aggregate_descriptions = other.aggregate_descriptions;
destination_type = other.destination_type;
destination_name = other.destination_name;
return * this;
}
TTLDescription TTLDescription::getTTLFromAST(
const ASTPtr & definition_ast,
const ColumnsDescription & columns,
@ -195,4 +230,34 @@ TTLDescription TTLDescription::getTTLFromAST(
return result;
}
TTLTableDescription TTLTableDescription::getTTLForTableFromAST(
const ASTPtr & definition_ast,
const ColumnsDescription & columns,
const Context & context,
const KeyDescription & primary_key)
{
TTLTableDescription result;
if (!definition_ast)
return result;
result.definition_ast = definition_ast->clone();
bool seen_delete_ttl = false;
for (const auto & ttl_element_ptr : definition_ast->children)
{
auto ttl = TTLDescription::getTTLFromAST(ttl_element_ptr, columns, context, primary_key);
if (ttl.destination_type == DataDestinationType::DELETE)
{
if (seen_delete_ttl)
throw Exception("More than one DELETE TTL expression is not allowed", ErrorCodes::BAD_TTL_EXPRESSION);
result.rows_ttl = ttl;
seen_delete_ttl = true;
}
else
result.move_ttl.emplace_back(std::move(ttl));
}
return result;
}
}

View File

@ -5,7 +5,6 @@
#include <Storages/KeyDescription.h>
#include <Interpreters/ExpressionActions.h>
#include <Interpreters/AggregateDescription.h>
#include <Storages/StorageInMemoryMetadata.h>
#include <Storages/TTLMode.h>
namespace DB
@ -75,6 +74,10 @@ struct TTLDescription
/// Parse TTL structure from definition. Able to parse both column and table
/// TTLs.
static TTLDescription getTTLFromAST(const ASTPtr & definition_ast, const ColumnsDescription & columns, const Context & context, const KeyDescription & primary_key);
TTLDescription() = default;
TTLDescription(const TTLDescription & other);
TTLDescription & operator=(const TTLDescription & other);
};
/// Mapping from column name to column TTL
@ -94,6 +97,9 @@ struct TTLTableDescription
/// Moving data TTL (to other disks or volumes)
TTLDescriptions move_ttl;
static TTLTableDescription getTTLForTableFromAST(
const ASTPtr & definition_ast, const ColumnsDescription & columns, const Context & context, const KeyDescription & primary_key);
};
}