2018-11-13 13:48:53 +00:00
# include <Storages/StorageMaterializedView.h>
2023-02-12 19:17:55 +00:00
# include <Storages/MaterializedView/RefreshTask.h>
2018-03-16 01:23:37 +00:00
# include <Parsers/ASTSelectWithUnionQuery.h>
2017-04-01 09:19:00 +00:00
# include <Parsers/ASTCreateQuery.h>
2013-11-08 17:43:03 +00:00
2024-03-19 16:04:29 +00:00
# include <Access/Common/AccessFlags.h>
2017-05-23 18:33:48 +00:00
# include <Interpreters/Context.h>
2024-03-19 16:04:29 +00:00
# include <Interpreters/DatabaseCatalog.h>
2017-04-01 09:19:00 +00:00
# include <Interpreters/InterpreterCreateQuery.h>
# include <Interpreters/InterpreterDropQuery.h>
2023-02-12 19:17:55 +00:00
# include <Interpreters/InterpreterInsertQuery.h>
2019-05-07 06:54:55 +00:00
# include <Interpreters/InterpreterRenameQuery.h>
2023-03-10 12:52:27 +00:00
# include <Interpreters/InterpreterSelectWithUnionQuery.h>
2021-01-20 16:36:18 +00:00
# include <Interpreters/getHeaderForProcessingStage.h>
2024-03-19 16:04:29 +00:00
# include <Interpreters/getTableExpressions.h>
2013-11-08 17:43:03 +00:00
2020-01-29 17:44:16 +00:00
# include <Storages/AlterCommands.h>
2017-12-30 00:36:06 +00:00
# include <Storages/StorageFactory.h>
2019-12-11 13:09:46 +00:00
# include <Storages/ReadInOrderOptimizer.h>
2020-06-05 11:54:54 +00:00
# include <Storages/SelectQueryDescription.h>
2017-12-30 00:36:06 +00:00
2017-07-13 20:58:19 +00:00
# include <Common/typeid_cast.h>
2020-10-16 00:13:17 +00:00
# include <Common/checkStackSize.h>
2023-12-20 05:14:08 +00:00
# include <Core/ServerSettings.h>
2022-05-20 19:49:31 +00:00
# include <QueryPipeline/Pipe.h>
2021-09-08 18:29:38 +00:00
# include <Processors/QueryPlan/QueryPlan.h>
2021-01-20 16:36:18 +00:00
# include <Processors/QueryPlan/ExpressionStep.h>
2021-03-04 17:38:12 +00:00
# include <Processors/QueryPlan/BuildQueryPipelineSettings.h>
# include <Processors/QueryPlan/Optimizations/QueryPlanOptimizationSettings.h>
2021-07-23 19:33:59 +00:00
# include <Processors/Sinks/SinkToStorage.h>
2013-11-15 09:43:50 +00:00
2022-05-31 09:33:23 +00:00
# include <Backups/BackupEntriesCollector.h>
2022-02-22 17:05:52 +00:00
2013-11-08 17:43:03 +00:00
namespace DB
{
2016-01-11 21:46:36 +00:00
namespace ErrorCodes
{
2020-10-16 00:19:45 +00:00
extern const int BAD_ARGUMENTS ;
2020-02-25 18:02:41 +00:00
extern const int NOT_IMPLEMENTED ;
2017-08-18 20:56:19 +00:00
extern const int INCORRECT_QUERY ;
2018-03-16 01:23:37 +00:00
extern const int QUERY_IS_NOT_SUPPORTED_IN_MATERIALIZED_VIEW ;
2023-12-20 05:14:08 +00:00
extern const int TOO_MANY_MATERIALIZED_VIEWS ;
2024-03-23 18:07:58 +00:00
extern const int NO_SUCH_COLUMN_IN_TABLE ;
2023-02-12 19:17:55 +00:00
}
2023-02-12 19:17:55 +00:00
namespace ActionLocks
{
extern const StorageActionBlockType ViewRefresh ;
2016-01-11 21:46:36 +00:00
}
2020-03-18 17:38:52 +00:00
static inline String generateInnerTableName ( const StorageID & view_id )
2019-05-07 06:54:55 +00:00
{
2020-03-18 17:38:52 +00:00
if ( view_id . hasUUID ( ) )
2020-03-20 00:07:52 +00:00
return " .inner_id. " + toString ( view_id . uuid ) ;
2020-03-18 17:38:52 +00:00
return " .inner. " + view_id . getTableName ( ) ;
2019-05-07 06:54:55 +00:00
}
2016-01-11 21:46:36 +00:00
2021-06-12 09:27:05 +00:00
/// Remove columns from target_header that does not exists in src_header
static void removeNonCommonColumns ( const Block & src_header , Block & target_header )
{
std : : set < size_t > target_only_positions ;
for ( const auto & column : target_header )
{
if ( ! src_header . has ( column . name ) )
target_only_positions . insert ( target_header . getPositionByName ( column . name ) ) ;
}
target_header . erase ( target_only_positions ) ;
}
2018-03-16 01:23:37 +00:00
2024-03-21 20:52:17 +00:00
namespace
{
void checkTargetTableHasQueryOutputColumns ( const ColumnsDescription & target_table_columns , const ColumnsDescription & select_query_output_columns )
{
for ( const auto & column : select_query_output_columns )
if ( ! target_table_columns . has ( column . name ) )
throw Exception ( ErrorCodes : : NO_SUCH_COLUMN_IN_TABLE , " Column {} does not exist in the materialized view's inner table " , column . name ) ;
}
}
2014-09-30 03:08:47 +00:00
StorageMaterializedView : : StorageMaterializedView (
2019-12-04 16:06:55 +00:00
const StorageID & table_id_ ,
2021-04-10 23:33:54 +00:00
ContextPtr local_context ,
2017-09-17 18:49:43 +00:00
const ASTCreateQuery & query ,
2018-03-06 20:18:34 +00:00
const ColumnsDescription & columns_ ,
2024-02-19 22:57:35 +00:00
LoadingStrictnessLevel mode ,
2021-11-03 20:32:29 +00:00
const String & comment )
2021-05-31 14:49:02 +00:00
: IStorage ( table_id_ ) , WithMutableContext ( local_context - > getGlobalContext ( ) )
2013-11-08 17:43:03 +00:00
{
2020-06-19 15:39:41 +00:00
StorageInMemoryMetadata storage_metadata ;
storage_metadata . setColumns ( columns_ ) ;
2024-04-07 17:27:10 +00:00
auto * storage_def = query . storage ;
2024-04-05 09:53:32 +00:00
if ( storage_def & & storage_def - > primary_key )
storage_metadata . primary_key = KeyDescription : : getKeyFromAST ( storage_def - > primary_key - > ptr ( ) ,
storage_metadata . columns ,
local_context - > getGlobalContext ( ) ) ;
2024-05-03 20:50:49 +00:00
if ( query . sql_security )
storage_metadata . setSQLSecurity ( query . sql_security - > as < ASTSQLSecurity & > ( ) ) ;
2024-05-03 14:23:45 +00:00
2024-05-03 20:50:49 +00:00
/// Materialized view doesn't support SQL SECURITY INVOKER.
if ( storage_metadata . sql_security_type = = SQLSecurityType : : INVOKER )
2024-05-03 14:23:45 +00:00
throw Exception ( ErrorCodes : : QUERY_IS_NOT_SUPPORTED_IN_MATERIALIZED_VIEW , " SQL SECURITY INVOKER can't be specified for MATERIALIZED VIEW " ) ;
2017-09-17 18:49:43 +00:00
if ( ! query . select )
2023-01-23 21:13:58 +00:00
throw Exception ( ErrorCodes : : INCORRECT_QUERY , " SELECT query is not specified for {} " , getName ( ) ) ;
2017-08-18 20:56:19 +00:00
2019-12-10 19:48:16 +00:00
/// If the destination table is not set, use inner table
2019-12-10 19:48:16 +00:00
has_inner_table = query . to_table_id . empty ( ) ;
2019-12-10 19:48:16 +00:00
if ( has_inner_table & & ! query . storage )
2023-01-23 21:13:58 +00:00
throw Exception ( ErrorCodes : : INCORRECT_QUERY ,
" You must specify where to save results of a MaterializedView query: "
" either ENGINE or an existing table in a TO clause " ) ;
2017-08-18 20:56:19 +00:00
2023-11-29 02:32:41 +00:00
auto select = SelectQueryDescription : : getSelectQueryFromASTForMatView ( query . select - > clone ( ) , query . refresh_strategy ! = nullptr , local_context ) ;
2024-01-15 12:22:31 +00:00
if ( select . select_table_id )
{
auto select_table_dependent_views = DatabaseCatalog : : instance ( ) . getDependentViews ( select . select_table_id ) ;
2023-12-20 05:14:08 +00:00
2024-01-15 12:22:31 +00:00
auto max_materialized_views_count_for_table = getContext ( ) - > getServerSettings ( ) . max_materialized_views_count_for_table ;
if ( max_materialized_views_count_for_table & & select_table_dependent_views . size ( ) > = max_materialized_views_count_for_table )
throw Exception ( ErrorCodes : : TOO_MANY_MATERIALIZED_VIEWS ,
" Too many materialized views, maximum: {} " , max_materialized_views_count_for_table ) ;
}
2018-02-25 06:34:20 +00:00
2020-06-19 15:39:41 +00:00
storage_metadata . setSelectQuery ( select ) ;
2021-11-03 20:32:29 +00:00
if ( ! comment . empty ( ) )
storage_metadata . setComment ( comment ) ;
2023-11-29 02:32:41 +00:00
if ( query . refresh_strategy )
storage_metadata . setRefresh ( query . refresh_strategy - > clone ( ) ) ;
2021-11-03 20:32:29 +00:00
2020-06-19 15:39:41 +00:00
setInMemoryMetadata ( storage_metadata ) ;
2020-01-31 17:12:18 +00:00
2021-04-13 20:14:05 +00:00
bool point_to_itself_by_uuid = has_inner_table & & query . to_inner_uuid ! = UUIDHelpers : : Nil
& & query . to_inner_uuid = = table_id_ . uuid ;
bool point_to_itself_by_name = ! has_inner_table & & query . to_table_id . database_name = = table_id_ . database_name
& & query . to_table_id . table_name = = table_id_ . table_name ;
2021-04-13 19:13:26 +00:00
if ( point_to_itself_by_uuid | | point_to_itself_by_name )
throw Exception ( ErrorCodes : : BAD_ARGUMENTS , " Materialized view {} cannot point to itself " , table_id_ . getFullTableName ( ) ) ;
2019-12-10 19:48:16 +00:00
if ( ! has_inner_table )
2020-10-16 00:19:45 +00:00
{
2019-12-10 19:48:16 +00:00
target_table_id = query . to_table_id ;
2020-10-16 00:19:45 +00:00
}
2024-02-19 22:57:35 +00:00
else if ( LoadingStrictnessLevel : : ATTACH < = mode )
2017-10-21 20:08:49 +00:00
{
2019-12-10 19:48:16 +00:00
/// If there is an ATTACH request, then the internal table must already be created.
2021-03-08 17:26:38 +00:00
target_table_id = StorageID ( getStorageID ( ) . database_name , generateInnerTableName ( getStorageID ( ) ) , query . to_inner_uuid ) ;
2017-10-21 20:08:49 +00:00
}
else
2013-11-08 17:43:03 +00:00
{
2017-06-25 00:17:08 +00:00
/// We will create a query to create an internal table.
2021-04-10 23:33:54 +00:00
auto create_context = Context : : createCopy ( local_context ) ;
2016-05-28 15:42:22 +00:00
auto manual_create_query = std : : make_shared < ASTCreateQuery > ( ) ;
2021-09-06 22:13:54 +00:00
manual_create_query - > setDatabase ( getStorageID ( ) . database_name ) ;
manual_create_query - > setTable ( generateInnerTableName ( getStorageID ( ) ) ) ;
2021-03-08 17:26:38 +00:00
manual_create_query - > uuid = query . to_inner_uuid ;
2019-02-05 14:50:25 +00:00
auto new_columns_list = std : : make_shared < ASTColumns > ( ) ;
new_columns_list - > set ( new_columns_list - > columns , query . columns_list - > columns - > ptr ( ) ) ;
manual_create_query - > set ( manual_create_query - > columns_list , new_columns_list ) ;
2017-10-25 19:52:32 +00:00
manual_create_query - > set ( manual_create_query - > storage , query . storage - > ptr ( ) ) ;
2017-04-01 07:20:54 +00:00
2021-01-26 17:51:25 +00:00
InterpreterCreateQuery create_interpreter ( manual_create_query , create_context ) ;
2019-12-10 19:48:16 +00:00
create_interpreter . setInternal ( true ) ;
create_interpreter . execute ( ) ;
2021-09-06 22:13:54 +00:00
target_table_id = DatabaseCatalog : : instance ( ) . getTable ( { manual_create_query - > getDatabase ( ) , manual_create_query - > getTable ( ) } , getContext ( ) ) - > getStorageID ( ) ;
2013-11-13 14:39:48 +00:00
}
2023-02-12 19:17:55 +00:00
if ( query . refresh_strategy )
2023-12-22 00:56:39 +00:00
{
2023-02-12 19:17:55 +00:00
refresher = RefreshTask : : create (
* this ,
getContext ( ) ,
* query . refresh_strategy ) ;
2024-02-19 22:57:35 +00:00
refresh_on_start = mode < LoadingStrictnessLevel : : ATTACH & & ! query . is_create_empty ;
2023-12-22 00:56:39 +00:00
}
2013-11-08 17:43:03 +00:00
}
2021-04-10 23:33:54 +00:00
QueryProcessingStage : : Enum StorageMaterializedView : : getQueryProcessingStage (
2021-04-22 13:32:17 +00:00
ContextPtr local_context ,
QueryProcessingStage : : Enum to_stage ,
2021-07-09 03:15:41 +00:00
const StorageSnapshotPtr & ,
2021-04-22 13:32:17 +00:00
SelectQueryInfo & query_info ) const
2018-04-19 14:47:09 +00:00
{
2022-02-09 20:38:38 +00:00
const auto & target_metadata = getTargetTable ( ) - > getInMemoryMetadataPtr ( ) ;
2022-03-17 17:26:18 +00:00
return getTargetTable ( ) - > getQueryProcessingStage ( local_context , to_stage , getTargetTable ( ) - > getStorageSnapshot ( target_metadata , local_context ) , query_info ) ;
2018-04-19 14:47:09 +00:00
}
2024-02-29 18:01:54 +00:00
StorageSnapshotPtr StorageMaterializedView : : getStorageSnapshot ( const StorageMetadataPtr & metadata_snapshot , ContextPtr ) const
{
/// We cannot set virtuals at table creation because target table may not exist at that time.
2024-03-01 22:29:56 +00:00
return std : : make_shared < StorageSnapshot > ( * this , metadata_snapshot , getTargetTable ( ) - > getVirtualsPtr ( ) ) ;
2024-02-29 18:01:54 +00:00
}
2020-09-29 16:21:58 +00:00
void StorageMaterializedView : : read (
QueryPlan & query_plan ,
2013-11-08 17:43:03 +00:00
const Names & column_names ,
2021-07-09 03:15:41 +00:00
const StorageSnapshotPtr & storage_snapshot ,
2020-09-20 17:52:17 +00:00
SelectQueryInfo & query_info ,
2021-04-10 23:33:54 +00:00
ContextPtr local_context ,
2018-04-19 14:47:09 +00:00
QueryProcessingStage : : Enum processed_stage ,
2019-02-18 23:38:44 +00:00
const size_t max_block_size ,
2022-10-07 10:46:45 +00:00
const size_t num_streams )
2013-11-08 17:43:03 +00:00
{
2024-02-28 00:00:17 +00:00
auto context = getInMemoryMetadataPtr ( ) - > getSQLSecurityOverriddenContext ( local_context ) ;
2018-11-28 18:40:02 +00:00
auto storage = getTargetTable ( ) ;
2024-02-28 00:00:17 +00:00
auto lock = storage - > lockForShare ( context - > getCurrentQueryId ( ) , context - > getSettingsRef ( ) . lock_acquire_timeout ) ;
2021-01-20 16:36:18 +00:00
auto target_metadata_snapshot = storage - > getInMemoryMetadataPtr ( ) ;
2024-02-28 00:00:17 +00:00
auto target_storage_snapshot = storage - > getStorageSnapshot ( target_metadata_snapshot , context ) ;
2020-04-09 18:10:27 +00:00
2020-05-13 13:49:10 +00:00
if ( query_info . order_optimizer )
2024-02-28 00:00:17 +00:00
query_info . input_order_info = query_info . order_optimizer - > getInputOrder ( target_metadata_snapshot , context ) ;
2019-12-11 13:09:46 +00:00
2024-02-28 00:00:17 +00:00
if ( ! getInMemoryMetadataPtr ( ) - > select . select_table_id . empty ( ) )
context - > checkAccess ( AccessType : : SELECT , getInMemoryMetadataPtr ( ) - > select . select_table_id , column_names ) ;
2019-12-11 13:09:46 +00:00
2024-02-28 00:00:17 +00:00
auto storage_id = storage - > getStorageID ( ) ;
2024-05-03 14:23:45 +00:00
2024-05-03 20:50:49 +00:00
/// TODO: remove sql_security_type check after we turn `ignore_empty_sql_security_in_create_view_query=false`
2024-02-28 00:00:17 +00:00
/// We don't need to check access if the inner table was created automatically.
2024-05-03 20:50:49 +00:00
if ( ! has_inner_table & & ! storage_id . empty ( ) & & getInMemoryMetadataPtr ( ) - > sql_security_type )
2024-02-28 00:00:17 +00:00
context - > checkAccess ( AccessType : : SELECT , storage_id , column_names ) ;
storage - > read ( query_plan , column_names , target_storage_snapshot , query_info , context , processed_stage , max_block_size , num_streams ) ;
2020-10-22 10:31:10 +00:00
2020-10-22 12:01:22 +00:00
if ( query_plan . isInitialized ( ) )
{
2024-02-28 00:00:17 +00:00
auto mv_header = getHeaderForProcessingStage ( column_names , storage_snapshot , query_info , context , processed_stage ) ;
2021-01-21 12:57:18 +00:00
auto target_header = query_plan . getCurrentDataStream ( ) . header ;
2021-06-02 06:28:12 +00:00
/// No need to convert columns that does not exists in MV
2021-06-12 09:27:05 +00:00
removeNonCommonColumns ( mv_header , target_header ) ;
2021-06-02 06:28:12 +00:00
2021-06-02 07:31:28 +00:00
/// No need to convert columns that does not exists in the result header.
///
/// Distributed storage may process query up to the specific stage, and
/// so the result header may not include all the columns from the
/// materialized view.
2021-06-12 09:27:05 +00:00
removeNonCommonColumns ( target_header , mv_header ) ;
2021-06-02 07:31:28 +00:00
2021-01-20 16:36:18 +00:00
if ( ! blocksHaveEqualStructure ( mv_header , target_header ) )
{
auto converting_actions = ActionsDAG : : makeConvertingActions ( target_header . getColumnsWithTypeAndName ( ) ,
mv_header . getColumnsWithTypeAndName ( ) ,
ActionsDAG : : MatchColumnsMode : : Name ) ;
2023-10-17 14:02:30 +00:00
/* Leave columns outside from materialized view structure as is.
* They may be added in case of distributed query with JOIN .
* In that case underlying table returns joined columns as well .
*/
converting_actions - > projectInput ( false ) ;
2021-01-20 16:36:18 +00:00
auto converting_step = std : : make_unique < ExpressionStep > ( query_plan . getCurrentDataStream ( ) , converting_actions ) ;
converting_step - > setStepDescription ( " Convert target table structure to MaterializedView structure " ) ;
query_plan . addStep ( std : : move ( converting_step ) ) ;
}
2022-05-20 19:49:31 +00:00
query_plan . addStorageHolder ( storage ) ;
query_plan . addTableLock ( std : : move ( lock ) ) ;
2020-10-22 12:01:22 +00:00
}
2013-11-08 17:43:03 +00:00
}
2023-06-07 18:33:08 +00:00
SinkToStoragePtr StorageMaterializedView : : write ( const ASTPtr & query , const StorageMetadataPtr & /*metadata_snapshot*/ , ContextPtr local_context , bool async_insert )
2013-11-08 17:43:03 +00:00
{
2024-02-28 00:00:17 +00:00
auto context = getInMemoryMetadataPtr ( ) - > getSQLSecurityOverriddenContext ( local_context ) ;
2018-11-28 18:40:02 +00:00
auto storage = getTargetTable ( ) ;
2024-02-28 00:00:17 +00:00
auto lock = storage - > lockForShare ( context - > getCurrentQueryId ( ) , context - > getSettingsRef ( ) . lock_acquire_timeout ) ;
2020-06-15 19:08:58 +00:00
auto metadata_snapshot = storage - > getInMemoryMetadataPtr ( ) ;
2024-02-28 00:00:17 +00:00
auto storage_id = storage - > getStorageID ( ) ;
2024-05-03 14:23:45 +00:00
2024-05-03 20:50:49 +00:00
/// TODO: remove sql_security_type check after we turn `ignore_empty_sql_security_in_create_view_query=false`
2024-02-28 00:00:17 +00:00
/// We don't need to check access if the inner table was created automatically.
2024-05-03 20:50:49 +00:00
if ( ! has_inner_table & & ! storage_id . empty ( ) & & getInMemoryMetadataPtr ( ) - > sql_security_type )
2024-02-28 00:00:17 +00:00
{
auto query_sample_block = InterpreterInsertQuery : : getSampleBlock ( query - > as < ASTInsertQuery & > ( ) , storage , metadata_snapshot , context ) ;
context - > checkAccess ( AccessType : : INSERT , storage_id , query_sample_block . getNames ( ) ) ;
}
auto sink = storage - > write ( query , metadata_snapshot , context , async_insert ) ;
2020-06-15 19:08:58 +00:00
2021-07-23 19:33:59 +00:00
sink - > addTableLock ( lock ) ;
return sink ;
2013-11-08 17:43:03 +00:00
}
2017-04-01 07:20:54 +00:00
2020-01-22 11:30:11 +00:00
void StorageMaterializedView : : drop ( )
2018-04-21 00:35:20 +00:00
{
2019-12-03 16:25:32 +00:00
auto table_id = getStorageID ( ) ;
2020-06-17 14:06:22 +00:00
const auto & select_query = getInMemoryMetadataPtr ( ) - > getSelectQuery ( ) ;
2020-06-05 11:54:54 +00:00
if ( ! select_query . select_table_id . empty ( ) )
2022-12-02 14:05:46 +00:00
DatabaseCatalog : : instance ( ) . removeViewDependency ( select_query . select_table_id , table_id ) ;
2018-04-21 00:35:20 +00:00
2023-04-01 12:26:00 +00:00
/// Sync flag and the setting make sense for Atomic databases only.
/// However, with Atomic databases, IStorage::drop() can be called only from a background task in DatabaseCatalog.
/// Running synchronous DROP from that task leads to deadlock.
/// Usually dropInnerTableIfAny is no-op, because the inner table is dropped before enqueueing a drop task for the MV itself.
/// But there's a race condition with SYSTEM RESTART REPLICA: the inner table might be detached due to RESTART.
/// In this case, dropInnerTableIfAny will not find the inner table and will not drop it during executions of DROP query for the MV itself.
/// DDLGuard does not protect from that, because RESTART REPLICA acquires DDLGuard for the inner table name,
/// but DROP acquires DDLGuard for the name of MV. And we cannot acquire second DDLGuard for the inner name in DROP,
/// because it may lead to lock-order-inversion (DDLGuards must be acquired in lexicographical order).
dropInnerTableIfAny ( /* sync */ false , getContext ( ) ) ;
2020-10-07 19:09:51 +00:00
}
2022-06-23 07:59:13 +00:00
void StorageMaterializedView : : dropInnerTableIfAny ( bool sync , ContextPtr local_context )
2020-10-07 19:09:51 +00:00
{
2023-04-01 12:26:00 +00:00
/// We will use `sync` argument wneh this function is called from a DROP query
/// and will ignore database_atomic_wait_for_drop_and_detach_synchronously when it's called from drop task.
2023-08-16 22:42:51 +00:00
/// See the comment in StorageMaterializedView::drop.
/// DDL queries with StorageMaterializedView are fundamentally broken.
/// Best-effort to make them work: the inner table name is almost always less than the MV name (so it's safe to lock DDLGuard)
2023-02-12 19:17:55 +00:00
auto inner_table_id = getTargetTableId ( ) ;
bool may_lock_ddl_guard = getStorageID ( ) . getQualifiedName ( ) < inner_table_id . getQualifiedName ( ) ;
2018-06-09 18:17:27 +00:00
if ( has_inner_table & & tryGetTargetTable ( ) )
2023-02-12 19:17:55 +00:00
InterpreterDropQuery : : executeDropQuery ( ASTDropQuery : : Kind : : Drop , getContext ( ) , local_context , inner_table_id ,
2023-08-16 22:42:51 +00:00
sync , /* ignore_sync_setting */ true , may_lock_ddl_guard ) ;
2018-06-09 15:48:22 +00:00
}
2021-04-10 23:33:54 +00:00
void StorageMaterializedView : : truncate ( const ASTPtr & , const StorageMetadataPtr & , ContextPtr local_context , TableExclusiveLockHolder & )
2018-06-09 15:48:22 +00:00
{
if ( has_inner_table )
2023-02-12 19:17:55 +00:00
InterpreterDropQuery : : executeDropQuery ( ASTDropQuery : : Kind : : Truncate , getContext ( ) , local_context , getTargetTableId ( ) , true ) ;
2018-04-21 00:35:20 +00:00
}
2018-03-20 13:42:44 +00:00
void StorageMaterializedView : : checkStatementCanBeForwarded ( ) const
{
if ( ! has_inner_table )
2023-01-23 21:13:58 +00:00
throw Exception ( ErrorCodes : : INCORRECT_QUERY , " MATERIALIZED VIEW targets existing table {}. "
2023-02-12 19:17:55 +00:00
" Execute the statement directly on it. " , getTargetTableId ( ) . getNameForLogs ( ) ) ;
2018-03-20 13:42:44 +00:00
}
2020-06-17 13:39:26 +00:00
bool StorageMaterializedView : : optimize (
const ASTPtr & query ,
const StorageMetadataPtr & /*metadata_snapshot*/ ,
const ASTPtr & partition ,
bool final ,
bool deduplicate ,
2020-12-01 09:10:12 +00:00
const Names & deduplicate_by_columns ,
2023-02-16 13:03:16 +00:00
bool cleanup ,
2021-04-10 23:33:54 +00:00
ContextPtr local_context )
2015-04-10 17:09:16 +00:00
{
2018-03-20 13:42:44 +00:00
checkStatementCanBeForwarded ( ) ;
2020-06-17 13:39:26 +00:00
auto storage_ptr = getTargetTable ( ) ;
auto metadata_snapshot = storage_ptr - > getInMemoryMetadataPtr ( ) ;
2023-12-29 14:46:24 +00:00
return storage_ptr - > optimize ( query , metadata_snapshot , partition , final , deduplicate , deduplicate_by_columns , cleanup , local_context ) ;
2023-02-12 19:17:55 +00:00
}
2023-12-04 23:23:11 +00:00
std : : tuple < ContextMutablePtr , std : : shared_ptr < ASTInsertQuery > > StorageMaterializedView : : prepareRefresh ( ) const
2023-02-12 19:17:55 +00:00
{
2024-02-28 00:00:17 +00:00
auto refresh_context = getInMemoryMetadataPtr ( ) - > getSQLSecurityOverriddenContext ( getContext ( ) ) ;
2023-12-04 23:23:11 +00:00
/// Generate a random query id.
refresh_context - > setCurrentQueryId ( " " ) ;
CurrentThread : : QueryScope query_scope ( refresh_context ) ;
2023-02-12 19:17:55 +00:00
auto inner_table_id = getTargetTableId ( ) ;
auto new_table_name = " .tmp " + generateInnerTableName ( getStorageID ( ) ) ;
auto db = DatabaseCatalog : : instance ( ) . getDatabase ( inner_table_id . database_name ) ;
auto create_table_query = db - > getCreateTableQuery ( inner_table_id . table_name , getContext ( ) ) ;
auto & create_query = create_table_query - > as < ASTCreateQuery & > ( ) ;
create_query . setTable ( new_table_name ) ;
create_query . setDatabase ( db - > getDatabaseName ( ) ) ;
create_query . create_or_replace = true ;
create_query . replace_table = true ;
create_query . uuid = UUIDHelpers : : Nil ;
2023-12-04 23:23:11 +00:00
InterpreterCreateQuery create_interpreter ( create_table_query , refresh_context ) ;
2023-02-12 19:17:55 +00:00
create_interpreter . setInternal ( true ) ;
create_interpreter . execute ( ) ;
2023-12-04 23:23:11 +00:00
StorageID fresh_table = DatabaseCatalog : : instance ( ) . getTable ( { create_query . getDatabase ( ) , create_query . getTable ( ) } , getContext ( ) ) - > getStorageID ( ) ;
2023-02-15 02:58:26 +00:00
2023-02-12 19:17:55 +00:00
auto insert_query = std : : make_shared < ASTInsertQuery > ( ) ;
insert_query - > select = getInMemoryMetadataPtr ( ) - > getSelectQuery ( ) . select_query ;
2023-12-04 23:23:11 +00:00
insert_query - > setTable ( fresh_table . table_name ) ;
insert_query - > setDatabase ( fresh_table . database_name ) ;
insert_query - > table_id = fresh_table ;
return { refresh_context , insert_query } ;
2023-02-12 19:17:55 +00:00
}
2023-12-04 23:23:11 +00:00
StorageID StorageMaterializedView : : exchangeTargetTable ( StorageID fresh_table , ContextPtr refresh_context )
2023-02-12 19:17:55 +00:00
{
2023-02-15 02:58:26 +00:00
auto stale_table_id = getTargetTableId ( ) ;
2023-02-12 19:17:55 +00:00
2023-02-15 02:58:26 +00:00
auto db = DatabaseCatalog : : instance ( ) . getDatabase ( stale_table_id . database_name ) ;
auto target_db = DatabaseCatalog : : instance ( ) . getDatabase ( fresh_table . database_name ) ;
2023-02-12 19:17:55 +00:00
2023-12-04 23:23:11 +00:00
CurrentThread : : QueryScope query_scope ( refresh_context ) ;
2023-02-12 19:17:55 +00:00
target_db - > renameTable (
2023-12-04 23:23:11 +00:00
refresh_context , fresh_table . table_name , * db , stale_table_id . table_name , /*exchange=*/ true , /*dictionary=*/ false ) ;
2023-02-12 19:17:55 +00:00
2023-02-15 05:04:13 +00:00
std : : swap ( stale_table_id . database_name , fresh_table . database_name ) ;
std : : swap ( stale_table_id . table_name , fresh_table . table_name ) ;
setTargetTableId ( std : : move ( fresh_table ) ) ;
2023-02-15 02:58:26 +00:00
return stale_table_id ;
2013-11-08 17:43:03 +00:00
}
2020-01-29 17:44:16 +00:00
void StorageMaterializedView : : alter (
const AlterCommands & params ,
2021-04-10 23:33:54 +00:00
ContextPtr local_context ,
2021-10-25 17:49:49 +00:00
AlterLockHolder & )
2020-01-29 17:44:16 +00:00
{
auto table_id = getStorageID ( ) ;
2020-06-09 21:22:01 +00:00
StorageInMemoryMetadata new_metadata = getInMemoryMetadata ( ) ;
2020-06-17 14:06:22 +00:00
StorageInMemoryMetadata old_metadata = getInMemoryMetadata ( ) ;
2021-04-10 23:33:54 +00:00
params . apply ( new_metadata , local_context ) ;
2020-01-31 17:12:18 +00:00
2023-11-28 12:17:55 +00:00
const auto & new_select = new_metadata . select ;
const auto & old_select = old_metadata . getSelectQuery ( ) ;
2020-01-31 17:12:18 +00:00
2023-11-28 12:17:55 +00:00
DatabaseCatalog : : instance ( ) . updateViewDependency ( old_select . select_table_id , table_id , new_select . select_table_id , table_id ) ;
2023-03-10 12:52:27 +00:00
2024-03-23 18:07:58 +00:00
new_metadata . setSelectQuery ( new_select ) ;
2023-03-10 12:52:27 +00:00
2024-03-23 18:10:42 +00:00
/// Check the materialized view's inner table structure.
2024-03-23 18:07:58 +00:00
if ( has_inner_table )
{
2024-03-21 20:52:17 +00:00
/// If this materialized view has an inner table it should always have the same columns as this materialized view.
/// Try to find mistakes in the select query (it shouldn't have columns which are not in the inner table).
auto target_table_metadata = getTargetTable ( ) - > getInMemoryMetadataPtr ( ) ;
const auto & select_query_output_columns = new_metadata . columns ; /// AlterCommands::alter() analyzed the query and assigned `new_metadata.columns` before.
checkTargetTableHasQueryOutputColumns ( target_table_metadata - > columns , select_query_output_columns ) ;
/// We need to copy the target table's columns (after checkTargetTableHasQueryOutputColumns() they can be still different - e.g. the data types of those columns can differ).
new_metadata . columns = target_table_metadata - > columns ;
2020-01-31 17:12:18 +00:00
}
2021-04-10 23:33:54 +00:00
DatabaseCatalog : : instance ( ) . getDatabase ( table_id . database_name ) - > alterTable ( local_context , table_id , new_metadata ) ;
2020-06-15 16:55:33 +00:00
setInMemoryMetadata ( new_metadata ) ;
2023-11-29 02:32:41 +00:00
if ( refresher )
refresher - > alterRefreshParams ( new_metadata . refresh - > as < const ASTRefreshStrategy & > ( ) ) ;
2020-01-29 17:44:16 +00:00
}
2023-11-28 13:01:31 +00:00
void StorageMaterializedView : : checkAlterIsPossible ( const AlterCommands & commands , ContextPtr /*local_context*/ ) const
2020-01-29 17:44:16 +00:00
{
2023-11-28 12:17:55 +00:00
for ( const auto & command : commands )
2020-01-29 17:44:16 +00:00
{
2024-02-28 00:00:17 +00:00
if ( command . type = = AlterCommand : : MODIFY_SQL_SECURITY )
2020-01-29 17:44:16 +00:00
{
2024-02-28 00:00:17 +00:00
if ( command . sql_security - > as < ASTSQLSecurity & > ( ) . type = = SQLSecurityType : : INVOKER )
throw Exception ( ErrorCodes : : QUERY_IS_NOT_SUPPORTED_IN_MATERIALIZED_VIEW , " SQL SECURITY INVOKER can't be specified for MATERIALIZED VIEW " ) ;
2023-11-29 02:32:41 +00:00
continue ;
2020-01-29 17:44:16 +00:00
}
2024-02-28 00:00:17 +00:00
else if ( command . isCommentAlter ( ) )
2023-11-29 02:32:41 +00:00
continue ;
2024-02-28 00:00:17 +00:00
else if ( command . type = = AlterCommand : : MODIFY_QUERY )
2023-11-29 02:32:41 +00:00
continue ;
2024-02-28 00:00:17 +00:00
else if ( command . type = = AlterCommand : : MODIFY_REFRESH & & refresher )
continue ;
2023-11-29 02:32:41 +00:00
throw Exception ( ErrorCodes : : NOT_IMPLEMENTED , " Alter of type '{}' is not supported by storage {} " ,
2024-02-28 00:00:17 +00:00
command . type , getName ( ) ) ;
2020-01-29 17:44:16 +00:00
}
2024-02-28 00:00:17 +00:00
2020-01-29 17:44:16 +00:00
}
2021-02-25 10:07:48 +00:00
void StorageMaterializedView : : checkMutationIsPossible ( const MutationCommands & commands , const Settings & settings ) const
{
checkStatementCanBeForwarded ( ) ;
getTargetTable ( ) - > checkMutationIsPossible ( commands , settings ) ;
}
2020-08-03 13:54:14 +00:00
Pipe StorageMaterializedView : : alterPartition (
2021-04-10 23:33:54 +00:00
const StorageMetadataPtr & metadata_snapshot , const PartitionCommands & commands , ContextPtr local_context )
2018-03-20 13:42:44 +00:00
{
checkStatementCanBeForwarded ( ) ;
2021-04-10 23:33:54 +00:00
return getTargetTable ( ) - > alterPartition ( metadata_snapshot , commands , local_context ) ;
2018-03-20 13:42:44 +00:00
}
2020-07-14 08:19:39 +00:00
void StorageMaterializedView : : checkAlterPartitionIsPossible (
2023-10-13 14:22:18 +00:00
const PartitionCommands & commands , const StorageMetadataPtr & metadata_snapshot ,
const Settings & settings , ContextPtr local_context ) const
2020-07-14 08:19:39 +00:00
{
checkStatementCanBeForwarded ( ) ;
2023-10-13 14:22:18 +00:00
getTargetTable ( ) - > checkAlterPartitionIsPossible ( commands , metadata_snapshot , settings , local_context ) ;
2020-07-14 08:19:39 +00:00
}
2023-01-30 17:38:28 +00:00
void StorageMaterializedView : : mutate ( const MutationCommands & commands , ContextPtr local_context )
2018-08-20 16:28:30 +00:00
{
checkStatementCanBeForwarded ( ) ;
2023-01-30 17:38:28 +00:00
getTargetTable ( ) - > mutate ( commands , local_context ) ;
2018-08-20 16:28:30 +00:00
}
2020-04-07 14:05:51 +00:00
void StorageMaterializedView : : renameInMemory ( const StorageID & new_table_id )
2019-05-07 06:54:55 +00:00
{
2020-03-18 17:38:52 +00:00
auto old_table_id = getStorageID ( ) ;
2023-02-12 19:17:55 +00:00
auto inner_table_id = getTargetTableId ( ) ;
2020-06-17 14:06:22 +00:00
auto metadata_snapshot = getInMemoryMetadataPtr ( ) ;
2020-04-10 01:35:37 +00:00
bool from_atomic_to_atomic_database = old_table_id . hasUUID ( ) & & new_table_id . hasUUID ( ) ;
2020-04-07 14:05:51 +00:00
2021-05-19 18:53:31 +00:00
if ( ! from_atomic_to_atomic_database & & has_inner_table & & tryGetTargetTable ( ) )
2019-05-07 06:54:55 +00:00
{
2020-04-08 01:02:00 +00:00
auto new_target_table_name = generateInnerTableName ( new_table_id ) ;
2019-05-07 06:54:55 +00:00
2024-03-08 11:54:20 +00:00
ASTRenameQuery : : Elements rename_elements ;
2023-02-12 19:17:55 +00:00
assert ( inner_table_id . database_name = = old_table_id . database_name ) ;
2019-05-07 06:54:55 +00:00
2023-02-14 13:01:06 +00:00
ASTRenameQuery : : Element elem
{
ASTRenameQuery : : Table
{
2023-02-12 19:17:55 +00:00
inner_table_id . database_name . empty ( ) ? nullptr : std : : make_shared < ASTIdentifier > ( inner_table_id . database_name ) ,
std : : make_shared < ASTIdentifier > ( inner_table_id . table_name )
2023-02-14 13:01:06 +00:00
} ,
ASTRenameQuery : : Table
{
2023-02-14 22:17:23 +00:00
new_table_id . database_name . empty ( ) ? nullptr : std : : make_shared < ASTIdentifier > ( new_table_id . database_name ) ,
2023-02-14 13:01:06 +00:00
std : : make_shared < ASTIdentifier > ( new_target_table_name )
}
} ;
2024-03-08 11:54:20 +00:00
rename_elements . emplace_back ( std : : move ( elem ) ) ;
2019-05-07 06:54:55 +00:00
2024-03-08 11:54:20 +00:00
auto rename = std : : make_shared < ASTRenameQuery > ( std : : move ( rename_elements ) ) ;
2021-04-10 23:33:54 +00:00
InterpreterRenameQuery ( rename , getContext ( ) ) . execute ( ) ;
2023-02-12 19:17:55 +00:00
updateTargetTableId ( new_table_id . database_name , new_target_table_name ) ;
2019-05-07 06:54:55 +00:00
}
2019-05-09 06:12:02 +00:00
2020-04-07 14:05:51 +00:00
IStorage : : renameInMemory ( new_table_id ) ;
2021-05-19 18:53:31 +00:00
if ( from_atomic_to_atomic_database & & has_inner_table )
{
2023-02-12 19:17:55 +00:00
assert ( inner_table_id . database_name = = old_table_id . database_name ) ;
updateTargetTableId ( new_table_id . database_name , std : : nullopt ) ;
2021-05-19 18:53:31 +00:00
}
2020-06-17 14:06:22 +00:00
const auto & select_query = metadata_snapshot - > getSelectQuery ( ) ;
2024-03-23 18:10:42 +00:00
/// TODO: Actually, we don't need to update dependency if MV has UUID, but then db and table name will be outdated
2022-12-02 14:05:46 +00:00
DatabaseCatalog : : instance ( ) . updateViewDependency ( select_query . select_table_id , old_table_id , select_query . select_table_id , getStorageID ( ) ) ;
2023-11-29 02:32:41 +00:00
if ( refresher )
refresher - > rename ( new_table_id ) ;
2019-05-07 06:54:55 +00:00
}
2022-05-16 09:04:37 +00:00
void StorageMaterializedView : : startup ( )
{
auto metadata_snapshot = getInMemoryMetadataPtr ( ) ;
const auto & select_query = metadata_snapshot - > getSelectQuery ( ) ;
if ( ! select_query . select_table_id . empty ( ) )
2022-12-02 14:05:46 +00:00
DatabaseCatalog : : instance ( ) . addViewDependency ( select_query . select_table_id , getStorageID ( ) ) ;
2023-02-12 19:17:55 +00:00
if ( refresher )
{
2023-11-23 05:08:44 +00:00
refresher - > initializeAndStart ( std : : static_pointer_cast < StorageMaterializedView > ( shared_from_this ( ) ) ) ;
2023-12-22 00:56:39 +00:00
if ( refresh_on_start )
refresher - > run ( ) ;
2023-02-12 19:17:55 +00:00
}
2022-05-16 09:04:37 +00:00
}
2023-11-06 14:40:01 +00:00
void StorageMaterializedView : : shutdown ( bool )
2017-12-16 01:41:06 +00:00
{
2023-02-12 19:17:55 +00:00
if ( refresher )
2023-11-24 01:32:45 +00:00
refresher - > shutdown ( ) ;
2023-02-12 19:17:55 +00:00
2020-06-17 14:06:22 +00:00
auto metadata_snapshot = getInMemoryMetadataPtr ( ) ;
const auto & select_query = metadata_snapshot - > getSelectQuery ( ) ;
2017-12-16 01:41:06 +00:00
/// Make sure the dependency is removed after DETACH TABLE
2020-06-05 11:54:54 +00:00
if ( ! select_query . select_table_id . empty ( ) )
2022-12-02 14:05:46 +00:00
DatabaseCatalog : : instance ( ) . removeViewDependency ( select_query . select_table_id , getStorageID ( ) ) ;
2017-12-16 01:41:06 +00:00
}
2017-10-21 20:08:49 +00:00
StoragePtr StorageMaterializedView : : getTargetTable ( ) const
2017-03-11 00:27:59 +00:00
{
2020-10-16 00:13:17 +00:00
checkStackSize ( ) ;
2023-02-12 19:17:55 +00:00
return DatabaseCatalog : : instance ( ) . getTable ( getTargetTableId ( ) , getContext ( ) ) ;
2017-03-11 00:27:59 +00:00
}
2018-06-09 18:17:27 +00:00
StoragePtr StorageMaterializedView : : tryGetTargetTable ( ) const
{
2020-10-16 00:13:17 +00:00
checkStackSize ( ) ;
2023-02-12 19:17:55 +00:00
return DatabaseCatalog : : instance ( ) . tryGetTable ( getTargetTableId ( ) , getContext ( ) ) ;
2022-01-10 07:53:41 +00:00
}
2019-04-04 13:13:59 +00:00
Strings StorageMaterializedView : : getDataPaths ( ) const
2018-06-09 18:17:27 +00:00
{
if ( auto table = tryGetTargetTable ( ) )
2019-04-04 13:13:59 +00:00
return table - > getDataPaths ( ) ;
2018-06-09 18:17:27 +00:00
return { } ;
}
2022-05-31 09:33:23 +00:00
void StorageMaterializedView : : backupData ( BackupEntriesCollector & backup_entries_collector , const String & data_path_in_backup , const std : : optional < ASTs > & partitions )
2022-02-22 17:05:52 +00:00
{
2022-05-31 09:33:23 +00:00
/// We backup the target table's data only if it's inner.
2022-05-29 19:53:56 +00:00
if ( hasInnerTable ( ) )
2023-11-06 18:05:12 +00:00
{
if ( auto table = tryGetTargetTable ( ) )
table - > backupData ( backup_entries_collector , data_path_in_backup , partitions ) ;
else
2024-01-23 17:04:50 +00:00
LOG_WARNING ( getLogger ( " StorageMaterializedView " ) ,
2023-11-06 18:05:12 +00:00
" Inner table does not exist, will not backup any data " ) ;
}
2022-02-22 17:05:52 +00:00
}
2022-05-31 09:33:23 +00:00
void StorageMaterializedView : : restoreDataFromBackup ( RestorerFromBackup & restorer , const String & data_path_in_backup , const std : : optional < ASTs > & partitions )
2022-02-22 17:05:52 +00:00
{
2022-05-31 09:33:23 +00:00
if ( hasInnerTable ( ) )
return getTargetTable ( ) - > restoreDataFromBackup ( restorer , data_path_in_backup , partitions ) ;
2022-02-22 17:05:52 +00:00
}
2022-06-29 12:42:23 +00:00
bool StorageMaterializedView : : supportsBackupPartition ( ) const
{
if ( hasInnerTable ( ) )
return getTargetTable ( ) - > supportsBackupPartition ( ) ;
return false ;
}
2022-05-02 22:01:11 +00:00
std : : optional < UInt64 > StorageMaterializedView : : totalRows ( const Settings & settings ) const
{
2022-05-03 17:55:45 +00:00
if ( hasInnerTable ( ) )
{
if ( auto table = tryGetTargetTable ( ) )
return table - > totalRows ( settings ) ;
}
return { } ;
2022-05-02 22:01:11 +00:00
}
std : : optional < UInt64 > StorageMaterializedView : : totalBytes ( const Settings & settings ) const
{
2022-05-03 17:55:45 +00:00
if ( hasInnerTable ( ) )
{
if ( auto table = tryGetTargetTable ( ) )
return table - > totalBytes ( settings ) ;
}
return { } ;
2022-05-02 22:01:11 +00:00
}
2023-11-24 04:45:59 +00:00
std : : optional < UInt64 > StorageMaterializedView : : totalBytesUncompressed ( const Settings & settings ) const
{
if ( hasInnerTable ( ) )
{
if ( auto table = tryGetTargetTable ( ) )
return table - > totalBytesUncompressed ( settings ) ;
}
return { } ;
}
2019-04-08 05:13:16 +00:00
ActionLock StorageMaterializedView : : getActionLock ( StorageActionBlockType type )
{
2023-02-12 19:17:55 +00:00
if ( type = = ActionLocks : : ViewRefresh & & refresher )
refresher - > stop ( ) ;
2021-05-31 13:38:33 +00:00
if ( has_inner_table )
{
if ( auto target_table = tryGetTargetTable ( ) )
return target_table - > getActionLock ( type ) ;
}
return ActionLock { } ;
2019-04-08 05:13:16 +00:00
}
2023-10-16 15:17:55 +00:00
bool StorageMaterializedView : : isRemote ( ) const
{
if ( auto table = tryGetTargetTable ( ) )
return table - > isRemote ( ) ;
return false ;
}
2023-02-12 19:17:55 +00:00
void StorageMaterializedView : : onActionLockRemove ( StorageActionBlockType action_type )
{
if ( action_type = = ActionLocks : : ViewRefresh & & refresher )
refresher - > start ( ) ;
}
DB : : StorageID StorageMaterializedView : : getTargetTableId ( ) const
{
std : : lock_guard guard ( target_table_id_mutex ) ;
return target_table_id ;
}
void StorageMaterializedView : : setTargetTableId ( DB : : StorageID id )
{
std : : lock_guard guard ( target_table_id_mutex ) ;
target_table_id = std : : move ( id ) ;
}
void StorageMaterializedView : : updateTargetTableId ( std : : optional < String > database_name , std : : optional < String > table_name )
{
std : : lock_guard guard ( target_table_id_mutex ) ;
if ( database_name )
target_table_id . database_name = * std : : move ( database_name ) ;
if ( table_name )
target_table_id . table_name = * std : : move ( table_name ) ;
}
2017-12-30 00:36:06 +00:00
void registerStorageMaterializedView ( StorageFactory & factory )
{
factory . registerStorage ( " MaterializedView " , [ ] ( const StorageFactory : : Arguments & args )
{
/// Pass local_context here to convey setting for inner table
2022-04-19 20:47:29 +00:00
return std : : make_shared < StorageMaterializedView > (
2021-04-10 23:33:54 +00:00
args . table_id , args . getLocalContext ( ) , args . query ,
2024-02-19 22:57:35 +00:00
args . columns , args . mode , args . comment ) ;
2017-12-30 00:36:06 +00:00
} ) ;
}
2013-11-08 17:43:03 +00:00
}