2019-02-01 16:36:40 +00:00
|
|
|
#include <Common/typeid_cast.h>
|
2019-03-12 13:17:17 +00:00
|
|
|
#include <Functions/FunctionsComparison.h>
|
|
|
|
#include <Functions/FunctionsLogical.h>
|
2019-02-01 16:36:40 +00:00
|
|
|
#include <Interpreters/CrossToInnerJoinVisitor.h>
|
|
|
|
#include <Interpreters/DatabaseAndTableWithAlias.h>
|
|
|
|
#include <Interpreters/IdentifierSemantic.h>
|
2019-10-23 13:59:03 +00:00
|
|
|
#include <Interpreters/misc.h>
|
2019-02-01 16:36:40 +00:00
|
|
|
#include <Parsers/ASTSelectQuery.h>
|
2020-03-08 11:07:05 +00:00
|
|
|
#include <Parsers/ASTSubquery.h>
|
2019-02-01 16:36:40 +00:00
|
|
|
#include <Parsers/ASTTablesInSelectQuery.h>
|
|
|
|
#include <Parsers/ASTIdentifier.h>
|
2019-02-04 18:45:31 +00:00
|
|
|
#include <Parsers/ASTFunction.h>
|
2019-02-01 16:36:40 +00:00
|
|
|
#include <Parsers/ASTExpressionList.h>
|
|
|
|
#include <Parsers/ParserTablesInSelectQuery.h>
|
|
|
|
#include <Parsers/ExpressionListParsers.h>
|
|
|
|
#include <Parsers/parseQuery.h>
|
|
|
|
#include <IO/WriteHelpers.h>
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
|
|
|
extern const int LOGICAL_ERROR;
|
2019-03-11 19:45:04 +00:00
|
|
|
extern const int NOT_IMPLEMENTED;
|
2019-02-01 16:36:40 +00:00
|
|
|
}
|
|
|
|
|
2019-03-11 19:45:04 +00:00
|
|
|
namespace
|
|
|
|
{
|
|
|
|
|
2020-03-08 11:07:05 +00:00
|
|
|
struct JoinedElement
|
2019-03-11 19:45:04 +00:00
|
|
|
{
|
2020-03-18 03:27:32 +00:00
|
|
|
explicit JoinedElement(const ASTTablesInSelectQueryElement & table_element)
|
2020-03-08 11:07:05 +00:00
|
|
|
: element(table_element)
|
2019-03-11 19:45:04 +00:00
|
|
|
{
|
2020-03-08 11:07:05 +00:00
|
|
|
if (element.table_join)
|
|
|
|
join = element.table_join->as<ASTTableJoin>();
|
|
|
|
}
|
2019-03-11 19:45:04 +00:00
|
|
|
|
2020-03-08 11:07:05 +00:00
|
|
|
void checkTableName(const DatabaseAndTableWithAlias & table, const String & current_database) const
|
|
|
|
{
|
|
|
|
if (!element.table_expression)
|
|
|
|
throw Exception("Not a table expression in JOIN (ARRAY JOIN?)", ErrorCodes::LOGICAL_ERROR);
|
2019-03-11 19:45:04 +00:00
|
|
|
|
2020-03-08 11:07:05 +00:00
|
|
|
ASTTableExpression * table_expression = element.table_expression->as<ASTTableExpression>();
|
|
|
|
if (!table_expression)
|
|
|
|
throw Exception("Wrong table expression in JOIN", ErrorCodes::LOGICAL_ERROR);
|
2019-03-11 19:45:04 +00:00
|
|
|
|
2020-03-08 11:07:05 +00:00
|
|
|
if (!table.same(DatabaseAndTableWithAlias(*table_expression, current_database)))
|
|
|
|
throw Exception("Inconsistent table names", ErrorCodes::LOGICAL_ERROR);
|
2019-03-11 19:45:04 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
void rewriteCommaToCross()
|
|
|
|
{
|
2020-04-16 20:28:23 +00:00
|
|
|
if (join && join->kind == ASTTableJoin::Kind::Comma)
|
2019-03-11 19:45:04 +00:00
|
|
|
join->kind = ASTTableJoin::Kind::Cross;
|
|
|
|
}
|
|
|
|
|
2020-04-16 20:28:23 +00:00
|
|
|
bool rewriteCrossToInner(ASTPtr on_expression)
|
2020-03-08 11:07:05 +00:00
|
|
|
{
|
2020-04-16 20:28:23 +00:00
|
|
|
if (join->kind != ASTTableJoin::Kind::Cross)
|
|
|
|
return false;
|
|
|
|
|
2020-03-08 11:07:05 +00:00
|
|
|
join->kind = ASTTableJoin::Kind::Inner;
|
|
|
|
join->strictness = ASTTableJoin::Strictness::All;
|
|
|
|
|
|
|
|
join->on_expression = on_expression;
|
|
|
|
join->children.push_back(join->on_expression);
|
2020-04-16 20:28:23 +00:00
|
|
|
return true;
|
2020-03-08 11:07:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
ASTPtr arrayJoin() const { return element.array_join; }
|
|
|
|
const ASTTableJoin * tableJoin() const { return join; }
|
|
|
|
|
2019-03-11 19:45:04 +00:00
|
|
|
bool canAttachOnExpression() const { return join && !join->on_expression; }
|
2020-03-08 11:07:05 +00:00
|
|
|
bool hasUsing() const { return join && join->using_expression_list; }
|
|
|
|
|
|
|
|
private:
|
|
|
|
const ASTTablesInSelectQueryElement & element;
|
|
|
|
ASTTableJoin * join = nullptr;
|
2019-03-11 19:45:04 +00:00
|
|
|
};
|
|
|
|
|
2019-03-12 13:17:17 +00:00
|
|
|
bool isComparison(const String & name)
|
|
|
|
{
|
|
|
|
return name == NameEquals::name ||
|
|
|
|
name == NameNotEquals::name ||
|
|
|
|
name == NameLess::name ||
|
|
|
|
name == NameGreater::name ||
|
|
|
|
name == NameLessOrEquals::name ||
|
|
|
|
name == NameGreaterOrEquals::name;
|
2019-02-01 16:36:40 +00:00
|
|
|
}
|
|
|
|
|
2019-02-04 18:45:31 +00:00
|
|
|
/// It checks if where expression could be moved to JOIN ON expression partially or entirely.
|
|
|
|
class CheckExpressionVisitorData
|
2019-02-01 16:36:40 +00:00
|
|
|
{
|
2019-02-04 18:45:31 +00:00
|
|
|
public:
|
|
|
|
using TypeToVisit = const ASTFunction;
|
2019-02-01 16:36:40 +00:00
|
|
|
|
2020-03-08 11:07:05 +00:00
|
|
|
CheckExpressionVisitorData(const std::vector<JoinedElement> & tables_,
|
|
|
|
const std::vector<TableWithColumnNamesAndTypes> & tables_with_columns,
|
2020-03-18 21:38:27 +00:00
|
|
|
const Aliases & aliases_)
|
2019-10-16 17:33:53 +00:00
|
|
|
: joined_tables(tables_)
|
2020-03-08 11:07:05 +00:00
|
|
|
, tables(tables_with_columns)
|
|
|
|
, aliases(aliases_)
|
2019-03-11 19:45:04 +00:00
|
|
|
, ands_only(true)
|
2020-03-08 11:07:05 +00:00
|
|
|
{}
|
2019-02-04 18:45:31 +00:00
|
|
|
|
2019-10-21 16:22:54 +00:00
|
|
|
void visit(const ASTFunction & node, const ASTPtr & ast)
|
2019-02-04 18:45:31 +00:00
|
|
|
{
|
2019-03-11 19:45:04 +00:00
|
|
|
if (!ands_only)
|
|
|
|
return;
|
|
|
|
|
2019-03-12 13:17:17 +00:00
|
|
|
if (node.name == NameAnd::name)
|
2019-02-04 18:45:31 +00:00
|
|
|
{
|
|
|
|
if (!node.arguments || node.arguments->children.empty())
|
2020-01-11 09:50:41 +00:00
|
|
|
throw Exception("Logical error: function requires argument", ErrorCodes::LOGICAL_ERROR);
|
2019-02-04 18:45:31 +00:00
|
|
|
|
|
|
|
for (auto & child : node.arguments->children)
|
|
|
|
{
|
2019-03-11 13:22:51 +00:00
|
|
|
if (const auto * func = child->as<ASTFunction>())
|
2019-02-04 18:45:31 +00:00
|
|
|
visit(*func, child);
|
|
|
|
else
|
2019-03-11 19:45:04 +00:00
|
|
|
ands_only = false;
|
2019-02-04 18:45:31 +00:00
|
|
|
}
|
|
|
|
}
|
2019-03-12 13:17:17 +00:00
|
|
|
else if (node.name == NameEquals::name)
|
2019-02-04 18:45:31 +00:00
|
|
|
{
|
2019-03-11 19:45:04 +00:00
|
|
|
if (size_t min_table = canMoveEqualsToJoinOn(node))
|
|
|
|
asts_to_join_on[min_table].push_back(ast);
|
2019-02-04 18:45:31 +00:00
|
|
|
}
|
2019-03-12 13:17:17 +00:00
|
|
|
else if (isComparison(node.name))
|
2019-02-04 18:45:31 +00:00
|
|
|
{
|
2019-03-12 13:17:17 +00:00
|
|
|
/// leave other comparisons as is
|
2019-02-04 18:45:31 +00:00
|
|
|
}
|
2020-02-20 13:33:14 +00:00
|
|
|
else if (functionIsLikeOperator(node.name) || /// LIKE, NOT LIKE
|
|
|
|
functionIsInOperator(node.name)) /// IN, NOT IN
|
2020-02-19 17:22:16 +00:00
|
|
|
{
|
2020-02-20 13:33:14 +00:00
|
|
|
/// leave as is. It's not possible to make push down here cause of unknown aliases and not implemented JOIN predicates.
|
|
|
|
/// select a as b form t1, t2 where t1.x = t2.x and b in(42)
|
|
|
|
/// select a as b form t1 inner join t2 on t1.x = t2.x and b in(42)
|
2019-05-23 16:01:17 +00:00
|
|
|
}
|
2019-02-04 18:45:31 +00:00
|
|
|
else
|
2019-03-11 19:45:04 +00:00
|
|
|
{
|
|
|
|
ands_only = false;
|
|
|
|
asts_to_join_on.clear();
|
|
|
|
}
|
2019-02-04 18:45:31 +00:00
|
|
|
}
|
|
|
|
|
2019-03-11 19:45:04 +00:00
|
|
|
bool complex() const { return !ands_only; }
|
|
|
|
bool matchAny(size_t t) const { return asts_to_join_on.count(t); }
|
2019-02-04 18:45:31 +00:00
|
|
|
|
2019-03-11 19:45:04 +00:00
|
|
|
ASTPtr makeOnExpression(size_t table_pos)
|
2019-02-04 18:45:31 +00:00
|
|
|
{
|
2019-03-11 19:45:04 +00:00
|
|
|
if (!asts_to_join_on.count(table_pos))
|
|
|
|
return {};
|
|
|
|
|
|
|
|
std::vector<ASTPtr> & expressions = asts_to_join_on[table_pos];
|
|
|
|
|
|
|
|
if (expressions.size() == 1)
|
|
|
|
return expressions[0]->clone();
|
2019-02-04 18:45:31 +00:00
|
|
|
|
|
|
|
std::vector<ASTPtr> arguments;
|
2019-03-11 19:45:04 +00:00
|
|
|
arguments.reserve(expressions.size());
|
|
|
|
for (auto & ast : expressions)
|
2019-02-04 18:45:31 +00:00
|
|
|
arguments.emplace_back(ast->clone());
|
|
|
|
|
2019-03-12 13:17:17 +00:00
|
|
|
return makeASTFunction(NameAnd::name, std::move(arguments));
|
2019-02-04 18:45:31 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
private:
|
2020-03-08 11:07:05 +00:00
|
|
|
const std::vector<JoinedElement> & joined_tables;
|
|
|
|
const std::vector<TableWithColumnNamesAndTypes> & tables;
|
2019-03-11 19:45:04 +00:00
|
|
|
std::map<size_t, std::vector<ASTPtr>> asts_to_join_on;
|
2020-03-18 21:38:27 +00:00
|
|
|
const Aliases & aliases;
|
2019-03-11 19:45:04 +00:00
|
|
|
bool ands_only;
|
2019-02-04 18:45:31 +00:00
|
|
|
|
2019-03-11 19:45:04 +00:00
|
|
|
size_t canMoveEqualsToJoinOn(const ASTFunction & node)
|
2019-02-04 18:45:31 +00:00
|
|
|
{
|
|
|
|
if (!node.arguments)
|
2019-11-07 11:44:02 +00:00
|
|
|
throw Exception("Logical error: function requires arguments", ErrorCodes::LOGICAL_ERROR);
|
2019-02-04 18:45:31 +00:00
|
|
|
if (node.arguments->children.size() != 2)
|
|
|
|
return false;
|
2019-02-01 16:36:40 +00:00
|
|
|
|
2019-03-11 13:22:51 +00:00
|
|
|
const auto * left = node.arguments->children[0]->as<ASTIdentifier>();
|
|
|
|
const auto * right = node.arguments->children[1]->as<ASTIdentifier>();
|
2019-02-04 18:45:31 +00:00
|
|
|
if (!left || !right)
|
|
|
|
return false;
|
2019-02-01 16:36:40 +00:00
|
|
|
|
2020-03-08 11:07:05 +00:00
|
|
|
/// Moving expressions that use column aliases is not supported.
|
|
|
|
if (left->isShort() && aliases.count(left->shortName()))
|
|
|
|
return false;
|
|
|
|
if (right->isShort() && aliases.count(right->shortName()))
|
|
|
|
return false;
|
|
|
|
|
2019-02-04 18:45:31 +00:00
|
|
|
return checkIdentifiers(*left, *right);
|
|
|
|
}
|
|
|
|
|
|
|
|
/// Check if the identifiers are from different joined tables. If it's a self joint, tables should have aliases.
|
|
|
|
/// select * from t1 a cross join t2 b where a.x = b.x
|
2019-03-11 19:45:04 +00:00
|
|
|
/// @return table position to attach expression to or 0.
|
|
|
|
size_t checkIdentifiers(const ASTIdentifier & left, const ASTIdentifier & right)
|
2019-02-01 16:36:40 +00:00
|
|
|
{
|
2020-03-08 11:07:05 +00:00
|
|
|
std::optional<size_t> left_table_pos = IdentifierSemantic::getMembership(left);
|
|
|
|
if (!left_table_pos)
|
|
|
|
left_table_pos = IdentifierSemantic::chooseTable(left, tables);
|
2019-02-04 18:45:31 +00:00
|
|
|
|
2020-03-08 11:07:05 +00:00
|
|
|
std::optional<size_t> right_table_pos = IdentifierSemantic::getMembership(right);
|
|
|
|
if (!right_table_pos)
|
|
|
|
right_table_pos = IdentifierSemantic::chooseTable(right, tables);
|
2019-02-04 18:45:31 +00:00
|
|
|
|
2020-03-08 11:07:05 +00:00
|
|
|
if (left_table_pos && right_table_pos && (*left_table_pos != *right_table_pos))
|
2019-03-11 19:45:04 +00:00
|
|
|
{
|
2020-03-08 11:07:05 +00:00
|
|
|
size_t table_pos = std::max(*left_table_pos, *right_table_pos);
|
2019-10-16 17:33:53 +00:00
|
|
|
if (joined_tables[table_pos].canAttachOnExpression())
|
2019-03-11 19:45:04 +00:00
|
|
|
return table_pos;
|
|
|
|
}
|
|
|
|
return 0;
|
2019-02-01 16:36:40 +00:00
|
|
|
}
|
|
|
|
};
|
|
|
|
|
2020-04-14 18:50:00 +00:00
|
|
|
using CheckExpressionMatcher = ConstOneTypeMatcher<CheckExpressionVisitorData, NeedChild::none>;
|
2019-10-21 16:22:54 +00:00
|
|
|
using CheckExpressionVisitor = ConstInDepthNodeVisitor<CheckExpressionMatcher, true>;
|
2019-02-01 16:36:40 +00:00
|
|
|
|
|
|
|
|
2020-03-08 11:07:05 +00:00
|
|
|
bool getTables(ASTSelectQuery & select, std::vector<JoinedElement> & joined_tables, size_t & num_comma)
|
2019-02-01 16:36:40 +00:00
|
|
|
{
|
2019-04-09 14:22:35 +00:00
|
|
|
if (!select.tables())
|
2019-03-11 19:45:04 +00:00
|
|
|
return false;
|
2019-02-01 16:36:40 +00:00
|
|
|
|
2019-04-09 14:22:35 +00:00
|
|
|
const auto * tables = select.tables()->as<ASTTablesInSelectQuery>();
|
2019-02-01 16:36:40 +00:00
|
|
|
if (!tables)
|
2019-03-11 19:45:04 +00:00
|
|
|
return false;
|
2019-02-01 16:36:40 +00:00
|
|
|
|
|
|
|
size_t num_tables = tables->children.size();
|
2019-03-11 19:45:04 +00:00
|
|
|
if (num_tables < 2)
|
|
|
|
return false;
|
2019-02-01 16:36:40 +00:00
|
|
|
|
2019-03-11 19:45:04 +00:00
|
|
|
joined_tables.reserve(num_tables);
|
2019-04-08 12:35:26 +00:00
|
|
|
size_t num_array_join = 0;
|
|
|
|
size_t num_using = 0;
|
|
|
|
|
2020-04-22 05:39:31 +00:00
|
|
|
for (const auto & child : tables->children)
|
2019-02-01 16:36:40 +00:00
|
|
|
{
|
2020-04-22 05:39:31 +00:00
|
|
|
auto * table_element = child->as<ASTTablesInSelectQueryElement>();
|
2020-03-08 11:07:05 +00:00
|
|
|
if (!table_element)
|
|
|
|
throw Exception("Logical error: TablesInSelectQueryElement expected", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
|
|
|
|
joined_tables.emplace_back(JoinedElement(*table_element));
|
|
|
|
JoinedElement & t = joined_tables.back();
|
|
|
|
|
|
|
|
if (t.arrayJoin())
|
2019-04-08 12:35:26 +00:00
|
|
|
{
|
|
|
|
++num_array_join;
|
|
|
|
continue;
|
|
|
|
}
|
2019-02-01 16:36:40 +00:00
|
|
|
|
2020-03-08 11:07:05 +00:00
|
|
|
if (t.hasUsing())
|
2019-04-08 12:35:26 +00:00
|
|
|
{
|
|
|
|
++num_using;
|
|
|
|
continue;
|
|
|
|
}
|
2019-02-01 16:36:40 +00:00
|
|
|
|
2020-04-22 05:39:31 +00:00
|
|
|
if (const auto * join = t.tableJoin())
|
2020-03-08 11:07:05 +00:00
|
|
|
{
|
|
|
|
if (join->kind == ASTTableJoin::Kind::Cross ||
|
|
|
|
join->kind == ASTTableJoin::Kind::Comma)
|
|
|
|
{
|
|
|
|
if (!join->children.empty())
|
|
|
|
throw Exception("Logical error: CROSS JOIN has expressions", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
}
|
|
|
|
|
2019-03-11 19:45:04 +00:00
|
|
|
if (join->kind == ASTTableJoin::Kind::Comma)
|
|
|
|
++num_comma;
|
2020-03-08 11:07:05 +00:00
|
|
|
}
|
2019-02-01 16:36:40 +00:00
|
|
|
}
|
2019-04-08 12:35:26 +00:00
|
|
|
|
|
|
|
if (num_using && (num_tables - num_array_join) > 2)
|
|
|
|
throw Exception("Multiple CROSS/COMMA JOIN do not support USING", ErrorCodes::NOT_IMPLEMENTED);
|
|
|
|
|
2020-03-08 23:48:08 +00:00
|
|
|
return !(num_array_join || num_using);
|
2019-03-11 19:45:04 +00:00
|
|
|
}
|
2019-02-01 16:36:40 +00:00
|
|
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
2020-03-08 11:07:05 +00:00
|
|
|
bool CrossToInnerJoinMatcher::needChildVisit(ASTPtr & node, const ASTPtr &)
|
|
|
|
{
|
2020-03-08 23:48:08 +00:00
|
|
|
return !node->as<ASTSubquery>();
|
2020-03-08 11:07:05 +00:00
|
|
|
}
|
|
|
|
|
2019-02-22 13:33:56 +00:00
|
|
|
void CrossToInnerJoinMatcher::visit(ASTPtr & ast, Data & data)
|
2019-02-01 16:36:40 +00:00
|
|
|
{
|
2019-03-11 13:22:51 +00:00
|
|
|
if (auto * t = ast->as<ASTSelectQuery>())
|
2019-02-01 16:36:40 +00:00
|
|
|
visit(*t, ast, data);
|
|
|
|
}
|
|
|
|
|
2019-03-11 19:45:04 +00:00
|
|
|
void CrossToInnerJoinMatcher::visit(ASTSelectQuery & select, ASTPtr &, Data & data)
|
2019-02-01 16:36:40 +00:00
|
|
|
{
|
2019-03-11 19:45:04 +00:00
|
|
|
size_t num_comma = 0;
|
2020-03-08 11:07:05 +00:00
|
|
|
std::vector<JoinedElement> joined_tables;
|
2019-03-11 19:45:04 +00:00
|
|
|
if (!getTables(select, joined_tables, num_comma))
|
2019-02-01 16:36:40 +00:00
|
|
|
return;
|
|
|
|
|
2020-03-08 11:07:05 +00:00
|
|
|
/// Check if joined_tables are consistent with known tables_with_columns
|
|
|
|
{
|
|
|
|
if (joined_tables.size() != data.tables_with_columns.size())
|
|
|
|
throw Exception("Logical error: inconsistent number of tables", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
|
|
|
|
for (size_t i = 0; i < joined_tables.size(); ++i)
|
|
|
|
joined_tables[i].checkTableName(data.tables_with_columns[i].table, data.current_database);
|
|
|
|
}
|
|
|
|
|
2019-03-11 19:45:04 +00:00
|
|
|
/// COMMA to CROSS
|
2019-02-01 16:36:40 +00:00
|
|
|
|
2019-03-11 19:45:04 +00:00
|
|
|
if (num_comma)
|
2019-02-04 18:45:31 +00:00
|
|
|
{
|
2019-03-11 19:45:04 +00:00
|
|
|
for (auto & table : joined_tables)
|
|
|
|
table.rewriteCommaToCross();
|
|
|
|
}
|
|
|
|
|
|
|
|
/// CROSS to INNER
|
|
|
|
|
2019-04-09 14:22:35 +00:00
|
|
|
if (!select.where())
|
2019-03-11 19:45:04 +00:00
|
|
|
return;
|
2019-02-01 16:36:40 +00:00
|
|
|
|
2020-03-18 21:38:27 +00:00
|
|
|
CheckExpressionVisitor::Data visitor_data{joined_tables, data.tables_with_columns, data.aliases};
|
2019-10-21 16:22:54 +00:00
|
|
|
CheckExpressionVisitor(visitor_data).visit(select.where());
|
2019-02-01 16:36:40 +00:00
|
|
|
|
2019-03-11 19:45:04 +00:00
|
|
|
if (visitor_data.complex())
|
|
|
|
return;
|
2019-02-01 16:36:40 +00:00
|
|
|
|
2019-03-11 19:45:04 +00:00
|
|
|
for (size_t i = 1; i < joined_tables.size(); ++i)
|
|
|
|
{
|
|
|
|
if (visitor_data.matchAny(i))
|
|
|
|
{
|
2020-04-16 20:28:23 +00:00
|
|
|
if (joined_tables[i].rewriteCrossToInner(visitor_data.makeOnExpression(i)))
|
|
|
|
data.done = true;
|
2019-03-11 19:45:04 +00:00
|
|
|
}
|
2019-03-05 15:16:59 +00:00
|
|
|
}
|
2019-02-01 16:36:40 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
}
|