StorageReplicatedMergeTree.cpp 83.0 KB
Newer Older
M
Merge  
Michael Kolupaev 已提交
1
#include <DB/Storages/StorageReplicatedMergeTree.h>
M
Merge  
Michael Kolupaev 已提交
2
#include <DB/Storages/MergeTree/ReplicatedMergeTreeBlockOutputStream.h>
M
Merge  
Michael Kolupaev 已提交
3
#include <DB/Storages/MergeTree/ReplicatedMergeTreePartsExchange.h>
M
Merge  
Michael Kolupaev 已提交
4
#include <DB/Storages/MergeTree/MergeTreePartChecker.h>
M
Merge  
Michael Kolupaev 已提交
5 6 7
#include <DB/Parsers/formatAST.h>
#include <DB/IO/WriteBufferFromOStream.h>
#include <DB/IO/ReadBufferFromString.h>
M
Merge  
Michael Kolupaev 已提交
8
#include <DB/Interpreters/InterpreterAlterQuery.h>
M
Merge  
Michael Kolupaev 已提交
9
#include <DB/Common/VirtualColumnUtils.h>
M
Merge  
Michael Kolupaev 已提交
10
#include <time.h>
M
Merge  
Michael Kolupaev 已提交
11 12 13 14

namespace DB
{

M
Merge  
Michael Kolupaev 已提交
15

M
Merge  
Michael Kolupaev 已提交
16
const auto ERROR_SLEEP_MS = 1000;
M
Merge  
Michael Kolupaev 已提交
17
const auto MERGE_SELECTING_SLEEP_MS = 5 * 1000;
M
Merge  
Michael Kolupaev 已提交
18
const auto CLEANUP_SLEEP_MS = 30 * 1000;
M
Merge  
Michael Kolupaev 已提交
19

M
Merge  
Michael Kolupaev 已提交
20 21 22 23 24 25 26 27 28
/// Преобразовать число в строку формате суффиксов автоинкрементных нод в ZooKeeper.
static String padIndex(UInt64 index)
{
	String index_str = toString(index);
	while (index_str.size() < 10)
		index_str = '0' + index_str;
	return index_str;
}

M
Merge  
Michael Kolupaev 已提交
29

M
Merge  
Michael Kolupaev 已提交
30 31 32
StorageReplicatedMergeTree::StorageReplicatedMergeTree(
	const String & zookeeper_path_,
	const String & replica_name_,
M
Merge  
Michael Kolupaev 已提交
33
	bool attach,
M
Merge  
Michael Kolupaev 已提交
34 35
	const String & path_, const String & database_name_, const String & name_,
	NamesAndTypesListPtr columns_,
M
Merge  
Michael Kolupaev 已提交
36
	Context & context_,
M
Merge  
Michael Kolupaev 已提交
37 38 39 40 41 42 43 44
	ASTPtr & primary_expr_ast_,
	const String & date_column_name_,
	const ASTPtr & sampling_expression_,
	size_t index_granularity_,
	MergeTreeData::Mode mode_,
	const String & sign_column_,
	const MergeTreeSettings & settings_)
	:
45
	context(context_), zookeeper(context.getZooKeeper()), database_name(database_name_),
M
Merge  
Michael Kolupaev 已提交
46
	table_name(name_), full_path(path_ + escapeForFileName(table_name) + '/'), zookeeper_path(zookeeper_path_),
M
Merge  
Michael Kolupaev 已提交
47
	replica_name(replica_name_),
M
Merge  
Michael Kolupaev 已提交
48
	data(	full_path, columns_, context_, primary_expr_ast_, date_column_name_, sampling_expression_,
M
Merge  
Michael Kolupaev 已提交
49 50
			index_granularity_, mode_, sign_column_, settings_, database_name_ + "." + table_name, true,
			std::bind(&StorageReplicatedMergeTree::enqueuePartForCheck, this, std::placeholders::_1)),
M
Merge  
Michael Kolupaev 已提交
51
	reader(data), writer(data), merger(data), fetcher(data),
M
Merge  
Michael Kolupaev 已提交
52 53
	log(&Logger::get(database_name_ + "." + table_name + " (StorageReplicatedMergeTree)")),
	shutdown_event(false), permanent_shutdown_event(false)
M
Merge  
Michael Kolupaev 已提交
54
{
M
Merge  
Michael Kolupaev 已提交
55 56
	if (!zookeeper)
	{
M
Merge  
Michael Kolupaev 已提交
57 58 59
		if (!attach)
			throw Exception("Can't create replicated table without ZooKeeper", ErrorCodes::NO_ZOOKEEPER);

M
Merge  
Michael Kolupaev 已提交
60 61 62 63
		goReadOnly();
		return;
	}

M
Merge  
Michael Kolupaev 已提交
64 65 66
	if (!zookeeper_path.empty() && *zookeeper_path.rbegin() == '/')
		zookeeper_path.erase(zookeeper_path.end() - 1);
	replica_path = zookeeper_path + "/replicas/" + replica_name;
67

M
Merge  
Michael Kolupaev 已提交
68 69
	if (!attach)
	{
M
Merge  
Michael Kolupaev 已提交
70
		if (!zookeeper->exists(zookeeper_path))
M
Merge  
Michael Kolupaev 已提交
71 72
			createTable();

M
Merge  
Michael Kolupaev 已提交
73
		checkTableStructure(false);
M
Merge  
Michael Kolupaev 已提交
74
		createReplica();
M
Merge  
Michael Kolupaev 已提交
75 76 77
	}
	else
	{
M
Merge  
Michael Kolupaev 已提交
78 79 80 81 82 83
		bool skip_sanity_checks = false;
		if (zookeeper->exists(replica_path + "/flags/force_restore_data"))
		{
			skip_sanity_checks = true;
			zookeeper->remove(replica_path + "/flags/force_restore_data");

M
Merge  
Michael Kolupaev 已提交
84 85
			LOG_WARNING(log, "Skipping the limits on severity of changes to data parts and columns (flag "
				<< replica_path << "/flags/force_restore_data).");
M
Merge  
Michael Kolupaev 已提交
86 87 88 89
		}

		checkTableStructure(skip_sanity_checks);
		checkParts(skip_sanity_checks);
M
Merge  
Michael Kolupaev 已提交
90
	}
M
Merge  
Michael Kolupaev 已提交
91

M
Merge  
Michael Kolupaev 已提交
92
	initVirtualParts();
M
Merge  
Michael Kolupaev 已提交
93
	loadQueue();
M
Merge  
Michael Kolupaev 已提交
94 95 96 97 98 99

	String unreplicated_path = full_path + "unreplicated/";
	if (Poco::File(unreplicated_path).exists())
	{
		LOG_INFO(log, "Have unreplicated data");
		unreplicated_data.reset(new MergeTreeData(unreplicated_path, columns_, context_, primary_expr_ast_,
M
Merge  
Michael Kolupaev 已提交
100
			date_column_name_, sampling_expression_, index_granularity_, mode_, sign_column_, settings_,
M
Merge  
Michael Kolupaev 已提交
101
			database_name_ + "." + table_name + "[unreplicated]", false));
M
Merge  
Michael Kolupaev 已提交
102
		unreplicated_reader.reset(new MergeTreeDataSelectExecutor(*unreplicated_data));
M
Merge  
Michael Kolupaev 已提交
103
		unreplicated_merger.reset(new MergeTreeDataMerger(*unreplicated_data));
M
Merge  
Michael Kolupaev 已提交
104
	}
M
Merge  
Michael Kolupaev 已提交
105

M
Merge  
Michael Kolupaev 已提交
106 107 108 109 110 111
	/// Сгенерируем этому экземпляру случайный идентификатор.
	struct timespec times;
	if (clock_gettime(CLOCK_THREAD_CPUTIME_ID, &times))
		throwFromErrno("Cannot clock_gettime.", ErrorCodes::CANNOT_CLOCK_GETTIME);
	active_node_identifier = toString(times.tv_nsec);

M
Merge  
Michael Kolupaev 已提交
112
	restarting_thread = std::thread(&StorageReplicatedMergeTree::restartingThread, this);
M
Merge  
Michael Kolupaev 已提交
113 114 115 116 117
}

StoragePtr StorageReplicatedMergeTree::create(
	const String & zookeeper_path_,
	const String & replica_name_,
M
Merge  
Michael Kolupaev 已提交
118
	bool attach,
M
Merge  
Michael Kolupaev 已提交
119 120
	const String & path_, const String & database_name_, const String & name_,
	NamesAndTypesListPtr columns_,
M
Merge  
Michael Kolupaev 已提交
121
	Context & context_,
M
Merge  
Michael Kolupaev 已提交
122 123 124 125 126 127 128 129
	ASTPtr & primary_expr_ast_,
	const String & date_column_name_,
	const ASTPtr & sampling_expression_,
	size_t index_granularity_,
	MergeTreeData::Mode mode_,
	const String & sign_column_,
	const MergeTreeSettings & settings_)
{
M
Merge  
Michael Kolupaev 已提交
130
	StorageReplicatedMergeTree * res = new StorageReplicatedMergeTree(zookeeper_path_, replica_name_, attach,
M
Merge  
Michael Kolupaev 已提交
131
		path_, database_name_, name_, columns_, context_, primary_expr_ast_, date_column_name_, sampling_expression_,
M
Merge  
Michael Kolupaev 已提交
132 133
		index_granularity_, mode_, sign_column_, settings_);
	StoragePtr res_ptr = res->thisPtr();
M
Merge  
Michael Kolupaev 已提交
134 135 136
	if (!res->is_read_only)
	{
		String endpoint_name = "ReplicatedMergeTree:" + res->replica_path;
M
Merge  
Michael Kolupaev 已提交
137
		InterserverIOEndpointPtr endpoint = new ReplicatedMergeTreePartsServer(res->data, *res);
M
Merge  
Michael Kolupaev 已提交
138 139
		res->endpoint_holder = new InterserverIOEndpointHolder(endpoint_name, endpoint, res->context.getInterserverIOHandler());
	}
M
Merge  
Michael Kolupaev 已提交
140
	return res_ptr;
M
Merge  
Michael Kolupaev 已提交
141 142
}

M
Merge  
Michael Kolupaev 已提交
143 144 145 146 147 148 149 150
static String formattedAST(const ASTPtr & ast)
{
	if (!ast)
		return "";
	std::stringstream ss;
	formatAST(*ast, ss, 0, false, true);
	return ss.str();
}
M
Merge  
Michael Kolupaev 已提交
151

M
Merge  
Michael Kolupaev 已提交
152
void StorageReplicatedMergeTree::createTable()
M
Merge  
Michael Kolupaev 已提交
153
{
M
Merge  
Michael Kolupaev 已提交
154 155
	LOG_DEBUG(log, "Creating table " << zookeeper_path);

M
Merge  
Michael Kolupaev 已提交
156
	zookeeper->create(zookeeper_path, "", zkutil::CreateMode::Persistent);
M
Merge  
Michael Kolupaev 已提交
157

M
Merge  
Michael Kolupaev 已提交
158
	/// Запишем метаданные таблицы, чтобы реплики могли сверять с ними параметры таблицы.
M
Merge  
Michael Kolupaev 已提交
159 160 161 162 163 164 165 166
	std::stringstream metadata;
	metadata << "metadata format version: 1" << std::endl;
	metadata << "date column: " << data.date_column_name << std::endl;
	metadata << "sampling expression: " << formattedAST(data.sampling_expression) << std::endl;
	metadata << "index granularity: " << data.index_granularity << std::endl;
	metadata << "mode: " << static_cast<int>(data.mode) << std::endl;
	metadata << "sign column: " << data.sign_column << std::endl;
	metadata << "primary key: " << formattedAST(data.primary_expr_ast) << std::endl;
M
Merge  
Michael Kolupaev 已提交
167

M
Merge  
Michael Kolupaev 已提交
168
	zookeeper->create(zookeeper_path + "/metadata", metadata.str(), zkutil::CreateMode::Persistent);
M
Merge  
Michael Kolupaev 已提交
169
	zookeeper->create(zookeeper_path + "/columns", data.getColumnsList().toString(), zkutil::CreateMode::Persistent);
M
Merge  
Michael Kolupaev 已提交
170

M
Merge  
Michael Kolupaev 已提交
171
	zookeeper->create(zookeeper_path + "/log", "", zkutil::CreateMode::Persistent);
M
Merge  
Michael Kolupaev 已提交
172 173
	zookeeper->create(zookeeper_path + "/blocks", "", zkutil::CreateMode::Persistent);
	zookeeper->create(zookeeper_path + "/block_numbers", "", zkutil::CreateMode::Persistent);
M
Merge  
Michael Kolupaev 已提交
174
	zookeeper->create(zookeeper_path + "/nonincrement_block_numbers", "", zkutil::CreateMode::Persistent);
M
Merge  
Michael Kolupaev 已提交
175 176
	zookeeper->create(zookeeper_path + "/leader_election", "", zkutil::CreateMode::Persistent);
	zookeeper->create(zookeeper_path + "/temp", "", zkutil::CreateMode::Persistent);
M
Merge  
Michael Kolupaev 已提交
177
	/// Создадим replicas в последнюю очередь, чтобы нельзя было добавить реплику, пока все остальные ноды не созданы.
M
Merge  
Michael Kolupaev 已提交
178
	zookeeper->create(zookeeper_path + "/replicas", "", zkutil::CreateMode::Persistent);
M
Merge  
Michael Kolupaev 已提交
179
}
M
Merge  
Michael Kolupaev 已提交
180

M
Merge  
Michael Kolupaev 已提交
181 182
/** Проверить, что список столбцов и настройки таблицы совпадают с указанными в ZK (/metadata).
	* Если нет - бросить исключение.
M
Merge  
Michael Kolupaev 已提交
183
	*/
M
Merge  
Michael Kolupaev 已提交
184
void StorageReplicatedMergeTree::checkTableStructure(bool skip_sanity_checks)
M
Merge  
Michael Kolupaev 已提交
185
{
M
Merge  
Michael Kolupaev 已提交
186
	String metadata_str = zookeeper->get(zookeeper_path + "/metadata");
M
Merge  
Michael Kolupaev 已提交
187 188 189 190 191 192 193 194 195 196 197 198 199
	ReadBufferFromString buf(metadata_str);
	assertString("metadata format version: 1", buf);
	assertString("\ndate column: ", buf);
	assertString(data.date_column_name, buf);
	assertString("\nsampling expression: ", buf);
	assertString(formattedAST(data.sampling_expression), buf);
	assertString("\nindex granularity: ", buf);
	assertString(toString(data.index_granularity), buf);
	assertString("\nmode: ", buf);
	assertString(toString(static_cast<int>(data.mode)), buf);
	assertString("\nsign column: ", buf);
	assertString(data.sign_column, buf);
	assertString("\nprimary key: ", buf);
M
Merge  
Michael Kolupaev 已提交
200 201
	/// NOTE: Можно сделать менее строгую проверку совпадения выражений, чтобы таблицы не ломались от небольших изменений
	///       в коде formatAST.
M
Merge  
Michael Kolupaev 已提交
202
	assertString(formattedAST(data.primary_expr_ast), buf);
M
Merge  
Michael Kolupaev 已提交
203
	assertString("\n", buf);
M
Merge  
Michael Kolupaev 已提交
204
	assertEOF(buf);
M
Merge  
Michael Kolupaev 已提交
205 206 207 208

	zkutil::Stat stat;
	auto columns = NamesAndTypesList::parse(zookeeper->get(zookeeper_path + "/columns", &stat), context.getDataTypeFactory());
	columns_version = stat.version;
M
Merge  
Michael Kolupaev 已提交
209
	if (columns != data.getColumnsList())
M
Merge  
Michael Kolupaev 已提交
210
	{
M
Merge  
Michael Kolupaev 已提交
211
		if (data.getColumnsList().sizeOfDifference(columns) <= 2 || skip_sanity_checks)
M
Merge  
Michael Kolupaev 已提交
212
		{
M
Merge  
Michael Kolupaev 已提交
213
			LOG_WARNING(log, "Table structure in ZooKeeper is a little different from local table structure. Assuming ALTER.");
214 215 216 217

			/// Без всяких блокировок, потому что таблица еще не создана.
			InterpreterAlterQuery::updateMetadata(database_name, table_name, columns, context);
			data.setColumnsList(columns);
M
Merge  
Michael Kolupaev 已提交
218
		}
M
Merge  
Michael Kolupaev 已提交
219
		else
M
Merge  
Michael Kolupaev 已提交
220
		{
M
Merge  
Michael Kolupaev 已提交
221 222
			throw Exception("Table structure in ZooKeeper is very different from local table structure.",
							ErrorCodes::INCOMPATIBLE_COLUMNS);
M
Merge  
Michael Kolupaev 已提交
223
		}
M
Merge  
Michael Kolupaev 已提交
224 225
	}
}
M
Merge  
Michael Kolupaev 已提交
226

M
Merge  
Michael Kolupaev 已提交
227 228
void StorageReplicatedMergeTree::createReplica()
{
M
Merge  
Michael Kolupaev 已提交
229
	LOG_DEBUG(log, "Creating replica " << replica_path);
M
Merge  
Michael Kolupaev 已提交
230

M
Merge  
Michael Kolupaev 已提交
231 232 233 234 235 236 237
	/** Запомним список других реплик.
	  * NOTE: Здесь есть race condition. Если почти одновременно добавить нескольких реплик, сразу же начиная в них писать,
	  *       небольшая часть данных может не реплицироваться.
	  */
	Strings replicas = zookeeper->getChildren(zookeeper_path + "/replicas");

	/// Создадим пустую реплику.
M
Merge  
Michael Kolupaev 已提交
238
	zookeeper->create(replica_path, "", zkutil::CreateMode::Persistent);
M
Merge  
Michael Kolupaev 已提交
239
	zookeeper->create(replica_path + "/columns", data.getColumnsList().toString(), zkutil::CreateMode::Persistent);
M
Merge  
Michael Kolupaev 已提交
240
	zookeeper->create(replica_path + "/host", "", zkutil::CreateMode::Persistent);
M
Merge  
Michael Kolupaev 已提交
241
	zookeeper->create(replica_path + "/log_pointer", "", zkutil::CreateMode::Persistent);
M
Merge  
Michael Kolupaev 已提交
242 243
	zookeeper->create(replica_path + "/queue", "", zkutil::CreateMode::Persistent);
	zookeeper->create(replica_path + "/parts", "", zkutil::CreateMode::Persistent);
M
Merge  
Michael Kolupaev 已提交
244
	zookeeper->create(replica_path + "/flags", "", zkutil::CreateMode::Persistent);
M
Merge  
Michael Kolupaev 已提交
245

M
Merge  
Michael Kolupaev 已提交
246 247 248 249 250
	/** Нужно изменить данные ноды /replicas на что угодно, чтобы поток, удаляющий старые записи в логе,
	  *  споткнулся об это изменение и не удалил записи, которые мы еще не прочитали.
	  */
	zookeeper->set(zookeeper_path + "/replicas", "last added replica: " + replica_name);

M
Merge  
Michael Kolupaev 已提交
251
	if (replicas.empty())
M
Merge  
Michael Kolupaev 已提交
252 253
	{
		LOG_DEBUG(log, "No other replicas");
M
Merge  
Michael Kolupaev 已提交
254
		return;
M
Merge  
Michael Kolupaev 已提交
255
	}
M
Merge  
Michael Kolupaev 已提交
256

M
Merge  
Michael Kolupaev 已提交
257 258
	/// "Эталонная" реплика, у которой мы возьмем информацию о множестве кусков, очередь и указатель на лог.
	String source_replica = replicas[rand() % replicas.size()];
M
Merge  
Michael Kolupaev 已提交
259

M
Merge  
Michael Kolupaev 已提交
260 261
	LOG_INFO(log, "Will mimic " << source_replica);

M
Merge  
Michael Kolupaev 已提交
262 263 264 265
	String source_path = zookeeper_path + "/replicas/" + source_replica;

	/// Порядок следующих трех действий важен. Записи в логе могут продублироваться, но не могут потеряться.

M
Merge  
Michael Kolupaev 已提交
266 267
	/// Скопируем у эталонной реплики ссылку на лог.
	zookeeper->set(replica_path + "/log_pointer", zookeeper->get(source_path + "/log_pointer"));
M
Merge  
Michael Kolupaev 已提交
268 269 270 271 272 273 274 275 276 277 278 279 280 281 282 283 284 285 286 287 288 289 290 291 292 293 294 295 296 297

	/// Запомним очередь эталонной реплики.
	Strings source_queue_names = zookeeper->getChildren(source_path + "/queue");
	std::sort(source_queue_names.begin(), source_queue_names.end());
	Strings source_queue;
	for (const String & entry_name : source_queue_names)
	{
		String entry;
		if (!zookeeper->tryGet(source_path + "/queue/" + entry_name, entry))
			continue;
		source_queue.push_back(entry);
	}

	/// Добавим в очередь задания на получение всех активных кусков, которые есть у эталонной реплики.
	Strings parts = zookeeper->getChildren(source_path + "/parts");
	ActiveDataPartSet active_parts_set;
	for (const String & part : parts)
	{
		active_parts_set.add(part);
	}
	Strings active_parts = active_parts_set.getParts();
	for (const String & name : active_parts)
	{
		LogEntry log_entry;
		log_entry.type = LogEntry::GET_PART;
		log_entry.source_replica = "";
		log_entry.new_part_name = name;

		zookeeper->create(replica_path + "/queue/queue-", log_entry.toString(), zkutil::CreateMode::PersistentSequential);
	}
M
Merge  
Michael Kolupaev 已提交
298
	LOG_DEBUG(log, "Queued " << active_parts.size() << " parts to be fetched");
M
Merge  
Michael Kolupaev 已提交
299 300 301 302 303 304

	/// Добавим в очередь содержимое очереди эталонной реплики.
	for (const String & entry : source_queue)
	{
		zookeeper->create(replica_path + "/queue/queue-", entry, zkutil::CreateMode::PersistentSequential);
	}
M
Merge  
Michael Kolupaev 已提交
305
	LOG_DEBUG(log, "Copied " << source_queue.size() << " queue entries");
M
Merge  
Michael Kolupaev 已提交
306
}
M
Merge  
Michael Kolupaev 已提交
307

M
Merge  
Michael Kolupaev 已提交
308 309 310 311 312
void StorageReplicatedMergeTree::activateReplica()
{
	std::stringstream host;
	host << "host: " << context.getInterserverIOHost() << std::endl;
	host << "port: " << context.getInterserverIOPort() << std::endl;
M
Merge  
Michael Kolupaev 已提交
313

M
Merge  
Michael Kolupaev 已提交
314 315 316 317 318 319 320 321 322
	/** Если нода отмечена как активная, но отметка сделана в этом же экземпляре, удалим ее.
	  * Такое возможно только при истечении сессии в ZooKeeper.
	  * Здесь есть небольшой race condition (можем удалить не ту ноду, для которой сделали tryGet),
	  *  но он крайне маловероятен при нормальном использовании.
	  */
	String data;
	if (zookeeper->tryGet(replica_path + "/is_active", data) && data == active_node_identifier)
		zookeeper->tryRemove(replica_path + "/is_active");

M
Merge  
Michael Kolupaev 已提交
323
	/// Одновременно объявим, что эта реплика активна, и обновим хост.
M
Merge  
Michael Kolupaev 已提交
324
	zkutil::Ops ops;
M
Merge  
Michael Kolupaev 已提交
325
	ops.push_back(new zkutil::Op::Create(replica_path + "/is_active", "", zookeeper->getDefaultACL(), zkutil::CreateMode::Ephemeral));
M
Merge  
Michael Kolupaev 已提交
326
	ops.push_back(new zkutil::Op::SetData(replica_path + "/host", host.str(), -1));
M
Merge  
Michael Kolupaev 已提交
327 328 329

	try
	{
M
Merge  
Michael Kolupaev 已提交
330
		zookeeper->multi(ops);
M
Merge  
Michael Kolupaev 已提交
331 332 333
	}
	catch (zkutil::KeeperException & e)
	{
334
		if (e.code == ZNODEEXISTS)
M
Merge  
Michael Kolupaev 已提交
335 336 337 338 339
			throw Exception("Replica " + replica_path + " appears to be already active. If you're sure it's not, "
				"try again in a minute or remove znode " + replica_path + "/is_active manually", ErrorCodes::REPLICA_IS_ALREADY_ACTIVE);

		throw;
	}
M
Merge  
Michael Kolupaev 已提交
340

M
Merge  
Michael Kolupaev 已提交
341
	replica_is_active_node = zkutil::EphemeralNodeHolder::existing(replica_path + "/is_active", *zookeeper);
M
Merge  
Michael Kolupaev 已提交
342
}
M
Merge  
Michael Kolupaev 已提交
343

M
Merge  
Michael Kolupaev 已提交
344
void StorageReplicatedMergeTree::checkParts(bool skip_sanity_checks)
M
Merge  
Michael Kolupaev 已提交
345
{
M
Merge  
Michael Kolupaev 已提交
346
	Strings expected_parts_vec = zookeeper->getChildren(replica_path + "/parts");
M
Merge  
Michael Kolupaev 已提交
347 348

	/// Куски в ZK.
M
Merge  
Michael Kolupaev 已提交
349 350
	NameSet expected_parts(expected_parts_vec.begin(), expected_parts_vec.end());

M
Merge  
Michael Kolupaev 已提交
351
	MergeTreeData::DataParts parts = data.getAllDataParts();
M
Merge  
Michael Kolupaev 已提交
352

M
Merge  
Michael Kolupaev 已提交
353
	/// Локальные куски, которых нет в ZK.
M
Merge  
Michael Kolupaev 已提交
354
	MergeTreeData::DataParts unexpected_parts;
M
Merge  
Michael Kolupaev 已提交
355

M
Merge  
Michael Kolupaev 已提交
356 357 358 359 360 361 362 363
	for (const auto & part : parts)
	{
		if (expected_parts.count(part->name))
		{
			expected_parts.erase(part->name);
		}
		else
		{
M
Merge  
Michael Kolupaev 已提交
364
			unexpected_parts.insert(part);
M
Merge  
Michael Kolupaev 已提交
365 366 367
		}
	}

M
Merge  
Michael Kolupaev 已提交
368
	/// Какие локальные куски добавить в ZK.
M
Merge  
Michael Kolupaev 已提交
369
	MergeTreeData::DataPartsVector parts_to_add;
M
Merge  
Michael Kolupaev 已提交
370 371 372 373

	/// Какие куски нужно забрать с других реплик.
	Strings parts_to_fetch;

M
Merge  
Michael Kolupaev 已提交
374 375
	for (const String & missing_name : expected_parts)
	{
M
Merge  
Michael Kolupaev 已提交
376
		/// Если локально не хватает какого-то куска, но есть покрывающий его кусок, можно заменить в ZK недостающий покрывающим.
M
Merge  
Michael Kolupaev 已提交
377
		auto containing = data.getActiveContainingPart(missing_name);
M
Merge  
Michael Kolupaev 已提交
378 379 380 381 382 383 384 385 386 387
		if (containing)
		{
			LOG_ERROR(log, "Ignoring missing local part " << missing_name << " because part " << containing->name << " exists");
			if (unexpected_parts.count(containing))
			{
				parts_to_add.push_back(containing);
				unexpected_parts.erase(containing);
			}
		}
		else
M
Merge  
Michael Kolupaev 已提交
388
		{
M
Merge  
Michael Kolupaev 已提交
389
			parts_to_fetch.push_back(missing_name);
M
Merge  
Michael Kolupaev 已提交
390 391
		}
	}
M
Merge  
Michael Kolupaev 已提交
392

M
Merge  
Michael Kolupaev 已提交
393 394 395 396 397 398
	for (const String & name : parts_to_fetch)
		expected_parts.erase(name);

	String sanity_report =
		"There are " + toString(unexpected_parts.size()) + " unexpected parts, "
					 + toString(parts_to_add.size()) + " unexpectedly merged parts, "
M
Merge  
Michael Kolupaev 已提交
399
					 + toString(expected_parts.size()) + " missing obsolete parts, "
M
Merge  
Michael Kolupaev 已提交
400 401 402
					 + toString(parts_to_fetch.size()) + " missing parts";
	bool insane =
		parts_to_add.size() > 2 ||
M
Merge  
Michael Kolupaev 已提交
403
		unexpected_parts.size() > 2 ||
M
Merge  
Michael Kolupaev 已提交
404 405 406
		expected_parts.size() > 20 ||
		parts_to_fetch.size() > 2;

M
Merge  
Michael Kolupaev 已提交
407
	if (insane && !skip_sanity_checks)
M
Merge  
Michael Kolupaev 已提交
408 409 410
	{
		throw Exception("The local set of parts of table " + getTableName() + " doesn't look like the set of parts in ZooKeeper. "
			+ sanity_report,
M
Merge  
Michael Kolupaev 已提交
411
			ErrorCodes::TOO_MANY_UNEXPECTED_DATA_PARTS);
M
Merge  
Michael Kolupaev 已提交
412 413
	}

M
Merge  
Michael Kolupaev 已提交
414 415 416 417 418
	if (insane)
	{
		LOG_WARNING(log, sanity_report);
	}

M
Merge  
Michael Kolupaev 已提交
419 420 421
	/// Добавим в ZK информацию о кусках, покрывающих недостающие куски.
	for (MergeTreeData::DataPartPtr part : parts_to_add)
	{
M
Merge  
Michael Kolupaev 已提交
422
		LOG_ERROR(log, "Adding unexpected local part to ZooKeeper: " << part->name);
M
Merge  
Michael Kolupaev 已提交
423

M
Merge  
Michael Kolupaev 已提交
424
		zkutil::Ops ops;
M
Merge  
Michael Kolupaev 已提交
425
		checkPartAndAddToZooKeeper(part, ops);
M
Merge  
Michael Kolupaev 已提交
426
		zookeeper->multi(ops);
M
Merge  
Michael Kolupaev 已提交
427
	}
M
Merge  
Michael Kolupaev 已提交
428

M
Merge  
Michael Kolupaev 已提交
429 430 431
	/// Удалим из ZK информацию о кусках, покрытых только что добавленными.
	for (const String & name : expected_parts)
	{
M
Merge  
Michael Kolupaev 已提交
432 433 434
		LOG_ERROR(log, "Removing unexpectedly merged local part from ZooKeeper: " << name);

		zkutil::Ops ops;
M
Merge  
Michael Kolupaev 已提交
435
		ops.push_back(new zkutil::Op::Remove(replica_path + "/parts/" + name + "/columns", -1));
M
Merge  
Michael Kolupaev 已提交
436 437 438 439 440 441 442 443 444 445 446 447
		ops.push_back(new zkutil::Op::Remove(replica_path + "/parts/" + name + "/checksums", -1));
		ops.push_back(new zkutil::Op::Remove(replica_path + "/parts/" + name, -1));
		zookeeper->multi(ops);
	}

	/// Добавим в очередь задание забрать недостающие куски с других реплик и уберем из ZK информацию, что они у нас есть.
	for (const String & name : parts_to_fetch)
	{
		LOG_ERROR(log, "Removing missing part from ZooKeeper and queueing a fetch: " << name);

		LogEntry log_entry;
		log_entry.type = LogEntry::GET_PART;
M
Merge  
Michael Kolupaev 已提交
448
		log_entry.source_replica = "";
M
Merge  
Michael Kolupaev 已提交
449 450 451
		log_entry.new_part_name = name;

		/// Полагаемся, что это происходит до загрузки очереди (loadQueue).
M
Merge  
Michael Kolupaev 已提交
452
		zkutil::Ops ops;
M
Merge  
Michael Kolupaev 已提交
453
		ops.push_back(new zkutil::Op::Remove(replica_path + "/parts/" + name + "/columns", -1));
M
Merge  
Michael Kolupaev 已提交
454 455
		ops.push_back(new zkutil::Op::Remove(replica_path + "/parts/" + name + "/checksums", -1));
		ops.push_back(new zkutil::Op::Remove(replica_path + "/parts/" + name, -1));
M
Merge  
Michael Kolupaev 已提交
456 457
		ops.push_back(new zkutil::Op::Create(
			replica_path + "/queue/queue-", log_entry.toString(), zookeeper->getDefaultACL(), zkutil::CreateMode::PersistentSequential));
M
Merge  
Michael Kolupaev 已提交
458
		zookeeper->multi(ops);
M
Merge  
Michael Kolupaev 已提交
459 460 461
	}

	/// Удалим лишние локальные куски.
M
Merge  
Michael Kolupaev 已提交
462 463
	for (MergeTreeData::DataPartPtr part : unexpected_parts)
	{
M
Merge  
Michael Kolupaev 已提交
464
		LOG_ERROR(log, "Renaming unexpected part " << part->name << " to ignored_" + part->name);
M
Merge  
Michael Kolupaev 已提交
465
		data.renameAndDetachPart(part, "ignored_", true);
M
Merge  
Michael Kolupaev 已提交
466 467
	}
}
M
Merge  
Michael Kolupaev 已提交
468

M
Merge  
Michael Kolupaev 已提交
469 470 471 472 473 474 475 476 477
void StorageReplicatedMergeTree::initVirtualParts()
{
	auto parts = data.getDataParts();
	for (const auto & part : parts)
	{
		virtual_parts.add(part->name);
	}
}

M
Merge  
Michael Kolupaev 已提交
478
void StorageReplicatedMergeTree::checkPartAndAddToZooKeeper(MergeTreeData::DataPartPtr part, zkutil::Ops & ops)
M
Merge  
Michael Kolupaev 已提交
479
{
480
	check(part->columns);
M
Merge  
Michael Kolupaev 已提交
481
	int expected_columns_version = columns_version;
482

M
Merge  
Michael Kolupaev 已提交
483 484 485 486 487
	Strings replicas = zookeeper->getChildren(zookeeper_path + "/replicas");
	std::random_shuffle(replicas.begin(), replicas.end());
	String expected_columns_str = part->columns.toString();

	for (const String & replica : replicas)
M
Merge  
Michael Kolupaev 已提交
488
	{
M
Merge  
Michael Kolupaev 已提交
489 490 491 492 493 494 495 496 497 498
		zkutil::Stat stat_before, stat_after;
		String columns_str;
		if (!zookeeper->tryGet(zookeeper_path + "/replicas/" + replica + "/parts/" + part->name + "/columns", columns_str, &stat_before))
			continue;
		if (columns_str != expected_columns_str)
		{
			LOG_INFO(log, "Not checking checksums of part " << part->name << " with replica " << replica
				<< " because columns are different");
			continue;
		}
M
Merge  
Michael Kolupaev 已提交
499
		String checksums_str;
M
Merge  
Michael Kolupaev 已提交
500 501 502 503 504
		/// Проверим, что версия ноды со столбцами не изменилась, пока мы читали checksums.
		/// Это гарантирует, что столбцы и чексуммы относятся к одним и тем же данным.
		if (!zookeeper->tryGet(zookeeper_path + "/replicas/" + replica + "/parts/" + part->name + "/checksums", checksums_str) ||
			!zookeeper->exists(zookeeper_path + "/replicas/" + replica + "/parts/" + part->name + "/columns", &stat_after) ||
			stat_before.version != stat_after.version)
M
Merge  
Michael Kolupaev 已提交
505
		{
M
Merge  
Michael Kolupaev 已提交
506 507 508
			LOG_INFO(log, "Not checking checksums of part " << part->name << " with replica " << replica
				<< " because part changed while we were reading its checksums");
			continue;
M
Merge  
Michael Kolupaev 已提交
509
		}
M
Merge  
Michael Kolupaev 已提交
510 511 512

		auto checksums = MergeTreeData::DataPart::Checksums::parse(checksums_str);
		checksums.checkEqual(part->checksums, true);
M
Merge  
Michael Kolupaev 已提交
513 514
	}

M
Merge  
Michael Kolupaev 已提交
515 516 517 518 519 520
	if (zookeeper->exists(replica_path + "/parts/" + part->name))
	{
		LOG_ERROR(log, "checkPartAndAddToZooKeeper: node " << replica_path + "/parts/" + part->name << " already exists");
		return;
	}

521 522 523
	ops.push_back(new zkutil::Op::Check(
		zookeeper_path + "/columns",
		expected_columns_version));
M
Merge  
Michael Kolupaev 已提交
524 525 526
	ops.push_back(new zkutil::Op::Create(
		replica_path + "/parts/" + part->name,
		"",
M
Merge  
Michael Kolupaev 已提交
527
		zookeeper->getDefaultACL(),
M
Merge  
Michael Kolupaev 已提交
528
		zkutil::CreateMode::Persistent));
M
Merge  
Michael Kolupaev 已提交
529 530 531 532 533
	ops.push_back(new zkutil::Op::Create(
		replica_path + "/parts/" + part->name + "/columns",
		part->columns.toString(),
		zookeeper->getDefaultACL(),
		zkutil::CreateMode::Persistent));
M
Merge  
Michael Kolupaev 已提交
534 535 536
	ops.push_back(new zkutil::Op::Create(
		replica_path + "/parts/" + part->name + "/checksums",
		part->checksums.toString(),
M
Merge  
Michael Kolupaev 已提交
537
		zookeeper->getDefaultACL(),
M
Merge  
Michael Kolupaev 已提交
538 539 540
		zkutil::CreateMode::Persistent));
}

M
Merge  
Michael Kolupaev 已提交
541 542
void StorageReplicatedMergeTree::clearOldParts()
{
M
Merge  
Michael Kolupaev 已提交
543 544
	auto table_lock = lockStructure(false);

M
Merge  
Michael Kolupaev 已提交
545 546
	MergeTreeData::DataPartsVector parts = data.grabOldParts();
	size_t count = parts.size();
M
Merge  
Michael Kolupaev 已提交
547

M
Merge  
Michael Kolupaev 已提交
548 549 550 551
	if (!count)
		return;

	try
M
Merge  
Michael Kolupaev 已提交
552
	{
M
Merge  
Michael Kolupaev 已提交
553 554 555 556
		while (!parts.empty())
		{
			MergeTreeData::DataPartPtr part = parts.back();

M
Merge  
Michael Kolupaev 已提交
557
			LOG_DEBUG(log, "Removing " << part->name);
M
Merge  
Michael Kolupaev 已提交
558

M
Merge  
Michael Kolupaev 已提交
559 560 561 562
			zkutil::Ops ops;
			ops.push_back(new zkutil::Op::Remove(replica_path + "/parts/" + part->name + "/columns", -1));
			ops.push_back(new zkutil::Op::Remove(replica_path + "/parts/" + part->name + "/checksums", -1));
			ops.push_back(new zkutil::Op::Remove(replica_path + "/parts/" + part->name, -1));
M
Merge  
Michael Kolupaev 已提交
563 564 565
			auto code = zookeeper->tryMulti(ops);
			if (code != ZOK)
				LOG_WARNING(log, "Couldn't remove " << part->name << " from ZooKeeper: " << zkutil::ZooKeeper::error2string(code));
M
Merge  
Michael Kolupaev 已提交
566 567 568 569 570 571 572 573 574

			part->remove();
			parts.pop_back();
		}
	}
	catch (...)
	{
		data.addOldParts(parts);
		throw;
M
Merge  
Michael Kolupaev 已提交
575 576
	}

M
Merge  
Michael Kolupaev 已提交
577
	LOG_DEBUG(log, "Removed " << count << " old parts");
M
Merge  
Michael Kolupaev 已提交
578 579
}

M
Merge  
Michael Kolupaev 已提交
580 581
void StorageReplicatedMergeTree::clearOldLogs()
{
M
Merge  
Michael Kolupaev 已提交
582 583 584 585 586 587 588 589 590 591 592
	zkutil::Stat stat;
	if (!zookeeper->exists(zookeeper_path + "/log", &stat))
		throw Exception(zookeeper_path + "/log doesn't exist", ErrorCodes::NOT_FOUND_NODE);

	int children_count = stat.numChildren;

	/// Будем ждать, пока накопятся в 1.1 раза больше записей, чем нужно.
	if (static_cast<double>(children_count) < data.settings.replicated_logs_to_keep * 1.1)
		return;

	Strings replicas = zookeeper->getChildren(zookeeper_path + "/replicas", &stat);
M
Merge  
Michael Kolupaev 已提交
593 594 595
	UInt64 min_pointer = std::numeric_limits<UInt64>::max();
	for (const String & replica : replicas)
	{
M
Merge  
Michael Kolupaev 已提交
596 597
		String pointer = zookeeper->get(zookeeper_path + "/replicas/" + replica + "/log_pointer");
		if (pointer.empty())
M
Merge  
Michael Kolupaev 已提交
598 599 600 601
			return;
		min_pointer = std::min(min_pointer, parse<UInt64>(pointer));
	}

M
Merge  
Michael Kolupaev 已提交
602
	Strings entries = zookeeper->getChildren(zookeeper_path + "/log");
M
Merge  
Michael Kolupaev 已提交
603
	std::sort(entries.begin(), entries.end());
M
Merge  
Michael Kolupaev 已提交
604

M
Merge  
Michael Kolupaev 已提交
605 606
	/// Не будем трогать последние replicated_logs_to_keep записей.
	entries.erase(entries.end() - std::min(entries.size(), data.settings.replicated_logs_to_keep), entries.end());
M
Merge  
Michael Kolupaev 已提交
607 608 609 610 611
	/// Не будем трогать записи, не меньшие min_pointer.
	entries.erase(std::lower_bound(entries.begin(), entries.end(), "log-" + padIndex(min_pointer)), entries.end());

	if (entries.empty())
		return;
M
Merge  
Michael Kolupaev 已提交
612

M
Merge  
Michael Kolupaev 已提交
613
	zkutil::Ops ops;
M
Merge  
Michael Kolupaev 已提交
614
	for (size_t i = 0; i < entries.size(); ++i)
M
Merge  
Michael Kolupaev 已提交
615
	{
M
Merge  
Michael Kolupaev 已提交
616 617 618
		ops.push_back(new zkutil::Op::Remove(zookeeper_path + "/log/" + entries[i], -1));

		if (ops.size() > 400 || i + 1 == entries.size())
M
Merge  
Michael Kolupaev 已提交
619 620 621 622 623 624 625 626
		{
			/// Одновременно с очисткой лога проверим, не добавилась ли реплика с тех пор, как мы получили список реплик.
			ops.push_back(new zkutil::Op::Check(zookeeper_path + "/replicas", stat.version));
			zookeeper->multi(ops);
			ops.clear();
		}
	}

M
Merge  
Michael Kolupaev 已提交
627
	LOG_DEBUG(log, "Removed " << entries.size() << " old log entries: " << entries.front() << " - " << entries.back());
M
Merge  
Michael Kolupaev 已提交
628 629 630 631 632
}

void StorageReplicatedMergeTree::clearOldBlocks()
{
	zkutil::Stat stat;
M
Merge  
Michael Kolupaev 已提交
633
	if (!zookeeper->exists(zookeeper_path + "/blocks", &stat))
M
Merge  
Michael Kolupaev 已提交
634 635
		throw Exception(zookeeper_path + "/blocks doesn't exist", ErrorCodes::NOT_FOUND_NODE);

636
	int children_count = stat.numChildren;
M
Merge  
Michael Kolupaev 已提交
637

M
Merge  
Michael Kolupaev 已提交
638
	/// Чтобы делать "асимптотически" меньше запросов exists, будем ждать, пока накопятся в 1.1 раза больше блоков, чем нужно.
M
Merge  
Michael Kolupaev 已提交
639
	if (static_cast<double>(children_count) < data.settings.replicated_deduplication_window * 1.1)
M
Merge  
Michael Kolupaev 已提交
640 641
		return;

M
Merge  
Michael Kolupaev 已提交
642
	LOG_TRACE(log, "Clearing about " << static_cast<size_t>(children_count) - data.settings.replicated_deduplication_window
M
Merge  
Michael Kolupaev 已提交
643
		<< " old blocks from ZooKeeper. This might take several minutes.");
M
Merge  
Michael Kolupaev 已提交
644

M
Merge  
Michael Kolupaev 已提交
645
	Strings blocks = zookeeper->getChildren(zookeeper_path + "/blocks");
M
Merge  
Michael Kolupaev 已提交
646 647 648 649 650 651

	std::vector<std::pair<Int64, String> > timed_blocks;

	for (const String & block : blocks)
	{
		zkutil::Stat stat;
M
Merge  
Michael Kolupaev 已提交
652
		zookeeper->exists(zookeeper_path + "/blocks/" + block, &stat);
653
		timed_blocks.push_back(std::make_pair(stat.czxid, block));
M
Merge  
Michael Kolupaev 已提交
654 655
	}

M
Merge  
Michael Kolupaev 已提交
656
	zkutil::Ops ops;
M
Merge  
Michael Kolupaev 已提交
657
	std::sort(timed_blocks.begin(), timed_blocks.end(), std::greater<std::pair<Int64, String>>());
M
Merge  
Michael Kolupaev 已提交
658 659
	for (size_t i = data.settings.replicated_deduplication_window; i <  timed_blocks.size(); ++i)
	{
M
Merge  
Michael Kolupaev 已提交
660
		ops.push_back(new zkutil::Op::Remove(zookeeper_path + "/blocks/" + timed_blocks[i].second + "/number", -1));
M
Merge  
Michael Kolupaev 已提交
661
		ops.push_back(new zkutil::Op::Remove(zookeeper_path + "/blocks/" + timed_blocks[i].second + "/columns", -1));
M
Merge  
Michael Kolupaev 已提交
662 663
		ops.push_back(new zkutil::Op::Remove(zookeeper_path + "/blocks/" + timed_blocks[i].second + "/checksums", -1));
		ops.push_back(new zkutil::Op::Remove(zookeeper_path + "/blocks/" + timed_blocks[i].second, -1));
M
Merge  
Michael Kolupaev 已提交
664
		if (ops.size() > 400 || i + 1 == timed_blocks.size())
M
Merge  
Michael Kolupaev 已提交
665 666 667 668
		{
			zookeeper->multi(ops);
			ops.clear();
		}
M
Merge  
Michael Kolupaev 已提交
669 670 671 672 673
	}

	LOG_TRACE(log, "Cleared " << blocks.size() - data.settings.replicated_deduplication_window << " old blocks from ZooKeeper");
}

M
Merge  
Michael Kolupaev 已提交
674 675
void StorageReplicatedMergeTree::loadQueue()
{
M
Merge  
Michael Kolupaev 已提交
676
	std::unique_lock<std::mutex> lock(queue_mutex);
M
Merge  
Michael Kolupaev 已提交
677

M
Merge  
Michael Kolupaev 已提交
678
	Strings children = zookeeper->getChildren(replica_path + "/queue");
M
Merge  
Michael Kolupaev 已提交
679 680 681
	std::sort(children.begin(), children.end());
	for (const String & child : children)
	{
M
Merge  
Michael Kolupaev 已提交
682
		String s = zookeeper->get(replica_path + "/queue/" + child);
M
Merge  
Michael Kolupaev 已提交
683 684 685
		LogEntryPtr entry = LogEntry::parse(s);
		entry->znode_name = child;
		entry->addResultToVirtualParts(*this);
M
Merge  
Michael Kolupaev 已提交
686
		queue.push_back(entry);
M
Merge  
Michael Kolupaev 已提交
687 688 689
	}
}

M
Merge  
Michael Kolupaev 已提交
690
void StorageReplicatedMergeTree::pullLogsToQueue(zkutil::EventPtr next_update_event)
M
Merge  
Michael Kolupaev 已提交
691
{
M
Merge  
Michael Kolupaev 已提交
692
	std::unique_lock<std::mutex> lock(queue_mutex);
M
Merge  
Michael Kolupaev 已提交
693

M
Merge  
Michael Kolupaev 已提交
694 695
	String index_str = zookeeper->get(replica_path + "/log_pointer");
	UInt64 index;
M
Merge  
Michael Kolupaev 已提交
696

M
Merge  
Michael Kolupaev 已提交
697
	if (index_str.empty())
M
Merge  
Michael Kolupaev 已提交
698
	{
M
Merge  
Michael Kolupaev 已提交
699 700 701
		/// Если у нас еще нет указателя на лог, поставим указатель на первую запись в нем.
		Strings entries = zookeeper->getChildren(zookeeper_path + "/log");
		index = entries.empty() ? 0 : parse<UInt64>(std::min_element(entries.begin(), entries.end())->substr(strlen("log-")));
M
Merge  
Michael Kolupaev 已提交
702

M
Merge  
Michael Kolupaev 已提交
703
		zookeeper->set(replica_path + "/log_pointer", toString(index));
M
Merge  
Michael Kolupaev 已提交
704 705
	}
	else
M
Merge  
Michael Kolupaev 已提交
706
	{
M
Merge  
Michael Kolupaev 已提交
707
		index = parse<UInt64>(index_str);
M
Merge  
Michael Kolupaev 已提交
708
	}
M
Merge  
Michael Kolupaev 已提交
709

M
Merge  
Michael Kolupaev 已提交
710 711
	UInt64 first_index = index;

M
Merge  
Michael Kolupaev 已提交
712
	size_t count = 0;
M
Merge  
Michael Kolupaev 已提交
713 714
	String entry_str;
	while (zookeeper->tryGet(zookeeper_path + "/log/log-" + padIndex(index), entry_str))
M
Merge  
Michael Kolupaev 已提交
715
	{
M
Merge  
Michael Kolupaev 已提交
716
		++count;
M
Merge  
Michael Kolupaev 已提交
717
		++index;
M
Merge  
Michael Kolupaev 已提交
718

M
Merge  
Michael Kolupaev 已提交
719
		LogEntryPtr entry = LogEntry::parse(entry_str);
M
Merge  
Michael Kolupaev 已提交
720

M
Merge  
Michael Kolupaev 已提交
721 722 723
		/// Одновременно добавим запись в очередь и продвинем указатель на лог.
		zkutil::Ops ops;
		ops.push_back(new zkutil::Op::Create(
M
Merge  
Michael Kolupaev 已提交
724
			replica_path + "/queue/queue-", entry_str, zookeeper->getDefaultACL(), zkutil::CreateMode::PersistentSequential));
M
Merge  
Michael Kolupaev 已提交
725
		ops.push_back(new zkutil::Op::SetData(
M
Merge  
Michael Kolupaev 已提交
726
			replica_path + "/log_pointer", toString(index), -1));
M
Merge  
Michael Kolupaev 已提交
727
		auto results = zookeeper->multi(ops);
M
Merge  
Michael Kolupaev 已提交
728

729
		String path_created = dynamic_cast<zkutil::Op::Create &>(ops[0]).getPathCreated();
M
Merge  
Michael Kolupaev 已提交
730 731
		entry->znode_name = path_created.substr(path_created.find_last_of('/') + 1);
		entry->addResultToVirtualParts(*this);
M
Merge  
Michael Kolupaev 已提交
732
		queue.push_back(entry);
M
Merge  
Michael Kolupaev 已提交
733
	}
M
Merge  
Michael Kolupaev 已提交
734

M
Merge  
Michael Kolupaev 已提交
735 736 737 738
	if (next_update_event)
	{
		if (zookeeper->exists(zookeeper_path + "/log/log-" + padIndex(index), nullptr, next_update_event))
			next_update_event->set();
M
Merge  
Michael Kolupaev 已提交
739
	}
M
Merge  
Michael Kolupaev 已提交
740

M
Merge  
Michael Kolupaev 已提交
741 742 743
	if (!count)
		return;

M
Merge  
Michael Kolupaev 已提交
744 745
	if (queue_task_handle)
		queue_task_handle->wake();
M
Merge  
Michael Kolupaev 已提交
746

M
Merge  
Michael Kolupaev 已提交
747
	LOG_DEBUG(log, "Pulled " << count << " entries to queue: log-" << padIndex(first_index) << " - log-" << padIndex(index - 1));
M
Merge  
Michael Kolupaev 已提交
748 749
}

M
Merge  
Michael Kolupaev 已提交
750
bool StorageReplicatedMergeTree::shouldExecuteLogEntry(const LogEntry & entry)
M
Merge  
Michael Kolupaev 已提交
751
{
M
Merge  
Michael Kolupaev 已提交
752
	if ((entry.type == LogEntry::MERGE_PARTS || entry.type == LogEntry::GET_PART) && future_parts.count(entry.new_part_name))
M
Merge  
Michael Kolupaev 已提交
753 754 755 756 757 758
	{
		LOG_DEBUG(log, "Not executing log entry for part " << entry.new_part_name <<
			" because another log entry for the same part is being processed. This shouldn't happen often.");
		return false;
	}

M
Merge  
Michael Kolupaev 已提交
759 760 761 762 763 764 765 766 767 768 769
	if (entry.type == LogEntry::MERGE_PARTS)
	{
		/** Если какая-то из нужных частей сейчас передается или мерджится, подождем окончания этой операции.
		  * Иначе, даже если всех нужных частей для мерджа нет, нужно попытаться сделать мердж.
		  * Если каких-то частей не хватает, вместо мерджа будет попытка скачать кусок.
		  * Такая ситуация возможна, если получение какого-то куска пофейлилось, и его переместили в конец очереди.
		  */
		for (const auto & name : entry.parts_to_merge)
		{
			if (future_parts.count(name))
			{
M
Merge  
Michael Kolupaev 已提交
770
				LOG_TRACE(log, "Not merging into part " << entry.new_part_name << " because part " << name << " is not ready yet.");
M
Merge  
Michael Kolupaev 已提交
771 772 773 774 775 776
				return false;
			}
		}
	}

	return true;
M
Merge  
Michael Kolupaev 已提交
777 778
}

M
Merge  
Michael Kolupaev 已提交
779
bool StorageReplicatedMergeTree::executeLogEntry(const LogEntry & entry, BackgroundProcessingPool::Context & pool_context)
M
Merge  
Michael Kolupaev 已提交
780
{
M
Merge  
Michael Kolupaev 已提交
781 782 783
	if (entry.type == LogEntry::DROP_RANGE)
		return executeDropRange(entry);

M
Merge  
Michael Kolupaev 已提交
784 785 786 787
	if (entry.type == LogEntry::GET_PART ||
		entry.type == LogEntry::MERGE_PARTS)
	{
		/// Если у нас уже есть этот кусок или покрывающий его кусок, ничего делать не нужно.
M
Merge  
Michael Kolupaev 已提交
788
		MergeTreeData::DataPartPtr containing_part = data.getActiveContainingPart(entry.new_part_name);
M
Merge  
Michael Kolupaev 已提交
789 790

		/// Даже если кусок есть локально, его (в исключительных случаях) может не быть в zookeeper.
M
Merge  
Michael Kolupaev 已提交
791
		if (containing_part && zookeeper->exists(replica_path + "/parts/" + containing_part->name))
M
Merge  
Michael Kolupaev 已提交
792
		{
M
Merge  
Michael Kolupaev 已提交
793 794
			if (!(entry.type == LogEntry::GET_PART && entry.source_replica == replica_name))
				LOG_DEBUG(log, "Skipping action for part " + entry.new_part_name + " - part already exists");
M
Merge  
Michael Kolupaev 已提交
795
			return true;
M
Merge  
Michael Kolupaev 已提交
796
		}
M
Merge  
Michael Kolupaev 已提交
797 798
	}

M
Merge  
Michael Kolupaev 已提交
799
	if (entry.type == LogEntry::GET_PART && entry.source_replica == replica_name)
M
Merge  
Michael Kolupaev 已提交
800
		LOG_WARNING(log, "Part " << entry.new_part_name << " from own log doesn't exist.");
M
Merge  
Michael Kolupaev 已提交
801 802 803

	bool do_fetch = false;

M
Merge  
Michael Kolupaev 已提交
804 805
	if (entry.type == LogEntry::GET_PART)
	{
M
Merge  
Michael Kolupaev 已提交
806
		do_fetch = true;
M
Merge  
Michael Kolupaev 已提交
807 808 809
	}
	else if (entry.type == LogEntry::MERGE_PARTS)
	{
M
Merge  
Michael Kolupaev 已提交
810
		MergeTreeData::DataPartsVector parts;
811
		bool have_all_parts = true;
M
Merge  
Michael Kolupaev 已提交
812 813
		for (const String & name : entry.parts_to_merge)
		{
M
Merge  
Michael Kolupaev 已提交
814
			MergeTreeData::DataPartPtr part = data.getActiveContainingPart(name);
M
Merge  
Michael Kolupaev 已提交
815 816 817 818 819 820 821 822 823 824 825 826
			if (!part)
			{
				have_all_parts = false;
				break;
			}
			if (part->name != name)
			{
				LOG_ERROR(log, "Log and parts set look inconsistent: " << name << " is covered by " << part->name
					<< " but should be merged into " << entry.new_part_name);
				have_all_parts = false;
				break;
			}
M
Merge  
Michael Kolupaev 已提交
827 828
			parts.push_back(part);
		}
M
Merge  
Michael Kolupaev 已提交
829

M
Merge  
Michael Kolupaev 已提交
830
		if (!have_all_parts)
M
Merge  
Michael Kolupaev 已提交
831
		{
M
Merge  
Michael Kolupaev 已提交
832 833 834
			/// Если нет всех нужных кусков, попробуем взять у кого-нибудь уже помердженный кусок.
			do_fetch = true;
			LOG_DEBUG(log, "Don't have all parts for merge " << entry.new_part_name << "; will try to fetch it instead");
M
Merge  
Michael Kolupaev 已提交
835
		}
M
Merge  
Michael Kolupaev 已提交
836 837
		else
		{
M
Merge  
Michael Kolupaev 已提交
838 839 840
			/// Если собираемся сливать большие куски, увеличим счетчик потоков, сливающих большие куски.
			for (const auto & part : parts)
			{
M
Merge  
Michael Kolupaev 已提交
841
				if (part->size_in_bytes > data.settings.max_bytes_to_merge_parts_small)
M
Merge  
Michael Kolupaev 已提交
842 843 844 845 846 847 848
				{
					pool_context.incrementCounter("big merges");
					pool_context.incrementCounter("replicated big merges");
					break;
				}
			}

M
Merge  
Michael Kolupaev 已提交
849
			auto table_lock = lockStructure(false);
850

M
Merge  
Michael Kolupaev 已提交
851 852
			MergeTreeData::Transaction transaction;
			MergeTreeData::DataPartPtr part = merger.mergeParts(parts, entry.new_part_name, &transaction);
M
Merge  
Michael Kolupaev 已提交
853 854

			zkutil::Ops ops;
M
Merge  
Michael Kolupaev 已提交
855
			checkPartAndAddToZooKeeper(part, ops);
M
Merge  
Michael Kolupaev 已提交
856

M
Merge  
Michael Kolupaev 已提交
857
			zookeeper->multi(ops);
M
Merge  
Michael Kolupaev 已提交
858 859 860 861

			/** При ZCONNECTIONLOSS или ZOPERATIONTIMEOUT можем зря откатить локальные изменения кусков.
			  * Это не проблема, потому что в таком случае слияние останется в очереди, и мы попробуем снова.
			  */
M
Merge  
Michael Kolupaev 已提交
862
			transaction.commit();
M
Merge  
Michael Kolupaev 已提交
863
			merge_selecting_event.set();
M
Merge  
Michael Kolupaev 已提交
864 865 866

			ProfileEvents::increment(ProfileEvents::ReplicatedPartMerges);
		}
M
Merge  
Michael Kolupaev 已提交
867 868 869 870 871
	}
	else
	{
		throw Exception("Unexpected log entry type: " + toString(static_cast<int>(entry.type)));
	}
M
Merge  
Michael Kolupaev 已提交
872 873 874

	if (do_fetch)
	{
M
Merge  
Michael Kolupaev 已提交
875 876
		String replica;

M
Merge  
Michael Kolupaev 已提交
877 878
		try
		{
M
Merge  
Michael Kolupaev 已提交
879
			replica = findReplicaHavingPart(entry.new_part_name, true);
M
Merge  
Michael Kolupaev 已提交
880 881 882 883 884
			if (replica.empty())
			{
				ProfileEvents::increment(ProfileEvents::ReplicatedPartFailedFetches);
				throw Exception("No active replica has part " + entry.new_part_name, ErrorCodes::NO_REPLICA_HAS_PART);
			}
M
Merge  
Michael Kolupaev 已提交
885 886 887 888 889 890 891 892 893 894 895 896 897 898
			fetchPart(entry.new_part_name, replica);

			if (entry.type == LogEntry::MERGE_PARTS)
				ProfileEvents::increment(ProfileEvents::ReplicatedPartFetchesOfMerged);
		}
		catch (...)
		{
			/** Если не получилось скачать кусок, нужный для какого-то мерджа, лучше не пытаться получить другие куски для этого мерджа,
			  * а попытаться сразу получить помердженный кусок. Чтобы так получилось, переместим действия для получения остальных кусков
			  * для этого мерджа в конец очереди.
			  *
			  */
			try
			{
M
Merge  
Michael Kolupaev 已提交
899
				std::unique_lock<std::mutex> lock(queue_mutex);
M
Merge  
Michael Kolupaev 已提交
900 901 902 903 904 905

				/// Найдем действие по объединению этого куска с другими. Запомним других.
				StringSet parts_for_merge;
				LogEntries::iterator merge_entry;
				for (LogEntries::iterator it = queue.begin(); it != queue.end(); ++it)
				{
M
Merge  
Michael Kolupaev 已提交
906
					if ((*it)->type == LogEntry::MERGE_PARTS)
M
Merge  
Michael Kolupaev 已提交
907
					{
M
Merge  
Michael Kolupaev 已提交
908 909
						if (std::find((*it)->parts_to_merge.begin(), (*it)->parts_to_merge.end(), entry.new_part_name)
							!= (*it)->parts_to_merge.end())
M
Merge  
Michael Kolupaev 已提交
910
						{
M
Merge  
Michael Kolupaev 已提交
911
							parts_for_merge = StringSet((*it)->parts_to_merge.begin(), (*it)->parts_to_merge.end());
M
Merge  
Michael Kolupaev 已提交
912 913 914 915 916 917 918 919 920 921 922 923 924 925 926 927 928
							merge_entry = it;
							break;
						}
					}
				}

				if (!parts_for_merge.empty())
				{
					/// Переместим в конец очереди действия, получающие parts_for_merge.
					for (LogEntries::iterator it = queue.begin(); it != queue.end();)
					{
						auto it0 = it;
						++it;

						if (it0 == merge_entry)
							break;

M
Merge  
Michael Kolupaev 已提交
929 930
						if (((*it0)->type == LogEntry::MERGE_PARTS || (*it0)->type == LogEntry::GET_PART)
							&& parts_for_merge.count((*it0)->new_part_name))
M
Merge  
Michael Kolupaev 已提交
931 932 933 934
						{
							queue.splice(queue.end(), queue, it0, it);
						}
					}
M
Merge  
Michael Kolupaev 已提交
935 936 937 938 939 940 941 942 943

					/** Если этого куска ни у кого нет, но в очереди упоминается мердж с его участием, то наверно этот кусок такой старый,
					  *  что его все померджили и удалили. Не будем бросать исключение, чтобы queueTask лишний раз не спала.
					  */
					if (replica.empty())
					{
						LOG_INFO(log, "No replica has part " << entry.new_part_name << ". Will fetch merged part instead.");
						return false;
					}
M
Merge  
Michael Kolupaev 已提交
944
				}
M
Merge  
Michael Kolupaev 已提交
945 946 947 948

				/// Если ни у кого нет куска, и в очереди нет слияний с его участием, проверим, есть ли у кого-то покрывающий его.
				if (replica.empty())
					enqueuePartForCheck(entry.new_part_name);
M
Merge  
Michael Kolupaev 已提交
949 950 951 952 953 954 955 956 957
			}
			catch (...)
			{
				tryLogCurrentException(__PRETTY_FUNCTION__);
			}

			throw;
		}
	}
M
Merge  
Michael Kolupaev 已提交
958 959

	return true;
M
Merge  
Michael Kolupaev 已提交
960 961
}

M
Merge  
Michael Kolupaev 已提交
962 963
bool StorageReplicatedMergeTree::executeDropRange(const StorageReplicatedMergeTree::LogEntry & entry)
{
M
Merge  
Michael Kolupaev 已提交
964
	LOG_INFO(log, (entry.detach ? "Detaching" : "Removing") << " parts inside " << entry.new_part_name << ".");
M
Merge  
Michael Kolupaev 已提交
965 966 967 968 969 970 971 972 973 974 975 976 977 978 979 980 981 982 983 984 985 986 987 988 989 990 991 992 993

	{
		LogEntries to_wait;
		size_t removed_entries = 0;

		/// Удалим из очереди операции с кусками, содержащимися в удаляемом диапазоне.
		std::unique_lock<std::mutex> lock(queue_mutex);
		for (LogEntries::iterator it = queue.begin(); it != queue.end();)
		{
			if (((*it)->type == LogEntry::GET_PART || (*it)->type == LogEntry::MERGE_PARTS) &&
				ActiveDataPartSet::contains(entry.new_part_name, (*it)->new_part_name))
			{
				if ((*it)->currently_executing)
					to_wait.push_back(*it);
				auto code = zookeeper->tryRemove(replica_path + "/queue/" + (*it)->znode_name);
				if (code != ZOK)
					LOG_INFO(log, "Couldn't remove " << replica_path + "/queue/" + (*it)->znode_name << ": "
						<< zkutil::ZooKeeper::error2string(code));
				queue.erase(it++);
				++removed_entries;
			}
			else
				++it;
		}

		LOG_DEBUG(log, "Removed " << removed_entries << " entries from queue. "
			"Waiting for " << to_wait.size() << " entries that are currently executing.");

		/// Дождемся завершения операций с кусками, содержащимися в удаляемом диапазоне.
M
Merge  
Michael Kolupaev 已提交
994 995
		for (LogEntryPtr & entry : to_wait)
			entry->execution_complete.wait(lock, [&entry] { return !entry->currently_executing; });
M
Merge  
Michael Kolupaev 已提交
996 997
	}

M
Merge  
Michael Kolupaev 已提交
998
	LOG_DEBUG(log, (entry.detach ? "Detaching" : "Removing") << " parts.");
M
Merge  
Michael Kolupaev 已提交
999 1000 1001 1002 1003 1004 1005 1006 1007 1008 1009
	size_t removed_parts = 0;

	/// Удалим куски, содержащиеся в удаляемом диапазоне.
	auto parts = data.getDataParts();
	for (const auto & part : parts)
	{
		if (!ActiveDataPartSet::contains(entry.new_part_name, part->name))
			continue;
		LOG_DEBUG(log, "Removing part " << part->name);
		++removed_parts;

M
Merge  
Michael Kolupaev 已提交
1010 1011 1012 1013
		/// Если кусок удалять не нужно, надежнее переместить директорию до изменений в ZooKeeper.
		if (entry.detach)
			data.renameAndDetachPart(part);

M
Merge  
Michael Kolupaev 已提交
1014 1015 1016 1017 1018 1019
		zkutil::Ops ops;
		ops.push_back(new zkutil::Op::Remove(replica_path + "/parts/" + part->name + "/columns", -1));
		ops.push_back(new zkutil::Op::Remove(replica_path + "/parts/" + part->name + "/checksums", -1));
		ops.push_back(new zkutil::Op::Remove(replica_path + "/parts/" + part->name, -1));
		zookeeper->multi(ops);

M
Merge  
Michael Kolupaev 已提交
1020
		/// Если кусок нужно удалить, надежнее удалить директорию после изменений в ZooKeeper.
M
Merge  
Michael Kolupaev 已提交
1021 1022
		if (!entry.detach)
			data.replaceParts({part}, {}, false);
M
Merge  
Michael Kolupaev 已提交
1023 1024
	}

M
Merge  
Michael Kolupaev 已提交
1025
	LOG_INFO(log, (entry.detach ? "Detached " : "Removed ") << removed_parts << " parts inside " << entry.new_part_name << ".");
M
Merge  
Michael Kolupaev 已提交
1026 1027 1028

	if (unreplicated_data)
	{
M
Merge  
Michael Kolupaev 已提交
1029 1030
		Poco::ScopedLock<Poco::FastMutex> unreplicated_lock(unreplicated_mutex);

M
Merge  
Michael Kolupaev 已提交
1031 1032 1033 1034 1035 1036 1037 1038 1039 1040 1041 1042 1043 1044 1045
		removed_parts = 0;
		parts = unreplicated_data->getDataParts();
		for (const auto & part : parts)
		{
			if (!ActiveDataPartSet::contains(entry.new_part_name, part->name))
				continue;
			LOG_DEBUG(log, "Removing unreplicated part " << part->name);
			++removed_parts;

			if (entry.detach)
				unreplicated_data->renameAndDetachPart(part, "");
			else
				unreplicated_data->replaceParts({part}, {}, false);
		}
	}
M
Merge  
Michael Kolupaev 已提交
1046

M
Merge  
Michael Kolupaev 已提交
1047 1048 1049
	/// На месте удаленных кусков могут появиться новые, с другими данными.
	context.resetCaches();

M
Merge  
Michael Kolupaev 已提交
1050 1051 1052
	return true;
}

M
Merge  
Michael Kolupaev 已提交
1053 1054 1055 1056
void StorageReplicatedMergeTree::queueUpdatingThread()
{
	while (!shutdown_called)
	{
M
Merge  
Michael Kolupaev 已提交
1057 1058
		try
		{
M
Merge  
Michael Kolupaev 已提交
1059
			pullLogsToQueue(queue_updating_event);
M
Merge  
Michael Kolupaev 已提交
1060

M
Merge  
Michael Kolupaev 已提交
1061
			queue_updating_event->wait();
M
Merge  
Michael Kolupaev 已提交
1062 1063 1064 1065 1066
		}
		catch (...)
		{
			tryLogCurrentException(__PRETTY_FUNCTION__);

M
Merge  
Michael Kolupaev 已提交
1067 1068
			queue_updating_event->tryWait(ERROR_SLEEP_MS);
		}
M
Merge  
Michael Kolupaev 已提交
1069
	}
M
Merge  
Michael Kolupaev 已提交
1070 1071

	LOG_DEBUG(log, "queue updating thread finished");
M
Merge  
Michael Kolupaev 已提交
1072
}
M
Merge  
Michael Kolupaev 已提交
1073

M
Merge  
Michael Kolupaev 已提交
1074
bool StorageReplicatedMergeTree::queueTask(BackgroundProcessingPool::Context & pool_context)
M
Merge  
Michael Kolupaev 已提交
1075
{
M
Merge  
Michael Kolupaev 已提交
1076
	LogEntryPtr entry;
M
Merge  
Michael Kolupaev 已提交
1077

M
Merge  
Michael Kolupaev 已提交
1078 1079
	try
	{
M
Merge  
Michael Kolupaev 已提交
1080
		std::unique_lock<std::mutex> lock(queue_mutex);
M
Merge  
Michael Kolupaev 已提交
1081 1082
		bool empty = queue.empty();
		if (!empty)
M
Merge  
Michael Kolupaev 已提交
1083
		{
M
Merge  
Michael Kolupaev 已提交
1084
			for (LogEntries::iterator it = queue.begin(); it != queue.end(); ++it)
M
Merge  
Michael Kolupaev 已提交
1085
			{
M
Merge  
Michael Kolupaev 已提交
1086
				if (!(*it)->currently_executing && shouldExecuteLogEntry(**it))
M
Merge  
Michael Kolupaev 已提交
1087
				{
M
Merge  
Michael Kolupaev 已提交
1088
					entry = *it;
M
Merge  
Michael Kolupaev 已提交
1089
					entry->tagPartAsFuture(*this);
M
Merge  
Michael Kolupaev 已提交
1090
					queue.splice(queue.end(), queue, it);
M
Merge  
Michael Kolupaev 已提交
1091
					entry->currently_executing = true;
M
Merge  
Michael Kolupaev 已提交
1092
					break;
M
Merge  
Michael Kolupaev 已提交
1093
				}
M
Merge  
Michael Kolupaev 已提交
1094 1095
			}
		}
M
Merge  
Michael Kolupaev 已提交
1096 1097 1098 1099 1100
	}
	catch (...)
	{
		tryLogCurrentException(__PRETTY_FUNCTION__);
	}
M
Merge  
Michael Kolupaev 已提交
1101

M
Merge  
Michael Kolupaev 已提交
1102
	if (!entry)
M
Merge  
Michael Kolupaev 已提交
1103
		return false;
M
Merge  
Michael Kolupaev 已提交
1104

M
Merge  
Michael Kolupaev 已提交
1105
	bool exception = true;
M
Merge  
Michael Kolupaev 已提交
1106
	bool success = false;
M
Merge  
Michael Kolupaev 已提交
1107

M
Merge  
Michael Kolupaev 已提交
1108 1109
	try
	{
M
Merge  
Michael Kolupaev 已提交
1110
		if (executeLogEntry(*entry, pool_context))
M
Merge  
Michael Kolupaev 已提交
1111
		{
M
Merge  
Michael Kolupaev 已提交
1112
			auto code = zookeeper->tryRemove(replica_path + "/queue/" + entry->znode_name);
M
Merge  
Michael Kolupaev 已提交
1113

M
Merge  
Michael Kolupaev 已提交
1114
			if (code != ZOK)
M
Merge  
Michael Kolupaev 已提交
1115
				LOG_ERROR(log, "Couldn't remove " << replica_path + "/queue/" + entry->znode_name << ": "
M
Merge  
Michael Kolupaev 已提交
1116 1117 1118
					<< zkutil::ZooKeeper::error2string(code) + ". This shouldn't happen often.");

			success = true;
M
Merge  
Michael Kolupaev 已提交
1119
		}
M
Merge  
Michael Kolupaev 已提交
1120

M
Merge  
Michael Kolupaev 已提交
1121
		exception = false;
M
Merge  
Michael Kolupaev 已提交
1122 1123 1124 1125
	}
	catch (Exception & e)
	{
		if (e.code() == ErrorCodes::NO_REPLICA_HAS_PART)
M
Merge  
Michael Kolupaev 已提交
1126
			/// Если ни у кого нет нужного куска, наверно, просто не все реплики работают; не будем писать в лог с уровнем Error.
M
Merge  
Michael Kolupaev 已提交
1127 1128
			LOG_INFO(log, e.displayText());
		else
M
Merge  
Michael Kolupaev 已提交
1129
			tryLogCurrentException(__PRETTY_FUNCTION__);
M
Merge  
Michael Kolupaev 已提交
1130 1131 1132 1133 1134
	}
	catch (...)
	{
		tryLogCurrentException(__PRETTY_FUNCTION__);
	}
M
Merge  
Michael Kolupaev 已提交
1135

M
Merge  
Michael Kolupaev 已提交
1136 1137 1138 1139 1140 1141 1142 1143
	entry->future_part_tagger = nullptr;

	std::unique_lock<std::mutex> lock(queue_mutex);

	entry->currently_executing = false;
	entry->execution_complete.notify_all();

	if (success)
M
Merge  
Michael Kolupaev 已提交
1144
	{
M
Merge  
Michael Kolupaev 已提交
1145 1146 1147
		/// Удалим задание из очереди.
		/// Нельзя просто обратиться по заранее сохраненному итератору, потому что задание мог успеть удалить кто-то другой.
		for (LogEntries::iterator it = queue.end(); it != queue.begin();)
M
Merge  
Michael Kolupaev 已提交
1148
		{
M
Merge  
Michael Kolupaev 已提交
1149 1150 1151
			--it;
			if (*it == entry)
			{
M
Merge  
Michael Kolupaev 已提交
1152
				queue.erase(it);
M
Merge  
Michael Kolupaev 已提交
1153 1154
				break;
			}
M
Merge  
Michael Kolupaev 已提交
1155
		}
M
Merge  
Michael Kolupaev 已提交
1156
	}
M
Merge  
Michael Kolupaev 已提交
1157

M
Merge  
Michael Kolupaev 已提交
1158 1159
	/// Если не было исключения, не нужно спать.
	return !exception;
M
Merge  
Michael Kolupaev 已提交
1160 1161
}

M
Merge  
Michael Kolupaev 已提交
1162 1163
void StorageReplicatedMergeTree::mergeSelectingThread()
{
M
Merge  
Michael Kolupaev 已提交
1164
	bool need_pull = true;
M
Merge  
Michael Kolupaev 已提交
1165 1166 1167

	while (!shutdown_called && is_leader_node)
	{
M
Michael Kolupaev 已提交
1168
		bool success = false;
M
Merge  
Michael Kolupaev 已提交
1169

M
Michael Kolupaev 已提交
1170
		try
M
Merge  
Michael Kolupaev 已提交
1171
		{
M
Merge  
Michael Kolupaev 已提交
1172 1173
			std::unique_lock<std::mutex> merge_selecting_lock(merge_selecting_mutex);

M
Merge  
Michael Kolupaev 已提交
1174 1175 1176 1177 1178 1179 1180 1181
			if (need_pull)
			{
				/// Нужно загрузить новую запись в очередь перед тем, как выбирать куски для слияния.
				///  (чтобы кусок добавился в virtual_parts).
				pullLogsToQueue();
				need_pull = false;
			}

M
Michael Kolupaev 已提交
1182
			size_t merges_queued = 0;
M
Merge  
Michael Kolupaev 已提交
1183
			/// Есть ли в очереди или в фоновом потоке мердж крупных кусков.
M
Merge  
Michael Kolupaev 已提交
1184
			bool has_big_merge = context.getBackgroundPool().getCounter("replicated big merges") > 0;
M
Merge  
Michael Kolupaev 已提交
1185

M
Merge  
Michael Kolupaev 已提交
1186
			if (!has_big_merge)
M
Michael Kolupaev 已提交
1187
			{
M
Merge  
Michael Kolupaev 已提交
1188
				std::unique_lock<std::mutex> lock(queue_mutex);
M
Merge  
Michael Kolupaev 已提交
1189

M
Michael Kolupaev 已提交
1190
				for (const auto & entry : queue)
M
Merge  
Michael Kolupaev 已提交
1191
				{
M
Merge  
Michael Kolupaev 已提交
1192
					if (entry->type == LogEntry::MERGE_PARTS)
M
Merge  
Michael Kolupaev 已提交
1193
					{
M
Michael Kolupaev 已提交
1194
						++merges_queued;
M
Merge  
Michael Kolupaev 已提交
1195 1196 1197

						if (!has_big_merge)
						{
M
Merge  
Michael Kolupaev 已提交
1198
							for (const String & name : entry->parts_to_merge)
M
Merge  
Michael Kolupaev 已提交
1199
							{
M
Merge  
Michael Kolupaev 已提交
1200
								MergeTreeData::DataPartPtr part = data.getActiveContainingPart(name);
M
Merge  
Michael Kolupaev 已提交
1201 1202
								if (!part || part->name != name)
									continue;
M
Merge  
Michael Kolupaev 已提交
1203
								if (part->size_in_bytes > data.settings.max_bytes_to_merge_parts_small)
M
Merge  
Michael Kolupaev 已提交
1204 1205 1206 1207 1208 1209 1210 1211
								{
									has_big_merge = true;
									break;
								}
							}
						}
					}
				}
M
Michael Kolupaev 已提交
1212
			}
M
Merge  
Michael Kolupaev 已提交
1213

M
Merge  
Michael Kolupaev 已提交
1214
			do
M
Michael Kolupaev 已提交
1215
			{
M
Merge  
Michael Kolupaev 已提交
1216
				if (merges_queued >= data.settings.max_replicated_merges_in_queue)
M
Merge  
Michael Kolupaev 已提交
1217
					break;
M
Merge  
Michael Kolupaev 已提交
1218

M
Merge  
Michael Kolupaev 已提交
1219
				MergeTreeData::DataPartsVector parts;
M
Merge  
Michael Kolupaev 已提交
1220

M
Merge  
Michael Kolupaev 已提交
1221 1222 1223 1224
				String merged_name;
				auto can_merge = std::bind(
					&StorageReplicatedMergeTree::canMergeParts, this, std::placeholders::_1, std::placeholders::_2);

M
Merge  
Michael Kolupaev 已提交
1225 1226 1227
				if (!merger.selectPartsToMerge(parts, merged_name, MergeTreeDataMerger::NO_LIMIT,
												false, false, has_big_merge, can_merge) &&
					!merger.selectPartsToMerge(parts, merged_name, MergeTreeDataMerger::NO_LIMIT,
M
Merge  
Michael Kolupaev 已提交
1228
												true, false, has_big_merge, can_merge))
M
Merge  
Michael Kolupaev 已提交
1229
					break;
M
Merge  
Michael Kolupaev 已提交
1230

M
Merge  
Michael Kolupaev 已提交
1231 1232 1233 1234 1235 1236 1237 1238 1239 1240 1241 1242 1243 1244
				bool all_in_zk = true;
				for (const auto & part : parts)
				{
					/// Если о каком-то из кусков нет информации в ZK, не будем сливать.
					if (!zookeeper->exists(replica_path + "/parts/" + part->name))
					{
						LOG_WARNING(log, "Part " << part->name << " exists locally but not in ZooKeeper.");
						enqueuePartForCheck(part->name);
						all_in_zk = false;
					}
				}
				if (!all_in_zk)
					break;

M
Merge  
Michael Kolupaev 已提交
1245 1246 1247 1248
				LogEntry entry;
				entry.type = LogEntry::MERGE_PARTS;
				entry.source_replica = replica_name;
				entry.new_part_name = merged_name;
M
Merge  
Michael Kolupaev 已提交
1249

M
Merge  
Michael Kolupaev 已提交
1250 1251 1252
				for (const auto & part : parts)
				{
					entry.parts_to_merge.push_back(part->name);
M
Merge  
Michael Kolupaev 已提交
1253 1254
				}

M
Merge  
Michael Kolupaev 已提交
1255
				need_pull = true;
M
Merge  
Michael Kolupaev 已提交
1256

M
Merge  
Michael Kolupaev 已提交
1257
				zookeeper->create(zookeeper_path + "/log/log-", entry.toString(), zkutil::CreateMode::PersistentSequential);
M
Merge  
Michael Kolupaev 已提交
1258 1259 1260

				String month_name = parts[0]->name.substr(0, 6);
				for (size_t i = 0; i + 1 < parts.size(); ++i)
M
Merge  
Michael Kolupaev 已提交
1261
				{
M
Merge  
Michael Kolupaev 已提交
1262 1263 1264
					/// Уберем больше не нужные отметки о несуществующих блоках.
					for (UInt64 number = parts[i]->right + 1; number <= parts[i + 1]->left - 1; ++number)
					{
M
Merge  
Michael Kolupaev 已提交
1265 1266
						zookeeper->tryRemove(zookeeper_path +              "/block_numbers/" + month_name + "/block-" + padIndex(number));
						zookeeper->tryRemove(zookeeper_path + "/nonincrement_block_numbers/" + month_name + "/block-" + padIndex(number));
M
Merge  
Michael Kolupaev 已提交
1267
					}
M
Merge  
Michael Kolupaev 已提交
1268
				}
M
Merge  
Michael Kolupaev 已提交
1269 1270

				success = true;
M
Merge  
Michael Kolupaev 已提交
1271
			}
M
Merge  
Michael Kolupaev 已提交
1272
			while(false);
M
Merge  
Michael Kolupaev 已提交
1273 1274 1275 1276 1277 1278
		}
		catch (...)
		{
			tryLogCurrentException(__PRETTY_FUNCTION__);
		}

M
Merge  
Michael Kolupaev 已提交
1279
		if (shutdown_called || !is_leader_node)
M
Merge  
Michael Kolupaev 已提交
1280 1281
			break;

M
Merge  
Michael Kolupaev 已提交
1282
		if (!success)
M
Merge  
Michael Kolupaev 已提交
1283
			merge_selecting_event.tryWait(MERGE_SELECTING_SLEEP_MS);
M
Merge  
Michael Kolupaev 已提交
1284
	}
M
Merge  
Michael Kolupaev 已提交
1285 1286

	LOG_DEBUG(log, "merge selecting thread finished");
M
Merge  
Michael Kolupaev 已提交
1287 1288
}

M
Merge  
Michael Kolupaev 已提交
1289
void StorageReplicatedMergeTree::cleanupThread()
M
Merge  
Michael Kolupaev 已提交
1290
{
M
Merge  
Michael Kolupaev 已提交
1291
	while (!shutdown_called)
M
Merge  
Michael Kolupaev 已提交
1292 1293
	{
		try
M
Merge  
Michael Kolupaev 已提交
1294
		{
M
Merge  
Michael Kolupaev 已提交
1295 1296
			clearOldParts();

M
Merge  
Michael Kolupaev 已提交
1297 1298 1299
			if (unreplicated_data)
				unreplicated_data->clearOldParts();

M
Merge  
Michael Kolupaev 已提交
1300 1301 1302 1303 1304
			if (is_leader_node)
			{
				clearOldLogs();
				clearOldBlocks();
			}
M
Merge  
Michael Kolupaev 已提交
1305
		}
M
Merge  
Michael Kolupaev 已提交
1306 1307 1308 1309
		catch (...)
		{
			tryLogCurrentException(__PRETTY_FUNCTION__);
		}
M
Merge  
Michael Kolupaev 已提交
1310

M
Merge  
Michael Kolupaev 已提交
1311
		shutdown_event.tryWait(CLEANUP_SLEEP_MS);
M
Merge  
Michael Kolupaev 已提交
1312
	}
M
Merge  
Michael Kolupaev 已提交
1313 1314

	LOG_DEBUG(log, "cleanup thread finished");
M
Merge  
Michael Kolupaev 已提交
1315 1316
}

M
Merge  
Michael Kolupaev 已提交
1317 1318 1319 1320 1321 1322 1323 1324 1325 1326 1327 1328 1329 1330 1331 1332 1333 1334 1335 1336 1337 1338
void StorageReplicatedMergeTree::alterThread()
{
	bool force_recheck_parts = true;

	while (!shutdown_called)
	{
		try
		{
			zkutil::Stat stat;
			String columns_str = zookeeper->get(zookeeper_path + "/columns", &stat, alter_thread_event);
			NamesAndTypesList columns = NamesAndTypesList::parse(columns_str, context.getDataTypeFactory());

			bool changed = false;

			/// Проверим, что описание столбцов изменилось.
			/// Чтобы не останавливать лишний раз все запросы в таблицу, проверим сначала под локом на чтение.
			{
				auto table_lock = lockStructure(false);
				if (columns != data.getColumnsList())
					changed = true;
			}

M
Merge  
Michael Kolupaev 已提交
1339 1340
			MergeTreeData::DataParts parts;

M
Merge  
Michael Kolupaev 已提交
1341 1342 1343 1344 1345 1346 1347 1348 1349
			/// Если описание столбцов изменилось, обновим структуру таблицы локально.
			if (changed)
			{
				auto table_lock = lockStructureForAlter();
				if (columns != data.getColumnsList())
				{
					LOG_INFO(log, "Columns list changed in ZooKeeper. Applying changes locally.");
					InterpreterAlterQuery::updateMetadata(database_name, table_name, columns, context);
					data.setColumnsList(columns);
M
Merge  
Michael Kolupaev 已提交
1350 1351
					if (unreplicated_data)
						unreplicated_data->setColumnsList(columns);
M
Merge  
Michael Kolupaev 已提交
1352 1353
					columns_version = stat.version;
					LOG_INFO(log, "Applied changes to table.");
M
Merge  
Michael Kolupaev 已提交
1354 1355 1356

					/// Нужно получить список кусков под блокировкой таблицы, чтобы избежать race condition с мерджем.
					parts = data.getDataParts();
M
Merge  
Michael Kolupaev 已提交
1357 1358 1359 1360 1361 1362 1363 1364 1365 1366 1367 1368 1369 1370 1371
				}
				else
				{
					changed = false;
				}
			}

			/// Обновим куски.
			if (changed || force_recheck_parts)
			{
				if (changed)
					LOG_INFO(log, "ALTER-ing parts");

				int changed_parts = 0;

M
Merge  
Michael Kolupaev 已提交
1372 1373 1374 1375
				if (!changed)
					parts = data.getDataParts();

				auto table_lock = lockStructure(false);
M
Merge  
Michael Kolupaev 已提交
1376

M
Merge  
Michael Kolupaev 已提交
1377 1378 1379
				for (const MergeTreeData::DataPartPtr & part : parts)
				{
					/// Обновим кусок и запишем результат во временные файлы.
M
Merge  
Michael Kolupaev 已提交
1380 1381
					/// TODO: Можно пропускать проверку на слишком большие изменения, если в ZooKeeper есть, например,
					///  нода /flags/force_alter.
M
Merge  
Michael Kolupaev 已提交
1382 1383 1384 1385 1386 1387 1388 1389 1390 1391 1392 1393 1394 1395 1396 1397 1398
					auto transaction = data.alterDataPart(part, columns);

					if (!transaction)
						continue;

					++changed_parts;

					/// Обновим метаданные куска в ZooKeeper.
					zkutil::Ops ops;
					ops.push_back(new zkutil::Op::SetData(replica_path + "/parts/" + part->name + "/columns", part->columns.toString(), -1));
					ops.push_back(new zkutil::Op::SetData(replica_path + "/parts/" + part->name + "/checksums", part->checksums.toString(), -1));
					zookeeper->multi(ops);

					/// Применим изменения файлов.
					transaction->commit();
				}

M
Merge  
Michael Kolupaev 已提交
1399 1400 1401 1402 1403 1404 1405 1406 1407 1408 1409 1410 1411 1412 1413 1414 1415
				/// То же самое для нереплицируемых данных.
				if (unreplicated_data)
				{
					parts = unreplicated_data->getDataParts();

					for (const MergeTreeData::DataPartPtr & part : parts)
					{
						auto transaction = unreplicated_data->alterDataPart(part, columns);

						if (!transaction)
							continue;

						++changed_parts;

						transaction->commit();
					}
				}
M
Merge  
Michael Kolupaev 已提交
1416 1417 1418

				zookeeper->set(replica_path + "/columns", columns.toString());

M
Merge  
Michael Kolupaev 已提交
1419 1420 1421 1422 1423 1424 1425 1426 1427 1428 1429 1430 1431 1432 1433 1434 1435 1436 1437 1438
				if (changed || changed_parts != 0)
					LOG_INFO(log, "ALTER-ed " << changed_parts << " parts");
				force_recheck_parts = false;
			}

			alter_thread_event->wait();
		}
		catch (...)
		{
			tryLogCurrentException(__PRETTY_FUNCTION__);

			force_recheck_parts = true;

			alter_thread_event->tryWait(ERROR_SLEEP_MS);
		}
	}

	LOG_DEBUG(log, "alter thread finished");
}

M
Merge  
Michael Kolupaev 已提交
1439 1440 1441 1442
void StorageReplicatedMergeTree::removePartAndEnqueueFetch(const String & part_name)
{
	String part_path = replica_path + "/parts/" + part_name;

M
Merge  
Michael Kolupaev 已提交
1443 1444 1445 1446
	LogEntryPtr log_entry = new LogEntry;
	log_entry->type = LogEntry::GET_PART;
	log_entry->source_replica = "";
	log_entry->new_part_name = part_name;
M
Merge  
Michael Kolupaev 已提交
1447 1448 1449

	zkutil::Ops ops;
	ops.push_back(new zkutil::Op::Create(
M
Merge  
Michael Kolupaev 已提交
1450
		replica_path + "/queue/queue-", log_entry->toString(), zookeeper->getDefaultACL(),
M
Merge  
Michael Kolupaev 已提交
1451 1452 1453 1454 1455 1456 1457
		zkutil::CreateMode::PersistentSequential));
	ops.push_back(new zkutil::Op::Remove(part_path + "/checksums", -1));
	ops.push_back(new zkutil::Op::Remove(part_path + "/columns", -1));
	ops.push_back(new zkutil::Op::Remove(part_path, -1));
	auto results = zookeeper->multi(ops);

	{
M
Merge  
Michael Kolupaev 已提交
1458
		std::unique_lock<std::mutex> lock(queue_mutex);
M
Merge  
Michael Kolupaev 已提交
1459 1460

		String path_created = dynamic_cast<zkutil::Op::Create &>(ops[0]).getPathCreated();
M
Merge  
Michael Kolupaev 已提交
1461 1462
		log_entry->znode_name = path_created.substr(path_created.find_last_of('/') + 1);
		log_entry->addResultToVirtualParts(*this);
M
Merge  
Michael Kolupaev 已提交
1463 1464 1465 1466 1467 1468 1469 1470 1471 1472 1473 1474 1475 1476 1477
		queue.push_back(log_entry);
	}
}

void StorageReplicatedMergeTree::enqueuePartForCheck(const String & name)
{
	Poco::ScopedLock<Poco::FastMutex> lock(parts_to_check_mutex);

	if (parts_to_check_set.count(name))
		return;
	parts_to_check_queue.push_back(name);
	parts_to_check_set.insert(name);
	parts_to_check_event.set();
}

M
Merge  
Michael Kolupaev 已提交
1478 1479 1480 1481 1482 1483 1484 1485 1486 1487 1488 1489 1490 1491 1492 1493 1494 1495 1496 1497 1498 1499 1500 1501 1502 1503 1504 1505 1506 1507
void StorageReplicatedMergeTree::partCheckThread()
{
	while (!shutdown_called)
	{
		try
		{
			/// Достанем из очереди кусок для проверки.
			String part_name;
			{
				Poco::ScopedLock<Poco::FastMutex> lock(parts_to_check_mutex);
				if (parts_to_check_queue.empty())
				{
					if (!parts_to_check_set.empty())
					{
						LOG_ERROR(log, "Non-empty parts_to_check_set with empty parts_to_check_queue. This is a bug.");
						parts_to_check_set.clear();
					}
				}
				else
				{
					part_name = parts_to_check_queue.front();
				}
			}
			if (part_name.empty())
			{
				parts_to_check_event.wait();
				continue;
			}

			LOG_WARNING(log, "Checking part " << part_name);
M
Merge  
Michael Kolupaev 已提交
1508
			ProfileEvents::increment(ProfileEvents::ReplicatedPartChecks);
M
Merge  
Michael Kolupaev 已提交
1509

M
Merge  
Michael Kolupaev 已提交
1510
			auto part = data.getActiveContainingPart(part_name);
M
Merge  
Michael Kolupaev 已提交
1511 1512 1513 1514 1515 1516 1517 1518 1519 1520
			String part_path = replica_path + "/parts/" + part_name;

			/// Этого или покрывающего куска у нас нет.
			if (!part)
			{
				/// Если кусок есть в ZooKeeper, удалим его оттуда и добавим в очередь задание скачать его.
				if (zookeeper->exists(part_path))
				{
					LOG_WARNING(log, "Part " << part_name << " exists in ZooKeeper but not locally. "
						"Removing from ZooKeeper and queueing a fetch.");
M
Merge  
Michael Kolupaev 已提交
1521
					ProfileEvents::increment(ProfileEvents::ReplicatedPartChecksFailed);
M
Merge  
Michael Kolupaev 已提交
1522

M
Merge  
Michael Kolupaev 已提交
1523
					removePartAndEnqueueFetch(part_name);
M
Merge  
Michael Kolupaev 已提交
1524 1525 1526 1527
				}
				/// Если куска нет в ZooKeeper, проверим есть ли он хоть у кого-то.
				else
				{
M
Merge  
Michael Kolupaev 已提交
1528 1529
					ActiveDataPartSet::Part part_info;
					ActiveDataPartSet::parsePartName(part_name, part_info);
M
Merge  
Michael Kolupaev 已提交
1530

M
Merge  
Michael Kolupaev 已提交
1531
					/** Будем проверять только куски, не полученные в результате слияния.
M
Merge  
Michael Kolupaev 已提交
1532
					  * Для кусков, полученных в результате слияния, такая проверка была бы некорректной,
M
Merge  
Michael Kolupaev 已提交
1533 1534
					  *  потому что слитого куска может еще ни у кого не быть.
					  */
M
Merge  
Michael Kolupaev 已提交
1535
					if (part_info.left == part_info.right)
M
Merge  
Michael Kolupaev 已提交
1536
					{
M
Merge  
Michael Kolupaev 已提交
1537 1538 1539 1540 1541
						LOG_WARNING(log, "Checking if anyone has part covering " << part_name << ".");

						bool found = false;
						Strings replicas = zookeeper->getChildren(zookeeper_path + "/replicas");
						for (const String & replica : replicas)
M
Merge  
Michael Kolupaev 已提交
1542
						{
M
Merge  
Michael Kolupaev 已提交
1543 1544
							Strings parts = zookeeper->getChildren(zookeeper_path + "/replicas/" + replica + "/parts");
							for (const String & part_on_replica : parts)
M
Merge  
Michael Kolupaev 已提交
1545
							{
M
Merge  
Michael Kolupaev 已提交
1546 1547 1548 1549 1550 1551
								if (part_on_replica == part_name || ActiveDataPartSet::contains(part_on_replica, part_name))
								{
									found = true;
									LOG_WARNING(log, "Found part " << part_on_replica << " on " << replica);
									break;
								}
M
Merge  
Michael Kolupaev 已提交
1552
							}
M
Merge  
Michael Kolupaev 已提交
1553 1554
							if (found)
								break;
M
Merge  
Michael Kolupaev 已提交
1555 1556
						}

M
Merge  
Michael Kolupaev 已提交
1557
						if (!found)
M
Merge  
Michael Kolupaev 已提交
1558
						{
M
Merge  
Michael Kolupaev 已提交
1559
							LOG_ERROR(log, "No replica has part covering " << part_name);
M
Merge  
Michael Kolupaev 已提交
1560 1561
							ProfileEvents::increment(ProfileEvents::ReplicatedPartChecksFailed);

M
Merge  
Michael Kolupaev 已提交
1562
							/// Если ни у кого нет такого куска, удалим его из нашей очереди.
M
Merge  
Michael Kolupaev 已提交
1563

M
Merge  
Michael Kolupaev 已提交
1564 1565
							bool was_in_queue = false;

M
Merge  
Michael Kolupaev 已提交
1566
							{
M
Merge  
Michael Kolupaev 已提交
1567
								std::unique_lock<std::mutex> lock(queue_mutex);
M
Merge  
Michael Kolupaev 已提交
1568 1569

								for (LogEntries::iterator it = queue.begin(); it != queue.end(); )
M
Merge  
Michael Kolupaev 已提交
1570
								{
M
Merge  
Michael Kolupaev 已提交
1571
									if ((*it)->new_part_name == part_name)
M
Merge  
Michael Kolupaev 已提交
1572
									{
M
Merge  
Michael Kolupaev 已提交
1573
										zookeeper->tryRemove(replica_path + "/queue/" + (*it)->znode_name);
M
Merge  
Michael Kolupaev 已提交
1574
										queue.erase(it++);
M
Merge  
Michael Kolupaev 已提交
1575
										was_in_queue = true;
M
Merge  
Michael Kolupaev 已提交
1576 1577 1578 1579 1580
									}
									else
									{
										++it;
									}
M
Merge  
Michael Kolupaev 已提交
1581 1582
								}
							}
M
Merge  
Michael Kolupaev 已提交
1583 1584

							if (was_in_queue)
M
Merge  
Michael Kolupaev 已提交
1585
							{
M
Merge  
Michael Kolupaev 已提交
1586 1587 1588 1589
								/** Такая ситуация возможна, если на всех репликах, где был кусок, он испортился.
								  * Например, у реплики, которая только что его записала, отключили питание, и данные не записались из кеша на диск.
								  */
								LOG_ERROR(log, "Part " << part_name << " is lost forever. Say goodbye to a piece of data!");
M
Merge  
Michael Kolupaev 已提交
1590 1591 1592 1593 1594 1595 1596 1597 1598 1599 1600 1601 1602

								/** Нужно добавить отсутствующий кусок в block_numbers, чтобы он не мешал слияниям.
								  * Вот только в сам block_numbers мы его добавить не можем - если так сделать,
								  *  ZooKeeper зачем-то пропустит один номер для автоинкремента,
								  *  и в номерах блоков все равно останется дырка.
								  * Специально из-за этого приходится отдельно иметь nonincrement_block_numbers.
								  */
								zookeeper->createIfNotExists(zookeeper_path + "/nonincrement_block_numbers", "");
								zookeeper->createIfNotExists(zookeeper_path + "/nonincrement_block_numbers/" + part_name.substr(0, 6), "");
								AbandonableLockInZooKeeper::createAbandonedIfNotExists(
									zookeeper_path + "/nonincrement_block_numbers/" + part_name.substr(0, 6) + "/block-" + padIndex(part_info.left),
									*zookeeper);
							}
M
Merge  
Michael Kolupaev 已提交
1603 1604
						}
					}
M
Merge  
Michael Kolupaev 已提交
1605 1606 1607 1608 1609
				}
			}
			/// У нас есть этот кусок, и он активен.
			else if (part->name == part_name)
			{
M
Merge  
Michael Kolupaev 已提交
1610 1611
				auto table_lock = lockStructure(false);

M
Merge  
Michael Kolupaev 已提交
1612 1613 1614
				/// Если кусок есть в ZooKeeper, сверим его данные с его чексуммами, а их с ZooKeeper.
				if (zookeeper->exists(replica_path + "/parts/" + part_name))
				{
M
Merge  
Michael Kolupaev 已提交
1615 1616 1617 1618 1619 1620 1621 1622 1623 1624 1625 1626 1627 1628 1629
					LOG_WARNING(log, "Checking data of part " << part_name << ".");

					try
					{
						auto zk_checksums = MergeTreeData::DataPart::Checksums::parse(
							zookeeper->get(replica_path + "/parts/" + part_name + "/checksums"));
						zk_checksums.checkEqual(part->checksums, true);

						auto zk_columns = NamesAndTypesList::parse(
							zookeeper->get(replica_path + "/parts/" + part_name + "/columns"), context.getDataTypeFactory());
						if (part->columns != zk_columns)
							throw Exception("Columns of local part " + part_name + " are different from ZooKeeper");

						MergeTreePartChecker::checkDataPart(
							data.getFullPath() + part_name, data.index_granularity, true, context.getDataTypeFactory());
M
Merge  
Michael Kolupaev 已提交
1630

M
Merge  
Michael Kolupaev 已提交
1631 1632 1633 1634 1635 1636
						LOG_INFO(log, "Part " << part_name << " looks good.");
					}
					catch (...)
					{
						tryLogCurrentException(__PRETTY_FUNCTION__);

M
Merge  
Michael Kolupaev 已提交
1637 1638
						LOG_ERROR(log, "Part " << part_name << " looks broken. Removing it and queueing a fetch.");
						ProfileEvents::increment(ProfileEvents::ReplicatedPartChecksFailed);
M
Merge  
Michael Kolupaev 已提交
1639 1640 1641 1642

						removePartAndEnqueueFetch(part_name);

						/// Удалим кусок локально.
M
Merge  
Michael Kolupaev 已提交
1643
						data.renameAndDetachPart(part, "broken_");
M
Merge  
Michael Kolupaev 已提交
1644
					}
M
Merge  
Michael Kolupaev 已提交
1645 1646
				}
				/// Если куска нет в ZooKeeper, удалим его локально.
M
Merge  
Michael Kolupaev 已提交
1647 1648 1649
				/// Возможно, кусок кто-то только что записал, и еще не успел добавить в ZK.
				/// Поэтому удаляем только если кусок старый (не очень надежно).
				else if (part->modification_time + 5 * 60 < time(0))
M
Merge  
Michael Kolupaev 已提交
1650
				{
M
Merge  
Michael Kolupaev 已提交
1651 1652
					ProfileEvents::increment(ProfileEvents::ReplicatedPartChecksFailed);

M
Merge  
Michael Kolupaev 已提交
1653 1654
					LOG_ERROR(log, "Unexpected part " << part_name << ". Removing.");
					data.renameAndDetachPart(part, "unexpected_");
M
Merge  
Michael Kolupaev 已提交
1655 1656 1657 1658 1659 1660 1661 1662 1663 1664 1665 1666 1667 1668 1669 1670 1671 1672 1673 1674 1675 1676 1677 1678 1679 1680 1681 1682 1683 1684 1685
				}
			}
			else
			{
				/// Если у нас есть покрывающий кусок, игнорируем все проблемы с этим куском.
				/// В худшем случае в лог еще old_parts_lifetime секунд будут валиться ошибки, пока кусок не удалится как старый.
			}

			/// Удалим кусок из очереди.
			{
				Poco::ScopedLock<Poco::FastMutex> lock(parts_to_check_mutex);
				if (parts_to_check_queue.empty() || parts_to_check_queue.front() != part_name)
				{
					LOG_ERROR(log, "Someone changed parts_to_check_queue.front(). This is a bug.");
				}
				else
				{
					parts_to_check_queue.pop_front();
					parts_to_check_set.erase(part_name);
				}
			}
		}
		catch (...)
		{
			tryLogCurrentException(__PRETTY_FUNCTION__);
			parts_to_check_event.tryWait(ERROR_SLEEP_MS);
		}
	}
}


M
Merge  
Michael Kolupaev 已提交
1686 1687
bool StorageReplicatedMergeTree::canMergeParts(const MergeTreeData::DataPartPtr & left, const MergeTreeData::DataPartPtr & right)
{
M
Merge  
Michael Kolupaev 已提交
1688 1689 1690
	/// Если какой-то из кусков уже собираются слить в больший, не соглашаемся его сливать.
	if (virtual_parts.getContainingPart(left->name) != left->name ||
		virtual_parts.getContainingPart(right->name) != right->name)
M
Merge  
Michael Kolupaev 已提交
1691 1692
		return false;

M
Merge  
Michael Kolupaev 已提交
1693 1694
	String month_name = left->name.substr(0, 6);

M
Merge  
Michael Kolupaev 已提交
1695 1696 1697
	/// Можно слить куски, если все номера между ними заброшены - не соответствуют никаким блокам.
	for (UInt64 number = left->right + 1; number <= right->left - 1; ++number)
	{
M
Merge  
Michael Kolupaev 已提交
1698 1699
		String path1 = zookeeper_path +              "/block_numbers/" + month_name + "/block-" + padIndex(number);
		String path2 = zookeeper_path + "/nonincrement_block_numbers/" + month_name + "/block-" + padIndex(number);
M
Merge  
Michael Kolupaev 已提交
1700

M
Merge  
Michael Kolupaev 已提交
1701 1702
		if (AbandonableLockInZooKeeper::check(path1, *zookeeper) != AbandonableLockInZooKeeper::ABANDONED &&
			AbandonableLockInZooKeeper::check(path2, *zookeeper) != AbandonableLockInZooKeeper::ABANDONED)
M
Merge  
Michael Kolupaev 已提交
1703 1704 1705 1706 1707 1708 1709 1710
			return false;
	}

	return true;
}

void StorageReplicatedMergeTree::becomeLeader()
{
M
Merge  
Michael Kolupaev 已提交
1711
	LOG_INFO(log, "Became leader");
M
Merge  
Michael Kolupaev 已提交
1712 1713 1714 1715
	is_leader_node = true;
	merge_selecting_thread = std::thread(&StorageReplicatedMergeTree::mergeSelectingThread, this);
}

M
Merge  
Michael Kolupaev 已提交
1716
String StorageReplicatedMergeTree::findReplicaHavingPart(const String & part_name, bool active)
M
Merge  
Michael Kolupaev 已提交
1717
{
M
Merge  
Michael Kolupaev 已提交
1718
	Strings replicas = zookeeper->getChildren(zookeeper_path + "/replicas");
M
Merge  
Michael Kolupaev 已提交
1719 1720 1721 1722 1723 1724

	/// Из реплик, у которых есть кусок, выберем одну равновероятно.
	std::random_shuffle(replicas.begin(), replicas.end());

	for (const String & replica : replicas)
	{
M
Merge  
Michael Kolupaev 已提交
1725 1726
		if (zookeeper->exists(zookeeper_path + "/replicas/" + replica + "/parts/" + part_name) &&
			(!active || zookeeper->exists(zookeeper_path + "/replicas/" + replica + "/is_active")))
M
Merge  
Michael Kolupaev 已提交
1727 1728 1729
			return replica;
	}

M
Merge  
Michael Kolupaev 已提交
1730
	return "";
M
Merge  
Michael Kolupaev 已提交
1731 1732 1733 1734
}

void StorageReplicatedMergeTree::fetchPart(const String & part_name, const String & replica_name)
{
M
Merge  
Michael Kolupaev 已提交
1735 1736
	LOG_DEBUG(log, "Fetching part " << part_name << " from " << replica_name);

M
Merge  
Michael Kolupaev 已提交
1737 1738 1739 1740 1741
	auto table_lock = lockStructure(true);

	String host;
	int port;

M
Merge  
Michael Kolupaev 已提交
1742
	String host_port_str = zookeeper->get(zookeeper_path + "/replicas/" + replica_name + "/host");
M
Merge  
Michael Kolupaev 已提交
1743 1744 1745 1746 1747 1748 1749 1750
	ReadBufferFromString buf(host_port_str);
	assertString("host: ", buf);
	readString(host, buf);
	assertString("\nport: ", buf);
	readText(port, buf);
	assertString("\n", buf);
	assertEOF(buf);

M
Merge  
Michael Kolupaev 已提交
1751
	MergeTreeData::MutableDataPartPtr part = fetcher.fetchPart(part_name, zookeeper_path + "/replicas/" + replica_name, host, port);
M
Merge  
Michael Kolupaev 已提交
1752 1753 1754

	MergeTreeData::Transaction transaction;
	auto removed_parts = data.renameTempPartAndReplace(part, nullptr, &transaction);
M
Merge  
Michael Kolupaev 已提交
1755

M
Merge  
Michael Kolupaev 已提交
1756
	zkutil::Ops ops;
M
Merge  
Michael Kolupaev 已提交
1757
	checkPartAndAddToZooKeeper(part, ops);
M
Michael Kolupaev 已提交
1758

M
Merge  
Michael Kolupaev 已提交
1759
	zookeeper->multi(ops);
M
Merge  
Michael Kolupaev 已提交
1760
	transaction.commit();
M
Merge  
Michael Kolupaev 已提交
1761
	merge_selecting_event.set();
M
Merge  
Michael Kolupaev 已提交
1762

M
Michael Kolupaev 已提交
1763 1764 1765 1766 1767 1768
	for (const auto & removed_part : removed_parts)
	{
		LOG_DEBUG(log, "Part " << removed_part->name << " is rendered obsolete by fetching part " << part_name);
		ProfileEvents::increment(ProfileEvents::ObsoleteReplicatedParts);
	}

M
Merge  
Michael Kolupaev 已提交
1769 1770
	ProfileEvents::increment(ProfileEvents::ReplicatedPartFetches);

M
Merge  
Michael Kolupaev 已提交
1771
	LOG_DEBUG(log, "Fetched part " << part_name << " from " << replica_name);
M
Merge  
Michael Kolupaev 已提交
1772
}
M
Merge  
Michael Kolupaev 已提交
1773

M
Merge  
Michael Kolupaev 已提交
1774 1775
void StorageReplicatedMergeTree::shutdown()
{
M
Merge  
Michael Kolupaev 已提交
1776
	if (permanent_shutdown_called)
M
Merge  
Michael Kolupaev 已提交
1777 1778 1779
	{
		if (restarting_thread.joinable())
			restarting_thread.join();
M
Merge  
Michael Kolupaev 已提交
1780
		return;
M
Merge  
Michael Kolupaev 已提交
1781
	}
M
Merge  
Michael Kolupaev 已提交
1782

M
Merge  
Michael Kolupaev 已提交
1783
	permanent_shutdown_called = true;
M
Merge  
Michael Kolupaev 已提交
1784
	permanent_shutdown_event.set();
M
Merge  
Michael Kolupaev 已提交
1785
	restarting_thread.join();
M
Merge  
Michael Kolupaev 已提交
1786 1787

	endpoint_holder = nullptr;
M
Merge  
Michael Kolupaev 已提交
1788 1789 1790 1791
}

void StorageReplicatedMergeTree::partialShutdown()
{
M
Merge  
Michael Kolupaev 已提交
1792
	leader_election = nullptr;
M
Merge  
Michael Kolupaev 已提交
1793
	shutdown_called = true;
M
Merge  
Michael Kolupaev 已提交
1794
	shutdown_event.set();
M
Merge  
Michael Kolupaev 已提交
1795 1796 1797
	merge_selecting_event.set();
	queue_updating_event->set();
	alter_thread_event->set();
M
Merge  
Michael Kolupaev 已提交
1798
	alter_query_event->set();
M
Merge  
Michael Kolupaev 已提交
1799
	parts_to_check_event.set();
M
Merge  
Michael Kolupaev 已提交
1800 1801
	replica_is_active_node = nullptr;

M
Merge  
Michael Kolupaev 已提交
1802 1803 1804 1805
	merger.cancelAll();
	if (unreplicated_merger)
		unreplicated_merger->cancelAll();

M
Merge  
Michael Kolupaev 已提交
1806
	LOG_TRACE(log, "Waiting for threads to finish");
M
Merge  
Michael Kolupaev 已提交
1807
	if (is_leader_node)
M
Merge  
Michael Kolupaev 已提交
1808
	{
M
Merge  
Michael Kolupaev 已提交
1809
		is_leader_node = false;
M
Merge  
Michael Kolupaev 已提交
1810 1811
		if (merge_selecting_thread.joinable())
			merge_selecting_thread.join();
M
Merge  
Michael Kolupaev 已提交
1812
	}
M
Merge  
Michael Kolupaev 已提交
1813 1814
	if (queue_updating_thread.joinable())
		queue_updating_thread.join();
M
Merge  
Michael Kolupaev 已提交
1815 1816 1817 1818
	if (cleanup_thread.joinable())
		cleanup_thread.join();
	if (alter_thread.joinable())
		alter_thread.join();
M
Merge  
Michael Kolupaev 已提交
1819 1820
	if (part_check_thread.joinable())
		part_check_thread.join();
M
Merge  
Michael Kolupaev 已提交
1821 1822
	if (queue_task_handle)
		context.getBackgroundPool().removeTask(queue_task_handle);
M
Merge  
Michael Kolupaev 已提交
1823
	queue_task_handle.reset();
M
Merge  
Michael Kolupaev 已提交
1824
	LOG_TRACE(log, "Threads finished");
M
Merge  
Michael Kolupaev 已提交
1825 1826
}

M
Merge  
Michael Kolupaev 已提交
1827 1828 1829 1830 1831 1832
void StorageReplicatedMergeTree::goReadOnly()
{
	LOG_INFO(log, "Going to read-only mode");

	is_read_only = true;
	permanent_shutdown_called = true;
M
Merge  
Michael Kolupaev 已提交
1833
	permanent_shutdown_event.set();
M
Merge  
Michael Kolupaev 已提交
1834

M
Merge  
Michael Kolupaev 已提交
1835
	partialShutdown();
M
Merge  
Michael Kolupaev 已提交
1836 1837
}

M
Merge  
Michael Kolupaev 已提交
1838 1839 1840
void StorageReplicatedMergeTree::startup()
{
	shutdown_called = false;
M
Merge  
Michael Kolupaev 已提交
1841
	shutdown_event.reset();
M
Merge  
Michael Kolupaev 已提交
1842

M
Merge  
Michael Kolupaev 已提交
1843 1844 1845 1846
	merger.uncancelAll();
	if (unreplicated_merger)
		unreplicated_merger->uncancelAll();

M
Merge  
Michael Kolupaev 已提交
1847 1848
	activateReplica();

M
Merge  
Michael Kolupaev 已提交
1849
	leader_election = new zkutil::LeaderElection(zookeeper_path + "/leader_election", *zookeeper,
M
Merge  
Michael Kolupaev 已提交
1850 1851 1852
		std::bind(&StorageReplicatedMergeTree::becomeLeader, this), replica_name);

	queue_updating_thread = std::thread(&StorageReplicatedMergeTree::queueUpdatingThread, this);
M
Merge  
Michael Kolupaev 已提交
1853
	cleanup_thread = std::thread(&StorageReplicatedMergeTree::cleanupThread, this);
M
Merge  
Michael Kolupaev 已提交
1854
	alter_thread = std::thread(&StorageReplicatedMergeTree::alterThread, this);
M
Merge  
Michael Kolupaev 已提交
1855
	part_check_thread = std::thread(&StorageReplicatedMergeTree::partCheckThread, this);
M
Merge  
Michael Kolupaev 已提交
1856 1857
	queue_task_handle = context.getBackgroundPool().addTask(
		std::bind(&StorageReplicatedMergeTree::queueTask, this, std::placeholders::_1));
M
Merge  
Michael Kolupaev 已提交
1858 1859
}

M
Merge  
Michael Kolupaev 已提交
1860 1861
void StorageReplicatedMergeTree::restartingThread()
{
M
Merge  
Michael Kolupaev 已提交
1862
	try
M
Merge  
Michael Kolupaev 已提交
1863
	{
M
Merge  
Michael Kolupaev 已提交
1864 1865 1866
		startup();

		while (!permanent_shutdown_called)
M
Merge  
Michael Kolupaev 已提交
1867
		{
M
Merge  
Michael Kolupaev 已提交
1868 1869 1870
			if (zookeeper->expired())
			{
				LOG_WARNING(log, "ZooKeeper session has expired. Switching to a new session.");
M
Merge  
Michael Kolupaev 已提交
1871

M
Merge  
Michael Kolupaev 已提交
1872
				/// Запретим писать в таблицу, пока подменяем zookeeper.
M
Merge  
Michael Kolupaev 已提交
1873
				LOG_TRACE(log, "Locking INSERTs");
M
Merge  
Michael Kolupaev 已提交
1874
				auto structure_lock = lockDataForAlter();
M
Merge  
Michael Kolupaev 已提交
1875
				LOG_TRACE(log, "Locked INSERTs");
M
Merge  
Michael Kolupaev 已提交
1876

M
Merge  
Michael Kolupaev 已提交
1877
				partialShutdown();
M
Merge  
Michael Kolupaev 已提交
1878

M
Merge  
Michael Kolupaev 已提交
1879 1880 1881 1882
				zookeeper = context.getZooKeeper();

				startup();
			}
M
Merge  
Michael Kolupaev 已提交
1883

M
Merge  
Michael Kolupaev 已提交
1884
			permanent_shutdown_event.tryWait(60 * 1000);
M
Merge  
Michael Kolupaev 已提交
1885 1886 1887 1888 1889
		}
	}
	catch (...)
	{
		tryLogCurrentException("StorageReplicatedMergeTree::restartingThread");
M
Merge  
Michael Kolupaev 已提交
1890 1891
		LOG_ERROR(log, "Exception in restartingThread. The storage will be read-only until server restart.");
		goReadOnly();
M
Merge  
Michael Kolupaev 已提交
1892
		LOG_DEBUG(log, "restarting thread finished");
M
Merge  
Michael Kolupaev 已提交
1893
		return;
M
Merge  
Michael Kolupaev 已提交
1894 1895
	}

M
Merge  
Michael Kolupaev 已提交
1896 1897 1898 1899 1900 1901 1902 1903 1904
	try
	{
		endpoint_holder = nullptr;
		partialShutdown();
	}
	catch (...)
	{
		tryLogCurrentException("StorageReplicatedMergeTree::restartingThread");
	}
M
Merge  
Michael Kolupaev 已提交
1905 1906

	LOG_DEBUG(log, "restarting thread finished");
M
Merge  
Michael Kolupaev 已提交
1907 1908
}

M
Merge  
Michael Kolupaev 已提交
1909 1910 1911 1912 1913 1914 1915 1916 1917 1918 1919 1920 1921 1922 1923 1924 1925 1926 1927 1928
StorageReplicatedMergeTree::~StorageReplicatedMergeTree()
{
	try
	{
		shutdown();
	}
	catch(...)
	{
		tryLogCurrentException("~StorageReplicatedMergeTree");
	}
}

BlockInputStreams StorageReplicatedMergeTree::read(
		const Names & column_names,
		ASTPtr query,
		const Settings & settings,
		QueryProcessingStage::Enum & processed_stage,
		size_t max_block_size,
		unsigned threads)
{
M
Merge  
Michael Kolupaev 已提交
1929 1930 1931 1932 1933 1934 1935 1936 1937 1938 1939 1940 1941 1942 1943 1944 1945
	Names virt_column_names;
	Names real_column_names;
	for (const auto & it : column_names)
		if (it == "_replicated")
			virt_column_names.push_back(it);
		else
			real_column_names.push_back(it);

	Block virtual_columns_block;
	ColumnUInt8 * column = new ColumnUInt8(2);
	ColumnPtr column_ptr = column;
	column->getData()[0] = 0;
	column->getData()[1] = 1;
	virtual_columns_block.insert(ColumnWithNameAndType(column_ptr, new DataTypeUInt8, "_replicated"));

	/// Если запрошен хотя бы один виртуальный столбец, пробуем индексировать
	if (!virt_column_names.empty())
1946
		VirtualColumnUtils::filterBlockWithQuery(query->clone(), virtual_columns_block, context);
M
Merge  
Michael Kolupaev 已提交
1947

1948
	std::multiset<UInt8> values = VirtualColumnUtils::extractSingleValueFromBlock<UInt8>(virtual_columns_block, "_replicated");
M
Merge  
Michael Kolupaev 已提交
1949

M
Merge  
Michael Kolupaev 已提交
1950 1951
	BlockInputStreams res;

M
Merge  
Michael Kolupaev 已提交
1952 1953 1954
	size_t part_index = 0;

	if (unreplicated_reader && values.count(0))
M
Merge  
Michael Kolupaev 已提交
1955
	{
M
Merge  
Michael Kolupaev 已提交
1956 1957
		res = unreplicated_reader->read(
			real_column_names, query, settings, processed_stage, max_block_size, threads, &part_index);
M
Merge  
Michael Kolupaev 已提交
1958 1959 1960 1961 1962 1963

		for (auto & virtual_column : virt_column_names)
		{
			if (virtual_column == "_replicated")
			{
				for (auto & stream : res)
M
Merge  
Michael Kolupaev 已提交
1964
					stream = new AddingConstColumnBlockInputStream<UInt8>(stream, new DataTypeUInt8, 0, "_replicated");
M
Merge  
Michael Kolupaev 已提交
1965 1966 1967 1968
			}
		}
	}

M
Merge  
Michael Kolupaev 已提交
1969
	if (values.count(1))
M
Merge  
Michael Kolupaev 已提交
1970
	{
M
Merge  
Michael Kolupaev 已提交
1971
		auto res2 = reader.read(real_column_names, query, settings, processed_stage, max_block_size, threads, &part_index);
M
Merge  
Michael Kolupaev 已提交
1972 1973 1974 1975 1976 1977

		for (auto & virtual_column : virt_column_names)
		{
			if (virtual_column == "_replicated")
			{
				for (auto & stream : res2)
M
Merge  
Michael Kolupaev 已提交
1978
					stream = new AddingConstColumnBlockInputStream<UInt8>(stream, new DataTypeUInt8, 1, "_replicated");
M
Merge  
Michael Kolupaev 已提交
1979 1980 1981
			}
		}

M
Merge  
Michael Kolupaev 已提交
1982
		res.insert(res.end(), res2.begin(), res2.end());
M
Merge  
Michael Kolupaev 已提交
1983 1984 1985
	}

	return res;
M
Merge  
Michael Kolupaev 已提交
1986 1987
}

M
Merge  
Michael Kolupaev 已提交
1988 1989
BlockOutputStreamPtr StorageReplicatedMergeTree::write(ASTPtr query)
{
M
Merge  
Michael Kolupaev 已提交
1990 1991 1992
	if (is_read_only)
		throw Exception("Table is in read only mode", ErrorCodes::TABLE_IS_READ_ONLY);

M
Merge  
Michael Kolupaev 已提交
1993
	String insert_id;
M
Merge  
Michael Kolupaev 已提交
1994 1995 1996
	if (query)
		if (ASTInsertQuery * insert = typeid_cast<ASTInsertQuery *>(&*query))
			insert_id = insert->insert_id;
M
Merge  
Michael Kolupaev 已提交
1997 1998 1999

	return new ReplicatedMergeTreeBlockOutputStream(*this, insert_id);
}
M
Merge  
Michael Kolupaev 已提交
2000

M
Merge  
Michael Kolupaev 已提交
2001 2002
bool StorageReplicatedMergeTree::optimize()
{
2003 2004
	/// Померджим какие-нибудь куски из директории unreplicated.
	/// TODO: Мерджить реплицируемые куски тоже.
M
Merge  
Michael Kolupaev 已提交
2005 2006 2007 2008

	if (!unreplicated_data)
		return false;

M
Merge  
Michael Kolupaev 已提交
2009 2010
	Poco::ScopedLock<Poco::FastMutex> lock(unreplicated_mutex);

M
Merge  
Michael Kolupaev 已提交
2011 2012 2013 2014 2015 2016 2017 2018 2019 2020 2021 2022
	unreplicated_data->clearOldParts();

	MergeTreeData::DataPartsVector parts;
	String merged_name;
	auto always_can_merge = [](const MergeTreeData::DataPartPtr &a, const MergeTreeData::DataPartPtr &b) { return true; };
	if (!unreplicated_merger->selectPartsToMerge(parts, merged_name, 0, true, true, false, always_can_merge))
		return false;

	unreplicated_merger->mergeParts(parts, merged_name);
	return true;
}

M
Merge  
Michael Kolupaev 已提交
2023 2024
void StorageReplicatedMergeTree::alter(const AlterCommands & params,
	const String & database_name, const String & table_name, Context & context)
M
Merge  
Michael Kolupaev 已提交
2025 2026 2027 2028 2029 2030 2031 2032 2033 2034 2035 2036 2037 2038 2039 2040 2041 2042 2043 2044 2045 2046 2047 2048 2049 2050 2051 2052 2053 2054 2055 2056 2057 2058 2059 2060 2061 2062 2063 2064 2065 2066 2067 2068 2069 2070 2071 2072 2073 2074 2075 2076 2077 2078 2079 2080 2081 2082 2083 2084 2085 2086 2087 2088 2089 2090 2091 2092
{
	LOG_DEBUG(log, "Doing ALTER");

	NamesAndTypesList new_columns;
	String new_columns_str;
	int new_columns_version;
	zkutil::Stat stat;

	{
		auto table_lock = lockStructureForAlter();

		data.checkAlter(params);

		new_columns = data.getColumnsList();
		params.apply(new_columns);

		new_columns_str = new_columns.toString();

		/// Делаем ALTER.
		zookeeper->set(zookeeper_path + "/columns", new_columns_str, -1, &stat);

		new_columns_version = stat.version;
	}

	LOG_DEBUG(log, "Updated columns in ZooKeeper. Waiting for replicas to apply changes.");

	/// Ждем, пока все реплики обновят данные.

	/// Подпишемся на изменения столбцов, чтобы перестать ждать, если кто-то еще сделает ALTER.
	if (!zookeeper->exists(zookeeper_path + "/columns", &stat, alter_query_event))
		throw Exception(zookeeper_path + "/columns doesn't exist", ErrorCodes::NOT_FOUND_NODE);
	if (stat.version != new_columns_version)
	{
		LOG_WARNING(log, zookeeper_path + "/columns changed before this ALTER finished; "
			"overlapping ALTER-s are fine but use caution with nontransitive changes");
		return;
	}

	Strings replicas = zookeeper->getChildren(zookeeper_path + "/replicas");
	for (const String & replica : replicas)
	{
		LOG_DEBUG(log, "Waiting for " << replica << " to apply changes");

		while (!shutdown_called)
		{
			String replica_columns_str;

			/// Реплику могли успеть удалить.
			if (!zookeeper->tryGet(zookeeper_path + "/replicas/" + replica + "/columns", replica_columns_str, &stat))
			{
				LOG_WARNING(log, replica << " was removed");
				break;
			}

			int replica_columns_version = stat.version;

			if (replica_columns_str == new_columns_str)
				break;

			if (!zookeeper->exists(zookeeper_path + "/columns", &stat))
				throw Exception(zookeeper_path + "/columns doesn't exist", ErrorCodes::NOT_FOUND_NODE);
			if (stat.version != new_columns_version)
			{
				LOG_WARNING(log, zookeeper_path + "/columns changed before ALTER finished; "
					"overlapping ALTER-s are fine but use caution with nontransitive changes");
				return;
			}

M
Merge  
Michael Kolupaev 已提交
2093
			if (!zookeeper->exists(zookeeper_path + "/replicas/" + replica + "/columns", &stat, alter_query_event))
M
Merge  
Michael Kolupaev 已提交
2094 2095 2096 2097 2098 2099 2100 2101 2102 2103 2104 2105 2106 2107 2108 2109 2110 2111
			{
				LOG_WARNING(log, replica << " was removed");
				break;
			}

			if (stat.version != replica_columns_version)
				continue;

			alter_query_event->wait();
		}

		if (shutdown_called)
			break;
	}

	LOG_DEBUG(log, "ALTER finished");
}

M
Merge  
Michael Kolupaev 已提交
2112 2113 2114 2115 2116 2117 2118 2119 2120 2121 2122 2123 2124 2125 2126 2127 2128 2129 2130 2131 2132 2133 2134 2135 2136 2137 2138 2139 2140 2141 2142 2143 2144 2145 2146 2147 2148 2149 2150 2151 2152 2153 2154 2155 2156 2157 2158 2159 2160 2161 2162 2163 2164 2165 2166 2167 2168 2169 2170 2171 2172 2173 2174 2175 2176 2177 2178 2179 2180 2181 2182 2183
static bool isValidMonthName(const String & s)
{
	if (s.size() != 6)
		return false;
	if (!std::all_of(s.begin(), s.end(), isdigit))
		return false;
	DayNum_t date = DateLUT::instance().toDayNum(OrderedIdentifier2Date(s + "01"));
	/// Не можем просто сравнить date с нулем, потому что 0 тоже валидный DayNum.
	return s == toString(Date2OrderedIdentifier(DateLUT::instance().fromDayNum(date)) / 100);
}

/// Название воображаемого куска, покрывающего все возможные куски в указанном месяце с номерами в указанном диапазоне.
static String getFakePartNameForDrop(const String & month_name, UInt64 left, UInt64 right)
{
	/// Диапазон дат - весь месяц.
	DateLUT & lut = DateLUT::instance();
	time_t start_time = OrderedIdentifier2Date(month_name + "01");
	DayNum_t left_date = lut.toDayNum(start_time);
	DayNum_t right_date = DayNum_t(static_cast<size_t>(left_date) + lut.daysInMonth(start_time) - 1);

	/// Уровень - right-left+1: кусок не мог образоваться в результате такого или большего количества слияний.
	return ActiveDataPartSet::getPartName(left_date, right_date, left, right, right - left + 1);
}

void StorageReplicatedMergeTree::dropPartition(const Field & field, bool detach)
{
	String month_name;

	if (field.getType() == Field::Types::UInt64)
		month_name = toString(field.get<UInt64>());
	else
		month_name = field.safeGet<String>();

	if (!isValidMonthName(month_name))
		throw Exception("Invalid partition format: " + month_name + ". Partition should consist of 6 digits: YYYYMM",
						ErrorCodes::INVALID_PARTITION_NAME);

	/// TODO: Делать запрос в лидера по TCP.
	if (!is_leader_node)
		throw Exception("DROP PARTITION can only be done on leader replica.", ErrorCodes::NOT_LEADER);


	/** Пропустим один номер в block_numbers для удаляемого месяца, и будем удалять только куски до этого номера.
	  * Это запретит мерджи удаляемых кусков с новыми вставляемыми данными.
	  * Инвариант: в логе не появятся слияния удаляемых кусков с другими кусками.
	  * NOTE: Если понадобится аналогично поддержать запрос DROP PART, для него придется придумать какой-нибудь новый механизм,
	  *        чтобы гарантировать этот инвариант.
	  */
	UInt64 right;

	{
		AbandonableLockInZooKeeper block_number_lock = allocateBlockNumber(month_name);
		right = block_number_lock.getNumber();
		block_number_lock.unlock();
	}

	/// Такого никогда не должно происходить.
	if (right == 0)
		return;
	--right;

	String fake_part_name = getFakePartNameForDrop(month_name, 0, right);

	/** Запретим выбирать для слияния удаляемые куски - сделаем вид, что их всех уже собираются слить в fake_part_name.
	  * Инвариант: после появления в логе записи DROP_RANGE, в логе не появятся слияния удаляемых кусков.
	  */
	{
		std::unique_lock<std::mutex> merge_selecting_lock(merge_selecting_mutex);

		virtual_parts.add(fake_part_name);
	}

M
Merge  
Michael Kolupaev 已提交
2184
	/// Наконец, добившись нужных инвариантов, можно положить запись в лог.
M
Merge  
Michael Kolupaev 已提交
2185 2186 2187 2188 2189 2190
	LogEntry entry;
	entry.type = LogEntry::DROP_RANGE;
	entry.source_replica = replica_name;
	entry.new_part_name = fake_part_name;
	entry.detach = detach;
	String log_znode_path = zookeeper->create(zookeeper_path + "/log/log-", entry.toString(), zkutil::CreateMode::PersistentSequential);
M
Merge  
Michael Kolupaev 已提交
2191
	entry.znode_name = log_znode_path.substr(log_znode_path.find_last_of('/') + 1);
M
Merge  
Michael Kolupaev 已提交
2192 2193 2194 2195 2196

	/// Дождемся, пока все реплики выполнят дроп.
	waitForAllReplicasToProcessLogEntry(log_znode_path, entry);
}

M
Merge  
Michael Kolupaev 已提交
2197 2198 2199 2200 2201
void StorageReplicatedMergeTree::attachPartition(const Field& partition, bool unreplicated, bool part)
{
	throw Exception("Not implemented", ErrorCodes::NOT_IMPLEMENTED);
}

M
Merge  
Michael Kolupaev 已提交
2202 2203
void StorageReplicatedMergeTree::drop()
{
M
Merge  
Michael Kolupaev 已提交
2204 2205 2206
	if (!zookeeper)
		throw Exception("Can't drop replicated table without ZooKeeper", ErrorCodes::NO_ZOOKEEPER);

M
Merge  
Michael Kolupaev 已提交
2207 2208
	shutdown();

M
Merge  
Michael Kolupaev 已提交
2209
	LOG_INFO(log, "Removing replica " << replica_path);
M
Merge  
Michael Kolupaev 已提交
2210
	replica_is_active_node = nullptr;
M
Merge  
Michael Kolupaev 已提交
2211 2212
	zookeeper->tryRemoveRecursive(replica_path);

M
Merge  
Michael Kolupaev 已提交
2213
	/// Проверяем, что zookeeper_path существует: его могла удалить другая реплика после выполнения предыдущей строки.
M
Merge  
Michael Kolupaev 已提交
2214 2215
	Strings replicas;
	if (zookeeper->tryGetChildren(zookeeper_path + "/replicas", replicas) == ZOK && replicas.empty())
M
Merge  
Michael Kolupaev 已提交
2216 2217
	{
		LOG_INFO(log, "Removing table " << zookeeper_path << " (this might take several minutes)");
M
Merge  
Michael Kolupaev 已提交
2218
		zookeeper->tryRemoveRecursive(zookeeper_path);
M
Merge  
Michael Kolupaev 已提交
2219
	}
M
Merge  
Michael Kolupaev 已提交
2220 2221

	data.dropAllData();
M
Merge  
Michael Kolupaev 已提交
2222 2223
}

M
Merge  
Michael Kolupaev 已提交
2224 2225 2226 2227 2228 2229 2230 2231 2232 2233 2234 2235 2236 2237 2238
void StorageReplicatedMergeTree::rename(const String & new_path_to_db, const String & new_database_name, const String & new_table_name)
{
	std::string new_full_path = new_path_to_db + escapeForFileName(new_table_name) + '/';

	data.setPath(new_full_path, true);
	if (unreplicated_data)
		unreplicated_data->setPath(new_full_path + "unreplicated/", false);

	database_name = new_database_name;
	table_name = new_table_name;
	full_path = new_full_path;

	/// TODO: Можно обновить названия логгеров.
}

M
Merge  
Michael Kolupaev 已提交
2239 2240 2241 2242 2243 2244 2245 2246 2247 2248 2249 2250 2251 2252 2253 2254 2255 2256 2257 2258 2259 2260 2261 2262 2263 2264
AbandonableLockInZooKeeper StorageReplicatedMergeTree::allocateBlockNumber(const String & month_name)
{
	String month_path = zookeeper_path + "/block_numbers/" + month_name;
	if (!zookeeper->exists(month_path))
	{
		/// Создадим в block_numbers ноду для месяца и пропустим в ней 200 значений инкремента.
		/// Нужно, чтобы в будущем при необходимости можно было добавить данные в начало.
		zkutil::Ops ops;
		auto acl = zookeeper->getDefaultACL();
		ops.push_back(new zkutil::Op::Create(month_path, "", acl, zkutil::CreateMode::Persistent));
		for (size_t i = 0; i < 200; ++i)
		{
			ops.push_back(new zkutil::Op::Create(month_path + "/skip_increment", "", acl, zkutil::CreateMode::Persistent));
			ops.push_back(new zkutil::Op::Remove(month_path + "/skip_increment", -1));
		}
		/// Игнорируем ошибки - не получиться могло только если кто-то еще выполнил эту строчку раньше нас.
		zookeeper->tryMulti(ops);
	}

	return AbandonableLockInZooKeeper(
		zookeeper_path + "/block_numbers/" + month_name + "/block-",
		zookeeper_path + "/temp", *zookeeper);
}

void StorageReplicatedMergeTree::waitForAllReplicasToProcessLogEntry(const String & log_znode_path, const LogEntry & entry)
{
M
Merge  
Michael Kolupaev 已提交
2265 2266
	LOG_DEBUG(log, "Waiting for all replicas to process " << entry.znode_name);

M
Merge  
Michael Kolupaev 已提交
2267 2268 2269 2270 2271 2272
	UInt64 log_index = parse<UInt64>(log_znode_path.substr(log_znode_path.size() - 10));
	String log_entry_str = entry.toString();

	Strings replicas = zookeeper->getChildren(zookeeper_path + "/replicas");
	for (const String & replica : replicas)
	{
M
Merge  
Michael Kolupaev 已提交
2273 2274
		LOG_DEBUG(log, "Waiting for " << replica << " to pull " << entry.znode_name << " to queue");

M
Merge  
Michael Kolupaev 已提交
2275 2276 2277 2278 2279 2280 2281 2282 2283 2284 2285 2286
		/// Дождемся, пока запись попадет в очередь реплики.
		while (true)
		{
			zkutil::EventPtr event = new Poco::Event;

			String pointer = zookeeper->get(zookeeper_path + "/replicas/" + replica + "/log_pointer", nullptr, event);
			if (!pointer.empty() && parse<UInt64>(pointer) > log_index)
				break;

			event->wait();
		}

M
Merge  
Michael Kolupaev 已提交
2287 2288
		LOG_DEBUG(log, "Looking for " << entry.znode_name << " in " << replica << " queue");

M
Merge  
Michael Kolupaev 已提交
2289 2290 2291 2292 2293 2294 2295
		/// Найдем запись в очереди реплики.
		Strings queue_entries = zookeeper->getChildren(zookeeper_path + "/replicas/" + replica + "/queue");
		String entry_to_wait_for;

		for (const String & entry_name : queue_entries)
		{
			String queue_entry_str;
M
Merge  
Michael Kolupaev 已提交
2296 2297
			bool exists = zookeeper->tryGet(zookeeper_path + "/replicas/" + replica + "/queue/" + entry_name, queue_entry_str);
			if (exists && queue_entry_str == log_entry_str)
M
Merge  
Michael Kolupaev 已提交
2298 2299 2300 2301 2302 2303 2304 2305 2306 2307
			{
				entry_to_wait_for = entry_name;
				break;
			}
		}

		/// Пока искали запись, ее уже выполнили и удалили.
		if (entry_to_wait_for.empty())
			continue;

M
Merge  
Michael Kolupaev 已提交
2308 2309
		LOG_DEBUG(log, "Waiting for " << entry_to_wait_for << " to disappear from " << replica << " queue");

M
Merge  
Michael Kolupaev 已提交
2310 2311 2312 2313 2314 2315 2316
		/// Дождемся, пока запись исчезнет из очереди реплики.
		while (true)
		{
			zkutil::EventPtr event = new Poco::Event;

			String unused;
			/// get вместо exists, чтобы не утек watch, если ноды уже нет.
M
Merge  
Michael Kolupaev 已提交
2317
			if (!zookeeper->tryGet(zookeeper_path + "/replicas/" + replica + "/queue/" + entry_to_wait_for, unused, nullptr, event))
M
Merge  
Michael Kolupaev 已提交
2318 2319 2320 2321 2322
				break;

			event->wait();
		}
	}
M
Merge  
Michael Kolupaev 已提交
2323 2324

	LOG_DEBUG(log, "Finished waiting for all replicas to process " << entry.znode_name);
M
Merge  
Michael Kolupaev 已提交
2325 2326 2327
}


M
Merge  
Michael Kolupaev 已提交
2328 2329 2330
void StorageReplicatedMergeTree::LogEntry::writeText(WriteBuffer & out) const
{
	writeString("format version: 1\n", out);
M
Merge  
Michael Kolupaev 已提交
2331 2332 2333
	writeString("source replica: ", out);
	writeString(source_replica, out);
	writeString("\n", out);
M
Merge  
Michael Kolupaev 已提交
2334 2335 2336 2337 2338 2339 2340 2341 2342 2343 2344 2345 2346 2347 2348 2349
	switch (type)
	{
		case GET_PART:
			writeString("get\n", out);
			writeString(new_part_name, out);
			break;
		case MERGE_PARTS:
			writeString("merge\n", out);
			for (const String & s : parts_to_merge)
			{
				writeString(s, out);
				writeString("\n", out);
			}
			writeString("into\n", out);
			writeString(new_part_name, out);
			break;
M
Merge  
Michael Kolupaev 已提交
2350
		case DROP_RANGE:
M
Merge  
Michael Kolupaev 已提交
2351 2352 2353 2354
			if (detach)
				writeString("detach\n", out);
			else
				writeString("drop\n", out);
M
Merge  
Michael Kolupaev 已提交
2355 2356
			writeString(new_part_name, out);
			break;
M
Merge  
Michael Kolupaev 已提交
2357 2358 2359 2360 2361 2362 2363 2364 2365
	}
	writeString("\n", out);
}

void StorageReplicatedMergeTree::LogEntry::readText(ReadBuffer & in)
{
	String type_str;

	assertString("format version: 1\n", in);
M
Michael Kolupaev 已提交
2366
	assertString("source replica: ", in);
M
Merge  
Michael Kolupaev 已提交
2367
	readString(source_replica, in);
M
Merge  
Michael Kolupaev 已提交
2368
	assertString("\n", in);
M
Michael Kolupaev 已提交
2369 2370
	readString(type_str, in);
	assertString("\n", in);
M
Merge  
Michael Kolupaev 已提交
2371 2372 2373 2374 2375 2376 2377 2378 2379 2380 2381 2382 2383 2384 2385 2386 2387 2388 2389 2390

	if (type_str == "get")
	{
		type = GET_PART;
		readString(new_part_name, in);
	}
	else if (type_str == "merge")
	{
		type = MERGE_PARTS;
		while (true)
		{
			String s;
			readString(s, in);
			assertString("\n", in);
			if (s == "into")
				break;
			parts_to_merge.push_back(s);
		}
		readString(new_part_name, in);
	}
M
Merge  
Michael Kolupaev 已提交
2391
	else if (type_str == "drop" || type_str == "detach")
M
Merge  
Michael Kolupaev 已提交
2392 2393
	{
		type = DROP_RANGE;
M
Merge  
Michael Kolupaev 已提交
2394
		detach = type_str == "detach";
M
Merge  
Michael Kolupaev 已提交
2395 2396
		readString(new_part_name, in);
	}
M
Merge  
Michael Kolupaev 已提交
2397 2398 2399
	assertString("\n", in);
}

M
Merge  
Michael Kolupaev 已提交
2400
}