|
16 | 16 | #include <Storages/extractTableFunctionFromSelectQuery.h>
|
17 | 17 | #include <Storages/ObjectStorage/StorageObjectStorageStableTaskDistributor.h>
|
18 | 18 |
|
19 |
| - |
20 | 19 | namespace DB
|
21 | 20 | {
|
22 | 21 | namespace Setting
|
@@ -177,13 +176,28 @@ void StorageObjectStorageCluster::updateQueryToSendIfNeeded(
|
177 | 176 | }
|
178 | 177 |
|
179 | 178 | RemoteQueryExecutor::Extension StorageObjectStorageCluster::getTaskIteratorExtension(
|
180 |
| - const ActionsDAG::Node * predicate, const ContextPtr & local_context, const size_t number_of_replicas) const |
| 179 | + const ActionsDAG::Node * predicate, |
| 180 | + const ContextPtr & local_context, |
| 181 | + ClusterPtr cluster) const |
181 | 182 | {
|
182 | 183 | auto iterator = StorageObjectStorageSource::createFileIterator(
|
183 | 184 | configuration, configuration->getQuerySettings(local_context), object_storage, /* distributed_processing */false,
|
184 | 185 | local_context, predicate, {}, virtual_columns, nullptr, local_context->getFileProgressCallback(), /*ignore_archive_globs=*/true, /*skip_object_metadata=*/true);
|
185 | 186 |
|
186 |
| - auto task_distributor = std::make_shared<StorageObjectStorageStableTaskDistributor>(iterator, number_of_replicas); |
| 187 | + std::vector<std::string> ids_of_hosts; |
| 188 | + for (const auto & shard : cluster->getShardsInfo()) |
| 189 | + { |
| 190 | + if (shard.per_replica_pools.empty()) |
| 191 | + throw Exception(ErrorCodes::LOGICAL_ERROR, "Cluster {} with empty shard {}", cluster->getName(), shard.shard_num); |
| 192 | + for (const auto & replica : shard.per_replica_pools) |
| 193 | + { |
| 194 | + if (!replica) |
| 195 | + throw Exception(ErrorCodes::LOGICAL_ERROR, "Cluster {}, shard {} with empty node", cluster->getName(), shard.shard_num); |
| 196 | + ids_of_hosts.push_back(replica->getAddress()); |
| 197 | + } |
| 198 | + } |
| 199 | + |
| 200 | + auto task_distributor = std::make_shared<StorageObjectStorageStableTaskDistributor>(iterator, ids_of_hosts); |
187 | 201 |
|
188 | 202 | auto callback = std::make_shared<TaskIterator>(
|
189 | 203 | [task_distributor](size_t number_of_current_replica) mutable -> String {
|
|
0 commit comments