2019-05-31 04:03:46 +00:00
# include "StorageMergeTree.h"
2018-12-25 23:13:30 +00:00
2019-05-31 04:03:46 +00:00
# include <Databases/IDatabase.h>
2017-04-01 09:19:00 +00:00
# include <Common/escapeForFileName.h>
2017-07-13 20:58:19 +00:00
# include <Common/typeid_cast.h>
2019-03-09 18:24:11 +00:00
# include <Common/ThreadPool.h>
2017-04-01 09:19:00 +00:00
# include <Interpreters/InterpreterAlterQuery.h>
# include <Interpreters/PartLog.h>
2020-05-20 20:16:32 +00:00
# include <Interpreters/MutationsInterpreter.h>
# include <Interpreters/Context.h>
2021-05-17 11:14:09 +00:00
# include <Interpreters/TransactionLog.h>
2019-07-03 13:17:19 +00:00
# include <Parsers/ASTCheckQuery.h>
2017-04-01 09:19:00 +00:00
# include <Parsers/ASTFunction.h>
2017-09-06 20:34:26 +00:00
# include <Parsers/ASTLiteral.h>
2019-07-03 13:17:19 +00:00
# include <Parsers/ASTPartition.h>
2019-07-24 12:56:39 +00:00
# include <Parsers/ASTSetQuery.h>
2018-12-25 23:13:30 +00:00
# include <Parsers/queryToString.h>
2017-04-14 12:40:48 +00:00
# include <Storages/MergeTree/MergeTreeData.h>
2017-08-14 18:16:11 +00:00
# include <Storages/MergeTree/ActiveDataPartSet.h>
2018-12-25 23:13:30 +00:00
# include <Storages/AlterCommands.h>
2018-12-25 23:18:07 +00:00
# include <Storages/PartitionCommands.h>
2018-12-25 23:13:30 +00:00
# include <Storages/MergeTree/MergeTreeBlockOutputStream.h>
2020-04-20 01:38:38 +00:00
# include <Storages/MergeTree/MergeTreeDataPartInMemory.h>
2020-10-22 06:18:10 +00:00
# include <Storages/MergeTree/PartitionPruner.h>
2018-12-25 23:13:30 +00:00
# include <Storages/MergeTree/MergeList.h>
2019-07-03 08:49:52 +00:00
# include <Storages/MergeTree/checkDataPart.h>
2019-10-20 09:12:42 +00:00
# include <Processors/Pipe.h>
2021-03-04 17:38:12 +00:00
# include <Processors/QueryPlan/BuildQueryPipelineSettings.h>
# include <Processors/QueryPlan/Optimizations/QueryPlanOptimizationSettings.h>
2020-05-20 20:16:32 +00:00
# include <optional>
2012-07-19 20:32:10 +00:00
2020-10-13 14:25:42 +00:00
namespace CurrentMetrics
{
extern const Metric BackgroundPoolTask ;
}
2016-01-11 21:46:36 +00:00
2012-07-17 20:04:39 +00:00
namespace DB
{
2016-01-11 21:46:36 +00:00
namespace ErrorCodes
{
2020-02-25 18:02:41 +00:00
extern const int NOT_IMPLEMENTED ;
extern const int LOGICAL_ERROR ;
extern const int NOT_ENOUGH_SPACE ;
2017-04-01 07:20:54 +00:00
extern const int BAD_ARGUMENTS ;
2017-06-19 20:47:02 +00:00
extern const int INCORRECT_DATA ;
2018-01-12 17:30:21 +00:00
extern const int CANNOT_ASSIGN_OPTIMIZE ;
2020-08-15 05:21:02 +00:00
extern const int TIMEOUT_EXCEEDED ;
2020-10-16 11:58:47 +00:00
extern const int UNKNOWN_POLICY ;
2020-11-03 09:24:10 +00:00
extern const int NO_SUCH_DATA_PART ;
2018-05-21 13:49:54 +00:00
}
namespace ActionLocks
{
extern const StorageActionBlockType PartsMerge ;
2019-08-01 15:36:12 +00:00
extern const StorageActionBlockType PartsTTLMerge ;
2019-09-03 14:50:49 +00:00
extern const StorageActionBlockType PartsMove ;
2016-01-11 21:46:36 +00:00
}
2014-09-30 03:08:47 +00:00
StorageMergeTree : : StorageMergeTree (
2019-12-04 16:06:55 +00:00
const StorageID & table_id_ ,
2019-10-28 20:12:14 +00:00
const String & relative_data_path_ ,
2020-06-09 17:28:29 +00:00
const StorageInMemoryMetadata & metadata_ ,
2017-04-01 07:20:54 +00:00
bool attach ,
2021-05-31 14:49:02 +00:00
ContextMutablePtr context_ ,
2017-09-08 18:11:09 +00:00
const String & date_column_name ,
2019-05-03 02:00:57 +00:00
const MergingParams & merging_params_ ,
2019-08-26 14:24:29 +00:00
std : : unique_ptr < MergeTreeSettings > storage_settings_ ,
2017-09-19 20:42:42 +00:00
bool has_force_restore_data_flag )
2019-12-27 16:34:50 +00:00
: MergeTreeData (
2019-12-30 11:08:09 +00:00
table_id_ ,
2019-12-27 16:34:50 +00:00
relative_data_path_ ,
2020-06-09 17:28:29 +00:00
metadata_ ,
2019-12-27 16:34:50 +00:00
context_ ,
date_column_name ,
merging_params_ ,
std : : move ( storage_settings_ ) ,
2019-12-30 11:08:09 +00:00
false , /// require_part_metadata
2019-12-27 16:34:50 +00:00
attach )
, reader ( * this )
, writer ( * this )
2021-04-10 23:33:54 +00:00
, merger_mutator ( * this , getContext ( ) - > getSettingsRef ( ) . background_pool_size )
, background_executor ( * this , getContext ( ) )
, background_moves_executor ( * this , getContext ( ) )
2020-10-14 14:56:42 +00:00
2014-03-13 12:48:07 +00:00
{
2019-05-03 02:00:57 +00:00
loadDataParts ( has_force_restore_data_flag ) ;
2017-06-19 20:47:02 +00:00
2019-05-03 02:00:57 +00:00
if ( ! attach & & ! getDataParts ( ) . empty ( ) )
2018-03-10 19:57:13 +00:00
throw Exception ( " Data directory for table already containing data parts - probably it was unclean DROP table or manual intervention. You must either clear directory by hand or use ATTACH TABLE instead of CREATE TABLE if you need to use that parts. " , ErrorCodes : : INCORRECT_DATA ) ;
2017-05-31 15:01:25 +00:00
2019-05-03 02:00:57 +00:00
increment . set ( getMaxBlockNumber ( ) ) ;
2018-07-11 12:43:55 +00:00
loadMutations ( ) ;
2021-04-02 11:46:42 +00:00
2021-04-06 10:14:44 +00:00
loadDeduplicationLog ( ) ;
2014-03-13 12:48:07 +00:00
}
2012-07-17 20:04:39 +00:00
2017-04-01 07:20:54 +00:00
2017-06-06 17:06:14 +00:00
void StorageMergeTree : : startup ( )
{
2019-05-03 02:00:57 +00:00
clearOldPartsFromFilesystem ( ) ;
2020-05-27 20:05:55 +00:00
clearOldWriteAheadLogs ( ) ;
2020-11-11 16:18:21 +00:00
clearEmptyParts ( ) ;
2018-03-10 19:57:13 +00:00
/// Temporary directories contain incomplete results of merges (after forced restart)
/// and don't allow to reinitialize them, so delete each of them immediately
2019-05-03 02:00:57 +00:00
clearOldTemporaryDirectories ( 0 ) ;
2019-02-12 16:41:06 +00:00
/// NOTE background task will also do the above cleanups periodically.
time_after_previous_cleanup . restart ( ) ;
2020-05-17 04:12:33 +00:00
2020-06-03 22:11:06 +00:00
try
{
2020-10-13 14:25:42 +00:00
background_executor . start ( ) ;
2020-06-23 16:40:58 +00:00
startBackgroundMovesIfNeeded ( ) ;
2020-06-03 22:11:06 +00:00
}
catch ( . . . )
2020-05-17 04:12:33 +00:00
{
2020-06-03 22:11:06 +00:00
/// Exception safety: failed "startup" does not require a call to "shutdown" from the caller.
/// And it should be able to safely destroy table after exception in "startup" method.
2020-06-03 22:15:13 +00:00
/// It means that failed "startup" must not create any background tasks that we will have to wait.
2020-06-03 22:11:06 +00:00
try
{
shutdown ( ) ;
}
catch ( . . . )
{
std : : terminate ( ) ;
}
/// Note: after failed "startup", the table will be in a state that only allows to destroy the object.
throw ;
2020-05-17 04:12:33 +00:00
}
2013-02-06 11:26:35 +00:00
}
2014-11-12 10:37:47 +00:00
2013-09-30 01:29:19 +00:00
void StorageMergeTree : : shutdown ( )
2012-07-30 20:32:36 +00:00
{
2017-04-01 07:20:54 +00:00
if ( shutdown_called )
return ;
shutdown_called = true ;
2020-02-13 13:15:54 +00:00
2020-03-18 14:43:16 +00:00
/// Unlock all waiting mutations
2020-05-08 09:01:06 +00:00
{
2020-07-22 12:36:19 +00:00
std : : lock_guard lock ( mutation_wait_mutex ) ;
2020-05-08 09:01:06 +00:00
mutation_wait_event . notify_all ( ) ;
}
2020-03-18 14:43:16 +00:00
2020-06-08 18:08:55 +00:00
merger_mutator . merges_blocker . cancelForever ( ) ;
parts_mover . moves_blocker . cancelForever ( ) ;
2020-10-13 14:25:42 +00:00
background_executor . finish ( ) ;
2020-10-14 14:56:42 +00:00
background_moves_executor . finish ( ) ;
2020-06-08 18:08:55 +00:00
2020-02-13 13:15:54 +00:00
try
{
2020-06-08 18:08:55 +00:00
/// We clear all old parts after stopping all background operations.
/// It's important, because background operations can produce temporary
2021-05-03 10:50:44 +00:00
/// parts which will remove themselves in their destructors. If so, we
2020-06-08 18:08:55 +00:00
/// may have race condition between our remove call and background
/// process.
2020-02-13 13:15:54 +00:00
clearOldPartsFromFilesystem ( true ) ;
}
catch ( . . . )
{
/// Example: the case of readonly filesystem, we have failure removing old parts.
/// Should not prevent table shutdown.
tryLogCurrentException ( log ) ;
}
2012-07-18 19:44:04 +00:00
}
2014-03-13 12:48:07 +00:00
StorageMergeTree : : ~ StorageMergeTree ( )
{
2017-04-01 07:20:54 +00:00
shutdown ( ) ;
2014-03-13 12:48:07 +00:00
}
2012-07-18 19:44:04 +00:00
2020-10-01 17:34:22 +00:00
void StorageMergeTree : : read (
QueryPlan & query_plan ,
2017-04-01 07:20:54 +00:00
const Names & column_names ,
2020-06-16 14:25:08 +00:00
const StorageMetadataPtr & metadata_snapshot ,
2020-09-20 17:52:17 +00:00
SelectQueryInfo & query_info ,
2021-04-10 23:33:54 +00:00
ContextPtr local_context ,
2021-02-10 14:12:49 +00:00
QueryProcessingStage : : Enum processed_stage ,
2020-10-01 17:34:22 +00:00
size_t max_block_size ,
unsigned num_streams )
{
2021-02-10 14:12:49 +00:00
if ( auto plan = reader . read ( column_names , metadata_snapshot , query_info , local_context , max_block_size , num_streams , processed_stage ) )
2020-11-10 10:26:26 +00:00
query_plan = std : : move ( * plan ) ;
2020-10-01 17:34:22 +00:00
}
Pipe StorageMergeTree : : read (
const Names & column_names ,
const StorageMetadataPtr & metadata_snapshot ,
2020-11-10 12:02:22 +00:00
SelectQueryInfo & query_info ,
2021-04-10 23:33:54 +00:00
ContextPtr local_context ,
2020-10-01 17:34:22 +00:00
QueryProcessingStage : : Enum processed_stage ,
2019-02-18 23:38:44 +00:00
const size_t max_block_size ,
2017-06-02 15:54:39 +00:00
const unsigned num_streams )
2012-07-21 05:07:14 +00:00
{
2020-10-01 17:34:22 +00:00
QueryPlan plan ;
2021-04-10 23:33:54 +00:00
read ( plan , column_names , metadata_snapshot , query_info , local_context , processed_stage , max_block_size , num_streams ) ;
2021-03-04 17:38:12 +00:00
return plan . convertToPipe (
2021-04-10 23:33:54 +00:00
QueryPlanOptimizationSettings : : fromContext ( local_context ) ,
BuildQueryPipelineSettings : : fromContext ( local_context ) ) ;
2012-12-06 09:45:09 +00:00
}
2020-11-25 13:47:32 +00:00
std : : optional < UInt64 > StorageMergeTree : : totalRows ( const Settings & ) const
2019-10-28 17:27:43 +00:00
{
return getTotalActiveSizeInRows ( ) ;
}
2021-04-10 23:33:54 +00:00
std : : optional < UInt64 > StorageMergeTree : : totalRowsByPartitionPredicate ( const SelectQueryInfo & query_info , ContextPtr local_context ) const
2020-09-21 10:13:01 +00:00
{
2021-03-03 08:36:20 +00:00
auto parts = getDataPartsVector ( { DataPartState : : Committed } ) ;
2021-04-10 23:33:54 +00:00
return totalRowsByPartitionPredicateImpl ( query_info , local_context , parts ) ;
2020-09-21 10:13:01 +00:00
}
2020-11-25 13:47:32 +00:00
std : : optional < UInt64 > StorageMergeTree : : totalBytes ( const Settings & ) const
2020-03-29 08:50:27 +00:00
{
return getTotalActiveSizeInBytes ( ) ;
}
2021-04-10 23:33:54 +00:00
BlockOutputStreamPtr
StorageMergeTree : : write ( const ASTPtr & /*query*/ , const StorageMetadataPtr & metadata_snapshot , ContextPtr local_context )
2013-01-23 11:16:32 +00:00
{
2021-04-10 23:33:54 +00:00
const auto & settings = local_context - > getSettingsRef ( ) ;
2020-06-26 11:30:23 +00:00
return std : : make_shared < MergeTreeBlockOutputStream > (
2021-02-10 14:12:49 +00:00
* this , metadata_snapshot , settings . max_partitions_per_insert_block , local_context ) ;
2013-01-23 11:16:32 +00:00
}
2018-08-03 13:17:32 +00:00
void StorageMergeTree : : checkTableCanBeDropped ( ) const
2012-08-16 18:17:01 +00:00
{
2019-12-03 16:25:32 +00:00
auto table_id = getStorageID ( ) ;
2021-04-10 23:33:54 +00:00
getContext ( ) - > checkTableCanBeDropped ( table_id . database_name , table_id . table_name , getTotalActiveSizeInBytes ( ) ) ;
2017-01-23 19:18:25 +00:00
}
2017-01-19 19:11:12 +00:00
2020-01-22 11:30:11 +00:00
void StorageMergeTree : : drop ( )
2017-01-23 19:18:25 +00:00
{
2017-04-01 07:20:54 +00:00
shutdown ( ) ;
2019-05-03 02:00:57 +00:00
dropAllData ( ) ;
2018-04-21 00:35:20 +00:00
}
2021-04-10 23:33:54 +00:00
void StorageMergeTree : : truncate ( const ASTPtr & , const StorageMetadataPtr & , ContextPtr , TableExclusiveLockHolder & )
2018-04-21 00:35:20 +00:00
{
2018-06-09 15:48:22 +00:00
{
/// Asks to complete merges and does not allow them to start.
/// This protects against "revival" of data for a removed partition after completion of merge.
2020-08-15 05:21:02 +00:00
auto merge_blocker = stopMergesAndWait ( ) ;
2018-06-09 18:17:27 +00:00
2019-05-03 02:00:57 +00:00
auto parts_to_remove = getDataPartsVector ( ) ;
removePartsFromWorkingSet ( parts_to_remove , true ) ;
2018-04-21 00:35:20 +00:00
2020-05-23 22:24:01 +00:00
LOG_INFO ( log , " Removed {} parts. " , parts_to_remove . size ( ) ) ;
2018-06-09 15:48:22 +00:00
}
2019-07-29 09:15:46 +00:00
clearOldMutations ( true ) ;
2019-05-03 02:00:57 +00:00
clearOldPartsFromFilesystem ( ) ;
2013-08-07 13:07:42 +00:00
}
2018-04-21 00:35:20 +00:00
2016-05-16 23:04:03 +00:00
void StorageMergeTree : : alter (
2020-03-09 01:22:33 +00:00
const AlterCommands & commands ,
2021-04-10 23:33:54 +00:00
ContextPtr local_context ,
2020-06-18 16:10:47 +00:00
TableLockHolder & table_lock_holder )
2013-08-07 13:07:42 +00:00
{
2019-12-10 20:47:05 +00:00
auto table_id = getStorageID ( ) ;
2021-04-06 10:14:44 +00:00
auto old_storage_settings = getSettings ( ) ;
2019-08-26 14:50:34 +00:00
2020-06-09 17:28:29 +00:00
StorageInMemoryMetadata new_metadata = getInMemoryMetadata ( ) ;
2020-06-17 10:34:23 +00:00
StorageInMemoryMetadata old_metadata = getInMemoryMetadata ( ) ;
2021-04-10 23:33:54 +00:00
auto maybe_mutation_commands = commands . getMutationCommands ( new_metadata , local_context - > getSettingsRef ( ) . materialize_ttl_after_modify , local_context ) ;
2020-06-01 15:16:10 +00:00
String mutation_file_name ;
Int64 mutation_version = - 1 ;
2021-04-10 23:33:54 +00:00
commands . apply ( new_metadata , local_context ) ;
2019-08-27 09:34:53 +00:00
2020-06-09 17:28:29 +00:00
/// This alter can be performed at new_metadata level only
2020-03-12 15:41:34 +00:00
if ( commands . isSettingsAlter ( ) )
2019-12-23 16:44:50 +00:00
{
2020-06-09 17:28:29 +00:00
changeSettings ( new_metadata . settings_changes , table_lock_holder ) ;
2019-04-15 09:30:45 +00:00
2021-04-10 23:33:54 +00:00
DatabaseCatalog : : instance ( ) . getDatabase ( table_id . database_name ) - > alterTable ( local_context , table_id , new_metadata ) ;
2019-05-02 15:12:57 +00:00
}
2019-12-23 16:44:50 +00:00
else
{
2020-06-01 15:16:10 +00:00
{
2020-06-09 17:28:29 +00:00
changeSettings ( new_metadata . settings_changes , table_lock_holder ) ;
2020-06-18 17:09:06 +00:00
checkTTLExpressions ( new_metadata , old_metadata ) ;
2020-06-25 23:21:04 +00:00
/// Reinitialize primary key because primary key column types might have changed.
2020-06-17 10:34:23 +00:00
setProperties ( new_metadata , old_metadata ) ;
2019-12-23 16:44:50 +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-04-03 11:09:27 +00:00
2020-06-01 15:16:10 +00:00
if ( ! maybe_mutation_commands . empty ( ) )
mutation_version = startMutation ( maybe_mutation_commands , mutation_file_name ) ;
}
2019-12-23 16:44:50 +00:00
2020-03-18 10:02:57 +00:00
/// Always execute required mutations synchronously, because alters
/// should be executed in sequential order.
2020-03-11 15:51:04 +00:00
if ( ! maybe_mutation_commands . empty ( ) )
2020-04-03 11:09:27 +00:00
waitForMutation ( mutation_version , mutation_file_name ) ;
2019-12-23 16:44:50 +00:00
}
2021-04-06 10:14:44 +00:00
{
/// Some additional changes in settings
auto new_storage_settings = getSettings ( ) ;
if ( old_storage_settings - > non_replicated_deduplication_window ! = new_storage_settings - > non_replicated_deduplication_window )
{
/// We cannot place this check into settings sanityCheck because it depends on format_version.
/// sanityCheck must work event without storage.
if ( new_storage_settings - > non_replicated_deduplication_window ! = 0 & & format_version < MERGE_TREE_DATA_MIN_FORMAT_VERSION_WITH_CUSTOM_PARTITIONING )
throw Exception ( " Deduplication for non-replicated MergeTree in old syntax is not supported " , ErrorCodes : : BAD_ARGUMENTS ) ;
deduplication_log - > setDeduplicationWindowSize ( new_storage_settings - > non_replicated_deduplication_window ) ;
}
}
2014-03-20 13:00:42 +00:00
}
2016-09-02 04:03:40 +00:00
2019-09-02 11:35:53 +00:00
/// While exists, marks parts as 'currently_merging_mutating_parts' and reserves free space on filesystem.
2020-10-23 08:54:00 +00:00
StorageMergeTree : : CurrentlyMergingPartsTagger : : CurrentlyMergingPartsTagger (
FutureMergedMutatedPart & future_part_ ,
size_t total_size ,
StorageMergeTree & storage_ ,
const StorageMetadataPtr & metadata_snapshot ,
bool is_mutation )
: future_part ( future_part_ ) , storage ( storage_ )
2016-09-02 04:03:40 +00:00
{
2020-10-23 08:54:00 +00:00
/// Assume mutex is already locked, because this method is called from mergeTask.
2019-09-04 17:26:53 +00:00
2020-10-23 08:54:00 +00:00
/// if we mutate part, than we should reserve space on the same disk, because mutations possible can create hardlinks
if ( is_mutation )
reserved_space = storage . tryReserveSpace ( total_size , future_part_ . parts [ 0 ] - > volume ) ;
else
2017-04-01 07:20:54 +00:00
{
2020-10-23 08:54:00 +00:00
IMergeTreeDataPart : : TTLInfos ttl_infos ;
size_t max_volume_index = 0 ;
for ( auto & part_ptr : future_part_ . parts )
2019-10-31 10:40:11 +00:00
{
2020-10-23 08:54:00 +00:00
ttl_infos . update ( part_ptr - > ttl_infos ) ;
max_volume_index = std : : max ( max_volume_index , storage . getStoragePolicy ( ) - > getVolumeIndexByDisk ( part_ptr - > volume - > getDisk ( ) ) ) ;
2019-10-31 10:40:11 +00:00
}
2019-12-07 09:54:05 +00:00
2021-02-18 08:50:31 +00:00
reserved_space = storage . balancedReservation (
metadata_snapshot ,
total_size ,
max_volume_index ,
future_part . name ,
future_part . part_info ,
future_part . parts ,
& tagger ,
& ttl_infos ) ;
if ( ! reserved_space )
reserved_space
= storage . tryReserveSpacePreferringTTLRules ( metadata_snapshot , total_size , ttl_infos , time ( nullptr ) , max_volume_index ) ;
2017-04-01 07:20:54 +00:00
}
2021-02-18 08:50:31 +00:00
2020-10-23 08:54:00 +00:00
if ( ! reserved_space )
{
if ( is_mutation )
2021-02-18 08:50:31 +00:00
throw Exception ( " Not enough space for mutating part ' " + future_part . parts [ 0 ] - > name + " ' " , ErrorCodes : : NOT_ENOUGH_SPACE ) ;
2020-10-23 08:54:00 +00:00
else
throw Exception ( " Not enough space for merging parts " , ErrorCodes : : NOT_ENOUGH_SPACE ) ;
2017-04-01 07:20:54 +00:00
}
2020-10-23 08:54:00 +00:00
future_part_ . updatePath ( storage , reserved_space ) ;
2017-04-01 07:20:54 +00:00
2020-10-23 08:54:00 +00:00
for ( const auto & part : future_part . parts )
2017-04-01 07:20:54 +00:00
{
2020-10-23 08:54:00 +00:00
if ( storage . currently_merging_mutating_parts . count ( part ) )
throw Exception ( " Tagging already tagged part " + part - > name + " . This is a bug. " , ErrorCodes : : LOGICAL_ERROR ) ;
}
storage . currently_merging_mutating_parts . insert ( future_part . parts . begin ( ) , future_part . parts . end ( ) ) ;
}
2017-04-01 07:20:54 +00:00
2020-10-23 08:54:00 +00:00
StorageMergeTree : : CurrentlyMergingPartsTagger : : ~ CurrentlyMergingPartsTagger ( )
{
std : : lock_guard lock ( storage . currently_processing_in_background_mutex ) ;
2019-01-10 17:06:27 +00:00
2020-10-23 08:54:00 +00:00
for ( const auto & part : future_part . parts )
{
if ( ! storage . currently_merging_mutating_parts . count ( part ) )
std : : terminate ( ) ;
storage . currently_merging_mutating_parts . erase ( part ) ;
2017-04-01 07:20:54 +00:00
}
2016-09-02 04:03:40 +00:00
2020-10-23 08:54:00 +00:00
storage . currently_processing_in_background_condition . notify_all ( ) ;
}
2018-03-30 19:25:37 +00:00
2020-04-03 11:09:27 +00:00
Int64 StorageMergeTree : : startMutation ( const MutationCommands & commands , String & mutation_file_name )
2018-03-30 19:25:37 +00:00
{
2019-09-04 17:26:53 +00:00
/// Choose any disk, because when we load mutations we search them at each disk
/// where storage can be placed. See loadMutations().
2020-01-09 14:50:34 +00:00
auto disk = getStoragePolicy ( ) - > getAnyDisk ( ) ;
2019-12-16 15:51:15 +00:00
Int64 version ;
2020-10-15 10:54:50 +00:00
{
std : : lock_guard lock ( currently_processing_in_background_mutex ) ;
2020-03-18 14:43:16 +00:00
2020-10-15 10:54:50 +00:00
MergeTreeMutationEntry entry ( commands , disk , relative_data_path , insert_increment . get ( ) ) ;
version = increment . get ( ) ;
entry . commit ( version ) ;
mutation_file_name = entry . file_name ;
auto insertion = current_mutations_by_id . emplace ( mutation_file_name , std : : move ( entry ) ) ;
current_mutations_by_version . emplace ( version , insertion . first - > second ) ;
2019-12-16 15:51:15 +00:00
2020-10-15 10:54:50 +00:00
LOG_INFO ( log , " Added mutation: {} " , mutation_file_name ) ;
}
2020-10-15 07:43:50 +00:00
background_executor . triggerTask ( ) ;
2020-04-03 11:09:27 +00:00
return version ;
}
2020-03-18 10:02:57 +00:00
2020-07-22 19:29:54 +00:00
void StorageMergeTree : : updateMutationEntriesErrors ( FutureMergedMutatedPart result_part , bool is_successful , const String & exception_message )
{
/// Update the information about failed parts in the system.mutations table.
Int64 sources_data_version = result_part . parts . at ( 0 ) - > info . getDataVersion ( ) ;
Int64 result_data_version = result_part . part_info . getDataVersion ( ) ;
if ( sources_data_version ! = result_data_version )
{
std : : lock_guard lock ( currently_processing_in_background_mutex ) ;
auto mutations_begin_it = current_mutations_by_version . upper_bound ( sources_data_version ) ;
auto mutations_end_it = current_mutations_by_version . upper_bound ( result_data_version ) ;
for ( auto it = mutations_begin_it ; it ! = mutations_end_it ; + + it )
{
MergeTreeMutationEntry & entry = it - > second ;
if ( is_successful )
{
if ( ! entry . latest_failed_part . empty ( ) & & result_part . part_info . contains ( entry . latest_failed_part_info ) )
{
entry . latest_failed_part . clear ( ) ;
entry . latest_failed_part_info = MergeTreePartInfo ( ) ;
entry . latest_fail_time = 0 ;
entry . latest_fail_reason . clear ( ) ;
}
}
else
{
entry . latest_failed_part = result_part . parts . at ( 0 ) - > name ;
entry . latest_failed_part_info = result_part . parts . at ( 0 ) - > info ;
entry . latest_fail_time = time ( nullptr ) ;
entry . latest_fail_reason = exception_message ;
}
}
}
std : : unique_lock lock ( mutation_wait_mutex ) ;
mutation_wait_event . notify_all ( ) ;
}
2020-04-03 11:09:27 +00:00
void StorageMergeTree : : waitForMutation ( Int64 version , const String & file_name )
{
2020-05-23 22:24:01 +00:00
LOG_INFO ( log , " Waiting mutation: {} " , file_name ) ;
2020-07-22 12:36:19 +00:00
{
auto check = [ version , this ] ( )
{
if ( shutdown_called )
return true ;
2020-07-22 15:19:54 +00:00
auto mutation_status = getIncompleteMutationsStatus ( version ) ;
2020-07-22 12:36:19 +00:00
return ! mutation_status | | mutation_status - > is_done | | ! mutation_status - > latest_fail_reason . empty ( ) ;
} ;
std : : unique_lock lock ( mutation_wait_mutex ) ;
mutation_wait_event . wait ( lock , check ) ;
}
2020-07-31 12:22:32 +00:00
/// At least we have our current mutation
2020-07-31 11:37:16 +00:00
std : : set < String > mutation_ids ;
mutation_ids . insert ( file_name ) ;
2020-07-31 12:22:32 +00:00
2020-07-22 15:19:54 +00:00
auto mutation_status = getIncompleteMutationsStatus ( version , & mutation_ids ) ;
2021-02-10 14:12:49 +00:00
try
{
checkMutationStatus ( mutation_status , mutation_ids ) ;
}
catch ( . . . )
{
tryLogCurrentException ( __PRETTY_FUNCTION__ ) ;
throw ;
}
2020-07-22 12:36:19 +00:00
2020-05-23 22:24:01 +00:00
LOG_INFO ( log , " Mutation {} done " , file_name ) ;
2020-03-18 10:02:57 +00:00
}
2021-04-10 23:33:54 +00:00
void StorageMergeTree : : mutate ( const MutationCommands & commands , ContextPtr query_context )
2020-03-18 10:02:57 +00:00
{
2020-04-03 11:09:27 +00:00
String mutation_file_name ;
Int64 version = startMutation ( commands , mutation_file_name ) ;
2021-04-10 23:33:54 +00:00
if ( query_context - > getSettingsRef ( ) . mutations_sync > 0 )
2020-04-03 11:09:27 +00:00
waitForMutation ( version , mutation_file_name ) ;
2018-03-30 19:25:37 +00:00
}
2019-12-12 16:24:03 +00:00
namespace
{
struct PartVersionWithName
{
Int64 version ;
String name ;
} ;
bool comparator ( const PartVersionWithName & f , const PartVersionWithName & s )
{
return f . version < s . version ;
}
}
2020-07-31 11:37:16 +00:00
std : : optional < MergeTreeMutationStatus > StorageMergeTree : : getIncompleteMutationsStatus ( Int64 mutation_version , std : : set < String > * mutation_ids ) const
2019-12-16 15:51:15 +00:00
{
std : : lock_guard lock ( currently_processing_in_background_mutex ) ;
2020-07-22 15:19:54 +00:00
auto current_mutation_it = current_mutations_by_version . find ( mutation_version ) ;
2019-12-19 15:27:56 +00:00
/// Killed
2020-07-22 15:19:54 +00:00
if ( current_mutation_it = = current_mutations_by_version . end ( ) )
2020-07-22 12:36:19 +00:00
return { } ;
MergeTreeMutationStatus result { . is_done = false } ;
2020-07-22 15:19:54 +00:00
const auto & mutation_entry = current_mutation_it - > second ;
2019-12-19 15:27:56 +00:00
2019-12-16 15:51:15 +00:00
auto data_parts = getDataPartsVector ( ) ;
for ( const auto & data_part : data_parts )
2020-07-22 12:36:19 +00:00
{
2019-12-16 15:51:15 +00:00
if ( data_part - > info . getDataVersion ( ) < mutation_version )
2020-07-22 12:36:19 +00:00
{
if ( ! mutation_entry . latest_fail_reason . empty ( ) )
{
result . latest_failed_part = mutation_entry . latest_failed_part ;
result . latest_fail_reason = mutation_entry . latest_fail_reason ;
result . latest_fail_time = mutation_entry . latest_fail_time ;
2020-07-22 15:19:54 +00:00
/// Fill all mutations which failed with the same error
/// (we can execute several mutations together)
if ( mutation_ids )
{
auto mutations_begin_it = current_mutations_by_version . upper_bound ( data_part - > info . getDataVersion ( ) ) ;
for ( auto it = mutations_begin_it ; it ! = current_mutations_by_version . end ( ) ; + + it )
/// All mutations with the same failure
if ( it - > second . latest_fail_reason = = result . latest_fail_reason )
2020-07-31 11:37:16 +00:00
mutation_ids - > insert ( it - > second . file_name ) ;
2020-07-22 15:19:54 +00:00
}
2020-07-22 12:36:19 +00:00
}
return result ;
}
}
result . is_done = true ;
return result ;
2019-12-16 15:51:15 +00:00
}
2018-07-06 19:04:54 +00:00
std : : vector < MergeTreeMutationStatus > StorageMergeTree : : getMutationsStatus ( ) const
{
2019-08-15 09:43:31 +00:00
std : : lock_guard lock ( currently_processing_in_background_mutex ) ;
2018-07-06 19:04:54 +00:00
2019-12-12 16:24:03 +00:00
std : : vector < PartVersionWithName > part_versions_with_names ;
2019-05-03 02:00:57 +00:00
auto data_parts = getDataPartsVector ( ) ;
2019-12-12 16:24:03 +00:00
part_versions_with_names . reserve ( data_parts . size ( ) ) ;
2018-07-06 19:04:54 +00:00
for ( const auto & part : data_parts )
2019-12-12 16:24:03 +00:00
part_versions_with_names . emplace_back ( PartVersionWithName { part - > info . getDataVersion ( ) , part - > name } ) ;
std : : sort ( part_versions_with_names . begin ( ) , part_versions_with_names . end ( ) , comparator ) ;
2018-07-06 19:04:54 +00:00
std : : vector < MergeTreeMutationStatus > result ;
for ( const auto & kv : current_mutations_by_version )
{
Int64 mutation_version = kv . first ;
const MergeTreeMutationEntry & entry = kv . second ;
2019-12-12 16:24:03 +00:00
const PartVersionWithName needle { mutation_version , " " } ;
2018-07-06 19:04:54 +00:00
auto versions_it = std : : lower_bound (
2019-12-12 16:24:03 +00:00
part_versions_with_names . begin ( ) , part_versions_with_names . end ( ) , needle , comparator ) ;
size_t parts_to_do = versions_it - part_versions_with_names . begin ( ) ;
Names parts_to_do_names ;
parts_to_do_names . reserve ( parts_to_do ) ;
for ( size_t i = 0 ; i < parts_to_do ; + + i )
parts_to_do_names . push_back ( part_versions_with_names [ i ] . name ) ;
2018-07-06 19:04:54 +00:00
std : : map < String , Int64 > block_numbers_map ( { { " " , entry . block_number } } ) ;
for ( const MutationCommand & command : entry . commands )
{
2020-11-09 16:05:40 +00:00
WriteBufferFromOwnString buf ;
formatAST ( * command . ast , buf , false , true ) ;
2018-07-06 19:04:54 +00:00
result . push_back ( MergeTreeMutationStatus
{
2018-07-11 12:43:55 +00:00
entry . file_name ,
2020-11-09 16:05:40 +00:00
buf . str ( ) ,
2018-07-06 19:04:54 +00:00
entry . create_time ,
block_numbers_map ,
2019-12-12 16:24:03 +00:00
parts_to_do_names ,
2020-07-22 12:36:19 +00:00
/* is_done = */ parts_to_do_names . empty ( ) ,
2019-01-10 17:06:27 +00:00
entry . latest_failed_part ,
entry . latest_fail_time ,
entry . latest_fail_reason ,
2018-07-06 19:04:54 +00:00
} ) ;
}
}
return result ;
}
2018-07-11 12:43:55 +00:00
2019-02-04 13:04:02 +00:00
CancellationCode StorageMergeTree : : killMutation ( const String & mutation_id )
2019-01-10 18:19:29 +00:00
{
2020-05-23 22:24:01 +00:00
LOG_TRACE ( log , " Killing mutation {} " , mutation_id ) ;
2019-01-14 12:25:25 +00:00
std : : optional < MergeTreeMutationEntry > to_kill ;
{
2019-08-15 09:43:31 +00:00
std : : lock_guard lock ( currently_processing_in_background_mutex ) ;
2019-01-14 12:25:25 +00:00
auto it = current_mutations_by_id . find ( mutation_id ) ;
if ( it ! = current_mutations_by_id . end ( ) )
{
to_kill . emplace ( std : : move ( it - > second ) ) ;
current_mutations_by_id . erase ( it ) ;
current_mutations_by_version . erase ( to_kill - > block_number ) ;
}
}
2019-02-04 13:04:02 +00:00
if ( ! to_kill )
return CancellationCode : : NotFound ;
2021-04-10 23:33:54 +00:00
getContext ( ) - > getMergeList ( ) . cancelPartMutations ( { } , to_kill - > block_number ) ;
2019-02-04 13:04:02 +00:00
to_kill - > removeFile ( ) ;
2020-05-23 22:24:01 +00:00
LOG_TRACE ( log , " Cancelled part mutations and removed mutation file {} " , mutation_id ) ;
2020-05-08 09:01:06 +00:00
{
std : : lock_guard < std : : mutex > lock ( mutation_wait_mutex ) ;
mutation_wait_event . notify_all ( ) ;
}
2019-02-05 17:22:23 +00:00
/// Maybe there is another mutation that was blocked by the killed one. Try to execute it immediately.
2020-10-15 07:43:50 +00:00
background_executor . triggerTask ( ) ;
2019-02-05 17:22:23 +00:00
2019-02-04 13:04:02 +00:00
return CancellationCode : : CancelSent ;
2019-01-10 18:19:29 +00:00
}
2021-04-06 10:14:44 +00:00
void StorageMergeTree : : loadDeduplicationLog ( )
{
auto settings = getSettings ( ) ;
if ( settings - > non_replicated_deduplication_window ! = 0 & & format_version < MERGE_TREE_DATA_MIN_FORMAT_VERSION_WITH_CUSTOM_PARTITIONING )
throw Exception ( " Deduplication for non-replicated MergeTree in old syntax is not supported " , ErrorCodes : : BAD_ARGUMENTS ) ;
std : : string path = getDataPaths ( ) [ 0 ] + " /deduplication_logs " ;
deduplication_log = std : : make_unique < MergeTreeDeduplicationLog > ( path , settings - > non_replicated_deduplication_window , format_version ) ;
deduplication_log - > load ( ) ;
}
2018-07-11 12:43:55 +00:00
void StorageMergeTree : : loadMutations ( )
{
2020-02-27 16:47:40 +00:00
for ( const auto & [ path , disk ] : getRelativeDataPathsWithDisks ( ) )
2018-07-11 12:43:55 +00:00
{
2020-02-27 16:47:40 +00:00
for ( auto it = disk - > iterateDirectory ( path ) ; it - > isValid ( ) ; it - > next ( ) )
2018-07-11 12:43:55 +00:00
{
2020-02-27 16:47:40 +00:00
if ( startsWith ( it - > name ( ) , " mutation_ " ) )
2019-04-04 13:13:59 +00:00
{
2020-02-28 17:14:55 +00:00
MergeTreeMutationEntry entry ( disk , path , it - > name ( ) ) ;
2019-04-04 13:13:59 +00:00
Int64 block_number = entry . block_number ;
2020-05-23 22:24:01 +00:00
LOG_DEBUG ( log , " Loading mutation: {} entry, commands size: {} " , it - > name ( ) , entry . commands . size ( ) ) ;
2020-02-27 16:47:40 +00:00
auto insertion = current_mutations_by_id . emplace ( it - > name ( ) , std : : move ( entry ) ) ;
2019-04-04 13:13:59 +00:00
current_mutations_by_version . emplace ( block_number , insertion . first - > second ) ;
}
2020-02-27 16:47:40 +00:00
else if ( startsWith ( it - > name ( ) , " tmp_mutation_ " ) )
2019-04-04 13:13:59 +00:00
{
2021-01-14 16:24:13 +00:00
disk - > removeFile ( it - > path ( ) ) ;
2019-04-04 13:13:59 +00:00
}
2018-07-11 12:43:55 +00:00
}
}
if ( ! current_mutations_by_version . empty ( ) )
increment . value = std : : max ( Int64 ( increment . value . load ( ) ) , current_mutations_by_version . rbegin ( ) - > first ) ;
}
2018-07-06 19:04:54 +00:00
2020-10-23 08:54:00 +00:00
std : : shared_ptr < StorageMergeTree : : MergeMutateSelectedEntry > StorageMergeTree : : selectPartsToMerge (
2021-05-17 11:14:09 +00:00
const StorageMetadataPtr & metadata_snapshot ,
bool aggressive , const String & partition_id ,
bool final ,
String * out_disable_reason ,
TableLockHolder & /* table_lock_holder */ ,
const MergeTreeTransactionPtr & txn ,
bool optimize_skip_merged_partitions ,
SelectPartsDecision * select_decision_out )
2014-03-13 12:48:07 +00:00
{
2020-09-30 12:40:46 +00:00
std : : unique_lock lock ( currently_processing_in_background_mutex ) ;
2020-09-04 06:55:19 +00:00
auto data_settings = getSettings ( ) ;
2014-03-13 12:48:07 +00:00
2019-01-13 22:02:33 +00:00
FutureMergedMutatedPart future_part ;
2017-08-16 19:24:50 +00:00
2020-11-02 14:38:18 +00:00
if ( storage_settings . get ( ) - > assign_part_uuids )
future_part . uuid = UUIDHelpers : : generateV4 ( ) ;
2017-08-16 19:24:50 +00:00
2019-08-15 09:43:31 +00:00
/// You must call destructor with unlocked `currently_processing_in_background_mutex`.
2020-09-30 12:40:46 +00:00
CurrentlyMergingPartsTaggerPtr merging_tagger ;
2020-09-04 10:08:09 +00:00
MergeList : : EntryPtr merge_entry ;
2014-03-13 12:48:07 +00:00
2021-05-18 17:07:29 +00:00
auto can_merge = [ this , & lock ] ( const DataPartPtr & left , const DataPartPtr & right , const MergeTreeTransaction * tx , String * ) - > bool
2017-04-01 07:20:54 +00:00
{
2021-05-18 17:07:29 +00:00
if ( tx )
{
/// Cannot merge parts if some of them is not visible in current snapshot
/// TODO We can use simplified visibility rules (without CSN lookup) here
if ( left & & ! left - > versions . isVisible ( * tx ) )
return false ;
if ( right & & ! right - > versions . isVisible ( * tx ) )
return false ;
}
2021-05-10 18:03:37 +00:00
/// This predicate is checked for the first part of each range.
2020-09-30 12:40:46 +00:00
/// (left = nullptr, right = "first part of partition")
if ( ! left )
return ! currently_merging_mutating_parts . count ( right ) ;
return ! currently_merging_mutating_parts . count ( left ) & & ! currently_merging_mutating_parts . count ( right )
2021-02-10 14:12:49 +00:00
& & getCurrentMutationVersion ( left , lock ) = = getCurrentMutationVersion ( right , lock ) & & partsContainSameProjections ( left , right ) ;
2020-09-30 12:40:46 +00:00
} ;
2014-03-27 11:30:54 +00:00
2020-11-11 10:34:32 +00:00
SelectPartsDecision select_decision = SelectPartsDecision : : CANNOT_SELECT ;
2016-08-13 01:59:09 +00:00
2020-09-30 12:40:46 +00:00
if ( partition_id . empty ( ) )
{
UInt64 max_source_parts_size = merger_mutator . getMaxSourcePartsSizeForMerge ( ) ;
bool merge_with_ttl_allowed = getTotalMergesWithTTLInMergeList ( ) < data_settings - > max_number_of_merges_with_ttl_in_pool ;
2016-05-16 18:43:38 +00:00
2020-09-30 12:40:46 +00:00
/// TTL requirements is much more strict than for regular merge, so
/// if regular not possible, than merge with ttl is not also not
/// possible.
if ( max_source_parts_size > 0 )
2017-04-01 07:20:54 +00:00
{
2020-11-10 19:58:21 +00:00
select_decision = merger_mutator . selectPartsToMerge (
2020-09-30 12:40:46 +00:00
future_part ,
aggressive ,
max_source_parts_size ,
can_merge ,
merge_with_ttl_allowed ,
2021-05-17 11:14:09 +00:00
txn ,
2020-09-30 12:40:46 +00:00
out_disable_reason ) ;
2017-04-01 07:20:54 +00:00
}
2020-09-30 12:40:46 +00:00
else if ( out_disable_reason )
* out_disable_reason = " Current value of max_source_parts_size is zero " ;
}
else
{
while ( true )
2017-04-01 07:20:54 +00:00
{
2020-09-30 12:40:46 +00:00
UInt64 disk_space = getStoragePolicy ( ) - > getMaxUnreservedFreeSpace ( ) ;
2020-11-10 19:58:21 +00:00
select_decision = merger_mutator . selectAllPartsToMergeWithinPartition (
2021-06-02 20:03:44 +00:00
future_part , disk_space , can_merge , partition_id , final , metadata_snapshot , txn , out_disable_reason , optimize_skip_merged_partitions ) ;
2021-04-25 07:08:19 +00:00
auto timeout_ms = getSettings ( ) - > lock_acquire_timeout_for_background_operations . totalMilliseconds ( ) ;
auto timeout = std : : chrono : : milliseconds ( timeout_ms ) ;
2020-09-30 12:40:46 +00:00
/// If final - we will wait for currently processing merges to finish and continue.
if ( final
2020-11-10 19:58:21 +00:00
& & select_decision ! = SelectPartsDecision : : SELECTED
2020-09-30 12:40:46 +00:00
& & ! currently_merging_mutating_parts . empty ( )
& & out_disable_reason
& & out_disable_reason - > empty ( ) )
2020-06-02 00:41:52 +00:00
{
2020-09-30 12:40:46 +00:00
LOG_DEBUG ( log , " Waiting for currently running merges ({} parts are merging right now) to perform OPTIMIZE FINAL " ,
2020-11-10 21:02:11 +00:00
currently_merging_mutating_parts . size ( ) ) ;
2020-06-02 00:41:52 +00:00
2021-04-25 07:08:19 +00:00
if ( std : : cv_status : : timeout = = currently_processing_in_background_condition . wait_for ( lock , timeout ) )
2020-06-02 00:41:52 +00:00
{
2021-04-25 07:08:19 +00:00
* out_disable_reason = fmt : : format ( " Timeout ({} ms) while waiting for already running merges before running OPTIMIZE with FINAL " , timeout_ms ) ;
2020-06-02 00:41:52 +00:00
break ;
2020-09-30 12:40:46 +00:00
}
2020-06-02 00:41:52 +00:00
}
2020-09-30 12:40:46 +00:00
else
break ;
2017-04-01 07:20:54 +00:00
}
2020-09-30 12:40:46 +00:00
}
2014-03-13 12:48:07 +00:00
2020-11-10 19:58:21 +00:00
/// In case of final we need to know the decision of select in StorageMergeTree::merge
/// to treat NOTHING_TO_MERGE as successful merge (otherwise optimize final will be uncompleted)
if ( select_decision_out )
* select_decision_out = select_decision ;
2020-11-10 14:42:56 +00:00
2020-11-10 19:58:21 +00:00
if ( select_decision ! = SelectPartsDecision : : SELECTED )
2020-09-30 12:40:46 +00:00
{
if ( out_disable_reason )
2019-07-12 13:39:16 +00:00
{
2020-09-30 12:40:46 +00:00
if ( ! out_disable_reason - > empty ( ) )
2020-01-09 15:35:45 +00:00
{
2020-09-30 12:40:46 +00:00
* out_disable_reason + = " . " ;
2020-01-09 15:35:45 +00:00
}
2020-09-30 12:40:46 +00:00
* out_disable_reason + = " Cannot select parts for optimization " ;
2019-07-12 13:39:16 +00:00
}
2016-05-16 18:43:38 +00:00
2020-09-30 12:40:46 +00:00
return { } ;
2017-04-01 07:20:54 +00:00
}
2014-03-13 12:48:07 +00:00
2021-01-27 11:56:12 +00:00
/// Account TTL merge here to avoid exceeding the max_number_of_merges_with_ttl_in_pool limit
if ( isTTLMergeType ( future_part . merge_type ) )
2021-04-10 23:33:54 +00:00
getContext ( ) - > getMergeList ( ) . bookMergeWithTTL ( ) ;
2021-01-27 11:56:12 +00:00
2020-10-23 08:54:00 +00:00
merging_tagger = std : : make_unique < CurrentlyMergingPartsTagger > ( future_part , MergeTreeDataMergerMutator : : estimateNeededDiskSpace ( future_part . parts ) , * this , metadata_snapshot , false ) ;
return std : : make_shared < MergeMutateSelectedEntry > ( future_part , std : : move ( merging_tagger ) , MutationCommands { } ) ;
2020-09-30 12:40:46 +00:00
}
bool StorageMergeTree : : merge (
bool aggressive ,
const String & partition_id ,
bool final ,
bool deduplicate ,
2020-12-01 09:10:12 +00:00
const Names & deduplicate_by_columns ,
2021-05-17 11:14:09 +00:00
const MergeTreeTransactionPtr & txn ,
2020-11-20 14:29:13 +00:00
String * out_disable_reason ,
2020-12-04 14:01:59 +00:00
bool optimize_skip_merged_partitions )
2020-09-30 12:40:46 +00:00
{
2020-10-20 21:10:55 +00:00
auto table_lock_holder = lockForShare ( RWLockImpl : : NO_QUERY , getSettings ( ) - > lock_acquire_timeout_for_background_operations ) ;
2020-09-30 12:40:46 +00:00
auto metadata_snapshot = getInMemoryMetadataPtr ( ) ;
2020-11-10 20:01:43 +00:00
SelectPartsDecision select_decision ;
2021-05-17 11:14:09 +00:00
auto merge_mutate_entry = selectPartsToMerge ( metadata_snapshot , aggressive , partition_id , final , out_disable_reason , table_lock_holder , txn , optimize_skip_merged_partitions , & select_decision ) ;
2020-11-10 20:01:43 +00:00
2020-11-19 22:22:40 +00:00
/// If there is nothing to merge then we treat this merge as successful (needed for optimize final optimization)
if ( select_decision = = SelectPartsDecision : : NOTHING_TO_MERGE )
2020-11-10 20:01:43 +00:00
return true ;
2020-09-30 12:40:46 +00:00
if ( ! merge_mutate_entry )
return false ;
2021-05-17 11:14:09 +00:00
return mergeSelectedParts ( metadata_snapshot , deduplicate , deduplicate_by_columns , * merge_mutate_entry , table_lock_holder , txn ) ;
2020-09-30 13:49:22 +00:00
}
2020-12-01 09:10:12 +00:00
bool StorageMergeTree : : mergeSelectedParts (
const StorageMetadataPtr & metadata_snapshot ,
bool deduplicate ,
const Names & deduplicate_by_columns ,
MergeMutateSelectedEntry & merge_mutate_entry ,
2021-05-17 11:14:09 +00:00
TableLockHolder & table_lock_holder ,
const MergeTreeTransactionPtr & txn )
2020-10-14 16:09:18 +00:00
{
2020-09-30 13:49:22 +00:00
auto & future_part = merge_mutate_entry . future_part ;
2017-04-01 07:20:54 +00:00
Stopwatch stopwatch ;
2019-05-03 02:00:57 +00:00
MutableDataPartPtr new_part ;
2020-10-13 14:25:42 +00:00
auto table_id = getStorageID ( ) ;
2021-04-10 23:33:54 +00:00
auto merge_list_entry = getContext ( ) - > getMergeList ( ) . insert ( table_id . database_name , table_id . table_name , future_part ) ;
2017-03-31 16:00:30 +00:00
2018-01-23 22:56:46 +00:00
auto write_part_log = [ & ] ( const ExecutionStatus & execution_status )
{
2019-09-03 11:32:25 +00:00
writePartLog (
PartLogElement : : MERGE_PARTS ,
execution_status ,
stopwatch . elapsed ( ) ,
future_part . name ,
new_part ,
future_part . parts ,
2020-10-13 14:25:42 +00:00
merge_list_entry . get ( ) ) ;
2018-01-23 22:56:46 +00:00
} ;
try
{
2018-03-30 19:25:37 +00:00
new_part = merger_mutator . mergePartsToTemporaryPart (
2021-02-10 14:12:49 +00:00
future_part ,
metadata_snapshot ,
* ( merge_list_entry ) ,
table_lock_holder ,
time ( nullptr ) ,
getContext ( ) ,
merge_mutate_entry . tagger - > reserved_space ,
deduplicate ,
deduplicate_by_columns ,
merging_params ) ;
2020-05-27 20:05:55 +00:00
2021-05-17 11:14:09 +00:00
merger_mutator . renameMergedTemporaryPart ( new_part , future_part . parts , txn , nullptr ) ;
2018-01-23 22:56:46 +00:00
write_part_log ( { } ) ;
}
catch ( . . . )
{
write_part_log ( ExecutionStatus : : fromCurrentException ( ) ) ;
throw ;
2017-04-01 07:20:54 +00:00
}
2017-03-07 17:13:54 +00:00
2017-04-01 07:20:54 +00:00
return true ;
2014-04-11 13:05:17 +00:00
}
2019-09-05 13:12:29 +00:00
bool StorageMergeTree : : partIsAssignedToBackgroundOperation ( const DataPartPtr & part ) const
{
std : : lock_guard background_processing_lock ( currently_processing_in_background_mutex ) ;
return currently_merging_mutating_parts . count ( part ) ;
2019-08-16 15:57:19 +00:00
}
2020-11-28 08:17:20 +00:00
std : : shared_ptr < StorageMergeTree : : MergeMutateSelectedEntry > StorageMergeTree : : selectPartsToMutate (
const StorageMetadataPtr & metadata_snapshot , String * /* disable_reason */ , TableLockHolder & /* table_lock_holder */ )
2019-06-19 17:56:41 +00:00
{
2020-09-30 12:40:46 +00:00
std : : lock_guard lock ( currently_processing_in_background_mutex ) ;
2021-04-10 23:33:54 +00:00
size_t max_ast_elements = getContext ( ) - > getSettingsRef ( ) . max_expanded_ast_elements ;
2019-07-29 10:21:15 +00:00
2019-01-13 22:02:33 +00:00
FutureMergedMutatedPart future_part ;
2020-11-02 14:38:18 +00:00
if ( storage_settings . get ( ) - > assign_part_uuids )
future_part . uuid = UUIDHelpers : : generateV4 ( ) ;
2018-07-09 15:34:11 +00:00
MutationCommands commands ;
2020-09-30 12:40:46 +00:00
CurrentlyMergingPartsTaggerPtr tagger ;
2018-07-09 15:34:11 +00:00
2020-09-30 12:40:46 +00:00
if ( current_mutations_by_version . empty ( ) )
return { } ;
2018-07-09 15:34:11 +00:00
2021-05-03 10:49:12 +00:00
size_t max_source_part_size = merger_mutator . getMaxSourcePartSizeForMutation ( ) ;
if ( max_source_part_size = = 0 )
{
LOG_DEBUG (
log ,
2021-05-03 10:50:44 +00:00
" Not enough idle threads to apply mutations at the moment. See settings 'number_of_free_entries_in_pool_to_execute_mutation' "
" and 'background_pool_size' " ) ;
2021-05-03 10:49:12 +00:00
return { } ;
}
2020-09-30 12:40:46 +00:00
auto mutations_end_it = current_mutations_by_version . end ( ) ;
for ( const auto & part : getDataPartsVector ( ) )
{
if ( currently_merging_mutating_parts . count ( part ) )
continue ;
2018-07-09 15:34:11 +00:00
2020-09-30 12:40:46 +00:00
auto mutations_begin_it = current_mutations_by_version . upper_bound ( part - > info . getDataVersion ( ) ) ;
if ( mutations_begin_it = = mutations_end_it )
continue ;
2018-07-09 15:34:11 +00:00
2020-09-30 12:40:46 +00:00
if ( max_source_part_size < part - > getBytesOnDisk ( ) )
{
2021-05-03 10:49:12 +00:00
LOG_DEBUG (
log ,
" Current max source part size for mutation is {} but part size {}. Will not mutate part {} yet " ,
max_source_part_size ,
part - > getBytesOnDisk ( ) ,
part - > name ) ;
2020-09-30 12:40:46 +00:00
continue ;
}
2018-07-09 15:34:11 +00:00
2020-09-30 12:40:46 +00:00
size_t current_ast_elements = 0 ;
for ( auto it = mutations_begin_it ; it ! = mutations_end_it ; + + it )
{
size_t commands_size = 0 ;
MutationCommands commands_for_size_validation ;
for ( const auto & command : it - > second . commands )
2019-07-29 08:55:08 +00:00
{
2020-09-30 12:40:46 +00:00
if ( command . type ! = MutationCommand : : Type : : DROP_COLUMN
& & command . type ! = MutationCommand : : Type : : DROP_INDEX
2021-02-10 14:12:49 +00:00
& & command . type ! = MutationCommand : : Type : : DROP_PROJECTION
2020-09-30 12:40:46 +00:00
& & command . type ! = MutationCommand : : Type : : RENAME_COLUMN )
2020-03-11 15:51:04 +00:00
{
2020-09-30 12:40:46 +00:00
commands_for_size_validation . push_back ( command ) ;
2020-03-11 15:51:04 +00:00
}
2020-09-30 12:40:46 +00:00
else
2020-03-11 15:51:04 +00:00
{
2020-09-30 12:40:46 +00:00
commands_size + = command . ast - > size ( ) ;
2020-03-11 15:51:04 +00:00
}
2019-07-29 08:55:08 +00:00
}
2018-07-09 15:34:11 +00:00
2020-09-30 12:40:46 +00:00
if ( ! commands_for_size_validation . empty ( ) )
{
MutationsInterpreter interpreter (
2021-04-10 23:33:54 +00:00
shared_from_this ( ) , metadata_snapshot , commands_for_size_validation , getContext ( ) , false ) ;
2020-09-30 12:40:46 +00:00
commands_size + = interpreter . evaluateCommandsSize ( ) ;
}
2018-07-09 15:34:11 +00:00
2020-09-30 12:40:46 +00:00
if ( current_ast_elements + commands_size > = max_ast_elements )
break ;
2018-07-09 15:34:11 +00:00
2020-09-30 12:40:46 +00:00
current_ast_elements + = commands_size ;
commands . insert ( commands . end ( ) , it - > second . commands . begin ( ) , it - > second . commands . end ( ) ) ;
2018-07-09 15:34:11 +00:00
}
2020-09-30 12:40:46 +00:00
auto new_part_info = part - > info ;
new_part_info . mutation = current_mutations_by_version . rbegin ( ) - > first ;
2018-07-09 15:34:11 +00:00
2020-09-30 12:40:46 +00:00
future_part . parts . push_back ( part ) ;
future_part . part_info = new_part_info ;
future_part . name = part - > getNewName ( new_part_info ) ;
future_part . type = part - > getType ( ) ;
2018-07-09 15:34:11 +00:00
2020-10-23 08:54:00 +00:00
tagger = std : : make_unique < CurrentlyMergingPartsTagger > ( future_part , MergeTreeDataMergerMutator : : estimateNeededDiskSpace ( { part } ) , * this , metadata_snapshot , true ) ;
return std : : make_shared < MergeMutateSelectedEntry > ( future_part , std : : move ( tagger ) , commands ) ;
2018-07-09 15:34:11 +00:00
}
2020-09-30 12:40:46 +00:00
return { } ;
}
2018-07-09 15:34:11 +00:00
2020-10-20 21:10:55 +00:00
bool StorageMergeTree : : mutateSelectedPart ( const StorageMetadataPtr & metadata_snapshot , MergeMutateSelectedEntry & merge_mutate_entry , TableLockHolder & table_lock_holder )
2020-09-30 13:49:22 +00:00
{
auto & future_part = merge_mutate_entry . future_part ;
2019-12-03 16:25:32 +00:00
auto table_id = getStorageID ( ) ;
2019-01-11 19:14:50 +00:00
2021-04-10 23:33:54 +00:00
auto merge_list_entry = getContext ( ) - > getMergeList ( ) . insert ( table_id . database_name , table_id . table_name , future_part ) ;
2018-07-09 15:34:11 +00:00
Stopwatch stopwatch ;
2019-05-03 02:00:57 +00:00
MutableDataPartPtr new_part ;
2018-07-09 15:34:11 +00:00
auto write_part_log = [ & ] ( const ExecutionStatus & execution_status )
{
2019-09-03 11:32:25 +00:00
writePartLog (
PartLogElement : : MUTATE_PART ,
execution_status ,
stopwatch . elapsed ( ) ,
future_part . name ,
new_part ,
future_part . parts ,
2020-10-13 14:25:42 +00:00
merge_list_entry . get ( ) ) ;
2018-07-09 15:34:11 +00:00
} ;
try
{
2020-06-16 12:19:21 +00:00
new_part = merger_mutator . mutatePartToTemporaryPart (
2020-10-13 14:25:42 +00:00
future_part , metadata_snapshot , merge_mutate_entry . commands , * ( merge_list_entry ) ,
2021-04-10 23:33:54 +00:00
time ( nullptr ) , getContext ( ) , merge_mutate_entry . tagger - > reserved_space , table_lock_holder ) ;
2019-08-21 10:09:29 +00:00
2021-04-08 17:20:45 +00:00
renameTempPartAndReplace ( new_part , nullptr ) ;
2020-01-22 13:24:20 +00:00
2020-07-22 19:29:54 +00:00
updateMutationEntriesErrors ( future_part , true , " " ) ;
2018-07-09 15:34:11 +00:00
write_part_log ( { } ) ;
}
catch ( . . . )
{
2020-07-22 19:29:54 +00:00
updateMutationEntriesErrors ( future_part , false , getCurrentExceptionMessage ( false ) ) ;
2018-07-09 15:34:11 +00:00
write_part_log ( ExecutionStatus : : fromCurrentException ( ) ) ;
throw ;
}
return true ;
}
2021-05-08 14:01:25 +00:00
std : : optional < JobAndPool > StorageMergeTree : : getDataProcessingJob ( ) //-V657
2014-03-13 12:48:07 +00:00
{
2017-04-01 07:20:54 +00:00
if ( shutdown_called )
2020-10-13 14:25:42 +00:00
return { } ;
2017-04-01 07:20:54 +00:00
2019-08-01 15:36:12 +00:00
if ( merger_mutator . merges_blocker . isCancelled ( ) )
2020-10-13 14:25:42 +00:00
return { } ;
2018-07-09 15:34:11 +00:00
2020-10-13 14:25:42 +00:00
auto metadata_snapshot = getInMemoryMetadataPtr ( ) ;
2020-10-23 08:54:00 +00:00
std : : shared_ptr < MergeMutateSelectedEntry > merge_entry , mutate_entry ;
2019-09-03 17:06:36 +00:00
2020-10-20 21:10:55 +00:00
auto share_lock = lockForShare ( RWLockImpl : : NO_QUERY , getSettings ( ) - > lock_acquire_timeout_for_background_operations ) ;
2021-05-18 17:07:29 +00:00
merge_entry = selectPartsToMerge ( metadata_snapshot , false , { } , false , nullptr , share_lock , nullptr ) ;
2020-10-13 14:25:42 +00:00
if ( ! merge_entry )
2020-10-20 21:10:55 +00:00
mutate_entry = selectPartsToMutate ( metadata_snapshot , nullptr , share_lock ) ;
2019-09-03 17:06:36 +00:00
2020-10-13 14:25:42 +00:00
if ( merge_entry | | mutate_entry )
{
2021-05-18 17:07:29 +00:00
return JobAndPool { [ this , metadata_snapshot , merge_entry , mutate_entry , share_lock ] ( ) mutable
2020-10-13 14:25:42 +00:00
{
2021-05-18 17:07:29 +00:00
merge_entry = { } ;
mutate_entry = { } ;
/// FIXME Transactions: do not begin transaction if we don't need it
auto txn = TransactionLog : : instance ( ) . beginTransaction ( ) ;
MergeTreeTransactionHolder autocommit { txn , true } ;
merge_entry = selectPartsToMerge ( metadata_snapshot , false , { } , false , nullptr , share_lock , txn ) ;
if ( ! merge_entry )
mutate_entry = selectPartsToMutate ( metadata_snapshot , nullptr , share_lock ) ;
2020-10-13 14:25:42 +00:00
if ( merge_entry )
2021-05-18 17:07:29 +00:00
return mergeSelectedParts ( metadata_snapshot , false , { } , * merge_entry , share_lock , txn ) ;
2020-10-13 14:25:42 +00:00
else if ( mutate_entry )
2021-02-11 11:46:18 +00:00
return mutateSelectedPart ( metadata_snapshot , * mutate_entry , share_lock ) ;
2021-05-18 17:07:29 +00:00
return true ;
//__builtin_unreachable();
2020-10-16 10:12:31 +00:00
} , PoolType : : MERGE_MUTATE } ;
2017-04-01 07:20:54 +00:00
}
2020-10-13 14:25:42 +00:00
else if ( auto lock = time_after_previous_cleanup . compareAndRestartDeferred ( 1 ) )
2017-04-01 07:20:54 +00:00
{
2020-10-20 21:10:55 +00:00
return JobAndPool { [ this , share_lock ] ( )
2017-04-01 07:20:54 +00:00
{
2020-10-20 21:10:55 +00:00
/// All use relative_data_path which changes during rename
/// so execute under share lock.
clearOldPartsFromFilesystem ( ) ;
clearOldTemporaryDirectories ( ) ;
clearOldWriteAheadLogs ( ) ;
2020-10-13 14:25:42 +00:00
clearOldMutations ( ) ;
2020-11-11 16:18:21 +00:00
clearEmptyParts ( ) ;
2021-02-11 11:46:18 +00:00
return true ;
2020-10-16 10:12:31 +00:00
} , PoolType : : MERGE_MUTATE } ;
2017-04-01 07:20:54 +00:00
}
2020-10-13 14:25:42 +00:00
return { } ;
2014-03-13 12:48:07 +00:00
}
2018-03-30 19:25:37 +00:00
Int64 StorageMergeTree : : getCurrentMutationVersion (
2019-05-03 02:00:57 +00:00
const DataPartPtr & part ,
2020-06-02 00:41:52 +00:00
std : : unique_lock < std : : mutex > & /* currently_processing_in_background_mutex_lock */ ) const
2018-03-30 19:25:37 +00:00
{
auto it = current_mutations_by_version . upper_bound ( part - > info . getDataVersion ( ) ) ;
if ( it = = current_mutations_by_version . begin ( ) )
return 0 ;
- - it ;
return it - > first ;
2018-08-10 04:02:56 +00:00
}
2018-03-30 19:25:37 +00:00
2019-07-29 09:15:46 +00:00
void StorageMergeTree : : clearOldMutations ( bool truncate )
2018-07-31 12:34:34 +00:00
{
2019-08-26 14:24:29 +00:00
const auto settings = getSettings ( ) ;
2019-07-29 09:15:46 +00:00
if ( ! truncate & & ! settings - > finished_mutations_to_keep )
2018-07-31 12:34:34 +00:00
return ;
std : : vector < MergeTreeMutationEntry > mutations_to_delete ;
{
2019-08-15 09:43:31 +00:00
std : : lock_guard lock ( currently_processing_in_background_mutex ) ;
2018-07-31 12:34:34 +00:00
2019-07-29 09:15:46 +00:00
if ( ! truncate & & current_mutations_by_version . size ( ) < = settings - > finished_mutations_to_keep )
2018-07-31 12:34:34 +00:00
return ;
2019-07-29 09:15:46 +00:00
auto end_it = current_mutations_by_version . end ( ) ;
2018-07-31 12:34:34 +00:00
auto begin_it = current_mutations_by_version . begin ( ) ;
2019-07-29 09:15:46 +00:00
size_t to_delete_count = std : : distance ( begin_it , end_it ) ;
2018-07-31 12:34:34 +00:00
2019-07-29 09:15:46 +00:00
if ( ! truncate )
{
if ( std : : optional < Int64 > min_version = getMinPartDataVersion ( ) )
end_it = current_mutations_by_version . upper_bound ( * min_version ) ;
2018-07-31 12:34:34 +00:00
2019-07-29 09:15:46 +00:00
size_t done_count = std : : distance ( begin_it , end_it ) ;
if ( done_count < = settings - > finished_mutations_to_keep )
return ;
2018-07-31 12:34:34 +00:00
2019-07-29 09:15:46 +00:00
to_delete_count = done_count - settings - > finished_mutations_to_keep ;
}
2018-07-31 12:34:34 +00:00
auto it = begin_it ;
for ( size_t i = 0 ; i < to_delete_count ; + + i )
{
mutations_to_delete . push_back ( std : : move ( it - > second ) ) ;
2019-01-14 12:25:25 +00:00
current_mutations_by_id . erase ( mutations_to_delete . back ( ) . file_name ) ;
2018-07-31 12:34:34 +00:00
it = current_mutations_by_version . erase ( it ) ;
}
}
for ( auto & mutation : mutations_to_delete )
{
2020-05-23 22:24:01 +00:00
LOG_TRACE ( log , " Removing mutation: {} " , mutation . file_name ) ;
2018-07-31 12:34:34 +00:00
mutation . removeFile ( ) ;
}
}
2018-03-30 19:25:37 +00:00
2017-09-06 20:34:26 +00:00
bool StorageMergeTree : : optimize (
2020-06-17 13:39:26 +00:00
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 ,
2021-04-10 23:33:54 +00:00
ContextPtr local_context )
2017-09-06 20:34:26 +00:00
{
2020-12-08 16:44:34 +00:00
if ( deduplicate )
{
if ( deduplicate_by_columns . empty ( ) )
LOG_DEBUG ( log , " DEDUPLICATE BY all columns " ) ;
else
LOG_DEBUG ( log , " DEDUPLICATE BY ('{}') " , fmt : : join ( deduplicate_by_columns , " ', ' " ) ) ;
}
2021-05-17 11:14:09 +00:00
auto txn = local_context - > getCurrentTransaction ( ) ;
2018-01-12 17:30:21 +00:00
String disable_reason ;
2018-07-05 17:32:14 +00:00
if ( ! partition & & final )
2018-01-12 17:30:21 +00:00
{
2019-05-03 02:00:57 +00:00
DataPartsVector data_parts = getDataPartsVector ( ) ;
2018-07-05 17:32:14 +00:00
std : : unordered_set < String > partition_ids ;
2019-05-03 02:00:57 +00:00
for ( const DataPartPtr & part : data_parts )
2018-07-05 18:45:18 +00:00
partition_ids . emplace ( part - > info . partition_id ) ;
2018-07-05 17:32:14 +00:00
for ( const String & partition_id : partition_ids )
{
2021-04-10 23:33:54 +00:00
if ( ! merge (
true ,
partition_id ,
true ,
deduplicate ,
deduplicate_by_columns ,
2021-05-17 11:14:09 +00:00
txn ,
2021-04-10 23:33:54 +00:00
& disable_reason ,
local_context - > getSettingsRef ( ) . optimize_skip_merged_partitions ) )
2018-07-05 17:32:14 +00:00
{
2020-11-10 18:22:26 +00:00
constexpr const char * message = " Cannot OPTIMIZE table: {} " ;
if ( disable_reason . empty ( ) )
disable_reason = " unknown reason " ;
LOG_INFO ( log , message , disable_reason ) ;
2019-07-12 13:39:16 +00:00
2021-04-10 23:33:54 +00:00
if ( local_context - > getSettingsRef ( ) . optimize_throw_if_noop )
2020-11-10 18:22:26 +00:00
throw Exception ( ErrorCodes : : CANNOT_ASSIGN_OPTIMIZE , message , disable_reason ) ;
2018-07-05 17:32:14 +00:00
return false ;
}
}
}
else
{
2019-01-04 12:10:00 +00:00
String partition_id ;
if ( partition )
2021-04-10 23:33:54 +00:00
partition_id = getPartitionIDFromQuery ( partition , local_context ) ;
if ( ! merge (
true ,
partition_id ,
final ,
deduplicate ,
deduplicate_by_columns ,
2021-05-17 11:14:09 +00:00
txn ,
2021-04-10 23:33:54 +00:00
& disable_reason ,
local_context - > getSettingsRef ( ) . optimize_skip_merged_partitions ) )
2018-07-05 17:32:14 +00:00
{
2020-11-10 18:22:26 +00:00
constexpr const char * message = " Cannot OPTIMIZE table: {} " ;
if ( disable_reason . empty ( ) )
disable_reason = " unknown reason " ;
LOG_INFO ( log , message , disable_reason ) ;
2019-07-12 13:39:16 +00:00
2021-04-10 23:33:54 +00:00
if ( local_context - > getSettingsRef ( ) . optimize_throw_if_noop )
2020-11-10 18:22:26 +00:00
throw Exception ( ErrorCodes : : CANNOT_ASSIGN_OPTIMIZE , message , disable_reason ) ;
2018-07-05 17:32:14 +00:00
return false ;
}
2018-01-12 17:30:21 +00:00
}
return true ;
2017-09-06 20:34:26 +00:00
}
2020-08-15 05:21:02 +00:00
ActionLock StorageMergeTree : : stopMergesAndWait ( )
{
/// Asks to complete merges and does not allow them to start.
/// This protects against "revival" of data for a removed partition after completion of merge.
auto merge_blocker = merger_mutator . merges_blocker . cancel ( ) ;
{
std : : unique_lock lock ( currently_processing_in_background_mutex ) ;
while ( ! currently_merging_mutating_parts . empty ( ) )
{
LOG_DEBUG ( log , " Waiting for currently running merges ({} parts are merging right now) " ,
currently_merging_mutating_parts . size ( ) ) ;
if ( std : : cv_status : : timeout = = currently_processing_in_background_condition . wait_for (
lock , std : : chrono : : seconds ( DBMS_DEFAULT_LOCK_ACQUIRE_TIMEOUT_SEC ) ) )
{
throw Exception ( " Timeout while waiting for already running merges " , ErrorCodes : : TIMEOUT_EXCEEDED ) ;
}
}
}
return merge_blocker ;
}
2020-12-22 13:46:09 +00:00
MergeTreeDataPartPtr StorageMergeTree : : outdatePart ( const String & part_name , bool force )
2014-10-03 17:57:01 +00:00
{
2020-12-22 13:46:09 +00:00
if ( force )
2017-04-01 07:20:54 +00:00
{
2020-12-22 13:46:09 +00:00
/// Forcefully stop merges and make part outdated
2020-08-15 05:21:02 +00:00
auto merge_blocker = stopMergesAndWait ( ) ;
2020-12-22 13:46:09 +00:00
auto part = getPartIfExists ( part_name , { MergeTreeDataPartState : : Committed } ) ;
if ( ! part )
throw Exception ( " Part " + part_name + " not found, won't try to drop it. " , ErrorCodes : : NO_SUCH_DATA_PART ) ;
removePartsFromWorkingSet ( { part } , true ) ;
return part ;
}
else
{
2018-05-21 13:49:54 +00:00
2020-12-22 13:46:09 +00:00
/// Wait merges selector
std : : unique_lock lock ( currently_processing_in_background_mutex ) ;
auto part = getPartIfExists ( part_name , { MergeTreeDataPartState : : Committed } ) ;
/// It's okay, part was already removed
if ( ! part )
return nullptr ;
2014-10-03 17:57:01 +00:00
2020-12-22 13:46:09 +00:00
/// Part will be "removed" by merge or mutation, it's OK in case of some
/// background cleanup processes like removing of empty parts.
if ( currently_merging_mutating_parts . count ( part ) )
return nullptr ;
removePartsFromWorkingSet ( { part } , true ) ;
return part ;
}
}
2021-05-17 14:26:36 +00:00
void StorageMergeTree : : dropPartNoWaitNoThrow ( const String & part_name )
2020-12-22 13:46:09 +00:00
{
2021-05-25 17:25:00 +00:00
if ( auto part = outdatePart ( part_name , /*force=*/ false ) )
dropPartsImpl ( { part } , /*detach=*/ false ) ;
2021-04-20 02:31:08 +00:00
2021-05-17 14:26:36 +00:00
/// Else nothing to do, part was removed in some different way
2021-04-20 02:31:08 +00:00
}
2021-05-17 14:26:36 +00:00
void StorageMergeTree : : dropPart ( const String & part_name , bool detach , ContextPtr /*query_context*/ )
2021-04-20 02:31:08 +00:00
{
2021-05-25 17:25:00 +00:00
if ( auto part = outdatePart ( part_name , /*force=*/ true ) )
2021-05-17 14:26:36 +00:00
dropPartsImpl ( { part } , detach ) ;
}
2020-12-22 13:46:09 +00:00
2021-05-17 14:26:36 +00:00
void StorageMergeTree : : dropPartition ( const ASTPtr & partition , bool detach , ContextPtr local_context )
{
2021-05-25 17:25:00 +00:00
DataPartsVector parts_to_remove ;
/// New scope controls lifetime of merge_blocker.
2020-12-22 13:46:09 +00:00
{
2021-05-25 17:25:00 +00:00
/// Asks to complete merges and does not allow them to start.
/// This protects against "revival" of data for a removed partition after completion of merge.
auto merge_blocker = stopMergesAndWait ( ) ;
String partition_id = getPartitionIDFromQuery ( partition , local_context ) ;
parts_to_remove = getDataPartsVectorInPartition ( MergeTreeDataPartState : : Committed , partition_id ) ;
2014-10-03 17:57:01 +00:00
2021-05-25 17:25:00 +00:00
/// TODO should we throw an exception if parts_to_remove is empty?
removePartsFromWorkingSet ( parts_to_remove , true ) ;
}
2020-12-22 13:46:09 +00:00
2021-04-22 20:07:00 +00:00
dropPartsImpl ( std : : move ( parts_to_remove ) , detach ) ;
2021-04-20 02:31:08 +00:00
}
2014-10-03 17:57:01 +00:00
2021-04-22 20:07:00 +00:00
void StorageMergeTree : : dropPartsImpl ( DataPartsVector & & parts_to_remove , bool detach )
2021-04-20 02:31:08 +00:00
{
auto metadata_snapshot = getInMemoryMetadataPtr ( ) ;
2018-05-21 13:49:54 +00:00
2021-04-20 02:31:08 +00:00
if ( detach )
{
/// If DETACH clone parts to detached/ directory
/// NOTE: no race with background cleanup until we hold pointers to parts
for ( const auto & part : parts_to_remove )
2021-04-02 11:46:42 +00:00
{
2021-04-20 02:31:08 +00:00
LOG_INFO ( log , " Detaching {} " , part - > relative_path ) ;
part - > makeCloneInDetached ( " " , metadata_snapshot ) ;
2021-04-02 11:46:42 +00:00
}
2021-04-20 02:31:08 +00:00
}
2021-04-02 11:46:42 +00:00
2021-04-20 02:31:08 +00:00
if ( deduplication_log )
{
for ( const auto & part : parts_to_remove )
deduplication_log - > dropPart ( part - > info ) ;
2017-04-01 07:20:54 +00:00
}
2014-10-03 17:57:01 +00:00
2021-04-20 02:31:08 +00:00
if ( detach )
LOG_INFO ( log , " Detached {} parts. " , parts_to_remove . size ( ) ) ;
else
LOG_INFO ( log , " Removed {} parts. " , parts_to_remove . size ( ) ) ;
2021-04-22 20:07:00 +00:00
/// Need to destroy part objects before clearing them from filesystem.
parts_to_remove . clear ( ) ;
2019-05-03 02:00:57 +00:00
clearOldPartsFromFilesystem ( ) ;
2014-10-03 17:57:01 +00:00
}
2014-10-03 18:41:16 +00:00
2020-08-15 06:56:47 +00:00
PartitionCommandsResultInfo StorageMergeTree : : attachPartition (
2020-11-11 13:34:07 +00:00
const ASTPtr & partition , const StorageMetadataPtr & /* metadata_snapshot */ ,
2021-04-10 23:33:54 +00:00
bool attach_part , ContextPtr local_context )
2014-10-03 18:41:16 +00:00
{
2020-07-28 15:10:36 +00:00
PartitionCommandsResultInfo results ;
2019-08-29 16:17:47 +00:00
PartsTemporaryRename renamed_parts ( * this , " detached/ " ) ;
2021-04-10 23:33:54 +00:00
MutableDataPartsVector loaded_parts = tryLoadPartsToAttach ( partition , attach_part , local_context , renamed_parts ) ;
2017-04-01 07:20:54 +00:00
2019-07-30 17:24:40 +00:00
for ( size_t i = 0 ; i < loaded_parts . size ( ) ; + + i )
2017-04-01 07:20:54 +00:00
{
2020-05-23 22:24:01 +00:00
LOG_INFO ( log , " Attaching part {} from {} " , loaded_parts [ i ] - > name , renamed_parts . old_and_new_names [ i ] . second ) ;
2020-07-28 15:10:36 +00:00
String old_name = renamed_parts . old_and_new_names [ i ] . first ;
2021-05-13 18:48:36 +00:00
renameTempPartAndAdd ( loaded_parts [ i ] , local_context - > getCurrentTransaction ( ) . get ( ) , & increment ) ;
2019-07-30 17:24:40 +00:00
renamed_parts . old_and_new_names [ i ] . first . clear ( ) ;
2020-07-28 15:10:36 +00:00
results . push_back ( PartitionCommandResultInfo {
. partition_id = loaded_parts [ i ] - > info . partition_id ,
. part_name = loaded_parts [ i ] - > name ,
. old_part_name = old_name ,
} ) ;
2017-04-01 07:20:54 +00:00
LOG_INFO ( log , " Finished attaching part " ) ;
}
2017-04-16 15:00:33 +00:00
/// New parts with other data may appear in place of deleted parts.
2021-04-10 23:33:54 +00:00
local_context - > dropCaches ( ) ;
2020-07-28 15:10:36 +00:00
return results ;
2014-10-03 18:41:16 +00:00
}
2021-04-10 23:33:54 +00:00
void StorageMergeTree : : replacePartitionFrom ( const StoragePtr & source_table , const ASTPtr & partition , bool replace , ContextPtr local_context )
2018-05-21 13:49:54 +00:00
{
2021-04-10 23:33:54 +00:00
auto lock1 = lockForShare ( local_context - > getCurrentQueryId ( ) , local_context - > getSettingsRef ( ) . lock_acquire_timeout ) ;
auto lock2 = source_table - > lockForShare ( local_context - > getCurrentQueryId ( ) , local_context - > getSettingsRef ( ) . lock_acquire_timeout ) ;
2020-06-17 10:34:23 +00:00
auto source_metadata_snapshot = source_table - > getInMemoryMetadataPtr ( ) ;
auto my_metadata_snapshot = getInMemoryMetadataPtr ( ) ;
2018-05-21 13:49:54 +00:00
Stopwatch watch ;
2020-06-17 10:34:23 +00:00
MergeTreeData & src_data = checkStructureAndGetMergeTreeData ( source_table , source_metadata_snapshot , my_metadata_snapshot ) ;
2021-04-10 23:33:54 +00:00
String partition_id = getPartitionIDFromQuery ( partition , local_context ) ;
2018-05-21 13:49:54 +00:00
2019-05-03 02:00:57 +00:00
DataPartsVector src_parts = src_data . getDataPartsVectorInPartition ( MergeTreeDataPartState : : Committed , partition_id ) ;
MutableDataPartsVector dst_parts ;
2018-05-21 13:49:54 +00:00
static const String TMP_PREFIX = " tmp_replace_from_ " ;
2019-05-03 02:00:57 +00:00
for ( const DataPartPtr & src_part : src_parts )
2018-05-21 13:49:54 +00:00
{
2019-06-19 16:16:13 +00:00
if ( ! canReplacePartition ( src_part ) )
throw Exception (
" Cannot replace partition ' " + partition_id + " ' because part ' " + src_part - > name + " ' has inconsistent granularity with table " ,
2019-12-30 14:46:02 +00:00
ErrorCodes : : BAD_ARGUMENTS ) ;
2019-06-19 16:16:13 +00:00
2018-05-21 13:49:54 +00:00
/// This will generate unique name in scope of current server process.
2019-05-03 02:00:57 +00:00
Int64 temp_index = insert_increment . get ( ) ;
2018-05-21 13:49:54 +00:00
MergeTreePartInfo dst_part_info ( partition_id , temp_index , temp_index , src_part - > info . level ) ;
2020-06-26 11:30:23 +00:00
dst_parts . emplace_back ( cloneAndLoadDataPartOnSameDisk ( src_part , TMP_PREFIX , dst_part_info , my_metadata_snapshot ) ) ;
2018-05-21 13:49:54 +00:00
}
/// ATTACH empty part set
if ( ! replace & & dst_parts . empty ( ) )
return ;
MergeTreePartInfo drop_range ;
if ( replace )
{
drop_range . partition_id = partition_id ;
drop_range . min_block = 0 ;
drop_range . max_block = increment . get ( ) ; // there will be a "hole" in block numbers
drop_range . level = std : : numeric_limits < decltype ( drop_range . level ) > : : max ( ) ;
}
/// Atomically add new parts and remove old ones
try
{
{
/// Here we use the transaction just like RAII since rare errors in renameTempPartAndReplace() are possible
/// and we should be able to rollback already added (Precomitted) parts
2021-06-04 09:26:47 +00:00
Transaction transaction ( * this , local_context - > getCurrentTransaction ( ) . get ( ) ) ;
2018-05-21 13:49:54 +00:00
2019-05-03 02:00:57 +00:00
auto data_parts_lock = lockParts ( ) ;
2018-05-21 13:49:54 +00:00
/// Populate transaction
2019-05-03 02:00:57 +00:00
for ( MutableDataPartPtr & part : dst_parts )
2021-05-13 18:48:36 +00:00
renameTempPartAndReplace ( part , local_context - > getCurrentTransaction ( ) . get ( ) , & increment , & transaction , data_parts_lock ) ;
2018-05-21 13:49:54 +00:00
transaction . commit ( & data_parts_lock ) ;
/// If it is REPLACE (not ATTACH), remove all parts which max_block_number less then min_block_number of the first new block
if ( replace )
2021-05-13 14:04:36 +00:00
removePartsInRangeFromWorkingSet ( drop_range , true , data_parts_lock ) ;
2018-05-21 13:49:54 +00:00
}
2021-04-10 23:33:54 +00:00
PartLog : : addNewParts ( getContext ( ) , dst_parts , watch . elapsed ( ) ) ;
2018-05-21 13:49:54 +00:00
}
catch ( . . . )
{
2021-04-10 23:33:54 +00:00
PartLog : : addNewParts ( getContext ( ) , dst_parts , watch . elapsed ( ) , ExecutionStatus : : fromCurrentException ( ) ) ;
2018-05-21 13:49:54 +00:00
throw ;
}
}
2021-04-10 23:33:54 +00:00
void StorageMergeTree : : movePartitionToTable ( const StoragePtr & dest_table , const ASTPtr & partition , ContextPtr local_context )
2019-07-26 08:42:17 +00:00
{
2021-04-10 23:33:54 +00:00
auto lock1 = lockForShare ( local_context - > getCurrentQueryId ( ) , local_context - > getSettingsRef ( ) . lock_acquire_timeout ) ;
auto lock2 = dest_table - > lockForShare ( local_context - > getCurrentQueryId ( ) , local_context - > getSettingsRef ( ) . lock_acquire_timeout ) ;
2019-09-17 09:00:20 +00:00
2019-07-26 09:35:47 +00:00
auto dest_table_storage = std : : dynamic_pointer_cast < StorageMergeTree > ( dest_table ) ;
if ( ! dest_table_storage )
2020-02-21 16:57:40 +00:00
throw Exception ( " Table " + getStorageID ( ) . getNameForLogs ( ) + " supports movePartitionToTable only for MergeTree family of table engines. "
2019-07-26 09:35:47 +00:00
" Got " + dest_table - > getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
2020-01-08 09:57:27 +00:00
if ( dest_table_storage - > getStoragePolicy ( ) ! = this - > getStoragePolicy ( ) )
2020-01-15 11:55:20 +00:00
throw Exception ( " Destination table " + dest_table_storage - > getStorageID ( ) . getNameForLogs ( ) +
" should have the same storage policy of source table " + getStorageID ( ) . getNameForLogs ( ) + " . " +
getStorageID ( ) . getNameForLogs ( ) + " : " + this - > getStoragePolicy ( ) - > getName ( ) + " , " +
2020-10-16 11:58:47 +00:00
dest_table_storage - > getStorageID ( ) . getNameForLogs ( ) + " : " + dest_table_storage - > getStoragePolicy ( ) - > getName ( ) , ErrorCodes : : UNKNOWN_POLICY ) ;
2020-06-17 10:34:23 +00:00
auto dest_metadata_snapshot = dest_table - > getInMemoryMetadataPtr ( ) ;
auto metadata_snapshot = getInMemoryMetadataPtr ( ) ;
2019-07-26 08:42:17 +00:00
Stopwatch watch ;
2020-06-17 10:34:23 +00:00
MergeTreeData & src_data = dest_table_storage - > checkStructureAndGetMergeTreeData ( * this , metadata_snapshot , dest_metadata_snapshot ) ;
2021-04-10 23:33:54 +00:00
String partition_id = getPartitionIDFromQuery ( partition , local_context ) ;
2019-07-26 08:42:17 +00:00
DataPartsVector src_parts = src_data . getDataPartsVectorInPartition ( MergeTreeDataPartState : : Committed , partition_id ) ;
MutableDataPartsVector dst_parts ;
2020-02-21 16:57:40 +00:00
static const String TMP_PREFIX = " tmp_move_from_ " ;
2019-07-26 08:42:17 +00:00
for ( const DataPartPtr & src_part : src_parts )
{
2019-07-26 09:35:47 +00:00
if ( ! dest_table_storage - > canReplacePartition ( src_part ) )
2019-07-26 08:42:17 +00:00
throw Exception (
2020-02-21 16:57:40 +00:00
" Cannot move partition ' " + partition_id + " ' because part ' " + src_part - > name + " ' has inconsistent granularity with table " ,
2019-07-26 08:42:17 +00:00
ErrorCodes : : LOGICAL_ERROR ) ;
/// This will generate unique name in scope of current server process.
Int64 temp_index = insert_increment . get ( ) ;
MergeTreePartInfo dst_part_info ( partition_id , temp_index , temp_index , src_part - > info . level ) ;
2020-06-26 11:30:23 +00:00
dst_parts . emplace_back ( dest_table_storage - > cloneAndLoadDataPartOnSameDisk ( src_part , TMP_PREFIX , dst_part_info , dest_metadata_snapshot ) ) ;
2019-07-26 08:42:17 +00:00
}
2020-02-21 16:57:40 +00:00
/// empty part set
2019-07-26 08:42:17 +00:00
if ( dst_parts . empty ( ) )
return ;
2020-02-21 16:57:40 +00:00
/// Move new parts to the destination table. NOTE It doesn't look atomic.
2019-07-26 08:42:17 +00:00
try
{
{
2021-06-04 09:26:47 +00:00
Transaction transaction ( * dest_table_storage , local_context - > getCurrentTransaction ( ) . get ( ) ) ;
2019-07-26 08:42:17 +00:00
2019-10-01 18:04:42 +00:00
auto src_data_parts_lock = lockParts ( ) ;
auto dest_data_parts_lock = dest_table_storage - > lockParts ( ) ;
2019-07-26 08:42:17 +00:00
2019-10-22 07:15:58 +00:00
std : : mutex mutex ;
DataPartsLock lock ( mutex ) ;
2019-07-26 08:42:17 +00:00
for ( MutableDataPartPtr & part : dst_parts )
2021-05-13 18:48:36 +00:00
dest_table_storage - > renameTempPartAndReplace ( part , local_context - > getCurrentTransaction ( ) . get ( ) , & dest_table_storage - > increment , & transaction , lock ) ;
2019-07-26 08:42:17 +00:00
2019-10-22 07:15:58 +00:00
removePartsFromWorkingSet ( src_parts , true , lock ) ;
transaction . commit ( & lock ) ;
2019-07-26 08:42:17 +00:00
}
2019-09-16 08:56:30 +00:00
clearOldMutations ( true ) ;
clearOldPartsFromFilesystem ( ) ;
2021-04-10 23:33:54 +00:00
PartLog : : addNewParts ( getContext ( ) , dst_parts , watch . elapsed ( ) ) ;
2019-07-26 08:42:17 +00:00
}
catch ( . . . )
{
2021-04-10 23:33:54 +00:00
PartLog : : addNewParts ( getContext ( ) , dst_parts , watch . elapsed ( ) , ExecutionStatus : : fromCurrentException ( ) ) ;
2019-07-26 08:42:17 +00:00
throw ;
}
}
2019-07-25 10:46:07 +00:00
2018-05-28 15:37:30 +00:00
ActionLock StorageMergeTree : : getActionLock ( StorageActionBlockType action_type )
2018-05-21 13:49:54 +00:00
{
if ( action_type = = ActionLocks : : PartsMerge )
2019-08-01 15:36:12 +00:00
return merger_mutator . merges_blocker . cancel ( ) ;
else if ( action_type = = ActionLocks : : PartsTTLMerge )
2019-09-03 14:50:49 +00:00
return merger_mutator . ttl_merges_blocker . cancel ( ) ;
else if ( action_type = = ActionLocks : : PartsMove )
return parts_mover . moves_blocker . cancel ( ) ;
2018-05-21 13:49:54 +00:00
return { } ;
}
2020-10-15 16:10:22 +00:00
void StorageMergeTree : : onActionLockRemove ( StorageActionBlockType action_type )
{
if ( action_type = = ActionLocks : : PartsMerge | | action_type = = ActionLocks : : PartsTTLMerge )
background_executor . triggerTask ( ) ;
else if ( action_type = = ActionLocks : : PartsMove )
background_moves_executor . triggerTask ( ) ;
}
2021-04-10 23:33:54 +00:00
CheckResults StorageMergeTree : : checkData ( const ASTPtr & query , ContextPtr local_context )
2019-07-03 08:49:52 +00:00
{
2019-07-03 13:17:19 +00:00
CheckResults results ;
DataPartsVector data_parts ;
if ( const auto & check_query = query - > as < ASTCheckQuery & > ( ) ; check_query . partition )
{
2021-04-10 23:33:54 +00:00
String partition_id = getPartitionIDFromQuery ( check_query . partition , local_context ) ;
2019-07-03 13:17:19 +00:00
data_parts = getDataPartsVectorInPartition ( MergeTreeDataPartState : : Committed , partition_id ) ;
}
else
data_parts = getDataPartsVector ( ) ;
2019-07-03 08:49:52 +00:00
for ( auto & part : data_parts )
2019-07-03 13:17:19 +00:00
{
2020-05-09 21:24:15 +00:00
auto disk = part - > volume - > getDisk ( ) ;
2020-02-28 17:14:55 +00:00
String part_path = part - > getFullRelativePath ( ) ;
2019-07-03 13:17:19 +00:00
/// If the checksums file is not present, calculate the checksums and write them to disk.
2021-05-05 15:10:14 +00:00
String checksums_path = fs : : path ( part_path ) / " checksums.txt " ;
String tmp_checksums_path = fs : : path ( part_path ) / " checksums.txt.tmp " ;
2020-06-03 18:59:18 +00:00
if ( part - > isStoredOnDisk ( ) & & ! disk - > exists ( checksums_path ) )
2019-07-03 13:17:19 +00:00
{
try
{
2020-01-13 14:53:32 +00:00
auto calculated_checksums = checkDataPart ( part , false ) ;
2019-07-09 09:02:52 +00:00
calculated_checksums . checkEqual ( part - > checksums , true ) ;
2020-02-28 17:14:55 +00:00
auto out = disk - > writeFile ( tmp_checksums_path , 4096 ) ;
part - > checksums . write ( * out ) ;
disk - > moveFile ( tmp_checksums_path , checksums_path ) ;
2019-07-03 20:51:13 +00:00
results . emplace_back ( part - > name , true , " Checksums recounted and written to disk. " ) ;
2019-07-03 13:17:19 +00:00
}
2019-07-09 09:02:52 +00:00
catch ( const Exception & ex )
2019-07-03 13:17:19 +00:00
{
2020-02-28 17:14:55 +00:00
if ( disk - > exists ( tmp_checksums_path ) )
2021-01-14 16:24:13 +00:00
disk - > removeFile ( tmp_checksums_path ) ;
2019-07-09 09:02:52 +00:00
2019-07-03 13:17:19 +00:00
results . emplace_back ( part - > name , false ,
2019-07-03 20:51:13 +00:00
" Check of part finished with error: ' " + ex . message ( ) + " ' " ) ;
2019-07-03 13:17:19 +00:00
}
}
else
{
try
{
2020-01-13 14:53:32 +00:00
checkDataPart ( part , true ) ;
2019-07-03 13:17:19 +00:00
results . emplace_back ( part - > name , true , " " ) ;
}
2019-07-09 09:02:52 +00:00
catch ( const Exception & ex )
2019-07-03 13:17:19 +00:00
{
results . emplace_back ( part - > name , false , ex . message ( ) ) ;
}
}
}
return results ;
2019-07-03 08:49:52 +00:00
}
2020-04-02 18:24:11 +00:00
2020-11-28 08:17:20 +00:00
MutationCommands StorageMergeTree : : getFirstAlterMutationCommandsForPart ( const DataPartPtr & part ) const
2020-04-02 18:24:11 +00:00
{
std : : lock_guard lock ( currently_processing_in_background_mutex ) ;
auto it = current_mutations_by_version . upper_bound ( part - > info . getDataVersion ( ) ) ;
if ( it = = current_mutations_by_version . end ( ) )
return { } ;
return it - > second . commands ;
}
2020-10-14 07:22:48 +00:00
void StorageMergeTree : : startBackgroundMovesIfNeeded ( )
{
2020-10-20 11:27:50 +00:00
if ( areBackgroundMovesNeeded ( ) )
background_moves_executor . start ( ) ;
2020-10-14 07:22:48 +00:00
}
2012-07-17 20:04:39 +00:00
}