@@ -30,12 +30,14 @@ namespace DB
3030namespace Setting
3131{
3232 extern const SettingsBool use_hive_partitioning;
33+ extern const SettingsUInt64 lock_object_storage_task_distribution_ms;
3334 extern const SettingsString object_storage_cluster;
3435}
3536
3637namespace ErrorCodes
3738{
3839 extern const int LOGICAL_ERROR;
40+ extern const int INVALID_SETTING_VALUE;
3941}
4042
4143String StorageObjectStorageCluster::getPathSample (ContextPtr context)
@@ -437,13 +439,43 @@ void StorageObjectStorageCluster::updateQueryToSendIfNeeded(
437439}
438440
439441RemoteQueryExecutor::Extension StorageObjectStorageCluster::getTaskIteratorExtension (
440- const ActionsDAG::Node * predicate, const ContextPtr & local_context, const size_t number_of_replicas) const
442+ const ActionsDAG::Node * predicate,
443+ const ContextPtr & local_context,
444+ ClusterPtr cluster) const
441445{
442446 auto iterator = StorageObjectStorageSource::createFileIterator (
443447 configuration, configuration->getQuerySettings (local_context), object_storage, /* distributed_processing */ false ,
444448 local_context, predicate, {}, getVirtualsList (), hive_partition_columns_to_read_from_file_path, nullptr , local_context->getFileProgressCallback (), /* ignore_archive_globs=*/ true , /* skip_object_metadata=*/ true );
445449
446- auto task_distributor = std::make_shared<StorageObjectStorageStableTaskDistributor>(iterator, number_of_replicas);
450+ std::vector<std::string> ids_of_hosts;
451+ for (const auto & shard : cluster->getShardsInfo ())
452+ {
453+ if (shard.per_replica_pools .empty ())
454+ throw Exception (ErrorCodes::LOGICAL_ERROR, " Cluster {} with empty shard {}" , cluster->getName (), shard.shard_num );
455+ for (const auto & replica : shard.per_replica_pools )
456+ {
457+ if (!replica)
458+ throw Exception (ErrorCodes::LOGICAL_ERROR, " Cluster {}, shard {} with empty node" , cluster->getName (), shard.shard_num );
459+ ids_of_hosts.push_back (replica->getAddress ());
460+ }
461+ }
462+
463+ uint64_t lock_object_storage_task_distribution_ms = local_context->getSettingsRef ()[Setting::lock_object_storage_task_distribution_ms];
464+
465+ // / Check value to avoid negative result after conversion in microseconds.
466+ // / Poco::Timestamp::TimeDiff is signed int 64.
467+ static const uint64_t lock_object_storage_task_distribution_ms_max = 0x0020000000000000ULL ;
468+ if (lock_object_storage_task_distribution_ms > lock_object_storage_task_distribution_ms_max)
469+ throw Exception (ErrorCodes::INVALID_SETTING_VALUE,
470+ " Value lock_object_storage_task_distribution_ms is too big: {}, allowed maximum is {}" ,
471+ lock_object_storage_task_distribution_ms,
472+ lock_object_storage_task_distribution_ms_max
473+ );
474+
475+ auto task_distributor = std::make_shared<StorageObjectStorageStableTaskDistributor>(
476+ iterator,
477+ ids_of_hosts,
478+ lock_object_storage_task_distribution_ms);
447479
448480 auto callback = std::make_shared<TaskIterator>(
449481 [task_distributor](size_t number_of_current_replica) mutable -> String
0 commit comments