2017-04-01 09:19:00 +00:00
|
|
|
#include <Interpreters/InJoinSubqueriesPreprocessor.h>
|
|
|
|
#include <Interpreters/Context.h>
|
2018-10-30 16:31:21 +00:00
|
|
|
#include <Interpreters/DatabaseAndTableWithAlias.h>
|
2019-01-25 15:42:24 +00:00
|
|
|
#include <Interpreters/IdentifierSemantic.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Storages/StorageDistributed.h>
|
2019-01-15 12:28:17 +00:00
|
|
|
#include <Parsers/ASTIdentifier.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Parsers/ASTSelectQuery.h>
|
|
|
|
#include <Parsers/ASTTablesInSelectQuery.h>
|
|
|
|
#include <Parsers/ASTFunction.h>
|
2017-07-13 20:58:19 +00:00
|
|
|
#include <Common/typeid_cast.h>
|
2017-01-04 02:37:47 +00:00
|
|
|
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
extern const int DISTRIBUTED_IN_JOIN_SUBQUERY_DENIED;
|
|
|
|
extern const int LOGICAL_ERROR;
|
2017-01-04 02:37:47 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
namespace
|
|
|
|
{
|
|
|
|
|
|
|
|
/** Call a function for each non-GLOBAL subquery in IN or JOIN.
|
|
|
|
* Pass to function: AST node with subquery, and AST node with corresponding IN function or JOIN.
|
|
|
|
* Consider only first-level subqueries (do not go recursively into subqueries).
|
|
|
|
*/
|
|
|
|
template <typename F>
|
|
|
|
void forEachNonGlobalSubquery(IAST * node, F && f)
|
|
|
|
{
|
2019-03-11 13:22:51 +00:00
|
|
|
if (auto * function = node->as<ASTFunction>())
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
|
|
|
if (function->name == "in" || function->name == "notIn")
|
|
|
|
{
|
|
|
|
f(function->arguments->children.at(1).get(), function, nullptr);
|
|
|
|
return;
|
|
|
|
}
|
|
|
|
|
|
|
|
/// Pass into other functions, as subquery could be in aggregate or in lambda functions.
|
|
|
|
}
|
2019-03-11 13:22:51 +00:00
|
|
|
else if (const auto * join = node->as<ASTTablesInSelectQueryElement>())
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
|
|
|
if (join->table_join && join->table_expression)
|
|
|
|
{
|
2019-03-11 13:22:51 +00:00
|
|
|
auto * table_join = join->table_join->as<ASTTableJoin>();
|
2019-03-11 12:49:39 +00:00
|
|
|
if (table_join->locality != ASTTableJoin::Locality::Global)
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2019-03-11 13:22:51 +00:00
|
|
|
auto & subquery = join->table_expression->as<ASTTableExpression>()->subquery;
|
2017-04-01 07:20:54 +00:00
|
|
|
if (subquery)
|
2019-03-11 12:49:39 +00:00
|
|
|
f(subquery.get(), nullptr, table_join);
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
return;
|
|
|
|
}
|
|
|
|
|
|
|
|
/// Pass into other kind of JOINs, as subquery could be in ARRAY JOIN.
|
|
|
|
}
|
|
|
|
|
|
|
|
/// Descent into all children, but not into subqueries of other kind (scalar subqueries), that are irrelevant to us.
|
|
|
|
for (auto & child : node->children)
|
2019-03-11 13:22:51 +00:00
|
|
|
if (!child->as<ASTSelectQuery>())
|
2017-04-01 07:20:54 +00:00
|
|
|
forEachNonGlobalSubquery(child.get(), f);
|
2017-01-04 02:37:47 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
/** Find all (ordinary) tables in any nesting level in AST.
|
|
|
|
*/
|
|
|
|
template <typename F>
|
|
|
|
void forEachTable(IAST * node, F && f)
|
|
|
|
{
|
2019-03-11 13:22:51 +00:00
|
|
|
if (auto * table_expression = node->as<ASTTableExpression>())
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
|
|
|
auto & database_and_table = table_expression->database_and_table_name;
|
|
|
|
if (database_and_table)
|
|
|
|
f(database_and_table);
|
|
|
|
}
|
|
|
|
|
|
|
|
for (auto & child : node->children)
|
|
|
|
forEachTable(child.get(), f);
|
2017-01-04 02:37:47 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
StoragePtr tryGetTable(const ASTPtr & database_and_table, const Context & context)
|
|
|
|
{
|
2019-01-14 18:15:04 +00:00
|
|
|
DatabaseAndTableWithAlias db_and_table(database_and_table);
|
2018-10-30 16:31:21 +00:00
|
|
|
return context.tryGetTable(db_and_table.database, db_and_table.table);
|
2017-01-04 02:37:47 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
void InJoinSubqueriesPreprocessor::process(ASTSelectQuery * query) const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!query)
|
|
|
|
return;
|
|
|
|
|
|
|
|
const SettingDistributedProductMode distributed_product_mode = context.getSettingsRef().distributed_product_mode;
|
|
|
|
|
|
|
|
if (distributed_product_mode == DistributedProductMode::ALLOW)
|
|
|
|
return;
|
|
|
|
|
|
|
|
if (!query->tables)
|
|
|
|
return;
|
|
|
|
|
2019-03-11 13:22:51 +00:00
|
|
|
const auto * tables_in_select_query = query->tables->as<ASTTablesInSelectQuery>();
|
2019-03-11 12:49:39 +00:00
|
|
|
if (tables_in_select_query->children.empty())
|
2017-04-01 07:20:54 +00:00
|
|
|
return;
|
|
|
|
|
2019-03-11 13:22:51 +00:00
|
|
|
const auto * tables_element = tables_in_select_query->children[0]->as<ASTTablesInSelectQueryElement>();
|
2019-03-11 12:49:39 +00:00
|
|
|
if (!tables_element->table_expression)
|
2017-04-01 07:20:54 +00:00
|
|
|
return;
|
|
|
|
|
2019-03-11 13:22:51 +00:00
|
|
|
const auto * table_expression = tables_element->table_expression->as<ASTTableExpression>();
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
/// If not ordinary table, skip it.
|
2018-06-04 14:17:24 +00:00
|
|
|
if (!table_expression->database_and_table_name)
|
2017-04-01 07:20:54 +00:00
|
|
|
return;
|
|
|
|
|
|
|
|
/// If not really distributed table, skip it.
|
2018-08-27 17:18:14 +00:00
|
|
|
{
|
|
|
|
StoragePtr storage = tryGetTable(table_expression->database_and_table_name, context);
|
|
|
|
if (!storage || !hasAtLeastTwoShards(*storage))
|
|
|
|
return;
|
|
|
|
}
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
forEachNonGlobalSubquery(query, [&] (IAST * subquery, IAST * function, IAST * table_join)
|
|
|
|
{
|
2018-10-31 17:31:04 +00:00
|
|
|
forEachTable(subquery, [&] (ASTPtr & database_and_table)
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
|
|
|
StoragePtr storage = tryGetTable(database_and_table, context);
|
|
|
|
|
|
|
|
if (!storage || !hasAtLeastTwoShards(*storage))
|
|
|
|
return;
|
|
|
|
|
|
|
|
if (distributed_product_mode == DistributedProductMode::DENY)
|
|
|
|
{
|
|
|
|
throw Exception("Double-distributed IN/JOIN subqueries is denied (distributed_product_mode = 'deny')."
|
|
|
|
" You may rewrite query to use local tables in subqueries, or use GLOBAL keyword, or set distributed_product_mode to suitable value.",
|
|
|
|
ErrorCodes::DISTRIBUTED_IN_JOIN_SUBQUERY_DENIED);
|
|
|
|
}
|
|
|
|
else if (distributed_product_mode == DistributedProductMode::GLOBAL)
|
|
|
|
{
|
|
|
|
if (function)
|
|
|
|
{
|
2019-03-11 13:22:51 +00:00
|
|
|
auto * concrete = function->as<ASTFunction>();
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
if (concrete->name == "in")
|
|
|
|
concrete->name = "globalIn";
|
|
|
|
else if (concrete->name == "notIn")
|
|
|
|
concrete->name = "globalNotIn";
|
|
|
|
else if (concrete->name == "globalIn" || concrete->name == "globalNotIn")
|
|
|
|
{
|
|
|
|
/// Already processed.
|
|
|
|
}
|
|
|
|
else
|
|
|
|
throw Exception("Logical error: unexpected function name " + concrete->name, ErrorCodes::LOGICAL_ERROR);
|
|
|
|
}
|
|
|
|
else if (table_join)
|
2019-03-11 13:22:51 +00:00
|
|
|
table_join->as<ASTTableJoin>()->locality = ASTTableJoin::Locality::Global;
|
2017-04-01 07:20:54 +00:00
|
|
|
else
|
|
|
|
throw Exception("Logical error: unexpected AST node", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
}
|
|
|
|
else if (distributed_product_mode == DistributedProductMode::LOCAL)
|
|
|
|
{
|
|
|
|
/// Convert distributed table to corresponding remote table.
|
|
|
|
|
|
|
|
std::string database;
|
|
|
|
std::string table;
|
|
|
|
std::tie(database, table) = getRemoteDatabaseAndTableName(*storage);
|
|
|
|
|
2019-01-15 12:28:17 +00:00
|
|
|
database_and_table = createTableIdentifier(database, table);
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
else
|
|
|
|
throw Exception("InJoinSubqueriesPreprocessor: unexpected value of 'distributed_product_mode' setting", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
});
|
|
|
|
});
|
2017-01-04 02:37:47 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
bool InJoinSubqueriesPreprocessor::hasAtLeastTwoShards(const IStorage & table) const
|
|
|
|
{
|
2017-11-04 16:46:14 +00:00
|
|
|
const StorageDistributed * distributed = dynamic_cast<const StorageDistributed *>(&table);
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!distributed)
|
|
|
|
return false;
|
2017-01-04 02:37:47 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return distributed->getShardCount() >= 2;
|
2017-01-04 02:37:47 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
std::pair<std::string, std::string>
|
|
|
|
InJoinSubqueriesPreprocessor::getRemoteDatabaseAndTableName(const IStorage & table) const
|
|
|
|
{
|
2017-11-04 16:46:14 +00:00
|
|
|
const StorageDistributed & distributed = dynamic_cast<const StorageDistributed &>(table);
|
2017-04-01 07:20:54 +00:00
|
|
|
return { distributed.getRemoteDatabaseName(), distributed.getRemoteTableName() };
|
2017-01-04 02:37:47 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
}
|