mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-21 15:12:02 +00:00
Enable predicate push-down optimization by default. (#4846)
* Enable predicate push-down optimization by default. * Forbid push-downs for some JOIN cases. * Fix existing tests * Forbid optimization if a select query has ARRAY JOIN on any side.
This commit is contained in:
parent
d8203ba664
commit
118bea2be6
@ -288,7 +288,7 @@ struct Settings
|
||||
M(SettingBool, log_query_settings, true, "Log query settings into the query_log.") \
|
||||
M(SettingBool, log_query_threads, true, "Log query threads into system.query_thread_log table. This setting have effect only when 'log_queries' is true.") \
|
||||
M(SettingLogsLevel, send_logs_level, LogsLevel::none, "Send server text logs with specified minimum level to client. Valid values: 'trace', 'debug', 'information', 'warning', 'error', 'none'") \
|
||||
M(SettingBool, enable_optimize_predicate_expression, 0, "If it is set to true, optimize predicates to subqueries.") \
|
||||
M(SettingBool, enable_optimize_predicate_expression, 1, "If it is set to true, optimize predicates to subqueries.") \
|
||||
\
|
||||
M(SettingUInt64, low_cardinality_max_dictionary_size, 8192, "Maximum size (in rows) of shared global dictionary for LowCardinality type.") \
|
||||
M(SettingBool, low_cardinality_use_single_dictionary_for_part, false, "LowCardinality type serialization setting. If is true, than will use additional keys when global dictionary overflows. Otherwise, will create several shared dictionaries.") \
|
||||
|
@ -281,7 +281,7 @@ InterpreterSelectQuery::InterpreterSelectQuery(
|
||||
{
|
||||
if (query_analyzer->isRewriteSubqueriesPredicate())
|
||||
{
|
||||
/// remake interpreter_subquery when PredicateOptimizer is rewrite subqueries and main table is subquery
|
||||
/// remake interpreter_subquery when PredicateOptimizer rewrites subqueries and main table is subquery
|
||||
if (is_subquery)
|
||||
interpreter_subquery = std::make_unique<InterpreterSelectWithUnionQuery>(
|
||||
table_expression,
|
||||
|
@ -34,15 +34,26 @@ namespace ErrorCodes
|
||||
extern const int UNKNOWN_ELEMENT_IN_AST;
|
||||
}
|
||||
|
||||
static constexpr auto and_function_name = "and";
|
||||
namespace
|
||||
{
|
||||
|
||||
constexpr auto and_function_name = "and";
|
||||
|
||||
String qualifiedName(ASTIdentifier * identifier, const String & prefix)
|
||||
{
|
||||
if (identifier->isShort())
|
||||
return prefix + identifier->getAliasOrColumnName();
|
||||
return identifier->getAliasOrColumnName();
|
||||
}
|
||||
|
||||
} // namespace
|
||||
|
||||
PredicateExpressionsOptimizer::PredicateExpressionsOptimizer(
|
||||
ASTSelectQuery * ast_select_, ExtractedSettings && settings_, const Context & context_)
|
||||
: ast_select(ast_select_), settings(settings_), context(context_)
|
||||
: ast_select(ast_select_), settings(settings_), context(context_)
|
||||
{
|
||||
}
|
||||
|
||||
|
||||
bool PredicateExpressionsOptimizer::optimize()
|
||||
{
|
||||
if (!settings.enable_optimize_predicate_expression || !ast_select || !ast_select->tables() || ast_select->tables()->children.empty())
|
||||
@ -51,6 +62,9 @@ bool PredicateExpressionsOptimizer::optimize()
|
||||
if (!ast_select->where() && !ast_select->prewhere())
|
||||
return false;
|
||||
|
||||
if (ast_select->array_join_expression_list())
|
||||
return false;
|
||||
|
||||
SubqueriesProjectionColumns all_subquery_projection_columns = getAllSubqueryProjectionColumns();
|
||||
|
||||
bool is_rewrite_subqueries = false;
|
||||
@ -83,7 +97,7 @@ bool PredicateExpressionsOptimizer::optimizeImpl(
|
||||
for (const auto & [subquery, projection_columns] : subqueries_projection_columns)
|
||||
{
|
||||
OptimizeKind optimize_kind = OptimizeKind::NONE;
|
||||
if (allowPushDown(subquery) && canPushDownOuterPredicate(projection_columns, outer_predicate_dependencies, optimize_kind))
|
||||
if (allowPushDown(subquery, outer_predicate, projection_columns, outer_predicate_dependencies, optimize_kind))
|
||||
{
|
||||
if (optimize_kind == OptimizeKind::NONE)
|
||||
optimize_kind = expression_kind;
|
||||
@ -115,9 +129,16 @@ bool PredicateExpressionsOptimizer::optimizeImpl(
|
||||
return is_rewrite_subquery;
|
||||
}
|
||||
|
||||
bool PredicateExpressionsOptimizer::allowPushDown(const ASTSelectQuery * subquery)
|
||||
bool PredicateExpressionsOptimizer::allowPushDown(
|
||||
const ASTSelectQuery * subquery,
|
||||
const ASTPtr &,
|
||||
const std::vector<ProjectionWithAlias> & projection_columns,
|
||||
const std::vector<IdentifierWithQualifier> & dependencies,
|
||||
OptimizeKind & optimize_kind)
|
||||
{
|
||||
if (subquery && !subquery->final() && !subquery->limitBy() && !subquery->limitLength() && !subquery->with())
|
||||
if (!subquery || subquery->final() || subquery->limitBy() || subquery->limitLength() || subquery->with())
|
||||
return false;
|
||||
else
|
||||
{
|
||||
ASTPtr expr_list = ast_select->select();
|
||||
ExtractFunctionVisitor::Data extract_data;
|
||||
@ -127,15 +148,92 @@ bool PredicateExpressionsOptimizer::allowPushDown(const ASTSelectQuery * subquer
|
||||
{
|
||||
const auto & function = FunctionFactory::instance().tryGet(subquery_function->name, context);
|
||||
|
||||
/// Skip lambda、tuple and other special functions
|
||||
/// Skip lambda, tuple and other special functions
|
||||
if (function && function->isStateful())
|
||||
return false;
|
||||
}
|
||||
|
||||
return true;
|
||||
}
|
||||
|
||||
return false;
|
||||
const auto * ast_join = ast_select->join();
|
||||
const ASTTableExpression * left_table_expr = nullptr;
|
||||
const ASTTableExpression * right_table_expr = nullptr;
|
||||
const ASTSelectQuery * left_subquery = nullptr;
|
||||
const ASTSelectQuery * right_subquery = nullptr;
|
||||
|
||||
if (ast_join)
|
||||
{
|
||||
left_table_expr = ast_select
|
||||
->tables()->as<ASTTablesInSelectQuery>()
|
||||
->children[0]->as<ASTTablesInSelectQueryElement>()
|
||||
->table_expression->as<ASTTableExpression>();
|
||||
right_table_expr = ast_select
|
||||
->tables()->as<ASTTablesInSelectQuery>()
|
||||
->children[1]->as<ASTTablesInSelectQueryElement>()
|
||||
->table_expression->as<ASTTableExpression>();
|
||||
|
||||
if (left_table_expr && left_table_expr->subquery)
|
||||
left_subquery = left_table_expr->subquery
|
||||
->children[0]->as<ASTSelectWithUnionQuery>()
|
||||
->list_of_selects->children[0]->as<ASTSelectQuery>();
|
||||
if (right_table_expr && right_table_expr->subquery)
|
||||
right_subquery = right_table_expr->subquery
|
||||
->children[0]->as<ASTSelectWithUnionQuery>()
|
||||
->list_of_selects->children[0]->as<ASTSelectQuery>();
|
||||
|
||||
/// NOTE: the syntactic way of pushdown has limitations and should be partially disabled in case of JOINs.
|
||||
/// Let's take a look at the query:
|
||||
///
|
||||
/// SELECT a, b FROM (SELECT 1 AS a) ANY LEFT JOIN (SELECT 1 AS a, 1 AS b) USING (a) WHERE b = 0
|
||||
///
|
||||
/// The result is empty - without pushdown. But the pushdown tends to modify it in this way:
|
||||
///
|
||||
/// SELECT a, b FROM (SELECT 1 AS a) ANY LEFT JOIN (SELECT 1 AS a, 1 AS b WHERE b = 0) USING (a) WHERE b = 0
|
||||
///
|
||||
/// That leads to the empty result in the right subquery and changes the whole outcome to (1, 0) or (1, NULL).
|
||||
/// It happens because the not-matching columns are replaced with a global default values on JOIN.
|
||||
/// Same is true for RIGHT JOIN and FULL JOIN.
|
||||
|
||||
/// Check right side for LEFT'o'FULL JOIN
|
||||
if (isLeftOrFull(ast_join->table_join->as<ASTTableJoin>()->kind) && right_subquery == subquery)
|
||||
return false;
|
||||
|
||||
/// Check left side for RIGHT'o'FULL JOIN
|
||||
if (isRightOrFull(ast_join->table_join->as<ASTTableJoin>()->kind) && left_subquery == subquery)
|
||||
return false;
|
||||
}
|
||||
|
||||
return checkDependencies(projection_columns, dependencies, optimize_kind);
|
||||
}
|
||||
|
||||
bool PredicateExpressionsOptimizer::checkDependencies(
|
||||
const std::vector<ProjectionWithAlias> & projection_columns,
|
||||
const std::vector<IdentifierWithQualifier> & dependencies,
|
||||
OptimizeKind & optimize_kind)
|
||||
{
|
||||
for (const auto & [identifier, prefix] : dependencies)
|
||||
{
|
||||
bool is_found = false;
|
||||
String qualified_name = qualifiedName(identifier, prefix);
|
||||
|
||||
for (const auto & [ast, alias] : projection_columns)
|
||||
{
|
||||
if (alias == qualified_name)
|
||||
{
|
||||
is_found = true;
|
||||
ASTPtr projection_column = ast;
|
||||
ExtractFunctionVisitor::Data extract_data;
|
||||
ExtractFunctionVisitor(extract_data).visit(projection_column);
|
||||
|
||||
if (!extract_data.aggregate_functions.empty())
|
||||
optimize_kind = OptimizeKind::PUSH_TO_HAVING;
|
||||
}
|
||||
}
|
||||
|
||||
if (!is_found)
|
||||
return false;
|
||||
}
|
||||
|
||||
return true;
|
||||
}
|
||||
|
||||
std::vector<ASTPtr> PredicateExpressionsOptimizer::splitConjunctionPredicate(const ASTPtr & predicate_expression)
|
||||
@ -194,44 +292,6 @@ PredicateExpressionsOptimizer::getDependenciesAndQualifiers(ASTPtr & expression,
|
||||
return dependencies;
|
||||
}
|
||||
|
||||
static String qualifiedName(ASTIdentifier * identifier, const String & prefix)
|
||||
{
|
||||
if (identifier->isShort())
|
||||
return prefix + identifier->getAliasOrColumnName();
|
||||
return identifier->getAliasOrColumnName();
|
||||
}
|
||||
|
||||
bool PredicateExpressionsOptimizer::canPushDownOuterPredicate(
|
||||
const std::vector<ProjectionWithAlias> & projection_columns,
|
||||
const std::vector<IdentifierWithQualifier> & dependencies,
|
||||
OptimizeKind & optimize_kind)
|
||||
{
|
||||
for (const auto & [identifier, prefix] : dependencies)
|
||||
{
|
||||
bool is_found = false;
|
||||
String qualified_name = qualifiedName(identifier, prefix);
|
||||
|
||||
for (const auto & [ast, alias] : projection_columns)
|
||||
{
|
||||
if (alias == qualified_name)
|
||||
{
|
||||
is_found = true;
|
||||
ASTPtr projection_column = ast;
|
||||
ExtractFunctionVisitor::Data extract_data;
|
||||
ExtractFunctionVisitor(extract_data).visit(projection_column);
|
||||
|
||||
if (!extract_data.aggregate_functions.empty())
|
||||
optimize_kind = OptimizeKind::PUSH_TO_HAVING;
|
||||
}
|
||||
}
|
||||
|
||||
if (!is_found)
|
||||
return false;
|
||||
}
|
||||
|
||||
return true;
|
||||
}
|
||||
|
||||
void PredicateExpressionsOptimizer::setNewAliasesForInnerPredicate(
|
||||
const std::vector<ProjectionWithAlias> & projection_columns,
|
||||
const std::vector<IdentifierWithQualifier> & dependencies)
|
||||
@ -340,9 +400,8 @@ ASTs PredicateExpressionsOptimizer::getSelectQueryProjectionColumns(ASTPtr & ast
|
||||
std::vector<DatabaseAndTableWithAlias> tables = getDatabaseAndTables(*select_query, context.getCurrentDatabase());
|
||||
|
||||
/// TODO: get tables from evaluateAsterisk instead of tablesOnly() to extract asterisks in general way
|
||||
NameSet source_columns;
|
||||
std::vector<TableWithColumnNames> tables_with_columns = TranslateQualifiedNamesVisitor::Data::tablesOnly(tables);
|
||||
TranslateQualifiedNamesVisitor::Data qn_visitor_data(source_columns, tables_with_columns, false);
|
||||
TranslateQualifiedNamesVisitor::Data qn_visitor_data({}, tables_with_columns, false);
|
||||
TranslateQualifiedNamesVisitor(qn_visitor_data).visit(ast);
|
||||
|
||||
QueryAliasesVisitor::Data query_aliases_data{aliases};
|
||||
|
@ -39,13 +39,15 @@ class PredicateExpressionsOptimizer
|
||||
|
||||
/// for PredicateExpressionsOptimizer
|
||||
const bool enable_optimize_predicate_expression;
|
||||
const bool join_use_nulls;
|
||||
|
||||
template<typename T>
|
||||
ExtractedSettings(const T & settings)
|
||||
: max_ast_depth(settings.max_ast_depth),
|
||||
max_expanded_ast_elements(settings.max_expanded_ast_elements),
|
||||
count_distinct_implementation(settings.count_distinct_implementation),
|
||||
enable_optimize_predicate_expression(settings.enable_optimize_predicate_expression)
|
||||
enable_optimize_predicate_expression(settings.enable_optimize_predicate_expression),
|
||||
join_use_nulls(settings.join_use_nulls)
|
||||
{}
|
||||
};
|
||||
|
||||
@ -78,11 +80,17 @@ private:
|
||||
|
||||
bool optimizeImpl(const ASTPtr & outer_expression, const SubqueriesProjectionColumns & subqueries_projection_columns, OptimizeKind optimize_kind);
|
||||
|
||||
bool allowPushDown(const ASTSelectQuery * subquery);
|
||||
bool allowPushDown(
|
||||
const ASTSelectQuery * subquery,
|
||||
const ASTPtr & outer_predicate,
|
||||
const std::vector<ProjectionWithAlias> & subquery_projection_columns,
|
||||
const std::vector<IdentifierWithQualifier> & outer_predicate_dependencies,
|
||||
OptimizeKind & optimize_kind);
|
||||
|
||||
bool canPushDownOuterPredicate(const std::vector<ProjectionWithAlias> & subquery_projection_columns,
|
||||
const std::vector<IdentifierWithQualifier> & outer_predicate_dependencies,
|
||||
OptimizeKind & optimize_kind);
|
||||
bool checkDependencies(
|
||||
const std::vector<ProjectionWithAlias> & projection_columns,
|
||||
const std::vector<IdentifierWithQualifier> & dependencies,
|
||||
OptimizeKind & optimize_kind);
|
||||
|
||||
void setNewAliasesForInnerPredicate(const std::vector<ProjectionWithAlias> & projection_columns,
|
||||
const std::vector<IdentifierWithQualifier> & inner_predicate_dependencies);
|
||||
|
@ -24,7 +24,7 @@ public:
|
||||
|
||||
struct Data
|
||||
{
|
||||
const NameSet & source_columns;
|
||||
NameSet source_columns;
|
||||
const std::vector<TableWithColumnNames> & tables;
|
||||
std::unordered_set<String> join_using_columns;
|
||||
bool has_columns;
|
||||
|
@ -216,11 +216,11 @@ static const ASTArrayJoin * getFirstArrayJoin(const ASTSelectQuery & select)
|
||||
static const ASTTablesInSelectQueryElement * getFirstTableJoin(const ASTSelectQuery & select)
|
||||
{
|
||||
if (!select.tables())
|
||||
return {};
|
||||
return nullptr;
|
||||
|
||||
const auto & tables_in_select_query = select.tables()->as<ASTTablesInSelectQuery &>();
|
||||
if (tables_in_select_query.children.empty())
|
||||
return {};
|
||||
return nullptr;
|
||||
|
||||
const ASTTablesInSelectQueryElement * joined_table = nullptr;
|
||||
for (const auto & child : tables_in_select_query.children)
|
||||
|
@ -82,12 +82,12 @@ struct ASTTableJoin : public IAST
|
||||
/// Join method.
|
||||
enum class Kind
|
||||
{
|
||||
Inner, /// Leave ony rows that was JOINed.
|
||||
Left, /// If in "right" table there is no corresponding rows, use default values instead.
|
||||
Inner, /// Leave only rows that was JOINed.
|
||||
Left, /// If in "right" table there is no corresponding rows, use default values instead.
|
||||
Right,
|
||||
Full,
|
||||
Cross, /// Direct product. Strictness and condition doesn't matter.
|
||||
Comma /// Same as direct product. Intended to be converted to INNER JOIN with conditions from WHERE.
|
||||
Cross, /// Direct product. Strictness and condition doesn't matter.
|
||||
Comma /// Same as direct product. Intended to be converted to INNER JOIN with conditions from WHERE.
|
||||
};
|
||||
|
||||
Locality locality = Locality::Unspecified;
|
||||
|
@ -3,9 +3,14 @@
|
||||
1
|
||||
1
|
||||
2000-01-01 1 test string 1 1
|
||||
-------Forbid push down-------
|
||||
SELECT count()\nFROM \n(\n SELECT \n [number] AS a, \n [number * 2] AS b\n FROM system.numbers \n LIMIT 1\n) AS t \nARRAY JOIN \n a, \n b\nWHERE NOT ignore(a + b)
|
||||
1
|
||||
SELECT \n a, \n b\nFROM \n(\n SELECT 1 AS a\n) \nANY LEFT JOIN \n(\n SELECT \n 1 AS a, \n 1 AS b\n) USING (a)\nWHERE b = 0
|
||||
SELECT \n a, \n b\nFROM \n(\n SELECT \n 1 AS a, \n 1 AS b\n) \nANY RIGHT JOIN \n(\n SELECT 1 AS a\n) USING (a)\nWHERE b = 0
|
||||
SELECT \n a, \n b\nFROM \n(\n SELECT 1 AS a\n) \nANY FULL OUTER JOIN \n(\n SELECT \n 1 AS a, \n 1 AS b\n) USING (a)\nWHERE b = 0
|
||||
SELECT \n a, \n b\nFROM \n(\n SELECT \n 1 AS a, \n 1 AS b\n) \nANY FULL OUTER JOIN \n(\n SELECT 1 AS a\n) USING (a)\nWHERE b = 0
|
||||
-------Need push down-------
|
||||
SELECT dummy\nFROM system.one \nANY LEFT JOIN \n(\n SELECT 0 AS dummy\n WHERE 1\n) USING (dummy)\nWHERE 1
|
||||
0
|
||||
SELECT toString(value) AS value\nFROM \n(\n SELECT 1 AS value\n WHERE toString(value) = \'1\'\n) \nWHERE value = \'1\'
|
||||
1
|
||||
SELECT id\nFROM \n(\n SELECT 1 AS id\n WHERE id = 1\n UNION ALL\n SELECT 2 AS `2`\n WHERE `2` = 1\n) \nWHERE id = 1
|
||||
@ -41,17 +46,17 @@ SELECT \n id, \n date, \n value\nFROM \n(\n SELECT \n id, \n
|
||||
SELECT \n date, \n id, \n name, \n value\nFROM \n(\n SELECT \n date, \n id, \n name, \n value\n FROM test.test \n WHERE id = 1\n UNION ALL\n SELECT \n date, \n id, \n name, \n value\n FROM test.test \n WHERE id = 1\n) \nWHERE id = 1
|
||||
2000-01-01 1 test string 1 1
|
||||
2000-01-01 1 test string 1 1
|
||||
SELECT \n date, \n id, \n name, \n value, \n date, \n name, \n value\nFROM \n(\n SELECT \n date, \n id, \n name, \n value\n FROM test.test \n WHERE id = 1\n) \nANY LEFT JOIN \n(\n SELECT *\n FROM test.test \n WHERE id = 1\n) USING (id)\nWHERE id = 1
|
||||
SELECT \n date, \n id, \n name, \n value, \n date, \n name, \n value\nFROM \n(\n SELECT \n date, \n id, \n name, \n value\n FROM test.test \n WHERE id = 1\n) \nANY LEFT JOIN \n(\n SELECT *\n FROM test.test \n) USING (id)\nWHERE id = 1
|
||||
2000-01-01 1 test string 1 1 2000-01-01 test string 1 1
|
||||
SELECT \n id, \n date, \n name, \n value\nFROM \n(\n SELECT toInt8(1) AS id\n) \nANY LEFT JOIN test.test USING (id)\nWHERE value = 1
|
||||
1 2000-01-01 test string 1 1
|
||||
SELECT value\nFROM \n(\n SELECT toInt8(1) AS id\n) \nANY LEFT JOIN test.test AS b USING (id)\nWHERE value = 1
|
||||
1
|
||||
SELECT \n date, \n id, \n name, \n value\nFROM \n(\n SELECT \n date, \n id, \n name, \n value, \n date, \n name, \n value\n FROM \n (\n SELECT \n date, \n id, \n name, \n value\n FROM test.test \n WHERE id = 1\n ) \n ANY LEFT JOIN \n (\n SELECT *\n FROM test.test \n WHERE id = 1\n ) USING (id)\n WHERE id = 1\n) \nWHERE id = 1
|
||||
SELECT \n date, \n id, \n name, \n value\nFROM \n(\n SELECT \n date, \n id, \n name, \n value, \n date, \n name, \n value\n FROM \n (\n SELECT \n date, \n id, \n name, \n value\n FROM test.test \n WHERE id = 1\n ) \n ANY LEFT JOIN \n (\n SELECT *\n FROM test.test \n ) USING (id)\n WHERE id = 1\n) \nWHERE id = 1
|
||||
2000-01-01 1 test string 1 1
|
||||
SELECT \n date, \n id, \n name, \n value, \n b.date, \n b.name, \n b.value\nFROM \n(\n SELECT \n date, \n id, \n name, \n value\n FROM test.test \n) \nANY LEFT JOIN \n(\n SELECT *\n FROM test.test \n WHERE id = 1\n) AS b USING (id)\nWHERE b.id = 1
|
||||
SELECT \n date, \n id, \n name, \n value, \n b.date, \n b.name, \n b.value\nFROM \n(\n SELECT \n date, \n id, \n name, \n value\n FROM test.test \n) \nANY LEFT JOIN \n(\n SELECT *\n FROM test.test \n) AS b USING (id)\nWHERE b.id = 1
|
||||
2000-01-01 1 test string 1 1 2000-01-01 test string 1 1
|
||||
SELECT \n id, \n date, \n name, \n value\nFROM \n(\n SELECT \n toInt8(1) AS id, \n toDate(\'2000-01-01\') AS date\n FROM system.numbers \n LIMIT 1\n) \nANY LEFT JOIN \n(\n SELECT *\n FROM test.test \n WHERE date = toDate(\'2000-01-01\')\n) AS b USING (date, id)\nWHERE b.date = toDate(\'2000-01-01\')
|
||||
SELECT \n id, \n date, \n name, \n value\nFROM \n(\n SELECT \n toInt8(1) AS id, \n toDate(\'2000-01-01\') AS date\n FROM system.numbers \n LIMIT 1\n) \nANY LEFT JOIN \n(\n SELECT *\n FROM test.test \n) AS b USING (date, id)\nWHERE b.date = toDate(\'2000-01-01\')
|
||||
1 2000-01-01 test string 1 1
|
||||
SELECT \n date, \n id, \n name, \n value, \n `b.date`, \n `b.id`, \n `b.name`, \n `b.value`\nFROM \n(\n SELECT \n date, \n id, \n name, \n value, \n b.date, \n b.id, \n b.name, \n b.value\n FROM \n (\n SELECT \n date, \n id, \n name, \n value\n FROM test.test \n WHERE id = 1\n ) AS a \n ANY LEFT JOIN \n (\n SELECT *\n FROM test.test \n ) AS b ON id = b.id\n WHERE id = 1\n) \nWHERE id = 1
|
||||
2000-01-01 1 test string 1 1 2000-01-01 1 test string 1 1
|
||||
|
@ -18,11 +18,28 @@ SELECT 1 AS id WHERE id = 1;
|
||||
SELECT arrayJoin([1,2,3]) AS id WHERE id = 1;
|
||||
SELECT * FROM test.test WHERE id = 1;
|
||||
|
||||
SELECT '-------Need push down-------';
|
||||
SELECT '-------Forbid push down-------';
|
||||
|
||||
-- Optimize predicate expressions without tables
|
||||
ANALYZE SELECT * FROM system.one ANY LEFT JOIN (SELECT 0 AS dummy) USING dummy WHERE 1;
|
||||
SELECT * FROM system.one ANY LEFT JOIN (SELECT 0 AS dummy) USING dummy WHERE 1;
|
||||
-- ARRAY JOIN
|
||||
ANALYZE SELECT count() FROM (SELECT [number] a, [number * 2] b FROM system.numbers LIMIT 1) AS t ARRAY JOIN a, b WHERE NOT ignore(a + b);
|
||||
SELECT count() FROM (SELECT [number] a, [number * 2] b FROM system.numbers LIMIT 1) AS t ARRAY JOIN a, b WHERE NOT ignore(a + b);
|
||||
|
||||
-- LEFT JOIN
|
||||
ANALYZE SELECT a, b FROM (SELECT 1 AS a) ANY LEFT JOIN (SELECT 1 AS a, 1 AS b) USING (a) WHERE b = 0;
|
||||
SELECT a, b FROM (SELECT 1 AS a) ANY LEFT JOIN (SELECT 1 AS a, 1 AS b) USING (a) WHERE b = 0;
|
||||
|
||||
-- RIGHT JOIN
|
||||
ANALYZE SELECT a, b FROM (SELECT 1 AS a, 1 as b) ANY RIGHT JOIN (SELECT 1 AS a) USING (a) WHERE b = 0;
|
||||
SELECT a, b FROM (SELECT 1 AS a, 1 as b) ANY RIGHT JOIN (SELECT 1 AS a) USING (a) WHERE b = 0;
|
||||
|
||||
-- FULL JOIN
|
||||
ANALYZE SELECT a, b FROM (SELECT 1 AS a) ANY FULL JOIN (SELECT 1 AS a, 1 AS b) USING (a) WHERE b = 0;
|
||||
SELECT a, b FROM (SELECT 1 AS a) ANY FULL JOIN (SELECT 1 AS a, 1 AS b) USING (a) WHERE b = 0;
|
||||
|
||||
ANALYZE SELECT a, b FROM (SELECT 1 AS a, 1 AS b) ANY FULL JOIN (SELECT 1 AS a) USING (a) WHERE b = 0;
|
||||
SELECT a, b FROM (SELECT 1 AS a) ANY FULL JOIN (SELECT 1 AS a, 1 AS b) USING (a) WHERE b = 0;
|
||||
|
||||
SELECT '-------Need push down-------';
|
||||
|
||||
ANALYZE SELECT toString(value) AS value FROM (SELECT 1 AS value) WHERE value = '1';
|
||||
SELECT toString(value) AS value FROM (SELECT 1 AS value) WHERE value = '1';
|
||||
|
@ -1,7 +1,6 @@
|
||||
SET compile_expressions = 1;
|
||||
SET min_count_to_compile_expression = 1;
|
||||
SET optimize_move_to_prewhere = 0;
|
||||
SET enable_optimize_predicate_expression=0;
|
||||
|
||||
DROP TABLE IF EXISTS dt;
|
||||
DROP TABLE IF EXISTS testx;
|
||||
|
@ -56,26 +56,26 @@ comma nullable
|
||||
1 1 1 1
|
||||
2 2 1 2
|
||||
cross
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nCROSS JOIN t2 \nWHERE a = t2.a
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nALL INNER JOIN t2 ON a = t2.a\nWHERE a = t2.a
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nCROSS JOIN \n(\n SELECT *\n FROM t2 \n) AS t2 \nWHERE a = t2.a
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nALL INNER JOIN \n(\n SELECT *\n FROM t2 \n) AS t2 ON a = t2.a\nWHERE a = t2.a
|
||||
cross nullable
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \n, t2 \nWHERE a = t2.a
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nALL INNER JOIN t2 ON a = t2.a\nWHERE a = t2.a
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \n, \n(\n SELECT *\n FROM t2 \n) AS t2 \nWHERE a = t2.a
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nALL INNER JOIN \n(\n SELECT *\n FROM t2 \n) AS t2 ON a = t2.a\nWHERE a = t2.a
|
||||
cross nullable vs not nullable
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nCROSS JOIN t2 \nWHERE a = t2.b
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nALL INNER JOIN t2 ON a = t2.b\nWHERE a = t2.b
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nCROSS JOIN \n(\n SELECT *\n FROM t2 \n) AS t2 \nWHERE a = t2.b
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nALL INNER JOIN \n(\n SELECT *\n FROM t2 \n) AS t2 ON a = t2.b\nWHERE a = t2.b
|
||||
cross self
|
||||
SELECT \n a, \n b, \n y.a, \n y.b\nFROM t1 AS x \nCROSS JOIN t1 AS y \nWHERE (a = y.a) AND (b = y.b)
|
||||
SELECT \n a, \n b, \n y.a, \n y.b\nFROM t1 AS x \nALL INNER JOIN t1 AS y ON (a = y.a) AND (b = y.b)\nWHERE (a = y.a) AND (b = y.b)
|
||||
cross one table expr
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nCROSS JOIN t2 \nWHERE a = b
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nCROSS JOIN t2 \nWHERE a = b
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nCROSS JOIN \n(\n SELECT *\n FROM t2 \n) AS t2 \nWHERE a = b
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nCROSS JOIN \n(\n SELECT *\n FROM t2 \n) AS t2 \nWHERE a = b
|
||||
cross multiple ands
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nCROSS JOIN t2 \nWHERE (a = t2.a) AND (b = t2.b)
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nALL INNER JOIN t2 ON (a = t2.a) AND (b = t2.b)\nWHERE (a = t2.a) AND (b = t2.b)
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nCROSS JOIN \n(\n SELECT *\n FROM t2 \n) AS t2 \nWHERE (a = t2.a) AND (b = t2.b)
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nALL INNER JOIN \n(\n SELECT *\n FROM t2 \n) AS t2 ON (a = t2.a) AND (b = t2.b)\nWHERE (a = t2.a) AND (b = t2.b)
|
||||
cross and inside and
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nCROSS JOIN t2 \nWHERE (a = t2.a) AND ((a = t2.a) AND ((a = t2.a) AND (b = t2.b)))
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nALL INNER JOIN t2 ON (a = t2.a) AND (a = t2.a) AND (a = t2.a) AND (b = t2.b)\nWHERE (a = t2.a) AND ((a = t2.a) AND ((a = t2.a) AND (b = t2.b)))
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nCROSS JOIN \n(\n SELECT *\n FROM t2 \n) AS t2 \nWHERE (a = t2.a) AND ((a = t2.a) AND ((a = t2.a) AND (b = t2.b)))
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nALL INNER JOIN \n(\n SELECT *\n FROM t2 \n) AS t2 ON (a = t2.a) AND (a = t2.a) AND (a = t2.a) AND (b = t2.b)\nWHERE (a = t2.a) AND ((a = t2.a) AND ((a = t2.a) AND (b = t2.b)))
|
||||
cross split conjunction
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nCROSS JOIN t2 \nWHERE (a = t2.a) AND (b = t2.b) AND (a >= 1) AND (t2.b > 0)
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nALL INNER JOIN t2 ON (a = t2.a) AND (b = t2.b)\nWHERE (a = t2.a) AND (b = t2.b) AND (a >= 1) AND (t2.b > 0)
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nCROSS JOIN \n(\n SELECT *\n FROM t2 \n WHERE b > 0\n) AS t2 \nWHERE (a = t2.a) AND (b = t2.b) AND (a >= 1) AND (t2.b > 0)
|
||||
SELECT \n a, \n b, \n t2.a, \n t2.b\nFROM t1 \nALL INNER JOIN \n(\n SELECT *\n FROM t2 \n WHERE b > 0\n) AS t2 ON (a = t2.a) AND (b = t2.b)\nWHERE (a = t2.a) AND (b = t2.b) AND (a >= 1) AND (t2.b > 0)
|
||||
|
@ -1,17 +1,17 @@
|
||||
SELECT a\nFROM t1 \nCROSS JOIN t2
|
||||
SELECT a\nFROM t1 \nALL INNER JOIN t2 ON a = t2.a\nWHERE a = t2.a
|
||||
SELECT a\nFROM t1 \nALL INNER JOIN t2 ON b = t2.b\nWHERE b = t2.b
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n a AS `--t1.a`, \n b, \n t2.a AS `--t2.a`, \n t2.b\n FROM t1 \n ALL INNER JOIN t2 ON `--t1.a` = `--t2.a`\n) \nALL INNER JOIN t3 ON `--t1.a` = a\nWHERE (`--t1.a` = `--t2.a`) AND (`--t1.a` = a)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n a AS `--t1.a`, \n b AS `--t1.b`, \n t2.a, \n t2.b AS `--t2.b`\n FROM t1 \n ALL INNER JOIN t2 ON `--t1.b` = `--t2.b`\n) \nALL INNER JOIN t3 ON `--t1.b` = b\nWHERE (`--t1.b` = `--t2.b`) AND (`--t1.b` = b)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n b, \n `--t2.a`, \n `t2.b`, \n a AS `--t3.a`, \n t3.b\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b, \n t2.a AS `--t2.a`, \n t2.b\n FROM t1 \n ALL INNER JOIN t2 ON `--t1.a` = `--t2.a`\n ) \n ALL INNER JOIN t3 ON `--t1.a` = `--t3.a`\n) \nALL INNER JOIN t4 ON `--t1.a` = a\nWHERE (`--t1.a` = `--t2.a`) AND (`--t1.a` = `--t3.a`) AND (`--t1.a` = a)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n `--t1.b`, \n `t2.a`, \n `--t2.b`, \n a, \n b AS `--t3.b`\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b AS `--t1.b`, \n t2.a, \n t2.b AS `--t2.b`\n FROM t1 \n ALL INNER JOIN t2 ON `--t1.b` = `--t2.b`\n ) \n ALL INNER JOIN t3 ON `--t1.b` = `--t3.b`\n) \nALL INNER JOIN t4 ON `--t1.b` = b\nWHERE (`--t1.b` = `--t2.b`) AND (`--t1.b` = `--t3.b`) AND (`--t1.b` = b)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n b, \n `--t2.a`, \n `t2.b`, \n a AS `--t3.a`, \n t3.b\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b, \n t2.a AS `--t2.a`, \n t2.b\n FROM t1 \n ALL INNER JOIN t2 ON `--t2.a` = `--t1.a`\n ) \n ALL INNER JOIN t3 ON `--t2.a` = `--t3.a`\n) \nALL INNER JOIN t4 ON `--t2.a` = a\nWHERE (`--t2.a` = `--t1.a`) AND (`--t2.a` = `--t3.a`) AND (`--t2.a` = a)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n b, \n `--t2.a`, \n `t2.b`, \n a AS `--t3.a`, \n t3.b\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b, \n t2.a AS `--t2.a`, \n t2.b\n FROM t1 \n CROSS JOIN t2 \n ) \n ALL INNER JOIN t3 ON (`--t3.a` = `--t1.a`) AND (`--t3.a` = `--t2.a`)\n) \nALL INNER JOIN t4 ON `--t3.a` = a\nWHERE (`--t3.a` = `--t1.a`) AND (`--t3.a` = `--t2.a`) AND (`--t3.a` = a)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n b, \n `--t2.a`, \n `t2.b`, \n a AS `--t3.a`, \n t3.b\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b, \n t2.a AS `--t2.a`, \n t2.b\n FROM t1 \n CROSS JOIN t2 \n ) \n CROSS JOIN t3 \n) \nALL INNER JOIN t4 ON (a = `--t1.a`) AND (a = `--t2.a`) AND (a = `--t3.a`)\nWHERE (a = `--t1.a`) AND (a = `--t2.a`) AND (a = `--t3.a`)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n b, \n `--t2.a`, \n `t2.b`, \n a AS `--t3.a`, \n t3.b\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b, \n t2.a AS `--t2.a`, \n t2.b\n FROM t1 \n ALL INNER JOIN t2 ON `--t1.a` = `--t2.a`\n ) \n ALL INNER JOIN t3 ON `--t2.a` = `--t3.a`\n) \nALL INNER JOIN t4 ON `--t3.a` = a\nWHERE (`--t1.a` = `--t2.a`) AND (`--t2.a` = `--t3.a`) AND (`--t3.a` = a)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n b, \n `t2.a`, \n `t2.b`, \n a, \n t3.b\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b, \n t2.a, \n t2.b\n FROM t1 \n CROSS JOIN t2 \n ) \n CROSS JOIN t3 \n) \nCROSS JOIN t4
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n b, \n `t2.a`, \n `t2.b`, \n a, \n t3.b\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b, \n t2.a, \n t2.b\n FROM t1 \n CROSS JOIN t2 \n ) \n CROSS JOIN t3 \n) \nCROSS JOIN t4
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n a AS `--t1.a`, \n b, \n t2.a AS `--t2.a`, \n t2.b\n FROM t1 \n ALL INNER JOIN t2 ON `--t1.a` = `--t2.a`\n) \nCROSS JOIN t3
|
||||
SELECT a\nFROM t1 \nCROSS JOIN \n(\n SELECT *\n FROM t2 \n) AS t2
|
||||
SELECT a\nFROM t1 \nALL INNER JOIN \n(\n SELECT *\n FROM t2 \n) AS t2 ON a = t2.a\nWHERE a = t2.a
|
||||
SELECT a\nFROM t1 \nALL INNER JOIN \n(\n SELECT *\n FROM t2 \n) AS t2 ON b = t2.b\nWHERE b = t2.b
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n a AS `--t1.a`, \n b, \n t2.a AS `--t2.a`, \n t2.b\n FROM t1 \n ALL INNER JOIN \n (\n SELECT *\n FROM t2 \n ) AS t2 ON `--t1.a` = `--t2.a`\n WHERE `--t1.a` = `--t2.a`\n) \nALL INNER JOIN \n(\n SELECT *\n FROM t3 \n) AS t3 ON `--t1.a` = a\nWHERE (`--t1.a` = `--t2.a`) AND (`--t1.a` = a)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n a AS `--t1.a`, \n b AS `--t1.b`, \n t2.a, \n t2.b AS `--t2.b`\n FROM t1 \n ALL INNER JOIN \n (\n SELECT *\n FROM t2 \n ) AS t2 ON `--t1.b` = `--t2.b`\n WHERE `--t1.b` = `--t2.b`\n) \nALL INNER JOIN \n(\n SELECT *\n FROM t3 \n) AS t3 ON `--t1.b` = b\nWHERE (`--t1.b` = `--t2.b`) AND (`--t1.b` = b)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n b, \n `--t2.a`, \n `t2.b`, \n a AS `--t3.a`, \n t3.b\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b, \n t2.a AS `--t2.a`, \n t2.b\n FROM t1 \n ALL INNER JOIN \n (\n SELECT *\n FROM t2 \n ) AS t2 ON `--t1.a` = `--t2.a`\n WHERE `--t1.a` = `--t2.a`\n ) \n ALL INNER JOIN \n (\n SELECT *\n FROM t3 \n ) AS t3 ON `--t1.a` = `--t3.a`\n WHERE (`--t1.a` = `--t3.a`) AND (`--t1.a` = `--t2.a`)\n) \nALL INNER JOIN \n(\n SELECT *\n FROM t4 \n) AS t4 ON `--t1.a` = a\nWHERE (`--t1.a` = `--t2.a`) AND (`--t1.a` = `--t3.a`) AND (`--t1.a` = a)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n `--t1.b`, \n `t2.a`, \n `--t2.b`, \n a, \n b AS `--t3.b`\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b AS `--t1.b`, \n t2.a, \n t2.b AS `--t2.b`\n FROM t1 \n ALL INNER JOIN \n (\n SELECT *\n FROM t2 \n ) AS t2 ON `--t1.b` = `--t2.b`\n WHERE `--t1.b` = `--t2.b`\n ) \n ALL INNER JOIN \n (\n SELECT *\n FROM t3 \n ) AS t3 ON `--t1.b` = `--t3.b`\n WHERE (`--t1.b` = `--t3.b`) AND (`--t1.b` = `--t2.b`)\n) \nALL INNER JOIN \n(\n SELECT *\n FROM t4 \n) AS t4 ON `--t1.b` = b\nWHERE (`--t1.b` = `--t2.b`) AND (`--t1.b` = `--t3.b`) AND (`--t1.b` = b)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n b, \n `--t2.a`, \n `t2.b`, \n a AS `--t3.a`, \n t3.b\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b, \n t2.a AS `--t2.a`, \n t2.b\n FROM t1 \n ALL INNER JOIN \n (\n SELECT *\n FROM t2 \n ) AS t2 ON `--t2.a` = `--t1.a`\n WHERE `--t2.a` = `--t1.a`\n ) \n ALL INNER JOIN \n (\n SELECT *\n FROM t3 \n ) AS t3 ON `--t2.a` = `--t3.a`\n WHERE (`--t2.a` = `--t3.a`) AND (`--t2.a` = `--t1.a`)\n) \nALL INNER JOIN \n(\n SELECT *\n FROM t4 \n) AS t4 ON `--t2.a` = a\nWHERE (`--t2.a` = `--t1.a`) AND (`--t2.a` = `--t3.a`) AND (`--t2.a` = a)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n b, \n `--t2.a`, \n `t2.b`, \n a AS `--t3.a`, \n t3.b\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b, \n t2.a AS `--t2.a`, \n t2.b\n FROM t1 \n CROSS JOIN \n (\n SELECT *\n FROM t2 \n ) AS t2 \n ) \n ALL INNER JOIN \n (\n SELECT *\n FROM t3 \n ) AS t3 ON (`--t3.a` = `--t1.a`) AND (`--t3.a` = `--t2.a`)\n WHERE (`--t3.a` = `--t2.a`) AND (`--t3.a` = `--t1.a`)\n) \nALL INNER JOIN \n(\n SELECT *\n FROM t4 \n) AS t4 ON `--t3.a` = a\nWHERE (`--t3.a` = `--t1.a`) AND (`--t3.a` = `--t2.a`) AND (`--t3.a` = a)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n b, \n `--t2.a`, \n `t2.b`, \n a AS `--t3.a`, \n t3.b\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b, \n t2.a AS `--t2.a`, \n t2.b\n FROM t1 \n CROSS JOIN \n (\n SELECT *\n FROM t2 \n ) AS t2 \n ) \n CROSS JOIN \n (\n SELECT *\n FROM t3 \n ) AS t3 \n) \nALL INNER JOIN \n(\n SELECT *\n FROM t4 \n) AS t4 ON (a = `--t1.a`) AND (a = `--t2.a`) AND (a = `--t3.a`)\nWHERE (a = `--t1.a`) AND (a = `--t2.a`) AND (a = `--t3.a`)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n b, \n `--t2.a`, \n `t2.b`, \n a AS `--t3.a`, \n t3.b\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b, \n t2.a AS `--t2.a`, \n t2.b\n FROM t1 \n ALL INNER JOIN \n (\n SELECT *\n FROM t2 \n ) AS t2 ON `--t1.a` = `--t2.a`\n WHERE `--t1.a` = `--t2.a`\n ) \n ALL INNER JOIN \n (\n SELECT *\n FROM t3 \n ) AS t3 ON `--t2.a` = `--t3.a`\n WHERE (`--t2.a` = `--t3.a`) AND (`--t1.a` = `--t2.a`)\n) \nALL INNER JOIN \n(\n SELECT *\n FROM t4 \n) AS t4 ON `--t3.a` = a\nWHERE (`--t1.a` = `--t2.a`) AND (`--t2.a` = `--t3.a`) AND (`--t3.a` = a)
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n b, \n `t2.a`, \n `t2.b`, \n a, \n t3.b\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b, \n t2.a, \n t2.b\n FROM t1 \n CROSS JOIN \n (\n SELECT *\n FROM t2 \n ) AS t2 \n ) \n CROSS JOIN \n (\n SELECT *\n FROM t3 \n ) AS t3 \n) \nCROSS JOIN \n(\n SELECT *\n FROM t4 \n) AS t4
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n `--t1.a`, \n b, \n `t2.a`, \n `t2.b`, \n a, \n t3.b\n FROM \n (\n SELECT \n a AS `--t1.a`, \n b, \n t2.a, \n t2.b\n FROM t1 \n CROSS JOIN \n (\n SELECT *\n FROM t2 \n ) AS t2 \n ) \n CROSS JOIN \n (\n SELECT *\n FROM t3 \n ) AS t3 \n) \nCROSS JOIN \n(\n SELECT *\n FROM t4 \n) AS t4
|
||||
SELECT `--t1.a` AS `t1.a`\nFROM \n(\n SELECT \n a AS `--t1.a`, \n b, \n t2.a AS `--t2.a`, \n t2.b\n FROM t1 \n ALL INNER JOIN \n (\n SELECT *\n FROM t2 \n ) AS t2 ON `--t1.a` = `--t2.a`\n) \nCROSS JOIN \n(\n SELECT *\n FROM t3 \n) AS t3
|
||||
SELECT * FROM t1, t2
|
||||
1 1 1 1
|
||||
1 1 1 \N
|
||||
|
@ -3,8 +3,6 @@ DROP TABLE IF EXISTS ANIMAL;
|
||||
CREATE TABLE ANIMAL ( ANIMAL Nullable(String) ) engine = TinyLog;
|
||||
INSERT INTO ANIMAL (ANIMAL) VALUES ('CAT'), ('FISH'), ('DOG'), ('HORSE'), ('BIRD');
|
||||
|
||||
set enable_optimize_predicate_expression = 0;
|
||||
|
||||
select * from (
|
||||
select x.b x, count(distinct x.c) ANIMAL
|
||||
from (
|
||||
|
Loading…
Reference in New Issue
Block a user