2017-07-28 12:58:24 +00:00
|
|
|
#include <Interpreters/ClusterProxy/executeQuery.h>
|
|
|
|
#include <Interpreters/ClusterProxy/IStreamFactory.h>
|
2019-03-22 12:08:30 +00:00
|
|
|
#include <Core/Settings.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Interpreters/Context.h>
|
|
|
|
#include <Interpreters/Cluster.h>
|
|
|
|
#include <Interpreters/IInterpreter.h>
|
2017-08-29 13:23:04 +00:00
|
|
|
#include <Interpreters/ProcessList.h>
|
2021-03-07 15:51:01 +00:00
|
|
|
#include <Interpreters/OptimizeShardingKeyRewriteInVisitor.h>
|
2021-10-16 14:03:50 +00:00
|
|
|
#include <QueryPipeline/Pipe.h>
|
2020-09-18 14:16:53 +00:00
|
|
|
#include <Processors/QueryPlan/QueryPlan.h>
|
2021-07-15 16:15:16 +00:00
|
|
|
#include <Processors/QueryPlan/ReadFromRemote.h>
|
2020-09-18 14:16:53 +00:00
|
|
|
#include <Processors/QueryPlan/UnionStep.h>
|
2020-09-10 19:55:36 +00:00
|
|
|
#include <Storages/SelectQueryInfo.h>
|
2021-07-31 07:45:26 +00:00
|
|
|
#include <DataTypes/DataTypesNumber.h>
|
2016-01-28 01:00:27 +00:00
|
|
|
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
2021-03-20 12:10:22 +00:00
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
|
|
|
extern const int TOO_LARGE_DISTRIBUTED_DEPTH;
|
|
|
|
}
|
|
|
|
|
2016-01-28 01:00:27 +00:00
|
|
|
namespace ClusterProxy
|
|
|
|
{
|
|
|
|
|
2021-05-31 14:49:02 +00:00
|
|
|
ContextMutablePtr updateSettingsForCluster(const Cluster & cluster, ContextPtr context, const Settings & settings, Poco::Logger * log)
|
2016-01-28 01:00:27 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
Settings new_settings = settings;
|
2018-03-11 00:15:26 +00:00
|
|
|
new_settings.queue_max_wait_ms = Cluster::saturate(new_settings.queue_max_wait_ms, settings.max_execution_time);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2020-10-02 22:28:46 +00:00
|
|
|
/// If "secret" (in remote_servers) is not in use,
|
|
|
|
/// user on the shard is not the same as the user on the initiator,
|
|
|
|
/// hence per-user limits should not be applied.
|
|
|
|
if (cluster.getSecret().empty())
|
|
|
|
{
|
|
|
|
/// Does not matter on remote servers, because queries are sent under different user.
|
|
|
|
new_settings.max_concurrent_queries_for_user = 0;
|
|
|
|
new_settings.max_memory_usage_for_user = 0;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2020-10-02 22:28:46 +00:00
|
|
|
/// Set as unchanged to avoid sending to remote server.
|
|
|
|
new_settings.max_concurrent_queries_for_user.changed = false;
|
|
|
|
new_settings.max_memory_usage_for_user.changed = false;
|
|
|
|
}
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2020-06-29 19:58:05 +00:00
|
|
|
if (settings.force_optimize_skip_unused_shards_nesting && settings.force_optimize_skip_unused_shards)
|
2020-03-22 22:42:44 +00:00
|
|
|
{
|
2020-06-21 11:29:54 +00:00
|
|
|
if (new_settings.force_optimize_skip_unused_shards_nesting == 1)
|
|
|
|
{
|
2020-06-21 18:29:11 +00:00
|
|
|
new_settings.force_optimize_skip_unused_shards = false;
|
2020-06-21 11:29:54 +00:00
|
|
|
new_settings.force_optimize_skip_unused_shards.changed = false;
|
|
|
|
|
|
|
|
if (log)
|
|
|
|
LOG_TRACE(log, "Disabling force_optimize_skip_unused_shards for nested queries (force_optimize_skip_unused_shards_nesting exceeded)");
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
2020-06-21 18:29:11 +00:00
|
|
|
--new_settings.force_optimize_skip_unused_shards_nesting.value;
|
2020-06-21 11:29:54 +00:00
|
|
|
new_settings.force_optimize_skip_unused_shards_nesting.changed = true;
|
2020-06-18 18:45:39 +00:00
|
|
|
|
2020-06-21 11:29:54 +00:00
|
|
|
if (log)
|
|
|
|
LOG_TRACE(log, "force_optimize_skip_unused_shards_nesting is now {}", new_settings.force_optimize_skip_unused_shards_nesting);
|
|
|
|
}
|
2020-03-22 22:42:44 +00:00
|
|
|
}
|
|
|
|
|
2020-06-29 19:58:05 +00:00
|
|
|
if (settings.optimize_skip_unused_shards_nesting && settings.optimize_skip_unused_shards)
|
2020-06-16 19:02:06 +00:00
|
|
|
{
|
2020-06-21 11:29:54 +00:00
|
|
|
if (new_settings.optimize_skip_unused_shards_nesting == 1)
|
|
|
|
{
|
2020-06-21 18:29:11 +00:00
|
|
|
new_settings.optimize_skip_unused_shards = false;
|
2020-06-21 11:29:54 +00:00
|
|
|
new_settings.optimize_skip_unused_shards.changed = false;
|
|
|
|
|
|
|
|
if (log)
|
|
|
|
LOG_TRACE(log, "Disabling optimize_skip_unused_shards for nested queries (optimize_skip_unused_shards_nesting exceeded)");
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
2020-06-21 18:29:11 +00:00
|
|
|
--new_settings.optimize_skip_unused_shards_nesting.value;
|
2020-06-21 11:29:54 +00:00
|
|
|
new_settings.optimize_skip_unused_shards_nesting.changed = true;
|
2020-06-18 18:45:39 +00:00
|
|
|
|
2020-06-21 11:29:54 +00:00
|
|
|
if (log)
|
|
|
|
LOG_TRACE(log, "optimize_skip_unused_shards_nesting is now {}", new_settings.optimize_skip_unused_shards_nesting);
|
|
|
|
}
|
2020-06-16 19:02:06 +00:00
|
|
|
}
|
|
|
|
|
2021-06-03 18:13:58 +00:00
|
|
|
if (settings.offset)
|
|
|
|
{
|
|
|
|
new_settings.offset = 0;
|
|
|
|
new_settings.offset.changed = false;
|
|
|
|
}
|
|
|
|
if (settings.limit)
|
|
|
|
{
|
|
|
|
new_settings.limit = 0;
|
|
|
|
new_settings.limit.changed = false;
|
|
|
|
}
|
|
|
|
|
2021-04-10 23:33:54 +00:00
|
|
|
auto new_context = Context::createCopy(context);
|
2020-12-11 18:10:36 +00:00
|
|
|
new_context->setSettings(new_settings);
|
2019-01-09 12:21:04 +00:00
|
|
|
return new_context;
|
|
|
|
}
|
|
|
|
|
2020-09-18 14:16:53 +00:00
|
|
|
void executeQuery(
|
|
|
|
QueryPlan & query_plan,
|
2021-07-15 16:15:16 +00:00
|
|
|
const Block & header,
|
|
|
|
QueryProcessingStage::Enum processed_stage,
|
|
|
|
const StorageID & main_table,
|
|
|
|
const ASTPtr & table_func_ptr,
|
2020-09-10 19:55:36 +00:00
|
|
|
IStreamFactory & stream_factory, Poco::Logger * log,
|
2021-03-07 15:51:01 +00:00
|
|
|
const ASTPtr & query_ast, ContextPtr context, const SelectQueryInfo & query_info,
|
|
|
|
const ExpressionActionsPtr & sharding_key_expr,
|
|
|
|
const std::string & sharding_key_column_name,
|
|
|
|
const ClusterPtr & not_optimized_cluster)
|
2019-01-09 12:21:04 +00:00
|
|
|
{
|
2020-06-18 18:45:39 +00:00
|
|
|
assert(log);
|
|
|
|
|
2021-04-10 23:33:54 +00:00
|
|
|
const Settings & settings = context->getSettingsRef();
|
2019-01-09 12:21:04 +00:00
|
|
|
|
2021-04-10 23:33:54 +00:00
|
|
|
if (settings.max_distributed_depth && context->getClientInfo().distributed_depth > settings.max_distributed_depth)
|
2021-03-20 12:10:22 +00:00
|
|
|
throw Exception("Maximum distributed depth exceeded", ErrorCodes::TOO_LARGE_DISTRIBUTED_DEPTH);
|
|
|
|
|
2020-09-18 14:16:53 +00:00
|
|
|
std::vector<QueryPlanPtr> plans;
|
2021-07-15 16:15:16 +00:00
|
|
|
IStreamFactory::Shards remote_shards;
|
2019-01-09 12:21:04 +00:00
|
|
|
|
2021-03-29 19:02:34 +00:00
|
|
|
auto new_context = updateSettingsForCluster(*query_info.getCluster(), context, settings, log);
|
2019-01-09 12:21:04 +00:00
|
|
|
|
2021-03-20 12:10:22 +00:00
|
|
|
new_context->getClientInfo().distributed_depth += 1;
|
|
|
|
|
2017-08-29 13:23:04 +00:00
|
|
|
ThrottlerPtr user_level_throttler;
|
2021-04-10 23:33:54 +00:00
|
|
|
if (auto * process_list_element = context->getProcessListElement())
|
2018-03-29 13:24:36 +00:00
|
|
|
user_level_throttler = process_list_element->getUserNetworkThrottler();
|
2017-08-29 13:23:04 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
/// Network bandwidth limit, if needed.
|
|
|
|
ThrottlerPtr throttler;
|
2018-03-11 00:15:26 +00:00
|
|
|
if (settings.max_network_bandwidth || settings.max_network_bytes)
|
2017-08-29 13:23:04 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
throttler = std::make_shared<Throttler>(
|
2018-03-11 00:15:26 +00:00
|
|
|
settings.max_network_bandwidth,
|
|
|
|
settings.max_network_bytes,
|
2017-08-29 13:23:04 +00:00
|
|
|
"Limit for bytes to send or receive over network exceeded.",
|
|
|
|
user_level_throttler);
|
|
|
|
}
|
2018-03-29 13:24:36 +00:00
|
|
|
else
|
2017-08-29 13:23:04 +00:00
|
|
|
throttler = user_level_throttler;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2021-03-29 19:02:34 +00:00
|
|
|
size_t shards = query_info.getCluster()->getShardCount();
|
|
|
|
for (const auto & shard_info : query_info.getCluster()->getShardsInfo())
|
2020-11-21 10:35:03 +00:00
|
|
|
{
|
2021-03-07 15:51:01 +00:00
|
|
|
ASTPtr query_ast_for_shard;
|
2021-03-29 19:02:34 +00:00
|
|
|
if (query_info.optimized_cluster && settings.optimize_skip_unused_shards_rewrite_in && shards > 1)
|
2021-03-07 15:51:01 +00:00
|
|
|
{
|
|
|
|
query_ast_for_shard = query_ast->clone();
|
|
|
|
|
|
|
|
OptimizeShardingKeyRewriteInVisitor::Data visitor_data{
|
|
|
|
sharding_key_expr,
|
2021-06-28 18:55:30 +00:00
|
|
|
sharding_key_expr->getSampleBlock().getByPosition(0).type,
|
2021-03-07 15:51:01 +00:00
|
|
|
sharding_key_column_name,
|
|
|
|
shard_info,
|
|
|
|
not_optimized_cluster->getSlotToShard(),
|
|
|
|
};
|
|
|
|
OptimizeShardingKeyRewriteInVisitor visitor(visitor_data);
|
|
|
|
visitor.visit(query_ast_for_shard);
|
|
|
|
}
|
|
|
|
else
|
|
|
|
query_ast_for_shard = query_ast;
|
|
|
|
|
|
|
|
stream_factory.createForShard(shard_info,
|
2021-07-15 16:15:16 +00:00
|
|
|
query_ast_for_shard, main_table, table_func_ptr,
|
2021-07-31 07:45:26 +00:00
|
|
|
new_context, plans, remote_shards, shards);
|
2020-11-21 10:35:03 +00:00
|
|
|
}
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2021-07-15 16:15:16 +00:00
|
|
|
if (!remote_shards.empty())
|
2020-09-18 14:16:53 +00:00
|
|
|
{
|
2021-07-31 07:45:26 +00:00
|
|
|
Scalars scalars = context->hasQueryContext() ? context->getQueryContext()->getScalars() : Scalars{};
|
|
|
|
scalars.emplace(
|
|
|
|
"_shard_count", Block{{DataTypeUInt32().createColumnConst(1, shards), std::make_shared<DataTypeUInt32>(), "_shard_count"}});
|
2021-07-15 16:15:16 +00:00
|
|
|
auto external_tables = context->getExternalTables();
|
2020-09-18 14:16:53 +00:00
|
|
|
|
|
|
|
auto plan = std::make_unique<QueryPlan>();
|
2021-07-15 16:15:16 +00:00
|
|
|
auto read_from_remote = std::make_unique<ReadFromRemote>(
|
|
|
|
std::move(remote_shards),
|
|
|
|
header,
|
|
|
|
processed_stage,
|
|
|
|
main_table,
|
|
|
|
table_func_ptr,
|
|
|
|
new_context,
|
|
|
|
throttler,
|
2021-07-31 07:45:26 +00:00
|
|
|
std::move(scalars),
|
2021-07-15 16:15:16 +00:00
|
|
|
std::move(external_tables),
|
2021-07-31 07:45:26 +00:00
|
|
|
log,
|
|
|
|
shards);
|
2021-07-15 16:15:16 +00:00
|
|
|
|
|
|
|
read_from_remote->setStepDescription("Read from remote replica");
|
2020-09-18 14:16:53 +00:00
|
|
|
plan->addStep(std::move(read_from_remote));
|
|
|
|
plans.emplace_back(std::move(plan));
|
|
|
|
}
|
|
|
|
|
|
|
|
if (plans.empty())
|
|
|
|
return;
|
|
|
|
|
|
|
|
if (plans.size() == 1)
|
|
|
|
{
|
|
|
|
query_plan = std::move(*plans.front());
|
|
|
|
return;
|
|
|
|
}
|
|
|
|
|
|
|
|
DataStreams input_streams;
|
|
|
|
input_streams.reserve(plans.size());
|
|
|
|
for (auto & plan : plans)
|
|
|
|
input_streams.emplace_back(plan->getCurrentDataStream());
|
|
|
|
|
2021-03-25 09:57:14 +00:00
|
|
|
auto union_step = std::make_unique<UnionStep>(std::move(input_streams));
|
2020-09-18 14:16:53 +00:00
|
|
|
query_plan.unitePlans(std::move(union_step), std::move(plans));
|
2016-01-28 01:00:27 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
}
|
|
|
|
|
|
|
|
}
|