11#include < Storages/IStorageCluster.h>
22
3+ #include < pcg_random.hpp>
4+ #include < Common/randomSeed.h>
5+
36#include < Common/Exception.h>
47#include < Core/Settings.h>
58#include < Core/QueryProcessingStage.h>
1316#include < Interpreters/AddDefaultDatabaseVisitor.h>
1417#include < Interpreters/TranslateQualifiedNamesVisitor.h>
1518#include < Interpreters/InterpreterSelectQueryAnalyzer.h>
19+ #include < Planner/Utils.h>
1620#include < Processors/Sources/NullSource.h>
1721#include < Processors/Sources/RemoteSource.h>
1822#include < QueryPipeline/narrowPipe.h>
2832#include < Analyzer/QueryNode.h>
2933#include < Analyzer/ColumnNode.h>
3034#include < Analyzer/InDepthQueryTreeVisitor.h>
35+ #include < Storages/StorageDistributed.h>
36+ #include < TableFunctions/TableFunctionFactory.h>
3137
3238#include < algorithm>
3339#include < memory>
@@ -47,6 +53,7 @@ namespace Setting
4753 extern const SettingsNonZeroUInt64 max_parallel_replicas;
4854 extern const SettingsObjectStorageClusterJoinMode object_storage_cluster_join_mode;
4955 extern const SettingsUInt64 object_storage_max_nodes;
56+ extern const SettingsBool object_storage_remote_initiator;
5057}
5158
5259namespace ErrorCodes
@@ -283,8 +290,9 @@ void IStorageCluster::read(
283290
284291 storage_snapshot->check (column_names);
285292
286- updateBeforeRead (context);
287- auto cluster = getClusterImpl (context, cluster_name_from_settings, context->getSettingsRef ()[Setting::object_storage_max_nodes]);
293+ const auto & settings = context->getSettingsRef ();
294+
295+ auto cluster = getClusterImpl (context, cluster_name_from_settings, settings[Setting::object_storage_max_nodes]);
288296
289297 // / Calculate the header. This is significant, because some columns could be thrown away in some cases like query with count(*)
290298
@@ -293,7 +301,7 @@ void IStorageCluster::read(
293301
294302 updateQueryWithJoinToSendIfNeeded (query_to_send, query_info.query_tree , context);
295303
296- if (context-> getSettingsRef () [Setting::allow_experimental_analyzer])
304+ if (settings [Setting::allow_experimental_analyzer])
297305 {
298306 sample_block = InterpreterSelectQueryAnalyzer::getSampleBlock (query_to_send, context, SelectQueryOptions (processed_stage));
299307 }
@@ -306,6 +314,17 @@ void IStorageCluster::read(
306314
307315 updateQueryToSendIfNeeded (query_to_send, storage_snapshot, context);
308316
317+ if (settings[Setting::object_storage_remote_initiator])
318+ {
319+ auto storage_and_context = convertToRemote (cluster, context, cluster_name_from_settings, query_to_send);
320+ auto src_distributed = std::dynamic_pointer_cast<StorageDistributed>(storage_and_context.storage );
321+ auto modified_query_info = query_info;
322+ modified_query_info.cluster = src_distributed->getCluster ();
323+ auto new_storage_snapshot = storage_and_context.storage ->getStorageSnapshot (storage_snapshot->metadata , storage_and_context.context );
324+ storage_and_context.storage ->read (query_plan, column_names, new_storage_snapshot, modified_query_info, storage_and_context.context , processed_stage, max_block_size, num_streams);
325+ return ;
326+ }
327+
309328 RestoreQualifiedNamesVisitor::Data data;
310329 data.distributed_table = DatabaseAndTableWithAlias (*getTableExpression (query_to_send->as <ASTSelectQuery &>(), 0 ));
311330 data.remote_table .database = context->getCurrentDatabase ();
@@ -333,6 +352,62 @@ void IStorageCluster::read(
333352 query_plan.addStep (std::move (reading));
334353}
335354
355+ IStorageCluster::RemoteCallVariables IStorageCluster::convertToRemote (
356+ ClusterPtr cluster,
357+ ContextPtr context,
358+ const std::string & cluster_name_from_settings,
359+ ASTPtr query_to_send)
360+ {
361+ auto host_addresses = cluster->getShardsAddresses ();
362+ if (host_addresses.empty ())
363+ throw Exception (ErrorCodes::LOGICAL_ERROR, " Empty cluster {}" , cluster_name_from_settings);
364+
365+ static pcg64 rng (randomSeed ());
366+ size_t shard_num = rng () % host_addresses.size ();
367+ auto shard_addresses = host_addresses[shard_num];
368+ // / After getClusterImpl each shard must have exactly 1 replica
369+ if (shard_addresses.size () != 1 )
370+ throw Exception (ErrorCodes::LOGICAL_ERROR, " Size of shard {} in cluster {} is not equal 1" , shard_num, cluster_name_from_settings);
371+ auto host_name = shard_addresses[0 ].toString ();
372+
373+ LOG_INFO (log, " Choose remote initiator '{}'" , host_name);
374+
375+ bool secure = shard_addresses[0 ].secure == Protocol::Secure::Enable;
376+ std::string remote_function_name = secure ? " remoteSecure" : " remote" ;
377+
378+ // / Clean object_storage_remote_initiator setting to avoid infinite remote call
379+ auto new_context = Context::createCopy (context);
380+ new_context->setSetting (" object_storage_remote_initiator" , false );
381+
382+ auto * select_query = query_to_send->as <ASTSelectQuery>();
383+ if (!select_query)
384+ throw Exception (ErrorCodes::LOGICAL_ERROR, " Expected SELECT query" );
385+
386+ auto query_settings = select_query->settings ();
387+ if (query_settings)
388+ {
389+ auto & settings_ast = query_settings->as <ASTSetQuery &>();
390+ if (settings_ast.changes .removeSetting (" object_storage_remote_initiator" ) && settings_ast.changes .empty ())
391+ {
392+ select_query->setExpression (ASTSelectQuery::Expression::SETTINGS, {});
393+ }
394+ }
395+
396+ ASTTableExpression * table_expression = extractTableExpressionASTPtrFromSelectQuery (query_to_send);
397+ if (!table_expression)
398+ throw Exception (ErrorCodes::LOGICAL_ERROR, " Can't find table expression" );
399+
400+ auto remote_query = makeASTFunction (remote_function_name, std::make_shared<ASTLiteral>(host_name), table_expression->table_function );
401+
402+ table_expression->table_function = remote_query;
403+
404+ auto remote_function = TableFunctionFactory::instance ().get (remote_query, new_context);
405+
406+ auto storage = remote_function->execute (query_to_send, new_context, remote_function_name);
407+
408+ return RemoteCallVariables{storage, new_context};
409+ }
410+
336411SinkToStoragePtr IStorageCluster::write (
337412 const ASTPtr & query,
338413 const StorageMetadataPtr & metadata_snapshot,
0 commit comments