2017-08-06 20:26:23 +00:00
|
|
|
#include <Interpreters/InterpreterSystemQuery.h>
|
2018-04-19 13:56:14 +00:00
|
|
|
#include <Common/DNSResolver.h>
|
2018-05-21 13:49:54 +00:00
|
|
|
#include <Common/ActionLock.h>
|
|
|
|
#include <Common/typeid_cast.h>
|
2018-06-05 19:46:49 +00:00
|
|
|
#include <Common/getNumberOfPhysicalCPUCores.h>
|
2020-11-30 14:30:55 +00:00
|
|
|
#include <Common/SymbolIndex.h>
|
2019-01-11 19:12:36 +00:00
|
|
|
#include <Common/ThreadPool.h>
|
2019-10-25 19:07:47 +00:00
|
|
|
#include <Common/escapeForFileName.h>
|
2021-01-07 19:19:33 +00:00
|
|
|
#include <Common/ShellCommand.h>
|
2023-03-22 07:49:22 +00:00
|
|
|
#include <Common/CurrentMetrics.h>
|
2023-04-25 20:07:04 +00:00
|
|
|
#include <Common/FailPoint.h>
|
2022-08-26 17:23:46 +00:00
|
|
|
#include <Interpreters/Cache/FileCacheFactory.h>
|
|
|
|
#include <Interpreters/Cache/FileCache.h>
|
2017-08-07 17:01:04 +00:00
|
|
|
#include <Interpreters/Context.h>
|
2020-02-10 13:10:17 +00:00
|
|
|
#include <Interpreters/DatabaseCatalog.h>
|
2019-09-26 10:41:33 +00:00
|
|
|
#include <Interpreters/ExternalDictionariesLoader.h>
|
2022-09-24 21:24:39 +00:00
|
|
|
#include <Functions/UserDefined/ExternalUserDefinedExecutableFunctionsLoader.h>
|
2017-08-24 18:19:06 +00:00
|
|
|
#include <Interpreters/EmbeddedDictionaries.h>
|
2018-05-21 13:49:54 +00:00
|
|
|
#include <Interpreters/ActionLocksManager.h>
|
|
|
|
#include <Interpreters/InterpreterDropQuery.h>
|
|
|
|
#include <Interpreters/InterpreterCreateQuery.h>
|
2020-03-30 23:36:23 +00:00
|
|
|
#include <Interpreters/InterpreterRenameQuery.h>
|
2018-06-13 19:01:07 +00:00
|
|
|
#include <Interpreters/QueryLog.h>
|
2020-11-03 13:47:26 +00:00
|
|
|
#include <Interpreters/executeDDLQueryOnCluster.h>
|
2018-06-13 19:01:07 +00:00
|
|
|
#include <Interpreters/PartLog.h>
|
|
|
|
#include <Interpreters/QueryThreadLog.h>
|
2021-06-18 13:44:08 +00:00
|
|
|
#include <Interpreters/QueryViewsLog.h>
|
2021-03-05 14:57:16 +00:00
|
|
|
#include <Interpreters/SessionLog.h>
|
2019-02-03 21:30:45 +00:00
|
|
|
#include <Interpreters/TraceLog.h>
|
2019-07-31 14:03:23 +00:00
|
|
|
#include <Interpreters/TextLog.h>
|
2019-08-13 14:31:46 +00:00
|
|
|
#include <Interpreters/MetricLog.h>
|
2020-06-10 19:17:30 +00:00
|
|
|
#include <Interpreters/AsynchronousMetricLog.h>
|
2020-10-22 16:47:20 +00:00
|
|
|
#include <Interpreters/OpenTelemetrySpanLog.h>
|
2021-07-09 14:05:35 +00:00
|
|
|
#include <Interpreters/ZooKeeperLog.h>
|
2022-04-30 05:00:40 +00:00
|
|
|
#include <Interpreters/FilesystemCacheLog.h>
|
2022-01-14 14:03:00 +00:00
|
|
|
#include <Interpreters/TransactionsInfoLog.h>
|
2022-02-05 16:33:42 +00:00
|
|
|
#include <Interpreters/ProcessorsProfileLog.h>
|
2022-10-03 18:52:14 +00:00
|
|
|
#include <Interpreters/AsynchronousInsertLog.h>
|
2021-06-13 12:38:57 +00:00
|
|
|
#include <Interpreters/JIT/CompiledExpressionCache.h>
|
2022-05-25 20:20:13 +00:00
|
|
|
#include <Interpreters/TransactionLog.h>
|
2023-04-25 21:30:03 +00:00
|
|
|
#include <Interpreters/AsynchronousInsertQueue.h>
|
2022-09-11 15:50:36 +00:00
|
|
|
#include <BridgeHelper/CatBoostLibraryBridgeHelper.h>
|
2022-09-16 11:19:39 +00:00
|
|
|
#include <Access/AccessControl.h>
|
2020-03-07 17:37:38 +00:00
|
|
|
#include <Access/ContextAccess.h>
|
2021-10-31 08:51:20 +00:00
|
|
|
#include <Access/Common/AllowedClientHosts.h>
|
2018-05-21 13:49:54 +00:00
|
|
|
#include <Databases/IDatabase.h>
|
2022-05-01 13:40:18 +00:00
|
|
|
#include <Databases/DatabaseReplicated.h>
|
2019-04-08 05:13:16 +00:00
|
|
|
#include <Storages/StorageDistributed.h>
|
2018-05-21 13:49:54 +00:00
|
|
|
#include <Storages/StorageReplicatedMergeTree.h>
|
2022-06-08 12:09:59 +00:00
|
|
|
#include <Storages/Freeze.h>
|
2018-05-21 13:49:54 +00:00
|
|
|
#include <Storages/StorageFactory.h>
|
2022-08-05 16:20:15 +00:00
|
|
|
#include <Storages/StorageFile.h>
|
|
|
|
#include <Storages/StorageS3.h>
|
|
|
|
#include <Storages/StorageURL.h>
|
|
|
|
#include <Storages/HDFS/StorageHDFS.h>
|
2017-08-06 20:26:23 +00:00
|
|
|
#include <Parsers/ASTSystemQuery.h>
|
2018-05-21 13:49:54 +00:00
|
|
|
#include <Parsers/ASTDropQuery.h>
|
|
|
|
#include <Parsers/ASTCreateQuery.h>
|
2021-09-20 14:23:10 +00:00
|
|
|
#include <Common/ThreadFuzzer.h>
|
2017-08-06 20:26:23 +00:00
|
|
|
#include <csignal>
|
2019-01-09 15:44:20 +00:00
|
|
|
#include <algorithm>
|
2023-01-27 01:10:40 +00:00
|
|
|
#include <unistd.h>
|
2017-08-06 20:26:23 +00:00
|
|
|
|
2023-02-03 13:30:52 +00:00
|
|
|
#if USE_AWS_S3
|
|
|
|
#include <IO/S3/Client.h>
|
|
|
|
#endif
|
|
|
|
|
2022-09-28 13:29:29 +00:00
|
|
|
#include "config.h"
|
2020-04-16 12:31:57 +00:00
|
|
|
|
2023-03-22 07:49:22 +00:00
|
|
|
namespace CurrentMetrics
|
|
|
|
{
|
|
|
|
extern const Metric RestartReplicaThreads;
|
|
|
|
extern const Metric RestartReplicaThreadsActive;
|
|
|
|
}
|
|
|
|
|
2017-08-04 15:54:00 +00:00
|
|
|
namespace DB
|
|
|
|
{
|
2022-12-30 13:47:30 +00:00
|
|
|
|
2017-08-06 20:26:23 +00:00
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
2020-02-25 18:02:41 +00:00
|
|
|
extern const int LOGICAL_ERROR;
|
2017-08-06 20:26:23 +00:00
|
|
|
extern const int BAD_ARGUMENTS;
|
|
|
|
extern const int CANNOT_KILL;
|
2017-08-07 17:01:04 +00:00
|
|
|
extern const int NOT_IMPLEMENTED;
|
2019-09-19 11:04:57 +00:00
|
|
|
extern const int TIMEOUT_EXCEEDED;
|
2020-06-23 12:01:51 +00:00
|
|
|
extern const int TABLE_WAS_NOT_DROPPED;
|
2017-08-06 20:26:23 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2018-05-21 13:49:54 +00:00
|
|
|
namespace ActionLocks
|
|
|
|
{
|
2023-08-03 10:34:43 +00:00
|
|
|
extern const StorageActionBlockType PartsMerge;
|
|
|
|
extern const StorageActionBlockType PartsFetch;
|
|
|
|
extern const StorageActionBlockType PartsSend;
|
|
|
|
extern const StorageActionBlockType ReplicationQueue;
|
|
|
|
extern const StorageActionBlockType DistributedSend;
|
|
|
|
extern const StorageActionBlockType PartsTTLMerge;
|
|
|
|
extern const StorageActionBlockType PartsMove;
|
|
|
|
extern const StorageActionBlockType PullReplicationLog;
|
2018-05-21 13:49:54 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2017-08-31 12:55:19 +00:00
|
|
|
namespace
|
|
|
|
{
|
2017-08-04 15:54:00 +00:00
|
|
|
|
2023-08-11 01:06:07 +00:00
|
|
|
/// Sequentially tries to execute all commands and throws exception with info about failed commands
|
|
|
|
void executeCommandsAndThrowIfError(std::vector<std::function<void()>> commands)
|
2017-08-24 18:19:06 +00:00
|
|
|
{
|
2023-08-11 01:06:07 +00:00
|
|
|
ExecutionStatus result(0);
|
|
|
|
for (auto & command : commands)
|
2017-08-24 18:19:06 +00:00
|
|
|
{
|
2023-08-11 01:06:07 +00:00
|
|
|
try
|
|
|
|
{
|
|
|
|
command();
|
|
|
|
}
|
|
|
|
catch (...)
|
|
|
|
{
|
|
|
|
ExecutionStatus current_result = ExecutionStatus::fromCurrentException();
|
2017-08-24 18:19:06 +00:00
|
|
|
|
2023-08-11 01:06:07 +00:00
|
|
|
if (result.code == 0)
|
|
|
|
result.code = current_result.code;
|
2017-08-24 18:19:06 +00:00
|
|
|
|
2023-08-11 01:06:07 +00:00
|
|
|
if (!current_result.message.empty())
|
|
|
|
{
|
|
|
|
if (!result.message.empty())
|
|
|
|
result.message += '\n';
|
|
|
|
result.message += current_result.message;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
2017-08-24 18:19:06 +00:00
|
|
|
|
2023-08-11 01:06:07 +00:00
|
|
|
if (result.code != 0)
|
|
|
|
throw Exception::createDeprecated(result.message, result.code);
|
2018-06-13 19:01:07 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2020-01-24 16:20:36 +00:00
|
|
|
AccessType getRequiredAccessType(StorageActionBlockType action_type)
|
|
|
|
{
|
|
|
|
if (action_type == ActionLocks::PartsMerge)
|
2020-04-03 11:54:50 +00:00
|
|
|
return AccessType::SYSTEM_MERGES;
|
2020-01-24 16:20:36 +00:00
|
|
|
else if (action_type == ActionLocks::PartsFetch)
|
2020-04-03 11:54:50 +00:00
|
|
|
return AccessType::SYSTEM_FETCHES;
|
2020-01-24 16:20:36 +00:00
|
|
|
else if (action_type == ActionLocks::PartsSend)
|
2020-04-03 11:54:50 +00:00
|
|
|
return AccessType::SYSTEM_REPLICATED_SENDS;
|
2020-01-24 16:20:36 +00:00
|
|
|
else if (action_type == ActionLocks::ReplicationQueue)
|
2020-04-03 11:54:50 +00:00
|
|
|
return AccessType::SYSTEM_REPLICATION_QUEUES;
|
2020-01-24 16:20:36 +00:00
|
|
|
else if (action_type == ActionLocks::DistributedSend)
|
2020-04-03 11:54:50 +00:00
|
|
|
return AccessType::SYSTEM_DISTRIBUTED_SENDS;
|
2020-01-24 16:20:36 +00:00
|
|
|
else if (action_type == ActionLocks::PartsTTLMerge)
|
2020-04-03 11:54:50 +00:00
|
|
|
return AccessType::SYSTEM_TTL_MERGES;
|
2020-01-24 16:20:36 +00:00
|
|
|
else if (action_type == ActionLocks::PartsMove)
|
2020-04-03 11:54:50 +00:00
|
|
|
return AccessType::SYSTEM_MOVES;
|
2023-08-03 10:34:43 +00:00
|
|
|
else if (action_type == ActionLocks::PullReplicationLog)
|
|
|
|
return AccessType::SYSTEM_PULLING_REPLICATION_LOG;
|
2020-01-24 16:20:36 +00:00
|
|
|
else
|
2023-01-23 21:13:58 +00:00
|
|
|
throw Exception(ErrorCodes::LOGICAL_ERROR, "Unknown action type: {}", std::to_string(action_type));
|
2020-01-24 16:20:36 +00:00
|
|
|
}
|
|
|
|
|
2021-06-20 08:24:43 +00:00
|
|
|
constexpr std::string_view table_is_not_replicated = "Table {} is not replicated";
|
|
|
|
|
2020-03-04 20:29:52 +00:00
|
|
|
}
|
2020-01-24 16:20:36 +00:00
|
|
|
|
2018-05-21 13:49:54 +00:00
|
|
|
/// Implements SYSTEM [START|STOP] <something action from ActionLocks>
|
2020-03-04 20:29:52 +00:00
|
|
|
void InterpreterSystemQuery::startStopAction(StorageActionBlockType action_type, bool start)
|
2018-05-21 13:49:54 +00:00
|
|
|
{
|
2021-04-10 23:33:54 +00:00
|
|
|
auto manager = getContext()->getActionLocksManager();
|
2018-05-21 13:49:54 +00:00
|
|
|
manager->cleanExpired();
|
|
|
|
|
2022-01-25 09:47:29 +00:00
|
|
|
auto access = getContext()->getAccess();
|
|
|
|
auto required_access_type = getRequiredAccessType(action_type);
|
|
|
|
|
2020-10-20 15:10:24 +00:00
|
|
|
if (volume_ptr && action_type == ActionLocks::PartsMerge)
|
|
|
|
{
|
2022-01-25 09:47:29 +00:00
|
|
|
access->checkAccess(required_access_type);
|
2020-10-20 15:10:24 +00:00
|
|
|
volume_ptr->setAvoidMergesUserOverride(!start);
|
|
|
|
}
|
|
|
|
else if (table_id)
|
2018-05-21 13:49:54 +00:00
|
|
|
{
|
2022-01-25 09:47:29 +00:00
|
|
|
access->checkAccess(required_access_type, table_id.database_name, table_id.table_name);
|
2021-04-10 23:33:54 +00:00
|
|
|
auto table = DatabaseCatalog::instance().tryGetTable(table_id, getContext());
|
2020-10-15 19:23:46 +00:00
|
|
|
if (table)
|
2020-10-15 16:10:22 +00:00
|
|
|
{
|
2020-10-15 19:23:46 +00:00
|
|
|
if (start)
|
|
|
|
{
|
|
|
|
manager->remove(table, action_type);
|
|
|
|
table->onActionLockRemove(action_type);
|
|
|
|
}
|
|
|
|
else
|
|
|
|
manager->add(table, action_type);
|
2020-10-15 16:10:22 +00:00
|
|
|
}
|
2018-05-21 13:49:54 +00:00
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
2020-02-10 13:10:17 +00:00
|
|
|
for (auto & elem : DatabaseCatalog::instance().getDatabases())
|
2020-01-24 16:20:36 +00:00
|
|
|
{
|
2022-09-09 16:27:19 +00:00
|
|
|
startStopActionInDatabase(action_type, start, elem.first, elem.second, getContext(), log);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
void InterpreterSystemQuery::startStopActionInDatabase(StorageActionBlockType action_type, bool start,
|
|
|
|
const String & database_name, const DatabasePtr & database,
|
|
|
|
const ContextPtr & local_context, Poco::Logger * log)
|
|
|
|
{
|
|
|
|
auto manager = local_context->getActionLocksManager();
|
|
|
|
auto access = local_context->getAccess();
|
|
|
|
auto required_access_type = getRequiredAccessType(action_type);
|
|
|
|
|
|
|
|
for (auto iterator = database->getTablesIterator(local_context); iterator->isValid(); iterator->next())
|
|
|
|
{
|
|
|
|
StoragePtr table = iterator->table();
|
|
|
|
if (!table)
|
|
|
|
continue;
|
|
|
|
|
|
|
|
if (!access->isGranted(required_access_type, database_name, iterator->name()))
|
|
|
|
{
|
|
|
|
LOG_INFO(log, "Access {} denied, skipping {}.{}", toString(required_access_type), database_name, iterator->name());
|
|
|
|
continue;
|
|
|
|
}
|
|
|
|
|
|
|
|
if (start)
|
|
|
|
{
|
|
|
|
manager->remove(table, action_type);
|
|
|
|
table->onActionLockRemove(action_type);
|
2020-01-24 16:20:36 +00:00
|
|
|
}
|
2022-09-09 16:27:19 +00:00
|
|
|
else
|
|
|
|
manager->add(table, action_type);
|
2018-05-21 13:49:54 +00:00
|
|
|
}
|
|
|
|
}
|
2020-01-24 16:20:36 +00:00
|
|
|
|
2020-03-04 20:29:52 +00:00
|
|
|
|
2021-05-31 14:49:02 +00:00
|
|
|
InterpreterSystemQuery::InterpreterSystemQuery(const ASTPtr & query_ptr_, ContextMutablePtr context_)
|
|
|
|
: WithMutableContext(context_), query_ptr(query_ptr_->clone()), log(&Poco::Logger::get("InterpreterSystemQuery"))
|
2019-11-11 01:11:32 +00:00
|
|
|
{
|
|
|
|
}
|
2017-08-31 12:55:19 +00:00
|
|
|
|
2017-08-06 20:26:23 +00:00
|
|
|
|
2017-08-04 15:54:00 +00:00
|
|
|
BlockIO InterpreterSystemQuery::execute()
|
|
|
|
{
|
2019-03-15 16:14:13 +00:00
|
|
|
auto & query = query_ptr->as<ASTSystemQuery &>();
|
2017-08-06 20:26:23 +00:00
|
|
|
|
2019-12-19 07:54:43 +00:00
|
|
|
if (!query.cluster.empty())
|
2022-04-22 12:15:29 +00:00
|
|
|
{
|
|
|
|
DDLQueryOnClusterParams params;
|
|
|
|
params.access_to_check = getRequiredAccessForDDLOnCluster();
|
|
|
|
return executeDDLQueryOnCluster(query_ptr, getContext(), params);
|
|
|
|
}
|
2019-12-19 07:54:43 +00:00
|
|
|
|
2017-08-06 20:26:23 +00:00
|
|
|
using Type = ASTSystemQuery::Type;
|
|
|
|
|
2018-03-26 14:12:07 +00:00
|
|
|
/// Use global context with fresh system profile settings
|
2021-04-10 23:33:54 +00:00
|
|
|
auto system_context = Context::createCopy(getContext()->getGlobalContext());
|
|
|
|
system_context->setSetting("profile", getContext()->getSystemProfileName());
|
2018-03-26 14:12:07 +00:00
|
|
|
|
2018-05-21 13:49:54 +00:00
|
|
|
/// Make canonical query for simpler processing
|
2021-06-20 08:24:43 +00:00
|
|
|
if (query.type == Type::RELOAD_DICTIONARY)
|
|
|
|
{
|
2021-11-11 13:28:18 +00:00
|
|
|
if (query.database)
|
2021-10-12 18:11:00 +00:00
|
|
|
query.setTable(query.getDatabase() + "." + query.getTable());
|
2021-06-20 08:24:43 +00:00
|
|
|
}
|
2021-11-11 13:28:18 +00:00
|
|
|
else if (query.table)
|
2021-06-20 08:24:43 +00:00
|
|
|
{
|
2021-10-12 18:11:00 +00:00
|
|
|
table_id = getContext()->resolveStorageID(StorageID(query.getDatabase(), query.getTable()), Context::ResolveOrdinary);
|
2021-06-20 08:24:43 +00:00
|
|
|
}
|
2018-05-21 13:49:54 +00:00
|
|
|
|
2019-12-15 13:48:11 +00:00
|
|
|
|
2022-06-08 12:09:59 +00:00
|
|
|
BlockIO result;
|
|
|
|
|
2020-10-20 15:10:24 +00:00
|
|
|
volume_ptr = {};
|
|
|
|
if (!query.storage_policy.empty() && !query.volume.empty())
|
2021-04-10 23:33:54 +00:00
|
|
|
volume_ptr = getContext()->getStoragePolicy(query.storage_policy)->getVolumeByName(query.volume);
|
2020-10-20 15:10:24 +00:00
|
|
|
|
2017-08-06 20:26:23 +00:00
|
|
|
switch (query.type)
|
|
|
|
{
|
|
|
|
case Type::SHUTDOWN:
|
2021-01-07 19:19:33 +00:00
|
|
|
{
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_SHUTDOWN);
|
2017-08-06 20:26:23 +00:00
|
|
|
if (kill(0, SIGTERM))
|
2017-08-31 12:55:19 +00:00
|
|
|
throwFromErrno("System call kill(0, SIGTERM) failed", ErrorCodes::CANNOT_KILL);
|
2017-08-06 20:26:23 +00:00
|
|
|
break;
|
2021-01-07 19:19:33 +00:00
|
|
|
}
|
2017-08-06 20:26:23 +00:00
|
|
|
case Type::KILL:
|
2021-01-07 19:19:33 +00:00
|
|
|
{
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_SHUTDOWN);
|
2021-01-07 18:36:38 +00:00
|
|
|
/// Exit with the same code as it is usually set by shell when process is terminated by SIGKILL.
|
|
|
|
/// It's better than doing 'raise' or 'kill', because they have no effect for 'init' process (with pid = 0, usually in Docker).
|
|
|
|
LOG_INFO(log, "Exit immediately as the SYSTEM KILL command has been issued.");
|
|
|
|
_exit(128 + SIGKILL);
|
|
|
|
// break; /// unreachable
|
2021-01-07 19:19:33 +00:00
|
|
|
}
|
|
|
|
case Type::SUSPEND:
|
|
|
|
{
|
2021-12-30 18:37:21 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_SHUTDOWN);
|
2021-01-07 19:19:33 +00:00
|
|
|
auto command = fmt::format("kill -STOP {0} && sleep {1} && kill -CONT {0}", getpid(), query.seconds);
|
|
|
|
LOG_DEBUG(log, "Will run {}", command);
|
|
|
|
auto res = ShellCommand::execute(command);
|
|
|
|
res->in.close();
|
|
|
|
WriteBufferFromOwnString out;
|
|
|
|
copyData(res->out, out);
|
|
|
|
copyData(res->err, out);
|
|
|
|
if (!out.str().empty())
|
2023-01-23 22:16:16 +00:00
|
|
|
LOG_DEBUG(log, "The command {} returned output: {}", command, out.str());
|
2021-01-07 19:22:41 +00:00
|
|
|
res->wait();
|
2017-08-06 20:26:23 +00:00
|
|
|
break;
|
2021-01-07 19:19:33 +00:00
|
|
|
}
|
2023-01-27 01:10:40 +00:00
|
|
|
case Type::SYNC_FILE_CACHE:
|
|
|
|
{
|
2023-01-29 01:43:24 +00:00
|
|
|
LOG_DEBUG(log, "Will perform 'sync' syscall (it can take time).");
|
2023-01-27 01:10:40 +00:00
|
|
|
sync();
|
|
|
|
break;
|
|
|
|
}
|
2017-08-06 20:26:23 +00:00
|
|
|
case Type::DROP_DNS_CACHE:
|
2021-01-07 19:19:33 +00:00
|
|
|
{
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_DNS_CACHE);
|
2018-04-19 13:56:14 +00:00
|
|
|
DNSResolver::instance().dropCache();
|
2018-01-15 14:13:19 +00:00
|
|
|
/// Reinitialize clusters to update their resolved_addresses
|
2021-04-10 23:33:54 +00:00
|
|
|
system_context->reloadClusterConfig();
|
2017-08-06 20:26:23 +00:00
|
|
|
break;
|
2021-01-07 19:19:33 +00:00
|
|
|
}
|
2017-08-06 20:26:23 +00:00
|
|
|
case Type::DROP_MARK_CACHE:
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_MARK_CACHE);
|
2023-06-29 10:02:41 +00:00
|
|
|
system_context->clearMarkCache();
|
2017-08-06 20:26:23 +00:00
|
|
|
break;
|
|
|
|
case Type::DROP_UNCOMPRESSED_CACHE:
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_UNCOMPRESSED_CACHE);
|
2023-06-29 10:02:41 +00:00
|
|
|
system_context->clearUncompressedCache();
|
2017-08-06 20:26:23 +00:00
|
|
|
break;
|
2021-08-21 17:27:22 +00:00
|
|
|
case Type::DROP_INDEX_MARK_CACHE:
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_MARK_CACHE);
|
2023-06-29 10:02:41 +00:00
|
|
|
system_context->clearIndexMarkCache();
|
2021-08-21 17:27:22 +00:00
|
|
|
break;
|
|
|
|
case Type::DROP_INDEX_UNCOMPRESSED_CACHE:
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_UNCOMPRESSED_CACHE);
|
2023-06-29 10:02:41 +00:00
|
|
|
system_context->clearIndexUncompressedCache();
|
2021-08-21 17:27:22 +00:00
|
|
|
break;
|
2021-03-28 01:10:30 +00:00
|
|
|
case Type::DROP_MMAP_CACHE:
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_MMAP_CACHE);
|
2023-06-29 10:02:41 +00:00
|
|
|
system_context->clearMMappedFileCache();
|
2021-03-28 01:10:30 +00:00
|
|
|
break;
|
2023-01-26 22:23:10 +00:00
|
|
|
case Type::DROP_QUERY_CACHE:
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_QUERY_CACHE);
|
2023-06-29 10:02:41 +00:00
|
|
|
getContext()->clearQueryCache();
|
2022-11-29 13:15:28 +00:00
|
|
|
break;
|
2018-08-30 16:31:20 +00:00
|
|
|
#if USE_EMBEDDED_COMPILER
|
2018-09-05 11:37:41 +00:00
|
|
|
case Type::DROP_COMPILED_EXPRESSION_CACHE:
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_COMPILED_EXPRESSION_CACHE);
|
2021-03-04 17:38:12 +00:00
|
|
|
if (auto * cache = CompiledExpressionCacheFactory::instance().tryGetCache())
|
|
|
|
cache->reset();
|
2018-08-30 16:31:20 +00:00
|
|
|
break;
|
|
|
|
#endif
|
2023-02-03 13:30:52 +00:00
|
|
|
#if USE_AWS_S3
|
|
|
|
case Type::DROP_S3_CLIENT_CACHE:
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_S3_CLIENT_CACHE);
|
|
|
|
S3::ClientCacheRegistry::instance().clearCacheForAll();
|
|
|
|
break;
|
|
|
|
#endif
|
|
|
|
|
2022-03-30 11:47:44 +00:00
|
|
|
case Type::DROP_FILESYSTEM_CACHE:
|
2022-03-21 11:30:25 +00:00
|
|
|
{
|
2022-06-22 16:00:15 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_FILESYSTEM_CACHE);
|
2023-04-15 11:08:49 +00:00
|
|
|
|
|
|
|
if (query.filesystem_cache_name.empty())
|
2022-03-21 11:30:25 +00:00
|
|
|
{
|
|
|
|
auto caches = FileCacheFactory::instance().getAll();
|
2022-03-21 13:56:38 +00:00
|
|
|
for (const auto & [_, cache_data] : caches)
|
2023-01-04 17:56:45 +00:00
|
|
|
cache_data->cache->removeAllReleasable();
|
2022-03-21 11:30:25 +00:00
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
2023-04-15 11:08:49 +00:00
|
|
|
auto cache = FileCacheFactory::instance().getByName(query.filesystem_cache_name).cache;
|
2023-07-30 11:39:54 +00:00
|
|
|
if (query.key_to_drop.empty())
|
|
|
|
{
|
|
|
|
cache->removeAllReleasable();
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
auto key = FileCacheKey::fromKeyString(query.key_to_drop);
|
|
|
|
if (query.offset_to_drop.has_value())
|
|
|
|
cache->removeFileSegment(key, query.offset_to_drop.value());
|
|
|
|
else
|
|
|
|
cache->removeKey(key);
|
|
|
|
}
|
2022-06-21 17:30:21 +00:00
|
|
|
}
|
2022-03-21 11:30:25 +00:00
|
|
|
break;
|
|
|
|
}
|
2022-08-05 16:20:15 +00:00
|
|
|
case Type::DROP_SCHEMA_CACHE:
|
|
|
|
{
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_SCHEMA_CACHE);
|
|
|
|
std::unordered_set<String> caches_to_drop;
|
|
|
|
if (query.schema_cache_storage.empty())
|
|
|
|
caches_to_drop = {"FILE", "S3", "HDFS", "URL"};
|
|
|
|
else
|
|
|
|
caches_to_drop = {query.schema_cache_storage};
|
|
|
|
|
|
|
|
if (caches_to_drop.contains("FILE"))
|
|
|
|
StorageFile::getSchemaCache(getContext()).clear();
|
2022-08-08 12:17:18 +00:00
|
|
|
#if USE_AWS_S3
|
2022-08-05 16:20:15 +00:00
|
|
|
if (caches_to_drop.contains("S3"))
|
|
|
|
StorageS3::getSchemaCache(getContext()).clear();
|
2022-08-08 12:17:18 +00:00
|
|
|
#endif
|
|
|
|
#if USE_HDFS
|
2022-08-05 16:20:15 +00:00
|
|
|
if (caches_to_drop.contains("HDFS"))
|
|
|
|
StorageHDFS::getSchemaCache(getContext()).clear();
|
2022-08-08 12:17:18 +00:00
|
|
|
#endif
|
2022-08-05 16:20:15 +00:00
|
|
|
if (caches_to_drop.contains("URL"))
|
|
|
|
StorageURL::getSchemaCache(getContext()).clear();
|
|
|
|
break;
|
|
|
|
}
|
2017-08-24 18:19:06 +00:00
|
|
|
case Type::RELOAD_DICTIONARY:
|
2021-01-07 19:19:33 +00:00
|
|
|
{
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_RELOAD_DICTIONARY);
|
2021-03-19 12:47:27 +00:00
|
|
|
|
2021-04-10 23:33:54 +00:00
|
|
|
auto & external_dictionaries_loader = system_context->getExternalDictionariesLoader();
|
2021-10-12 18:11:00 +00:00
|
|
|
external_dictionaries_loader.reloadDictionary(query.getTable(), getContext());
|
2021-03-19 12:47:27 +00:00
|
|
|
|
2020-03-03 08:32:58 +00:00
|
|
|
ExternalDictionariesLoader::resetAll();
|
2017-08-24 18:19:06 +00:00
|
|
|
break;
|
2021-01-07 19:19:33 +00:00
|
|
|
}
|
2017-08-24 18:19:06 +00:00
|
|
|
case Type::RELOAD_DICTIONARIES:
|
2021-01-07 19:19:33 +00:00
|
|
|
{
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_RELOAD_DICTIONARY);
|
2023-08-11 01:06:07 +00:00
|
|
|
executeCommandsAndThrowIfError({
|
2021-06-20 08:24:43 +00:00
|
|
|
[&] { system_context->getExternalDictionariesLoader().reloadAllTriedToLoad(); },
|
|
|
|
[&] { system_context->getEmbeddedDictionaries().reload(); }
|
2023-08-11 01:06:07 +00:00
|
|
|
});
|
2020-03-03 08:32:58 +00:00
|
|
|
ExternalDictionariesLoader::resetAll();
|
2017-08-24 18:19:06 +00:00
|
|
|
break;
|
2021-01-07 19:19:33 +00:00
|
|
|
}
|
2021-04-16 09:56:40 +00:00
|
|
|
case Type::RELOAD_MODEL:
|
|
|
|
{
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_RELOAD_MODEL);
|
2022-09-11 15:50:36 +00:00
|
|
|
auto bridge_helper = std::make_unique<CatBoostLibraryBridgeHelper>(getContext(), query.target_model);
|
|
|
|
bridge_helper->removeModel();
|
2021-04-16 09:56:40 +00:00
|
|
|
break;
|
|
|
|
}
|
|
|
|
case Type::RELOAD_MODELS:
|
|
|
|
{
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_RELOAD_MODEL);
|
2022-09-11 15:50:36 +00:00
|
|
|
auto bridge_helper = std::make_unique<CatBoostLibraryBridgeHelper>(getContext());
|
|
|
|
bridge_helper->removeAllModels();
|
2021-04-16 09:56:40 +00:00
|
|
|
break;
|
|
|
|
}
|
2021-09-17 13:04:44 +00:00
|
|
|
case Type::RELOAD_FUNCTION:
|
|
|
|
{
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_RELOAD_FUNCTION);
|
|
|
|
|
|
|
|
auto & external_user_defined_executable_functions_loader = system_context->getExternalUserDefinedExecutableFunctionsLoader();
|
|
|
|
external_user_defined_executable_functions_loader.reloadFunction(query.target_function);
|
|
|
|
break;
|
|
|
|
}
|
|
|
|
case Type::RELOAD_FUNCTIONS:
|
|
|
|
{
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_RELOAD_FUNCTION);
|
|
|
|
|
|
|
|
auto & external_user_defined_executable_functions_loader = system_context->getExternalUserDefinedExecutableFunctionsLoader();
|
|
|
|
external_user_defined_executable_functions_loader.reloadAllTriedToLoad();
|
|
|
|
break;
|
|
|
|
}
|
2018-03-26 14:12:07 +00:00
|
|
|
case Type::RELOAD_EMBEDDED_DICTIONARIES:
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_RELOAD_EMBEDDED_DICTIONARIES);
|
|
|
|
system_context->getEmbeddedDictionaries().reload();
|
2018-03-26 14:12:07 +00:00
|
|
|
break;
|
2018-03-13 10:41:47 +00:00
|
|
|
case Type::RELOAD_CONFIG:
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_RELOAD_CONFIG);
|
|
|
|
system_context->reloadConfig();
|
2018-03-13 10:41:47 +00:00
|
|
|
break;
|
2022-09-16 11:19:39 +00:00
|
|
|
case Type::RELOAD_USERS:
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_RELOAD_USERS);
|
|
|
|
system_context->getAccessControl().reload(AccessControl::ReloadMode::ALL);
|
|
|
|
break;
|
2017-08-07 17:01:04 +00:00
|
|
|
case Type::STOP_MERGES:
|
2020-03-04 20:29:52 +00:00
|
|
|
startStopAction(ActionLocks::PartsMerge, false);
|
2018-05-21 13:49:54 +00:00
|
|
|
break;
|
2017-08-07 17:01:04 +00:00
|
|
|
case Type::START_MERGES:
|
2020-03-04 20:29:52 +00:00
|
|
|
startStopAction(ActionLocks::PartsMerge, true);
|
2018-05-21 13:49:54 +00:00
|
|
|
break;
|
2019-08-01 15:36:12 +00:00
|
|
|
case Type::STOP_TTL_MERGES:
|
2020-03-04 20:29:52 +00:00
|
|
|
startStopAction(ActionLocks::PartsTTLMerge, false);
|
2019-08-01 15:36:12 +00:00
|
|
|
break;
|
|
|
|
case Type::START_TTL_MERGES:
|
2020-03-04 20:29:52 +00:00
|
|
|
startStopAction(ActionLocks::PartsTTLMerge, true);
|
2019-08-01 15:36:12 +00:00
|
|
|
break;
|
2019-09-03 14:50:49 +00:00
|
|
|
case Type::STOP_MOVES:
|
2020-03-04 20:29:52 +00:00
|
|
|
startStopAction(ActionLocks::PartsMove, false);
|
2019-09-03 14:50:49 +00:00
|
|
|
break;
|
|
|
|
case Type::START_MOVES:
|
2020-03-04 20:29:52 +00:00
|
|
|
startStopAction(ActionLocks::PartsMove, true);
|
2019-09-03 14:50:49 +00:00
|
|
|
break;
|
2018-05-21 13:49:54 +00:00
|
|
|
case Type::STOP_FETCHES:
|
2020-03-04 20:29:52 +00:00
|
|
|
startStopAction(ActionLocks::PartsFetch, false);
|
2018-05-21 13:49:54 +00:00
|
|
|
break;
|
|
|
|
case Type::START_FETCHES:
|
2020-03-04 20:29:52 +00:00
|
|
|
startStopAction(ActionLocks::PartsFetch, true);
|
2018-05-21 13:49:54 +00:00
|
|
|
break;
|
|
|
|
case Type::STOP_REPLICATED_SENDS:
|
2020-03-04 20:29:52 +00:00
|
|
|
startStopAction(ActionLocks::PartsSend, false);
|
2018-05-21 13:49:54 +00:00
|
|
|
break;
|
2019-02-02 11:28:43 +00:00
|
|
|
case Type::START_REPLICATED_SENDS:
|
2020-03-04 20:29:52 +00:00
|
|
|
startStopAction(ActionLocks::PartsSend, true);
|
2018-05-21 13:49:54 +00:00
|
|
|
break;
|
2017-08-07 17:01:04 +00:00
|
|
|
case Type::STOP_REPLICATION_QUEUES:
|
2020-03-04 20:29:52 +00:00
|
|
|
startStopAction(ActionLocks::ReplicationQueue, false);
|
2018-05-21 13:49:54 +00:00
|
|
|
break;
|
2017-08-07 17:01:04 +00:00
|
|
|
case Type::START_REPLICATION_QUEUES:
|
2020-03-04 20:29:52 +00:00
|
|
|
startStopAction(ActionLocks::ReplicationQueue, true);
|
2018-05-21 13:49:54 +00:00
|
|
|
break;
|
2019-04-22 15:11:16 +00:00
|
|
|
case Type::STOP_DISTRIBUTED_SENDS:
|
2020-03-04 20:29:52 +00:00
|
|
|
startStopAction(ActionLocks::DistributedSend, false);
|
2019-04-22 15:11:16 +00:00
|
|
|
break;
|
|
|
|
case Type::START_DISTRIBUTED_SENDS:
|
2020-03-04 20:29:52 +00:00
|
|
|
startStopAction(ActionLocks::DistributedSend, true);
|
2019-04-22 15:11:16 +00:00
|
|
|
break;
|
2023-08-03 10:34:43 +00:00
|
|
|
case Type::STOP_PULLING_REPLICATION_LOG:
|
|
|
|
startStopAction(ActionLocks::PullReplicationLog, false);
|
|
|
|
break;
|
|
|
|
case Type::START_PULLING_REPLICATION_LOG:
|
|
|
|
startStopAction(ActionLocks::PullReplicationLog, true);
|
|
|
|
break;
|
2020-05-17 12:44:22 +00:00
|
|
|
case Type::DROP_REPLICA:
|
|
|
|
dropReplica(query);
|
|
|
|
break;
|
2022-12-29 14:34:11 +00:00
|
|
|
case Type::DROP_DATABASE_REPLICA:
|
|
|
|
dropDatabaseReplica(query);
|
|
|
|
break;
|
2018-05-21 13:49:54 +00:00
|
|
|
case Type::SYNC_REPLICA:
|
2023-03-16 15:04:42 +00:00
|
|
|
syncReplica(query);
|
2018-05-21 13:49:54 +00:00
|
|
|
break;
|
2022-05-01 13:40:18 +00:00
|
|
|
case Type::SYNC_DATABASE_REPLICA:
|
|
|
|
syncReplicatedDatabase(query);
|
|
|
|
break;
|
2022-05-25 20:20:13 +00:00
|
|
|
case Type::SYNC_TRANSACTION_LOG:
|
|
|
|
syncTransactionLog();
|
|
|
|
break;
|
2019-05-10 04:19:02 +00:00
|
|
|
case Type::FLUSH_DISTRIBUTED:
|
|
|
|
flushDistributed(query);
|
2019-04-22 15:11:16 +00:00
|
|
|
break;
|
2018-05-21 13:49:54 +00:00
|
|
|
case Type::RESTART_REPLICAS:
|
|
|
|
restartReplicas(system_context);
|
|
|
|
break;
|
|
|
|
case Type::RESTART_REPLICA:
|
2022-01-25 09:47:29 +00:00
|
|
|
restartReplica(table_id, system_context);
|
2021-06-20 08:24:43 +00:00
|
|
|
break;
|
|
|
|
case Type::RESTORE_REPLICA:
|
|
|
|
restoreReplica();
|
2018-05-21 13:49:54 +00:00
|
|
|
break;
|
2022-12-12 14:33:42 +00:00
|
|
|
case Type::WAIT_LOADING_PARTS:
|
|
|
|
waitLoadingParts();
|
|
|
|
break;
|
2021-04-20 19:23:54 +00:00
|
|
|
case Type::RESTART_DISK:
|
|
|
|
restartDisk(query.disk);
|
2018-10-09 10:05:27 +00:00
|
|
|
case Type::FLUSH_LOGS:
|
2021-01-07 19:19:33 +00:00
|
|
|
{
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_FLUSH_LOGS);
|
2023-08-11 01:06:07 +00:00
|
|
|
|
|
|
|
auto logs = getContext()->getSystemLogs();
|
2023-08-11 12:45:12 +00:00
|
|
|
std::vector<std::function<void()>> commands;
|
|
|
|
commands.reserve(logs.size());
|
2023-08-11 01:06:07 +00:00
|
|
|
for (auto * system_log : logs)
|
2023-08-11 12:45:12 +00:00
|
|
|
commands.emplace_back([system_log] { system_log->flush(true); });
|
2023-08-11 01:06:07 +00:00
|
|
|
|
|
|
|
executeCommandsAndThrowIfError(commands);
|
2018-06-13 19:01:07 +00:00
|
|
|
break;
|
2021-01-07 19:19:33 +00:00
|
|
|
}
|
2023-07-26 17:58:41 +00:00
|
|
|
case Type::STOP_LISTEN:
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_LISTEN);
|
|
|
|
getContext()->stopServers(query.server_type);
|
|
|
|
break;
|
|
|
|
case Type::START_LISTEN:
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_LISTEN);
|
|
|
|
getContext()->startServers(query.server_type);
|
|
|
|
break;
|
2023-04-25 21:30:03 +00:00
|
|
|
case Type::FLUSH_ASYNC_INSERT_QUEUE:
|
|
|
|
{
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_FLUSH_ASYNC_INSERT_QUEUE);
|
|
|
|
auto * queue = getContext()->getAsynchronousInsertQueue();
|
|
|
|
if (!queue)
|
|
|
|
throw Exception(ErrorCodes::BAD_ARGUMENTS,
|
|
|
|
"Cannot flush asynchronous insert queue because it is not initialized");
|
|
|
|
|
|
|
|
queue->flushAll();
|
|
|
|
break;
|
|
|
|
}
|
2021-09-20 14:23:10 +00:00
|
|
|
case Type::STOP_THREAD_FUZZER:
|
2021-12-30 18:37:21 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_THREAD_FUZZER);
|
2021-09-20 14:23:10 +00:00
|
|
|
ThreadFuzzer::stop();
|
|
|
|
break;
|
|
|
|
case Type::START_THREAD_FUZZER:
|
2021-12-30 18:37:21 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_THREAD_FUZZER);
|
2021-09-20 14:23:10 +00:00
|
|
|
ThreadFuzzer::start();
|
|
|
|
break;
|
2022-06-08 12:09:59 +00:00
|
|
|
case Type::UNFREEZE:
|
|
|
|
{
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_UNFREEZE);
|
|
|
|
/// The result contains information about deleted parts as a table. It is for compatibility with ALTER TABLE UNFREEZE query.
|
2022-08-23 10:53:06 +00:00
|
|
|
result = Unfreezer(getContext()).systemUnfreeze(query.backup_name);
|
2022-06-08 12:09:59 +00:00
|
|
|
break;
|
|
|
|
}
|
2023-04-25 20:07:04 +00:00
|
|
|
case Type::ENABLE_FAILPOINT:
|
|
|
|
{
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_FAILPOINT);
|
2023-05-08 21:02:36 +00:00
|
|
|
FailPointInjection::enableFailPoint(query.fail_point_name);
|
2023-04-25 20:07:04 +00:00
|
|
|
break;
|
|
|
|
}
|
|
|
|
case Type::DISABLE_FAILPOINT:
|
|
|
|
{
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_FAILPOINT);
|
2023-05-08 21:02:36 +00:00
|
|
|
FailPointInjection::disableFailPoint(query.fail_point_name);
|
2023-04-25 20:07:04 +00:00
|
|
|
break;
|
|
|
|
}
|
2017-08-06 20:26:23 +00:00
|
|
|
default:
|
2023-01-23 21:13:58 +00:00
|
|
|
throw Exception(ErrorCodes::BAD_ARGUMENTS, "Unknown type of SYSTEM query");
|
2017-08-06 20:26:23 +00:00
|
|
|
}
|
|
|
|
|
2022-06-08 12:09:59 +00:00
|
|
|
return result;
|
2017-08-04 15:54:00 +00:00
|
|
|
}
|
|
|
|
|
2021-06-20 08:24:43 +00:00
|
|
|
void InterpreterSystemQuery::restoreReplica()
|
|
|
|
{
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_RESTORE_REPLICA, table_id);
|
|
|
|
|
|
|
|
const StoragePtr table_ptr = DatabaseCatalog::instance().getTable(table_id, getContext());
|
|
|
|
|
|
|
|
auto * const table_replicated_ptr = dynamic_cast<StorageReplicatedMergeTree *>(table_ptr.get());
|
|
|
|
|
|
|
|
if (table_replicated_ptr == nullptr)
|
|
|
|
throw Exception(ErrorCodes::BAD_ARGUMENTS, table_is_not_replicated.data(), table_id.getNameForLogs());
|
|
|
|
|
2022-01-20 18:55:59 +00:00
|
|
|
table_replicated_ptr->restoreMetadataInZooKeeper();
|
2021-06-20 08:24:43 +00:00
|
|
|
}
|
2017-08-06 20:26:23 +00:00
|
|
|
|
2021-05-31 14:49:02 +00:00
|
|
|
StoragePtr InterpreterSystemQuery::tryRestartReplica(const StorageID & replica, ContextMutablePtr system_context, bool need_ddl_guard)
|
2018-05-21 13:49:54 +00:00
|
|
|
{
|
2023-04-01 12:26:00 +00:00
|
|
|
LOG_TRACE(log, "Restarting replica {}", replica);
|
2021-06-20 08:24:43 +00:00
|
|
|
auto table_ddl_guard = need_ddl_guard
|
|
|
|
? DatabaseCatalog::instance().getDDLGuard(replica.getDatabaseName(), replica.getTableName())
|
|
|
|
: nullptr;
|
|
|
|
|
2021-04-10 23:33:54 +00:00
|
|
|
auto [database, table] = DatabaseCatalog::instance().tryGetDatabaseAndTable(replica, getContext());
|
2018-05-21 13:49:54 +00:00
|
|
|
ASTPtr create_ast;
|
|
|
|
|
|
|
|
/// Detach actions
|
2020-03-04 20:29:52 +00:00
|
|
|
if (!table || !dynamic_cast<const StorageReplicatedMergeTree *>(table.get()))
|
|
|
|
return nullptr;
|
2018-05-21 13:49:54 +00:00
|
|
|
|
2021-05-13 07:03:00 +00:00
|
|
|
table->flushAndShutdown();
|
2020-03-04 20:29:52 +00:00
|
|
|
{
|
2018-05-21 13:49:54 +00:00
|
|
|
/// If table was already dropped by anyone, an exception will be thrown
|
2021-04-10 23:33:54 +00:00
|
|
|
auto table_lock = table->lockExclusively(getContext()->getCurrentQueryId(), getContext()->getSettingsRef().lock_acquire_timeout);
|
|
|
|
create_ast = database->getCreateTableQuery(replica.table_name, getContext());
|
2018-05-21 13:49:54 +00:00
|
|
|
|
2021-11-18 10:45:12 +00:00
|
|
|
database->detachTable(system_context, replica.table_name);
|
2018-05-21 13:49:54 +00:00
|
|
|
}
|
2022-08-23 14:10:44 +00:00
|
|
|
UUID uuid = table->getStorageID().uuid;
|
2020-03-04 20:29:52 +00:00
|
|
|
table.reset();
|
2022-08-23 14:10:44 +00:00
|
|
|
database->waitDetachedTableNotInUse(uuid);
|
2018-05-21 13:49:54 +00:00
|
|
|
|
|
|
|
/// Attach actions
|
2020-03-04 20:29:52 +00:00
|
|
|
/// getCreateTableQuery must return canonical CREATE query representation, there are no need for AST postprocessing
|
|
|
|
auto & create = create_ast->as<ASTCreateQuery &>();
|
|
|
|
create.attach = true;
|
2018-05-21 13:49:54 +00:00
|
|
|
|
2021-04-16 10:15:35 +00:00
|
|
|
auto columns = InterpreterCreateQuery::getColumnsDescription(*create.columns_list->columns, system_context, true);
|
2020-03-04 20:29:52 +00:00
|
|
|
auto constraints = InterpreterCreateQuery::getConstraintsDescription(create.columns_list->constraints);
|
2020-04-02 01:00:21 +00:00
|
|
|
auto data_path = database->getTableDataPath(create);
|
2018-05-21 13:49:54 +00:00
|
|
|
|
2020-03-04 20:29:52 +00:00
|
|
|
table = StorageFactory::instance().get(create,
|
2020-04-02 01:00:21 +00:00
|
|
|
data_path,
|
2020-03-04 20:29:52 +00:00
|
|
|
system_context,
|
2021-04-10 23:33:54 +00:00
|
|
|
system_context->getGlobalContext(),
|
2020-03-04 20:29:52 +00:00
|
|
|
columns,
|
|
|
|
constraints,
|
|
|
|
false);
|
2018-05-21 13:49:54 +00:00
|
|
|
|
2021-11-10 13:13:27 +00:00
|
|
|
database->attachTable(system_context, replica.table_name, table, data_path);
|
2018-05-21 13:49:54 +00:00
|
|
|
|
2020-03-04 20:29:52 +00:00
|
|
|
table->startup();
|
2023-04-01 12:26:00 +00:00
|
|
|
LOG_TRACE(log, "Restarted replica {}", replica);
|
2020-03-04 20:29:52 +00:00
|
|
|
return table;
|
2018-05-21 13:49:54 +00:00
|
|
|
}
|
|
|
|
|
2022-01-25 09:47:29 +00:00
|
|
|
void InterpreterSystemQuery::restartReplica(const StorageID & replica, ContextMutablePtr system_context)
|
|
|
|
{
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_RESTART_REPLICA, replica);
|
|
|
|
if (!tryRestartReplica(replica, system_context))
|
|
|
|
throw Exception(ErrorCodes::BAD_ARGUMENTS, table_is_not_replicated.data(), replica.getNameForLogs());
|
|
|
|
}
|
|
|
|
|
2021-05-31 14:49:02 +00:00
|
|
|
void InterpreterSystemQuery::restartReplicas(ContextMutablePtr system_context)
|
2018-05-21 13:49:54 +00:00
|
|
|
{
|
2020-03-04 20:29:52 +00:00
|
|
|
std::vector<StorageID> replica_names;
|
2020-03-30 23:36:23 +00:00
|
|
|
auto & catalog = DatabaseCatalog::instance();
|
2018-05-21 13:49:54 +00:00
|
|
|
|
2022-01-25 09:47:29 +00:00
|
|
|
auto access = getContext()->getAccess();
|
|
|
|
bool access_is_granted_globally = access->isGranted(AccessType::SYSTEM_RESTART_REPLICA);
|
|
|
|
|
2020-03-30 23:36:23 +00:00
|
|
|
for (auto & elem : catalog.getDatabases())
|
2022-01-25 09:47:29 +00:00
|
|
|
{
|
2021-06-20 08:24:43 +00:00
|
|
|
for (auto it = elem.second->getTablesIterator(getContext()); it->isValid(); it->next())
|
2022-01-25 09:47:29 +00:00
|
|
|
{
|
2021-06-20 08:24:43 +00:00
|
|
|
if (dynamic_cast<const StorageReplicatedMergeTree *>(it->table().get()))
|
2022-01-25 09:47:29 +00:00
|
|
|
{
|
|
|
|
if (!access_is_granted_globally && !access->isGranted(AccessType::SYSTEM_RESTART_REPLICA, elem.first, it->name()))
|
|
|
|
{
|
|
|
|
LOG_INFO(log, "Access {} denied, skipping {}.{}", "SYSTEM RESTART REPLICA", elem.first, it->name());
|
|
|
|
continue;
|
|
|
|
}
|
2021-06-20 08:24:43 +00:00
|
|
|
replica_names.emplace_back(it->databaseName(), it->name());
|
2022-01-25 09:47:29 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
2018-05-21 13:49:54 +00:00
|
|
|
|
|
|
|
if (replica_names.empty())
|
|
|
|
return;
|
|
|
|
|
2020-03-30 23:36:23 +00:00
|
|
|
TableGuards guards;
|
2021-06-20 08:24:43 +00:00
|
|
|
|
2020-03-30 23:36:23 +00:00
|
|
|
for (const auto & name : replica_names)
|
|
|
|
guards.emplace(UniqueTableName{name.database_name, name.table_name}, nullptr);
|
2021-06-20 08:24:43 +00:00
|
|
|
|
2020-03-30 23:36:23 +00:00
|
|
|
for (auto & guard : guards)
|
|
|
|
guard.second = catalog.getDDLGuard(guard.first.database_name, guard.first.table_name);
|
|
|
|
|
2023-03-22 07:49:22 +00:00
|
|
|
size_t threads = std::min(static_cast<size_t>(getNumberOfPhysicalCPUCores()), replica_names.size());
|
2023-04-01 12:26:00 +00:00
|
|
|
LOG_DEBUG(log, "Will restart {} replicas using {} threads", replica_names.size(), threads);
|
2023-03-22 07:49:22 +00:00
|
|
|
ThreadPool pool(CurrentMetrics::RestartReplicaThreads, CurrentMetrics::RestartReplicaThreadsActive, threads);
|
2021-06-20 08:24:43 +00:00
|
|
|
|
2021-01-10 09:44:56 +00:00
|
|
|
for (auto & replica : replica_names)
|
|
|
|
{
|
|
|
|
pool.scheduleOrThrowOnError([&]() { tryRestartReplica(replica, system_context, false); });
|
|
|
|
}
|
2018-05-21 13:49:54 +00:00
|
|
|
pool.wait();
|
|
|
|
}
|
|
|
|
|
2020-05-17 12:44:22 +00:00
|
|
|
void InterpreterSystemQuery::dropReplica(ASTSystemQuery & query)
|
|
|
|
{
|
2020-06-23 12:01:51 +00:00
|
|
|
if (query.replica.empty())
|
2023-01-23 21:13:58 +00:00
|
|
|
throw Exception(ErrorCodes::BAD_ARGUMENTS, "Replica name is empty");
|
2020-06-22 15:01:40 +00:00
|
|
|
|
2020-05-17 12:44:22 +00:00
|
|
|
if (!table_id.empty())
|
|
|
|
{
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_REPLICA, table_id);
|
|
|
|
StoragePtr table = DatabaseCatalog::instance().getTable(table_id, getContext());
|
2020-05-17 12:44:22 +00:00
|
|
|
|
2020-06-23 12:01:51 +00:00
|
|
|
if (!dropReplicaImpl(query, table))
|
2021-06-20 08:24:43 +00:00
|
|
|
throw Exception(ErrorCodes::BAD_ARGUMENTS, table_is_not_replicated.data(), table_id.getNameForLogs());
|
2020-05-17 12:44:22 +00:00
|
|
|
}
|
2021-11-11 13:28:18 +00:00
|
|
|
else if (query.database)
|
2020-05-17 12:44:22 +00:00
|
|
|
{
|
2021-10-12 18:11:00 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_REPLICA, query.getDatabase());
|
|
|
|
DatabasePtr database = DatabaseCatalog::instance().getDatabase(query.getDatabase());
|
2021-04-10 23:33:54 +00:00
|
|
|
for (auto iterator = database->getTablesIterator(getContext()); iterator->isValid(); iterator->next())
|
2020-06-23 12:01:51 +00:00
|
|
|
dropReplicaImpl(query, iterator->table());
|
|
|
|
LOG_TRACE(log, "Dropped replica {} from database {}", query.replica, backQuoteIfNeed(database->getDatabaseName()));
|
|
|
|
}
|
|
|
|
else if (query.is_drop_whole_replica)
|
|
|
|
{
|
|
|
|
auto databases = DatabaseCatalog::instance().getDatabases();
|
2022-01-25 09:47:29 +00:00
|
|
|
auto access = getContext()->getAccess();
|
|
|
|
bool access_is_granted_globally = access->isGranted(AccessType::SYSTEM_DROP_REPLICA);
|
2020-06-23 12:01:51 +00:00
|
|
|
|
|
|
|
for (auto & elem : databases)
|
2020-05-24 13:53:23 +00:00
|
|
|
{
|
2020-06-23 12:01:51 +00:00
|
|
|
DatabasePtr & database = elem.second;
|
2021-04-10 23:33:54 +00:00
|
|
|
for (auto iterator = database->getTablesIterator(getContext()); iterator->isValid(); iterator->next())
|
2022-01-25 09:47:29 +00:00
|
|
|
{
|
|
|
|
if (!access_is_granted_globally && !access->isGranted(AccessType::SYSTEM_DROP_REPLICA, elem.first, iterator->name()))
|
|
|
|
{
|
|
|
|
LOG_INFO(log, "Access {} denied, skipping {}.{}", "SYSTEM DROP REPLICA", elem.first, iterator->name());
|
|
|
|
continue;
|
|
|
|
}
|
2020-06-23 12:01:51 +00:00
|
|
|
dropReplicaImpl(query, iterator->table());
|
2022-01-25 09:47:29 +00:00
|
|
|
}
|
2020-06-23 12:01:51 +00:00
|
|
|
LOG_TRACE(log, "Dropped replica {} from database {}", query.replica, backQuoteIfNeed(database->getDatabaseName()));
|
2020-05-24 13:53:23 +00:00
|
|
|
}
|
2020-06-05 07:03:51 +00:00
|
|
|
}
|
|
|
|
else if (!query.replica_zk_path.empty())
|
|
|
|
{
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_REPLICA);
|
2021-05-09 11:59:49 +00:00
|
|
|
String remote_replica_path = fs::path(query.replica_zk_path) / "replicas" / query.replica;
|
2020-06-22 15:01:40 +00:00
|
|
|
|
2020-06-23 12:01:51 +00:00
|
|
|
/// This check is actually redundant, but it may prevent from some user mistakes
|
|
|
|
for (auto & elem : DatabaseCatalog::instance().getDatabases())
|
2020-06-22 15:01:40 +00:00
|
|
|
{
|
|
|
|
DatabasePtr & database = elem.second;
|
2021-04-10 23:33:54 +00:00
|
|
|
for (auto iterator = database->getTablesIterator(getContext()); iterator->isValid(); iterator->next())
|
2020-06-22 15:01:40 +00:00
|
|
|
{
|
|
|
|
if (auto * storage_replicated = dynamic_cast<StorageReplicatedMergeTree *>(iterator->table().get()))
|
|
|
|
{
|
2022-12-22 13:31:42 +00:00
|
|
|
ReplicatedTableStatus status;
|
2020-06-22 15:01:40 +00:00
|
|
|
storage_replicated->getStatus(status);
|
2020-06-23 12:01:51 +00:00
|
|
|
if (status.zookeeper_path == query.replica_zk_path)
|
2023-01-23 21:13:58 +00:00
|
|
|
throw Exception(ErrorCodes::TABLE_WAS_NOT_DROPPED,
|
|
|
|
"There is a local table {}, which has the same table path in ZooKeeper. "
|
|
|
|
"Please check the path in query. "
|
|
|
|
"If you want to drop replica "
|
|
|
|
"of this table, use `DROP TABLE` "
|
|
|
|
"or `SYSTEM DROP REPLICA 'name' FROM db.table`",
|
|
|
|
storage_replicated->getStorageID().getNameForLogs());
|
2020-06-22 15:01:40 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2021-04-10 23:33:54 +00:00
|
|
|
auto zookeeper = getContext()->getZooKeeper();
|
2020-06-23 12:01:51 +00:00
|
|
|
|
|
|
|
bool looks_like_table_path = zookeeper->exists(query.replica_zk_path + "/replicas") ||
|
|
|
|
zookeeper->exists(query.replica_zk_path + "/dropped");
|
|
|
|
if (!looks_like_table_path)
|
2023-01-23 21:13:58 +00:00
|
|
|
throw Exception(ErrorCodes::TABLE_WAS_NOT_DROPPED, "Specified path {} does not look like a table path",
|
|
|
|
query.replica_zk_path);
|
2020-06-23 12:01:51 +00:00
|
|
|
|
2020-06-22 15:01:40 +00:00
|
|
|
if (zookeeper->exists(remote_replica_path + "/is_active"))
|
2023-01-23 21:13:58 +00:00
|
|
|
throw Exception(ErrorCodes::TABLE_WAS_NOT_DROPPED, "Can't remove replica: {}, because it's active", query.replica);
|
2020-06-22 15:01:40 +00:00
|
|
|
|
2020-06-23 12:01:51 +00:00
|
|
|
StorageReplicatedMergeTree::dropReplica(zookeeper, query.replica_zk_path, query.replica, log);
|
|
|
|
LOG_INFO(log, "Dropped replica {}", remote_replica_path);
|
2020-06-05 07:03:51 +00:00
|
|
|
}
|
2020-06-23 12:01:51 +00:00
|
|
|
else
|
2023-01-23 21:13:58 +00:00
|
|
|
throw Exception(ErrorCodes::LOGICAL_ERROR, "Invalid query");
|
2020-06-23 12:01:51 +00:00
|
|
|
}
|
2020-05-24 13:53:23 +00:00
|
|
|
|
2020-06-23 12:01:51 +00:00
|
|
|
bool InterpreterSystemQuery::dropReplicaImpl(ASTSystemQuery & query, const StoragePtr & table)
|
|
|
|
{
|
|
|
|
auto * storage_replicated = dynamic_cast<StorageReplicatedMergeTree *>(table.get());
|
|
|
|
if (!storage_replicated)
|
|
|
|
return false;
|
|
|
|
|
2022-12-22 13:31:42 +00:00
|
|
|
ReplicatedTableStatus status;
|
2020-06-23 12:01:51 +00:00
|
|
|
storage_replicated->getStatus(status);
|
|
|
|
|
|
|
|
/// Do not allow to drop local replicas and active remote replicas
|
|
|
|
if (query.replica == status.replica_name)
|
2023-01-23 21:13:58 +00:00
|
|
|
throw Exception(ErrorCodes::TABLE_WAS_NOT_DROPPED,
|
|
|
|
"We can't drop local replica, please use `DROP TABLE` if you want "
|
|
|
|
"to clean the data and drop this replica");
|
2020-06-23 12:01:51 +00:00
|
|
|
|
2023-05-05 11:11:11 +00:00
|
|
|
storage_replicated->dropReplica(status.zookeeper_path, query.replica, log);
|
2020-06-23 12:01:51 +00:00
|
|
|
LOG_TRACE(log, "Dropped replica {} of {}", query.replica, table->getStorageID().getNameForLogs());
|
|
|
|
|
|
|
|
return true;
|
2020-05-17 12:44:22 +00:00
|
|
|
}
|
|
|
|
|
2022-12-29 14:34:11 +00:00
|
|
|
void InterpreterSystemQuery::dropDatabaseReplica(ASTSystemQuery & query)
|
|
|
|
{
|
|
|
|
if (query.replica.empty())
|
2023-01-23 21:13:58 +00:00
|
|
|
throw Exception(ErrorCodes::BAD_ARGUMENTS, "Replica name is empty");
|
2022-12-29 14:34:11 +00:00
|
|
|
|
2023-04-13 08:32:44 +00:00
|
|
|
auto check_not_local_replica = [](const DatabaseReplicated * replicated, const ASTSystemQuery & query_)
|
2022-12-29 14:34:11 +00:00
|
|
|
{
|
2023-04-13 08:32:44 +00:00
|
|
|
if (!query_.replica_zk_path.empty() && fs::path(replicated->getZooKeeperPath()) != fs::path(query_.replica_zk_path))
|
2022-12-29 14:34:11 +00:00
|
|
|
return;
|
2023-04-24 13:54:03 +00:00
|
|
|
String full_replica_name = query_.shard.empty() ? query_.replica
|
|
|
|
: DatabaseReplicated::getFullReplicaName(query_.shard, query_.replica);
|
2023-04-07 16:26:23 +00:00
|
|
|
if (replicated->getFullReplicaName() != full_replica_name)
|
2022-12-29 14:34:11 +00:00
|
|
|
return;
|
|
|
|
|
|
|
|
throw Exception(ErrorCodes::TABLE_WAS_NOT_DROPPED, "There is a local database {}, which has the same path in ZooKeeper "
|
|
|
|
"and the same replica name. Please check the path in query. "
|
|
|
|
"If you want to drop replica of this database, use `DROP DATABASE`", replicated->getDatabaseName());
|
|
|
|
};
|
|
|
|
|
|
|
|
if (query.database)
|
|
|
|
{
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_REPLICA, query.getDatabase());
|
|
|
|
DatabasePtr database = DatabaseCatalog::instance().getDatabase(query.getDatabase());
|
|
|
|
if (auto * replicated = dynamic_cast<DatabaseReplicated *>(database.get()))
|
|
|
|
{
|
|
|
|
check_not_local_replica(replicated, query);
|
2023-04-07 16:26:23 +00:00
|
|
|
DatabaseReplicated::dropReplica(replicated, replicated->getZooKeeperPath(), query.shard, query.replica);
|
2022-12-29 14:34:11 +00:00
|
|
|
}
|
|
|
|
else
|
|
|
|
throw Exception(ErrorCodes::BAD_ARGUMENTS, "Database {} is not Replicated, cannot drop replica", query.getDatabase());
|
|
|
|
LOG_TRACE(log, "Dropped replica {} of Replicated database {}", query.replica, backQuoteIfNeed(database->getDatabaseName()));
|
|
|
|
}
|
|
|
|
else if (query.is_drop_whole_replica)
|
|
|
|
{
|
|
|
|
auto databases = DatabaseCatalog::instance().getDatabases();
|
|
|
|
auto access = getContext()->getAccess();
|
|
|
|
bool access_is_granted_globally = access->isGranted(AccessType::SYSTEM_DROP_REPLICA);
|
|
|
|
|
|
|
|
for (auto & elem : databases)
|
|
|
|
{
|
|
|
|
DatabasePtr & database = elem.second;
|
|
|
|
auto * replicated = dynamic_cast<DatabaseReplicated *>(database.get());
|
|
|
|
if (!replicated)
|
|
|
|
continue;
|
|
|
|
if (!access_is_granted_globally && !access->isGranted(AccessType::SYSTEM_DROP_REPLICA, elem.first))
|
|
|
|
{
|
|
|
|
LOG_INFO(log, "Access {} denied, skipping database {}", "SYSTEM DROP REPLICA", elem.first);
|
|
|
|
continue;
|
|
|
|
}
|
|
|
|
|
|
|
|
check_not_local_replica(replicated, query);
|
2023-04-07 16:26:23 +00:00
|
|
|
DatabaseReplicated::dropReplica(replicated, replicated->getZooKeeperPath(), query.shard, query.replica);
|
2022-12-29 14:34:11 +00:00
|
|
|
LOG_TRACE(log, "Dropped replica {} of Replicated database {}", query.replica, backQuoteIfNeed(database->getDatabaseName()));
|
|
|
|
}
|
|
|
|
}
|
|
|
|
else if (!query.replica_zk_path.empty())
|
|
|
|
{
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_DROP_REPLICA);
|
|
|
|
|
|
|
|
/// This check is actually redundant, but it may prevent from some user mistakes
|
|
|
|
for (auto & elem : DatabaseCatalog::instance().getDatabases())
|
|
|
|
if (auto * replicated = dynamic_cast<DatabaseReplicated *>(elem.second.get()))
|
|
|
|
check_not_local_replica(replicated, query);
|
|
|
|
|
2023-04-07 16:26:23 +00:00
|
|
|
DatabaseReplicated::dropReplica(nullptr, query.replica_zk_path, query.shard, query.replica);
|
2022-12-29 14:34:11 +00:00
|
|
|
LOG_INFO(log, "Dropped replica {} of Replicated database with path {}", query.replica, query.replica_zk_path);
|
|
|
|
}
|
|
|
|
else
|
2023-01-23 21:13:58 +00:00
|
|
|
throw Exception(ErrorCodes::LOGICAL_ERROR, "Invalid query");
|
2022-12-29 14:34:11 +00:00
|
|
|
}
|
|
|
|
|
2023-03-16 15:04:42 +00:00
|
|
|
void InterpreterSystemQuery::syncReplica(ASTSystemQuery & query)
|
2018-05-21 13:49:54 +00:00
|
|
|
{
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_SYNC_REPLICA, table_id);
|
|
|
|
StoragePtr table = DatabaseCatalog::instance().getTable(table_id, getContext());
|
2018-05-21 13:49:54 +00:00
|
|
|
|
2020-04-22 06:01:33 +00:00
|
|
|
if (auto * storage_replicated = dynamic_cast<StorageReplicatedMergeTree *>(table.get()))
|
2019-09-19 11:04:57 +00:00
|
|
|
{
|
2023-01-26 10:40:09 +00:00
|
|
|
LOG_TRACE(log, "Synchronizing entries in replica's queue with table's log and waiting for current last entry to be processed");
|
2023-03-16 15:04:42 +00:00
|
|
|
auto sync_timeout = getContext()->getSettingsRef().receive_timeout.totalMilliseconds();
|
2023-03-27 23:39:36 +00:00
|
|
|
if (!storage_replicated->waitForProcessingQueue(sync_timeout, query.sync_replica_mode))
|
2019-09-19 21:20:58 +00:00
|
|
|
{
|
2020-05-23 22:24:01 +00:00
|
|
|
LOG_ERROR(log, "SYNC REPLICA {}: Timed out!", table_id.getNameForLogs());
|
2022-05-01 13:40:18 +00:00
|
|
|
throw Exception(ErrorCodes::TIMEOUT_EXCEEDED, "SYNC REPLICA {}: command timed out. " \
|
|
|
|
"See the 'receive_timeout' setting", table_id.getNameForLogs());
|
2019-09-19 21:20:58 +00:00
|
|
|
}
|
2020-05-23 22:24:01 +00:00
|
|
|
LOG_TRACE(log, "SYNC REPLICA {}: OK", table_id.getNameForLogs());
|
2019-09-19 11:04:57 +00:00
|
|
|
}
|
2019-04-08 05:13:16 +00:00
|
|
|
else
|
2021-06-20 08:24:43 +00:00
|
|
|
throw Exception(ErrorCodes::BAD_ARGUMENTS, table_is_not_replicated.data(), table_id.getNameForLogs());
|
2018-05-21 13:49:54 +00:00
|
|
|
}
|
|
|
|
|
2022-12-12 14:33:42 +00:00
|
|
|
void InterpreterSystemQuery::waitLoadingParts()
|
|
|
|
{
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_WAIT_LOADING_PARTS, table_id);
|
|
|
|
StoragePtr table = DatabaseCatalog::instance().getTable(table_id, getContext());
|
|
|
|
|
|
|
|
if (auto * merge_tree = dynamic_cast<MergeTreeData *>(table.get()))
|
|
|
|
{
|
|
|
|
LOG_TRACE(log, "Waiting for loading of parts of table {}", table_id.getFullTableName());
|
|
|
|
merge_tree->waitForOutdatedPartsToBeLoaded();
|
|
|
|
LOG_TRACE(log, "Finished waiting for loading of parts of table {}", table_id.getFullTableName());
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
throw Exception(ErrorCodes::BAD_ARGUMENTS,
|
|
|
|
"Command WAIT LOADING PARTS is supported only for MergeTree table, but got: {}", table->getName());
|
|
|
|
}
|
|
|
|
}
|
2022-05-01 13:40:18 +00:00
|
|
|
|
|
|
|
void InterpreterSystemQuery::syncReplicatedDatabase(ASTSystemQuery & query)
|
|
|
|
{
|
|
|
|
const auto database_name = query.getDatabase();
|
2022-07-20 20:54:43 +00:00
|
|
|
auto guard = DatabaseCatalog::instance().getDDLGuard(database_name, "");
|
2022-05-01 13:40:18 +00:00
|
|
|
auto database = DatabaseCatalog::instance().getDatabase(database_name);
|
|
|
|
|
|
|
|
if (auto * ptr = typeid_cast<DatabaseReplicated *>(database.get()))
|
|
|
|
{
|
|
|
|
LOG_TRACE(log, "Synchronizing entries in the database replica's (name: {}) queue with the log", database_name);
|
|
|
|
if (!ptr->waitForReplicaToProcessAllEntries(getContext()->getSettingsRef().receive_timeout.totalMilliseconds()))
|
|
|
|
{
|
2022-07-20 20:54:43 +00:00
|
|
|
throw Exception(ErrorCodes::TIMEOUT_EXCEEDED, "SYNC DATABASE REPLICA {}: database is readonly or command timed out. " \
|
2022-05-01 13:40:18 +00:00
|
|
|
"See the 'receive_timeout' setting", database_name);
|
|
|
|
}
|
|
|
|
LOG_TRACE(log, "SYNC DATABASE REPLICA {}: OK", database_name);
|
|
|
|
}
|
|
|
|
else
|
|
|
|
throw Exception(ErrorCodes::BAD_ARGUMENTS, "SYSTEM SYNC DATABASE REPLICA query is intended to work only with Replicated engine");
|
|
|
|
}
|
|
|
|
|
|
|
|
|
2022-05-25 20:20:13 +00:00
|
|
|
void InterpreterSystemQuery::syncTransactionLog()
|
|
|
|
{
|
|
|
|
getContext()->checkTransactionsAreAllowed(/* explicit_tcl_query */ true);
|
|
|
|
TransactionLog::instance().sync();
|
|
|
|
}
|
|
|
|
|
|
|
|
|
2020-03-04 20:29:52 +00:00
|
|
|
void InterpreterSystemQuery::flushDistributed(ASTSystemQuery &)
|
2019-04-22 15:11:16 +00:00
|
|
|
{
|
2021-04-10 23:33:54 +00:00
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_FLUSH_DISTRIBUTED, table_id);
|
2019-04-22 15:11:16 +00:00
|
|
|
|
2021-04-10 23:33:54 +00:00
|
|
|
if (auto * storage_distributed = dynamic_cast<StorageDistributed *>(DatabaseCatalog::instance().getTable(table_id, getContext()).get()))
|
|
|
|
storage_distributed->flushClusterNodesAllData(getContext());
|
2019-04-22 15:11:16 +00:00
|
|
|
else
|
2023-01-23 21:13:58 +00:00
|
|
|
throw Exception(ErrorCodes::BAD_ARGUMENTS, "Table {} is not distributed", table_id.getNameForLogs());
|
2018-05-21 13:49:54 +00:00
|
|
|
}
|
|
|
|
|
2022-12-30 13:47:30 +00:00
|
|
|
[[noreturn]] void InterpreterSystemQuery::restartDisk(String &)
|
2021-04-20 19:23:54 +00:00
|
|
|
{
|
|
|
|
getContext()->checkAccess(AccessType::SYSTEM_RESTART_DISK);
|
2023-01-23 21:13:58 +00:00
|
|
|
throw Exception(ErrorCodes::NOT_IMPLEMENTED, "SYSTEM RESTART DISK is not supported");
|
2021-04-20 19:23:54 +00:00
|
|
|
}
|
|
|
|
|
2018-05-21 13:49:54 +00:00
|
|
|
|
2020-01-24 16:20:36 +00:00
|
|
|
AccessRightsElements InterpreterSystemQuery::getRequiredAccessForDDLOnCluster() const
|
|
|
|
{
|
|
|
|
const auto & query = query_ptr->as<const ASTSystemQuery &>();
|
|
|
|
using Type = ASTSystemQuery::Type;
|
|
|
|
AccessRightsElements required_access;
|
2021-06-20 08:24:43 +00:00
|
|
|
|
2020-01-24 16:20:36 +00:00
|
|
|
switch (query.type)
|
|
|
|
{
|
2022-04-19 15:51:23 +00:00
|
|
|
case Type::SHUTDOWN:
|
|
|
|
case Type::KILL:
|
2021-01-07 19:19:33 +00:00
|
|
|
case Type::SUSPEND:
|
2020-01-24 16:20:36 +00:00
|
|
|
{
|
2020-04-03 11:54:50 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_SHUTDOWN);
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
2022-04-19 15:51:23 +00:00
|
|
|
case Type::DROP_DNS_CACHE:
|
|
|
|
case Type::DROP_MARK_CACHE:
|
|
|
|
case Type::DROP_MMAP_CACHE:
|
2023-01-26 22:23:10 +00:00
|
|
|
case Type::DROP_QUERY_CACHE:
|
2020-01-24 16:20:36 +00:00
|
|
|
#if USE_EMBEDDED_COMPILER
|
2022-04-19 15:51:23 +00:00
|
|
|
case Type::DROP_COMPILED_EXPRESSION_CACHE:
|
2020-01-24 16:20:36 +00:00
|
|
|
#endif
|
|
|
|
case Type::DROP_UNCOMPRESSED_CACHE:
|
2021-08-21 17:27:22 +00:00
|
|
|
case Type::DROP_INDEX_MARK_CACHE:
|
|
|
|
case Type::DROP_INDEX_UNCOMPRESSED_CACHE:
|
2022-03-30 11:47:44 +00:00
|
|
|
case Type::DROP_FILESYSTEM_CACHE:
|
2022-08-05 16:20:15 +00:00
|
|
|
case Type::DROP_SCHEMA_CACHE:
|
2023-02-03 13:30:52 +00:00
|
|
|
#if USE_AWS_S3
|
|
|
|
case Type::DROP_S3_CLIENT_CACHE:
|
|
|
|
#endif
|
2020-01-24 16:20:36 +00:00
|
|
|
{
|
2020-04-03 11:54:50 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_DROP_CACHE);
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
2022-04-19 15:51:23 +00:00
|
|
|
case Type::RELOAD_DICTIONARY:
|
|
|
|
case Type::RELOAD_DICTIONARIES:
|
2020-01-24 16:20:36 +00:00
|
|
|
case Type::RELOAD_EMBEDDED_DICTIONARIES:
|
|
|
|
{
|
2020-04-03 11:54:50 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_RELOAD_DICTIONARY);
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
2022-04-19 15:51:23 +00:00
|
|
|
case Type::RELOAD_MODEL:
|
2021-04-16 09:56:40 +00:00
|
|
|
case Type::RELOAD_MODELS:
|
|
|
|
{
|
|
|
|
required_access.emplace_back(AccessType::SYSTEM_RELOAD_MODEL);
|
|
|
|
break;
|
|
|
|
}
|
2022-04-19 15:51:23 +00:00
|
|
|
case Type::RELOAD_FUNCTION:
|
2021-09-17 13:04:44 +00:00
|
|
|
case Type::RELOAD_FUNCTIONS:
|
|
|
|
{
|
|
|
|
required_access.emplace_back(AccessType::SYSTEM_RELOAD_FUNCTION);
|
|
|
|
break;
|
|
|
|
}
|
2020-01-24 16:20:36 +00:00
|
|
|
case Type::RELOAD_CONFIG:
|
|
|
|
{
|
2020-04-03 11:54:50 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_RELOAD_CONFIG);
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
2022-09-16 11:19:39 +00:00
|
|
|
case Type::RELOAD_USERS:
|
|
|
|
{
|
|
|
|
required_access.emplace_back(AccessType::SYSTEM_RELOAD_USERS);
|
|
|
|
break;
|
|
|
|
}
|
2022-04-19 15:51:23 +00:00
|
|
|
case Type::STOP_MERGES:
|
2020-01-24 16:20:36 +00:00
|
|
|
case Type::START_MERGES:
|
|
|
|
{
|
2021-11-11 13:28:18 +00:00
|
|
|
if (!query.table)
|
2020-04-03 11:54:50 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_MERGES);
|
2020-01-24 16:20:36 +00:00
|
|
|
else
|
2021-10-12 18:11:00 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_MERGES, query.getDatabase(), query.getTable());
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
2022-04-19 15:51:23 +00:00
|
|
|
case Type::STOP_TTL_MERGES:
|
2020-01-24 16:20:36 +00:00
|
|
|
case Type::START_TTL_MERGES:
|
|
|
|
{
|
2021-11-11 13:28:18 +00:00
|
|
|
if (!query.table)
|
2020-04-03 11:54:50 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_TTL_MERGES);
|
2020-01-24 16:20:36 +00:00
|
|
|
else
|
2021-10-12 18:11:00 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_TTL_MERGES, query.getDatabase(), query.getTable());
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
2022-04-19 15:51:23 +00:00
|
|
|
case Type::STOP_MOVES:
|
2020-01-24 16:20:36 +00:00
|
|
|
case Type::START_MOVES:
|
|
|
|
{
|
2021-11-11 13:28:18 +00:00
|
|
|
if (!query.table)
|
2020-04-03 11:54:50 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_MOVES);
|
2020-01-24 16:20:36 +00:00
|
|
|
else
|
2021-10-12 18:11:00 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_MOVES, query.getDatabase(), query.getTable());
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
2023-08-03 10:34:43 +00:00
|
|
|
case Type::STOP_PULLING_REPLICATION_LOG:
|
|
|
|
case Type::START_PULLING_REPLICATION_LOG:
|
|
|
|
{
|
|
|
|
if (!query.table)
|
|
|
|
required_access.emplace_back(AccessType::SYSTEM_PULLING_REPLICATION_LOG);
|
|
|
|
else
|
|
|
|
required_access.emplace_back(AccessType::SYSTEM_PULLING_REPLICATION_LOG, query.getDatabase(), query.getTable());
|
|
|
|
break;
|
|
|
|
}
|
2022-04-19 15:51:23 +00:00
|
|
|
case Type::STOP_FETCHES:
|
2020-01-24 16:20:36 +00:00
|
|
|
case Type::START_FETCHES:
|
|
|
|
{
|
2021-11-11 13:28:18 +00:00
|
|
|
if (!query.table)
|
2020-04-03 11:54:50 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_FETCHES);
|
2020-01-24 16:20:36 +00:00
|
|
|
else
|
2021-10-12 18:11:00 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_FETCHES, query.getDatabase(), query.getTable());
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
2022-04-19 15:51:23 +00:00
|
|
|
case Type::STOP_DISTRIBUTED_SENDS:
|
2020-01-24 16:20:36 +00:00
|
|
|
case Type::START_DISTRIBUTED_SENDS:
|
|
|
|
{
|
2021-11-11 13:28:18 +00:00
|
|
|
if (!query.table)
|
2020-04-03 11:54:50 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_DISTRIBUTED_SENDS);
|
2020-01-24 16:20:36 +00:00
|
|
|
else
|
2021-10-12 18:11:00 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_DISTRIBUTED_SENDS, query.getDatabase(), query.getTable());
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
2022-04-19 15:51:23 +00:00
|
|
|
case Type::STOP_REPLICATED_SENDS:
|
2020-01-24 16:20:36 +00:00
|
|
|
case Type::START_REPLICATED_SENDS:
|
|
|
|
{
|
2021-11-11 13:28:18 +00:00
|
|
|
if (!query.table)
|
2020-04-03 11:54:50 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_REPLICATED_SENDS);
|
2020-01-24 16:20:36 +00:00
|
|
|
else
|
2021-10-12 18:11:00 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_REPLICATED_SENDS, query.getDatabase(), query.getTable());
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
2022-04-19 15:51:23 +00:00
|
|
|
case Type::STOP_REPLICATION_QUEUES:
|
2020-01-24 16:20:36 +00:00
|
|
|
case Type::START_REPLICATION_QUEUES:
|
|
|
|
{
|
2021-11-11 13:28:18 +00:00
|
|
|
if (!query.table)
|
2020-04-03 11:54:50 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_REPLICATION_QUEUES);
|
2020-01-24 16:20:36 +00:00
|
|
|
else
|
2021-10-12 18:11:00 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_REPLICATION_QUEUES, query.getDatabase(), query.getTable());
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
2020-05-17 12:44:22 +00:00
|
|
|
case Type::DROP_REPLICA:
|
2022-12-29 14:34:11 +00:00
|
|
|
case Type::DROP_DATABASE_REPLICA:
|
2020-05-17 12:44:22 +00:00
|
|
|
{
|
2021-10-12 18:11:00 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_DROP_REPLICA, query.getDatabase(), query.getTable());
|
2020-05-17 12:44:22 +00:00
|
|
|
break;
|
|
|
|
}
|
2021-06-20 08:24:43 +00:00
|
|
|
case Type::RESTORE_REPLICA:
|
|
|
|
{
|
2021-10-12 18:11:00 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_RESTORE_REPLICA, query.getDatabase(), query.getTable());
|
2021-06-20 08:24:43 +00:00
|
|
|
break;
|
|
|
|
}
|
2020-01-24 16:20:36 +00:00
|
|
|
case Type::SYNC_REPLICA:
|
|
|
|
{
|
2021-10-12 18:11:00 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_SYNC_REPLICA, query.getDatabase(), query.getTable());
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
|
|
|
case Type::RESTART_REPLICA:
|
|
|
|
{
|
2021-10-12 18:11:00 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_RESTART_REPLICA, query.getDatabase(), query.getTable());
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
|
|
|
case Type::RESTART_REPLICAS:
|
|
|
|
{
|
2020-04-03 11:54:50 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_RESTART_REPLICA);
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
2022-12-12 14:33:42 +00:00
|
|
|
case Type::WAIT_LOADING_PARTS:
|
|
|
|
{
|
|
|
|
required_access.emplace_back(AccessType::SYSTEM_WAIT_LOADING_PARTS, query.getDatabase(), query.getTable());
|
|
|
|
break;
|
|
|
|
}
|
2022-05-01 13:40:18 +00:00
|
|
|
case Type::SYNC_DATABASE_REPLICA:
|
|
|
|
{
|
|
|
|
required_access.emplace_back(AccessType::SYSTEM_SYNC_DATABASE_REPLICA, query.getDatabase());
|
|
|
|
break;
|
|
|
|
}
|
2022-05-25 20:20:13 +00:00
|
|
|
case Type::SYNC_TRANSACTION_LOG:
|
|
|
|
{
|
|
|
|
required_access.emplace_back(AccessType::SYSTEM_SYNC_TRANSACTION_LOG);
|
|
|
|
break;
|
|
|
|
}
|
2020-01-24 16:20:36 +00:00
|
|
|
case Type::FLUSH_DISTRIBUTED:
|
|
|
|
{
|
2021-10-12 18:11:00 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_FLUSH_DISTRIBUTED, query.getDatabase(), query.getTable());
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
|
|
|
case Type::FLUSH_LOGS:
|
|
|
|
{
|
2020-04-03 11:54:50 +00:00
|
|
|
required_access.emplace_back(AccessType::SYSTEM_FLUSH_LOGS);
|
2020-01-24 16:20:36 +00:00
|
|
|
break;
|
|
|
|
}
|
2023-04-25 21:30:03 +00:00
|
|
|
case Type::FLUSH_ASYNC_INSERT_QUEUE:
|
|
|
|
{
|
|
|
|
required_access.emplace_back(AccessType::SYSTEM_FLUSH_ASYNC_INSERT_QUEUE);
|
|
|
|
break;
|
|
|
|
}
|
2021-04-20 19:23:54 +00:00
|
|
|
case Type::RESTART_DISK:
|
|
|
|
{
|
|
|
|
required_access.emplace_back(AccessType::SYSTEM_RESTART_DISK);
|
|
|
|
break;
|
|
|
|
}
|
2022-06-08 12:09:59 +00:00
|
|
|
case Type::UNFREEZE:
|
|
|
|
{
|
|
|
|
required_access.emplace_back(AccessType::SYSTEM_UNFREEZE);
|
|
|
|
break;
|
|
|
|
}
|
2023-01-27 01:10:40 +00:00
|
|
|
case Type::SYNC_FILE_CACHE:
|
|
|
|
{
|
|
|
|
required_access.emplace_back(AccessType::SYSTEM_SYNC_FILE_CACHE);
|
|
|
|
break;
|
|
|
|
}
|
2023-07-26 17:58:41 +00:00
|
|
|
case Type::STOP_LISTEN:
|
|
|
|
case Type::START_LISTEN:
|
|
|
|
{
|
|
|
|
required_access.emplace_back(AccessType::SYSTEM_LISTEN);
|
|
|
|
break;
|
|
|
|
}
|
2022-04-19 15:51:23 +00:00
|
|
|
case Type::STOP_THREAD_FUZZER:
|
|
|
|
case Type::START_THREAD_FUZZER:
|
2023-04-25 20:07:04 +00:00
|
|
|
case Type::ENABLE_FAILPOINT:
|
|
|
|
case Type::DISABLE_FAILPOINT:
|
2022-04-19 15:51:23 +00:00
|
|
|
case Type::UNKNOWN:
|
2020-01-24 16:20:36 +00:00
|
|
|
case Type::END: break;
|
|
|
|
}
|
|
|
|
return required_access;
|
|
|
|
}
|
|
|
|
|
2017-08-30 21:25:44 +00:00
|
|
|
}
|