mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-23 16:12:01 +00:00
Merge branch 'vavrusa-materialized-view-to'
This commit is contained in:
commit
fbef2a9816
@ -22,8 +22,6 @@ String getTableDefinitionFromCreateQuery(const ASTPtr & query)
|
||||
/// We remove everything that is not needed for ATTACH from the query.
|
||||
create.attach = true;
|
||||
create.database.clear();
|
||||
create.as_database.clear();
|
||||
create.as_table.clear();
|
||||
create.if_not_exists = false;
|
||||
create.is_populate = false;
|
||||
|
||||
@ -31,6 +29,13 @@ String getTableDefinitionFromCreateQuery(const ASTPtr & query)
|
||||
if (!create.is_view && !create.is_materialized_view)
|
||||
create.select = nullptr;
|
||||
|
||||
/// For "MATERIALIZED VIEW x TO y" it's necessary to save destination table
|
||||
if (engine != "MaterializedView" || create.inner_storage)
|
||||
{
|
||||
create.as_database.clear();
|
||||
create.as_table.clear();
|
||||
}
|
||||
|
||||
std::ostringstream statement_stream;
|
||||
formatAST(create, statement_stream, 0, false);
|
||||
statement_stream << '\n';
|
||||
@ -56,6 +61,8 @@ std::pair<String, StoragePtr> createTableFromDefinition(
|
||||
/// We do not directly use `InterpreterCreateQuery::execute`, because
|
||||
/// - the database has not been created yet;
|
||||
/// - the code is simpler, since the query is already brought to a suitable form.
|
||||
if (!ast_create_query.columns)
|
||||
throw Exception("Missing definition of columns.", ErrorCodes::EMPTY_LIST_OF_COLUMNS_PASSED);
|
||||
|
||||
InterpreterCreateQuery::ColumnsInfo columns_info = InterpreterCreateQuery::getColumnsInfo(*ast_create_query.columns, context);
|
||||
|
||||
|
@ -434,6 +434,12 @@ void InterpreterCreateQuery::setEngine(ASTCreateQuery & create) const
|
||||
storage_ast->set(storage_ast->engine, engine_ast);
|
||||
create.set(create.storage, storage_ast);
|
||||
}
|
||||
else if (create.is_temporary)
|
||||
set_engine("Memory");
|
||||
else if (create.is_view)
|
||||
set_engine("View");
|
||||
else if (create.is_materialized_view)
|
||||
set_engine("MaterializedView");
|
||||
else if (!create.as_table.empty())
|
||||
{
|
||||
/// NOTE Getting the structure from the table specified in the AS is done not atomically with the creation of the table.
|
||||
@ -470,6 +476,16 @@ BlockIO InterpreterCreateQuery::createTable(ASTCreateQuery & create)
|
||||
String data_path = path + "data/" + database_name_escaped + "/";
|
||||
String metadata_path = path + "metadata/" + database_name_escaped + "/" + table_name_escaped + ".sql";
|
||||
|
||||
// If this is a stub ATTACH query, read the query definition from the database
|
||||
if (create.attach && !create.storage && !create.columns)
|
||||
{
|
||||
// Table SQL definition is available even if the table is detached
|
||||
auto query = context.getCreateQuery(database_name, table_name);
|
||||
auto & as_create = typeid_cast<const ASTCreateQuery &>(*query);
|
||||
create = as_create; // Copy the saved create query, but use ATTACH instead of CREATE
|
||||
create.attach = true;
|
||||
}
|
||||
|
||||
std::unique_ptr<InterpreterSelectQuery> interpreter_select;
|
||||
Block as_select_sample;
|
||||
/// For `view` type tables, you may need `sample_block` to get the columns.
|
||||
|
@ -168,7 +168,8 @@ protected:
|
||||
|
||||
if (!as_table.empty())
|
||||
{
|
||||
settings.ostr << (settings.hilite ? hilite_keyword : "") << " AS " << (settings.hilite ? hilite_none : "")
|
||||
std::string what = (!is_materialized_view ? " AS " : " TO ");
|
||||
settings.ostr << (settings.hilite ? hilite_keyword : "") << what << (settings.hilite ? hilite_none : "")
|
||||
<< (!as_database.empty() ? backQuoteIfNeed(as_database) + "." : "") << backQuoteIfNeed(as_table);
|
||||
}
|
||||
|
||||
|
@ -219,6 +219,7 @@ bool ParserCreateQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
|
||||
bool is_materialized_view = false;
|
||||
bool is_populate = false;
|
||||
bool is_temporary = false;
|
||||
bool to_table = false;
|
||||
|
||||
if (!s_create.ignore(pos, expected))
|
||||
{
|
||||
@ -254,6 +255,23 @@ bool ParserCreateQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
|
||||
return false;
|
||||
}
|
||||
|
||||
// Shortcut for ATTACH a previously detached table
|
||||
if (attach && (!pos.isValid() || pos.get().type == TokenType::Semicolon))
|
||||
{
|
||||
auto query = std::make_shared<ASTCreateQuery>(StringRange(begin, pos));
|
||||
node = query;
|
||||
|
||||
query->attach = attach;
|
||||
query->if_not_exists = if_not_exists;
|
||||
|
||||
if (database)
|
||||
query->database = typeid_cast<ASTIdentifier &>(*database).name;
|
||||
if (table)
|
||||
query->table = typeid_cast<ASTIdentifier &>(*table).name;
|
||||
|
||||
return true;
|
||||
}
|
||||
|
||||
/// List of columns.
|
||||
if (s_lparen.ignore(pos, expected))
|
||||
{
|
||||
@ -341,6 +359,22 @@ bool ParserCreateQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
|
||||
return false;
|
||||
}
|
||||
|
||||
// TO [db.]table
|
||||
if (ParserKeyword{"TO"}.ignore(pos, expected))
|
||||
{
|
||||
to_table = true;
|
||||
|
||||
if (!name_p.parse(pos, as_table, expected))
|
||||
return false;
|
||||
|
||||
if (s_dot.ignore(pos, expected))
|
||||
{
|
||||
as_database = as_table;
|
||||
if (!name_p.parse(pos, as_table, expected))
|
||||
return false;
|
||||
}
|
||||
}
|
||||
|
||||
/// Optional - a list of columns can be specified. It must fully comply with SELECT.
|
||||
if (s_lparen.ignore(pos, expected))
|
||||
{
|
||||
@ -351,7 +385,7 @@ bool ParserCreateQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
|
||||
return false;
|
||||
}
|
||||
|
||||
if (is_materialized_view)
|
||||
if (is_materialized_view && !to_table)
|
||||
{
|
||||
/// Internal ENGINE for MATERIALIZED VIEW must be specified.
|
||||
if (!storage_p.parse(pos, storage, expected))
|
||||
|
@ -214,7 +214,7 @@ protected:
|
||||
* CREATE|ATTACH DATABASE db [ENGINE = engine]
|
||||
*
|
||||
* Or:
|
||||
* CREATE|ATTACH [MATERIALIZED] VIEW [IF NOT EXISTS] [db.]name [ENGINE = engine] [POPULATE] AS SELECT ...
|
||||
* CREATE|ATTACH [MATERIALIZED] VIEW [IF NOT EXISTS] [db.]name [TO [db.]name] [ENGINE = engine] [POPULATE] AS SELECT ...
|
||||
*/
|
||||
class ParserCreateQuery : public IParserBase
|
||||
{
|
||||
|
@ -70,8 +70,8 @@ StorageMaterializedView::StorageMaterializedView(
|
||||
if (!query.select)
|
||||
throw Exception("SELECT query is not specified for " + getName(), ErrorCodes::INCORRECT_QUERY);
|
||||
|
||||
if (!query.storage)
|
||||
throw Exception("ENGINE of MaterializedView must be specified explicitly", ErrorCodes::INCORRECT_QUERY);
|
||||
if (!query.storage && query.as_table.empty())
|
||||
throw Exception("ENGINE of MaterializedView should be specified explicitly", ErrorCodes::INCORRECT_QUERY);
|
||||
|
||||
extractDependentTable(*query.select, select_database_name, select_table_name);
|
||||
|
||||
@ -80,16 +80,28 @@ StorageMaterializedView::StorageMaterializedView(
|
||||
DatabaseAndTableName(select_database_name, select_table_name),
|
||||
DatabaseAndTableName(database_name, table_name));
|
||||
|
||||
String inner_table_name = getInnerTableName();
|
||||
// If the destination table is not set, use inner table
|
||||
if (!create.inner_storage)
|
||||
{
|
||||
target_database_name = create.as_database;
|
||||
target_table_name = create.as_table;
|
||||
}
|
||||
else
|
||||
{
|
||||
target_database_name = database_name;
|
||||
target_table_name = ".inner." + table_name;
|
||||
has_inner_table = true;
|
||||
}
|
||||
|
||||
inner_query = query.select->ptr();
|
||||
|
||||
/// If there is an ATTACH request, then the internal table must already be connected.
|
||||
if (!attach_)
|
||||
if (!attach_ && has_inner_table)
|
||||
{
|
||||
/// We will create a query to create an internal table.
|
||||
auto manual_create_query = std::make_shared<ASTCreateQuery>();
|
||||
manual_create_query->database = database_name;
|
||||
manual_create_query->table = inner_table_name;
|
||||
manual_create_query->database = target_database_name;
|
||||
manual_create_query->table = target_table_name;
|
||||
manual_create_query->set(manual_create_query->columns, query.columns->ptr());
|
||||
manual_create_query->set(manual_create_query->storage, query.storage->ptr());
|
||||
|
||||
@ -114,12 +126,12 @@ StorageMaterializedView::StorageMaterializedView(
|
||||
|
||||
NameAndTypePair StorageMaterializedView::getColumn(const String & column_name) const
|
||||
{
|
||||
return getInnerTable()->getColumn(column_name);
|
||||
return getTargetTable()->getColumn(column_name);
|
||||
}
|
||||
|
||||
bool StorageMaterializedView::hasColumn(const String & column_name) const
|
||||
{
|
||||
return getInnerTable()->hasColumn(column_name);
|
||||
return getTargetTable()->hasColumn(column_name);
|
||||
}
|
||||
|
||||
BlockInputStreams StorageMaterializedView::read(
|
||||
@ -130,12 +142,12 @@ BlockInputStreams StorageMaterializedView::read(
|
||||
const size_t max_block_size,
|
||||
const unsigned num_streams)
|
||||
{
|
||||
return getInnerTable()->read(column_names, query_info, context, processed_stage, max_block_size, num_streams);
|
||||
return getTargetTable()->read(column_names, query_info, context, processed_stage, max_block_size, num_streams);
|
||||
}
|
||||
|
||||
BlockOutputStreamPtr StorageMaterializedView::write(const ASTPtr & query, const Settings & settings)
|
||||
{
|
||||
return getInnerTable()->write(query, settings);
|
||||
return getTargetTable()->write(query, settings);
|
||||
}
|
||||
|
||||
void StorageMaterializedView::drop()
|
||||
@ -144,14 +156,12 @@ void StorageMaterializedView::drop()
|
||||
DatabaseAndTableName(select_database_name, select_table_name),
|
||||
DatabaseAndTableName(database_name, table_name));
|
||||
|
||||
auto inner_table_name = getInnerTableName();
|
||||
|
||||
if (global_context.tryGetTable(database_name, inner_table_name))
|
||||
if (has_inner_table && global_context.tryGetTable(target_database_name, target_table_name))
|
||||
{
|
||||
/// We create and execute `drop` query for internal table.
|
||||
auto drop_query = std::make_shared<ASTDropQuery>();
|
||||
drop_query->database = database_name;
|
||||
drop_query->table = inner_table_name;
|
||||
drop_query->database = target_database_name;
|
||||
drop_query->table = target_table_name;
|
||||
ASTPtr ast_drop_query = drop_query;
|
||||
InterpreterDropQuery drop_interpreter(ast_drop_query, global_context);
|
||||
drop_interpreter.execute();
|
||||
@ -160,12 +170,12 @@ void StorageMaterializedView::drop()
|
||||
|
||||
bool StorageMaterializedView::optimize(const ASTPtr & query, const ASTPtr & partition, bool final, bool deduplicate, const Context & context)
|
||||
{
|
||||
return getInnerTable()->optimize(query, partition, final, deduplicate, context);
|
||||
return getTargetTable()->optimize(query, partition, final, deduplicate, context);
|
||||
}
|
||||
|
||||
StoragePtr StorageMaterializedView::getInnerTable() const
|
||||
StoragePtr StorageMaterializedView::getTargetTable() const
|
||||
{
|
||||
return global_context.getTable(database_name, getInnerTableName());
|
||||
return global_context.getTable(target_database_name, target_table_name);
|
||||
}
|
||||
|
||||
}
|
||||
|
@ -20,18 +20,17 @@ public:
|
||||
std::string getName() const override { return "MaterializedView"; }
|
||||
std::string getTableName() const override { return table_name; }
|
||||
const NamesAndTypesList & getColumnsListImpl() const override { return *columns; }
|
||||
std::string getInnerTableName() const { return ".inner." + table_name; }
|
||||
ASTPtr getInnerQuery() const { return inner_query->clone(); };
|
||||
StoragePtr getInnerTable() const;
|
||||
StoragePtr getTargetTable() const;
|
||||
|
||||
NameAndTypePair getColumn(const String & column_name) const override;
|
||||
bool hasColumn(const String & column_name) const override;
|
||||
|
||||
bool supportsSampling() const override { return getInnerTable()->supportsSampling(); }
|
||||
bool supportsPrewhere() const override { return getInnerTable()->supportsPrewhere(); }
|
||||
bool supportsFinal() const override { return getInnerTable()->supportsFinal(); }
|
||||
bool supportsParallelReplicas() const override { return getInnerTable()->supportsParallelReplicas(); }
|
||||
bool supportsIndexForIn() const override { return getInnerTable()->supportsIndexForIn(); }
|
||||
bool supportsSampling() const override { return getTargetTable()->supportsSampling(); }
|
||||
bool supportsPrewhere() const override { return getTargetTable()->supportsPrewhere(); }
|
||||
bool supportsFinal() const override { return getTargetTable()->supportsFinal(); }
|
||||
bool supportsParallelReplicas() const override { return getTargetTable()->supportsParallelReplicas(); }
|
||||
bool supportsIndexForIn() const override { return getTargetTable()->supportsIndexForIn(); }
|
||||
|
||||
BlockOutputStreamPtr write(const ASTPtr & query, const Settings & settings) override;
|
||||
void drop() override;
|
||||
@ -48,11 +47,14 @@ public:
|
||||
private:
|
||||
String select_database_name;
|
||||
String select_table_name;
|
||||
String target_database_name;
|
||||
String target_table_name;
|
||||
String table_name;
|
||||
String database_name;
|
||||
ASTPtr inner_query;
|
||||
Context & global_context;
|
||||
NamesAndTypesListPtr columns;
|
||||
bool has_inner_table = false;
|
||||
|
||||
StorageMaterializedView(
|
||||
const String & table_name_,
|
||||
|
@ -0,0 +1,4 @@
|
||||
1
|
||||
2
|
||||
1
|
||||
2
|
@ -0,0 +1,23 @@
|
||||
DROP TABLE IF EXISTS test.src;
|
||||
DROP TABLE IF EXISTS test.dst;
|
||||
DROP TABLE IF EXISTS test.mv;
|
||||
|
||||
CREATE TABLE test.src (x UInt8) ENGINE = Null;
|
||||
CREATE TABLE test.dst (x UInt8) ENGINE = Memory();
|
||||
|
||||
CREATE MATERIALIZED VIEW test.mv TO test.dst AS SELECT * FROM test.src;
|
||||
INSERT INTO test.src VALUES (1), (2);
|
||||
|
||||
-- Detach MV and see if the data is still readable
|
||||
DETACH TABLE test.mv;
|
||||
SELECT * FROM test.dst;
|
||||
|
||||
-- Reattach MV (shortcut)
|
||||
ATTACH TABLE test.mv;
|
||||
|
||||
-- Drop the MV and see if the data is still readable
|
||||
DROP TABLE test.mv;
|
||||
SELECT * FROM test.dst;
|
||||
|
||||
DROP TABLE test.src;
|
||||
DROP TABLE test.dst;
|
@ -107,7 +107,7 @@ At the moment, ``ALTER`` queries for replicated tables are not supported yet.
|
||||
|
||||
CREATE VIEW
|
||||
~~~~~~~~~~~
|
||||
``CREATE [MATERIALIZED] VIEW [IF NOT EXISTS] [db.]name [ENGINE = engine] [POPULATE] AS SELECT ...``
|
||||
``CREATE [MATERIALIZED] VIEW [IF NOT EXISTS] [db.]name [TO [db.]name] [ENGINE = engine] [POPULATE] AS SELECT ...``
|
||||
|
||||
Creates a view. There are two types of views: normal and MATERIALIZED.
|
||||
|
||||
@ -133,7 +133,7 @@ This query is fully equivalent to using the subquery:
|
||||
|
||||
Materialized views store data transformed by the corresponding SELECT query.
|
||||
|
||||
When creating a materialized view, you have to specify ENGINE - the table engine for storing data.
|
||||
When creating a materialized view, you have to either specify ENGINE - the table engine for storing data, or target table for materialized results. By default, it uses the same engine as for the table that the SELECT query is made from.
|
||||
|
||||
A materialized view is arranged as follows: when inserting data to the table specified in SELECT, part of the inserted data is converted by this SELECT query, and the result is inserted in the view.
|
||||
|
||||
@ -142,6 +142,7 @@ If you specify POPULATE, the existing table data is inserted in the view when cr
|
||||
The SELECT query can contain DISTINCT, GROUP BY, ORDER BY, LIMIT ... Note that the corresponding conversions are performed independently on each block of inserted data. For example, if GROUP BY is set, data is aggregated during insertion, but only within a single packet of inserted data. The data won't be further aggregated. The exception is when using an ENGINE that independently performs data aggregation, such as SummingMergeTree.
|
||||
|
||||
The execution of ALTER queries on materialized views has not been fully developed, so they might be inconvenient.
|
||||
If the materialized view uses a ``TO [db.]name`` to specify a target table, it is possible to DETACH the view, ALTER the target table, and ATTACH the view again.
|
||||
|
||||
Views look the same as normal tables. For example, they are listed in the result of the SHOW TABLES query.
|
||||
|
||||
@ -154,6 +155,12 @@ The query is exactly the same as CREATE, except
|
||||
- The query doesn't create data on the disk, but assumes that data is already in the appropriate places, and just adds information about the table to the server.
|
||||
After executing an ATTACH query, the server will know about the existence of the table.
|
||||
|
||||
If the table has been previously detached and it's structure is known, it's possible to use shorthand form and omit structure definition:
|
||||
|
||||
.. code-block:: sql
|
||||
|
||||
ATTACH TABLE [IF NOT EXISTS] [db.]name
|
||||
|
||||
This query is used when starting the server. The server stores table metadata as files with ATTACH queries, which it simply runs at launch (with the exception of system tables, which are explicitly created on the server).
|
||||
|
||||
DROP
|
||||
|
Loading…
Reference in New Issue
Block a user