mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-23 16:12:01 +00:00
Merge pull request #53419 from ClickHouse/cleanup-parallel-replicas
Parallel replicas: remove unnecessary code
This commit is contained in:
commit
1e0cab13be
@ -60,9 +60,6 @@ public:
|
||||
/// (When there is a local replica with big delay).
|
||||
bool lazy = false;
|
||||
time_t local_delay = 0;
|
||||
|
||||
/// Set only if parallel reading from replicas is used.
|
||||
std::shared_ptr<ParallelReplicasReadingCoordinator> coordinator;
|
||||
};
|
||||
|
||||
using Shards = std::vector<Shard>;
|
||||
|
@ -28,7 +28,6 @@ namespace DB
|
||||
namespace ErrorCodes
|
||||
{
|
||||
extern const int TOO_LARGE_DISTRIBUTED_DEPTH;
|
||||
extern const int LOGICAL_ERROR;
|
||||
extern const int SUPPORT_IS_DISABLED;
|
||||
}
|
||||
|
||||
@ -281,7 +280,6 @@ void executeQueryWithParallelReplicas(
|
||||
auto all_replicas_count = std::min(static_cast<size_t>(settings.max_parallel_replicas), new_cluster->getShardCount());
|
||||
auto coordinator = std::make_shared<ParallelReplicasReadingCoordinator>(all_replicas_count);
|
||||
auto remote_plan = std::make_unique<QueryPlan>();
|
||||
auto plans = std::vector<QueryPlanPtr>();
|
||||
|
||||
/// This is a little bit weird, but we construct an "empty" coordinator without
|
||||
/// any specified reading/coordination method (like Default, InOrder, InReverseOrder)
|
||||
@ -309,20 +307,7 @@ void executeQueryWithParallelReplicas(
|
||||
&Poco::Logger::get("ReadFromParallelRemoteReplicasStep"),
|
||||
query_info.storage_limits);
|
||||
|
||||
remote_plan->addStep(std::move(read_from_remote));
|
||||
remote_plan->addInterpreterContext(context);
|
||||
plans.emplace_back(std::move(remote_plan));
|
||||
|
||||
if (std::all_of(plans.begin(), plans.end(), [](const QueryPlanPtr & plan) { return !plan; }))
|
||||
throw Exception(ErrorCodes::LOGICAL_ERROR, "No plans were generated for reading from shard. This is a bug");
|
||||
|
||||
DataStreams input_streams;
|
||||
input_streams.reserve(plans.size());
|
||||
for (const auto & plan : plans)
|
||||
input_streams.emplace_back(plan->getCurrentDataStream());
|
||||
|
||||
auto union_step = std::make_unique<UnionStep>(std::move(input_streams));
|
||||
query_plan.unitePlans(std::move(union_step), std::move(plans));
|
||||
query_plan.addStep(std::move(read_from_remote));
|
||||
}
|
||||
|
||||
}
|
||||
|
@ -118,8 +118,7 @@ ExpressionTransform
|
||||
MergingAggregatedBucketTransform × 4
|
||||
Resize 1 → 4
|
||||
GroupingAggregatedTransform 3 → 1
|
||||
(Union)
|
||||
(ReadFromRemoteParallelReplicas)
|
||||
(ReadFromRemoteParallelReplicas)
|
||||
select a, count() from pr_t group by a order by a limit 5 offset 500;
|
||||
500 1000
|
||||
501 1000
|
||||
|
Loading…
Reference in New Issue
Block a user