2016-06-07 08:23:15 +00:00
|
|
|
#pragma once
|
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Common/Exception.h>
|
2018-03-24 20:00:16 +00:00
|
|
|
#include <Common/ZooKeeper/Types.h>
|
2020-09-15 09:55:57 +00:00
|
|
|
#include <common/types.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <IO/WriteHelpers.h>
|
2020-02-11 13:41:26 +00:00
|
|
|
#include <Storages/MergeTree/MergeTreeDataPartType.h>
|
2020-09-03 13:00:13 +00:00
|
|
|
#include <Storages/MergeTree/MergeType.h>
|
2016-06-07 08:23:15 +00:00
|
|
|
|
|
|
|
#include <mutex>
|
|
|
|
#include <condition_variable>
|
|
|
|
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
|
|
|
class ReadBuffer;
|
|
|
|
class WriteBuffer;
|
|
|
|
class ReplicatedMergeTreeQueue;
|
|
|
|
|
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
extern const int LOGICAL_ERROR;
|
2016-06-07 08:23:15 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2017-04-16 15:00:33 +00:00
|
|
|
/// Record about what needs to be done. Only data (you can copy them).
|
2016-06-07 08:23:15 +00:00
|
|
|
struct ReplicatedMergeTreeLogEntryData
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
enum Type
|
|
|
|
{
|
2017-05-12 13:47:42 +00:00
|
|
|
EMPTY, /// Not used.
|
|
|
|
GET_PART, /// Get the part from another replica.
|
2021-02-14 22:59:13 +00:00
|
|
|
ATTACH_PART, /// Attach the part, possibly from our own replica (if found in /detached folder).
|
2021-02-15 15:06:48 +00:00
|
|
|
/// You may think of it as a GET_PART with some opmitisations as they're nearly identical.
|
2017-05-12 13:47:42 +00:00
|
|
|
MERGE_PARTS, /// Merge the parts.
|
2017-08-14 18:16:11 +00:00
|
|
|
DROP_RANGE, /// Delete the parts in the specified partition in the specified number range.
|
2020-03-17 13:49:50 +00:00
|
|
|
CLEAR_COLUMN, /// NOTE: Deprecated. Drop specific column from specified partition.
|
|
|
|
CLEAR_INDEX, /// NOTE: Deprecated. Drop specific index from specified partition.
|
2018-05-21 13:49:54 +00:00
|
|
|
REPLACE_RANGE, /// Drop certain range of partitions and replace them by new ones
|
2018-04-20 16:18:16 +00:00
|
|
|
MUTATE_PART, /// Apply one or several mutations to the part.
|
2020-01-28 17:15:22 +00:00
|
|
|
ALTER_METADATA, /// Apply alter modification according to global /metadata and /columns paths
|
2017-04-01 07:20:54 +00:00
|
|
|
};
|
|
|
|
|
2018-05-21 13:49:54 +00:00
|
|
|
static String typeToString(Type type)
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
|
|
|
switch (type)
|
|
|
|
{
|
2018-05-21 13:49:54 +00:00
|
|
|
case ReplicatedMergeTreeLogEntryData::GET_PART: return "GET_PART";
|
2021-02-14 22:59:13 +00:00
|
|
|
case ReplicatedMergeTreeLogEntryData::ATTACH_PART: return "ATTACH_PART";
|
2018-05-21 13:49:54 +00:00
|
|
|
case ReplicatedMergeTreeLogEntryData::MERGE_PARTS: return "MERGE_PARTS";
|
|
|
|
case ReplicatedMergeTreeLogEntryData::DROP_RANGE: return "DROP_RANGE";
|
|
|
|
case ReplicatedMergeTreeLogEntryData::CLEAR_COLUMN: return "CLEAR_COLUMN";
|
2019-05-09 14:25:18 +00:00
|
|
|
case ReplicatedMergeTreeLogEntryData::CLEAR_INDEX: return "CLEAR_INDEX";
|
2018-05-21 13:49:54 +00:00
|
|
|
case ReplicatedMergeTreeLogEntryData::REPLACE_RANGE: return "REPLACE_RANGE";
|
2018-05-28 15:37:30 +00:00
|
|
|
case ReplicatedMergeTreeLogEntryData::MUTATE_PART: return "MUTATE_PART";
|
2020-01-28 17:15:22 +00:00
|
|
|
case ReplicatedMergeTreeLogEntryData::ALTER_METADATA: return "ALTER_METADATA";
|
2017-04-01 07:20:54 +00:00
|
|
|
default:
|
|
|
|
throw Exception("Unknown log entry type: " + DB::toString<int>(type), ErrorCodes::LOGICAL_ERROR);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2018-05-21 13:49:54 +00:00
|
|
|
String typeToString() const
|
|
|
|
{
|
|
|
|
return typeToString(type);
|
|
|
|
}
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
void writeText(WriteBuffer & out) const;
|
|
|
|
void readText(ReadBuffer & in);
|
|
|
|
String toString() const;
|
|
|
|
|
|
|
|
String znode_name;
|
|
|
|
|
|
|
|
Type type = EMPTY;
|
2017-04-16 15:00:33 +00:00
|
|
|
String source_replica; /// Empty string means that this entry was added to the queue immediately, and not copied from the log.
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2021-02-14 22:59:13 +00:00
|
|
|
String part_checksum; /// Part checksum for ATTACH_PART, empty otherwise.
|
|
|
|
|
2018-05-21 13:49:54 +00:00
|
|
|
/// The name of resulting part for GET_PART and MERGE_PARTS
|
|
|
|
/// Part range for DROP_RANGE and CLEAR_COLUMN
|
2017-04-01 07:20:54 +00:00
|
|
|
String new_part_name;
|
2020-02-11 13:41:26 +00:00
|
|
|
MergeTreeDataPartType new_part_type;
|
2017-05-12 13:47:42 +00:00
|
|
|
String block_id; /// For parts of level zero, the block identifier for deduplication (node name in /blocks/).
|
|
|
|
mutable String actual_new_part_name; /// GET_PART could actually fetch a part covering 'new_part_name'.
|
2020-11-02 14:38:18 +00:00
|
|
|
UUID new_part_uuid = UUIDHelpers::Nil;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-04-20 16:18:16 +00:00
|
|
|
Strings source_parts;
|
2017-04-14 20:49:03 +00:00
|
|
|
bool deduplicate = false; /// Do deduplicate on merge
|
2020-12-01 09:10:12 +00:00
|
|
|
Strings deduplicate_by_columns = {}; // Which columns should be checked for duplicates, empty means 'all' (default).
|
2020-09-03 13:00:13 +00:00
|
|
|
MergeType merge_type = MergeType::REGULAR;
|
2017-06-16 16:47:09 +00:00
|
|
|
String column_name;
|
2019-05-09 14:25:18 +00:00
|
|
|
String index_name;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2017-04-16 15:00:33 +00:00
|
|
|
/// For DROP_RANGE, true means that the parts need not be deleted, but moved to the `detached` directory.
|
2017-04-01 07:20:54 +00:00
|
|
|
bool detach = false;
|
|
|
|
|
2018-05-21 13:49:54 +00:00
|
|
|
/// REPLACE PARTITION FROM command
|
|
|
|
struct ReplaceRangeEntry
|
|
|
|
{
|
|
|
|
String drop_range_part_name;
|
|
|
|
|
|
|
|
String from_database;
|
|
|
|
String from_table;
|
|
|
|
Strings src_part_names; // as in from_table
|
|
|
|
Strings new_part_names;
|
|
|
|
Strings part_names_checksums;
|
|
|
|
int columns_version;
|
|
|
|
|
|
|
|
void writeText(WriteBuffer & out) const;
|
|
|
|
void readText(ReadBuffer & in);
|
|
|
|
};
|
|
|
|
|
|
|
|
std::shared_ptr<ReplaceRangeEntry> replace_range_entry;
|
|
|
|
|
2020-02-13 16:16:09 +00:00
|
|
|
/// ALTER METADATA and MUTATE PART command
|
2020-01-28 17:15:22 +00:00
|
|
|
|
2020-02-13 16:16:09 +00:00
|
|
|
/// Version of metadata which will be set after this alter
|
|
|
|
/// Also present in MUTATE_PART command, to track mutations
|
|
|
|
/// required for complete alter execution.
|
2020-12-07 14:38:25 +00:00
|
|
|
int alter_version = -1; /// May be equal to -1, if it's normal mutation, not metadata update.
|
2020-02-13 16:16:09 +00:00
|
|
|
|
|
|
|
/// only ALTER METADATA command
|
2020-12-07 14:38:25 +00:00
|
|
|
bool have_mutation = false; /// If this alter requires additional mutation step, for data update
|
2020-02-13 16:16:09 +00:00
|
|
|
|
|
|
|
String columns_str; /// New columns data corresponding to alter_version
|
|
|
|
String metadata_str; /// New metadata corresponding to alter_version
|
2020-01-30 12:54:52 +00:00
|
|
|
|
2018-06-18 12:17:46 +00:00
|
|
|
/// Returns a set of parts that will appear after executing the entry + parts to block
|
|
|
|
/// selection of merges. These parts are added to queue.virtual_parts.
|
|
|
|
Strings getVirtualPartNames() const
|
2018-05-21 13:49:54 +00:00
|
|
|
{
|
2020-01-15 13:00:08 +00:00
|
|
|
/// Doesn't produce any part
|
2020-01-28 17:15:22 +00:00
|
|
|
if (type == ALTER_METADATA)
|
2020-01-15 13:00:08 +00:00
|
|
|
return {};
|
|
|
|
|
2018-06-18 12:17:46 +00:00
|
|
|
/// DROP_RANGE does not add a real part, but we must disable merges in that range
|
2018-05-23 12:29:36 +00:00
|
|
|
if (type == DROP_RANGE)
|
2018-05-21 13:49:54 +00:00
|
|
|
return {new_part_name};
|
|
|
|
|
2018-06-18 12:17:46 +00:00
|
|
|
/// Return {} because selection of merges in the partition where the column is cleared
|
|
|
|
/// should not be blocked (only execution of merges should be blocked).
|
2019-05-09 14:25:18 +00:00
|
|
|
if (type == CLEAR_COLUMN || type == CLEAR_INDEX)
|
2018-06-18 12:17:46 +00:00
|
|
|
return {};
|
|
|
|
|
2018-05-21 13:49:54 +00:00
|
|
|
if (type == REPLACE_RANGE)
|
|
|
|
{
|
|
|
|
Strings res = replace_range_entry->new_part_names;
|
|
|
|
res.emplace_back(replace_range_entry->drop_range_part_name);
|
|
|
|
return res;
|
|
|
|
}
|
|
|
|
|
|
|
|
return {new_part_name};
|
|
|
|
}
|
|
|
|
|
2018-06-18 12:17:46 +00:00
|
|
|
/// Returns set of parts that denote the block number ranges that should be blocked during the entry execution.
|
|
|
|
/// These parts are added to future_parts.
|
2018-05-23 12:29:36 +00:00
|
|
|
Strings getBlockingPartNames() const
|
|
|
|
{
|
2018-06-18 12:17:46 +00:00
|
|
|
Strings res = getVirtualPartNames();
|
2018-05-23 12:29:36 +00:00
|
|
|
|
|
|
|
if (type == CLEAR_COLUMN)
|
|
|
|
res.emplace_back(new_part_name);
|
|
|
|
|
|
|
|
return res;
|
|
|
|
}
|
|
|
|
|
2017-04-16 15:00:33 +00:00
|
|
|
/// Access under queue_mutex, see ReplicatedMergeTreeQueue.
|
|
|
|
bool currently_executing = false; /// Whether the action is executing now.
|
2020-08-11 20:58:39 +00:00
|
|
|
bool removed_by_other_entry = false;
|
2017-04-16 15:00:33 +00:00
|
|
|
/// These several fields are informational only (for viewing by the user using system tables).
|
|
|
|
/// Access under queue_mutex, see ReplicatedMergeTreeQueue.
|
|
|
|
size_t num_tries = 0; /// The number of attempts to perform the action (since the server started, including the running one).
|
|
|
|
std::exception_ptr exception; /// The last exception, in the case of an unsuccessful attempt to perform the action.
|
|
|
|
time_t last_attempt_time = 0; /// The time at which the last attempt was attempted to complete the action.
|
|
|
|
size_t num_postponed = 0; /// The number of times the action was postponed.
|
|
|
|
String postpone_reason; /// The reason why the action was postponed, if it was postponed.
|
|
|
|
time_t last_postpone_time = 0; /// The time of the last time the action was postponed.
|
|
|
|
|
|
|
|
/// Creation time or the time to copy from the general log to the queue of a particular replica.
|
2017-04-01 07:20:54 +00:00
|
|
|
time_t create_time = 0;
|
|
|
|
|
2017-04-16 15:00:33 +00:00
|
|
|
/// The quorum value (for GET_PART) is a non-zero value when the quorum write is enabled.
|
2017-04-01 07:20:54 +00:00
|
|
|
size_t quorum = 0;
|
2020-02-17 16:33:05 +00:00
|
|
|
|
|
|
|
/// If this MUTATE_PART entry caused by alter(modify/drop) query.
|
|
|
|
bool isAlterMutation() const
|
|
|
|
{
|
|
|
|
return type == MUTATE_PART && alter_version != -1;
|
|
|
|
}
|
2016-06-07 08:23:15 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
|
2018-05-23 14:33:55 +00:00
|
|
|
struct ReplicatedMergeTreeLogEntry : public ReplicatedMergeTreeLogEntryData, std::enable_shared_from_this<ReplicatedMergeTreeLogEntry>
|
2016-06-07 08:23:15 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
using Ptr = std::shared_ptr<ReplicatedMergeTreeLogEntry>;
|
2016-06-07 08:23:15 +00:00
|
|
|
|
2017-04-16 15:00:33 +00:00
|
|
|
std::condition_variable execution_complete; /// Awake when currently_executing becomes false.
|
2016-06-07 08:23:15 +00:00
|
|
|
|
2018-08-25 01:58:14 +00:00
|
|
|
static Ptr parse(const String & s, const Coordination::Stat & stat);
|
2016-06-07 08:23:15 +00:00
|
|
|
};
|
|
|
|
|
2018-05-23 14:33:55 +00:00
|
|
|
using ReplicatedMergeTreeLogEntryPtr = std::shared_ptr<ReplicatedMergeTreeLogEntry>;
|
|
|
|
|
2016-06-07 08:23:15 +00:00
|
|
|
|
|
|
|
}
|