Support ALIASed columns for right joined table

This commit is contained in:
vdimir 2021-06-24 17:57:21 +03:00
parent af7776554b
commit 241b64d02c
No known key found for this signature in database
GPG Key ID: F57B3E10A21DBB31
10 changed files with 48 additions and 18 deletions

View File

@ -900,7 +900,7 @@ JoinPtr SelectQueryExpressionAnalyzer::makeTableJoin(
* - this function shows the expression JOIN _data1.
*/
auto interpreter = interpretSubquery(
join_element.table_expression, getContext(), original_right_columns, query_options.copy().setWithMaterialized());
join_element.table_expression, getContext(), original_right_columns, query_options.copy().setWithAllColumns());
{
joined_plan = std::make_unique<QueryPlan>();
interpreter->buildQueryPlan(*joined_plan);

View File

@ -329,7 +329,7 @@ InterpreterSelectQuery::InterpreterSelectQuery(
metadata_snapshot = storage->getInMemoryMetadataPtr();
}
if (has_input || !joined_tables.resolveTables(options.with_materialized))
if (has_input || !joined_tables.resolveTables(options.with_all_cols))
joined_tables.makeFakeTable(storage, metadata_snapshot, source_header);
/// Rewrite JOINs
@ -338,7 +338,7 @@ InterpreterSelectQuery::InterpreterSelectQuery(
rewriteMultipleJoins(query_ptr, joined_tables.tablesWithColumns(), context->getCurrentDatabase(), context->getSettingsRef());
joined_tables.reset(getSelectQuery());
joined_tables.resolveTables(options.with_materialized);
joined_tables.resolveTables(options.with_all_cols);
if (storage && joined_tables.isLeftTableSubquery())
{

View File

@ -185,9 +185,9 @@ StoragePtr JoinedTables::getLeftTableStorage()
return DatabaseCatalog::instance().getTable(table_id, context);
}
bool JoinedTables::resolveTables(bool with_materialized)
bool JoinedTables::resolveTables(bool include_all_columns)
{
tables_with_columns = getDatabaseAndTablesWithColumns(table_expressions, context, with_materialized);
tables_with_columns = getDatabaseAndTablesWithColumns(table_expressions, context, include_all_columns);
if (tables_with_columns.size() != table_expressions.size())
throw Exception("Unexpected tables count", ErrorCodes::LOGICAL_ERROR);

View File

@ -30,7 +30,7 @@ public:
}
StoragePtr getLeftTableStorage();
bool resolveTables(bool with_materialized);
bool resolveTables(bool include_all_columns);
/// Make fake tables_with_columns[0] in case we have predefined input in InterpreterSelectQuery
void makeFakeTable(StoragePtr storage, const StorageMetadataPtr & metadata_snapshot, const Block & source_header);

View File

@ -42,7 +42,7 @@ struct SelectQueryOptions
bool ignore_alias = false;
bool is_internal = false;
bool is_subquery = false; // non-subquery can also have subquery_depth > 0, e.g. insert select
bool with_materialized = false; /// asterisk include materialized columns
bool with_all_cols = false; /// asterisk include materialized and aliased columns
SelectQueryOptions(
QueryProcessingStage::Enum stage = QueryProcessingStage::Complete,
@ -118,9 +118,9 @@ struct SelectQueryOptions
return *this;
}
SelectQueryOptions & setWithMaterialized(bool value = true)
SelectQueryOptions & setWithAllColumns(bool value = true)
{
with_materialized = value;
with_all_cols = value;
return *this;
}
};

View File

@ -897,9 +897,9 @@ TreeRewriterResultPtr TreeRewriter::analyzeSelect(
const auto & right_table = tables_with_columns[1];
auto & cols_from_joined = result.analyzed_join->columns_from_joined_table;
cols_from_joined = right_table.columns;
/// query can use materialized columns from right joined table, add it to columns_from_joined_table
cols_from_joined.insert(
cols_from_joined.end(), right_table.hidden_columns.begin(), right_table.hidden_columns.end());
/// query can use materialized or aliased columns from right joined table,
/// we want to request it for right table
cols_from_joined.insert(cols_from_joined.end(), right_table.hidden_columns.begin(), right_table.hidden_columns.end());
result.analyzed_join->deduplicateAndQualifyColumnNames(
source_columns_set, right_table.table.getQualifiedNamePrefix());

View File

@ -116,13 +116,13 @@ static NamesAndTypesList getColumnsFromTableExpression(
TablesWithColumns getDatabaseAndTablesWithColumns(
const ASTTableExprConstPtrs & table_expressions,
ContextPtr context,
bool add_materialized)
bool include_all)
{
TablesWithColumns tables_with_columns;
String current_database = context->getCurrentDatabase();
bool include_alias_cols = context->getSettingsRef().asterisk_include_alias_columns;
bool include_materialized_cols = add_materialized || context->getSettingsRef().asterisk_include_materialized_columns;
bool include_alias_cols = include_all || context->getSettingsRef().asterisk_include_alias_columns;
bool include_materialized_cols = include_all || context->getSettingsRef().asterisk_include_materialized_columns;
for (const ASTTableExpression * table_expression : table_expressions)
{

View File

@ -23,6 +23,6 @@ ASTPtr extractTableExpression(const ASTSelectQuery & select, size_t table_number
TablesWithColumns getDatabaseAndTablesWithColumns(
const ASTTableExprConstPtrs & table_expressions,
ContextPtr context,
bool add_materialized = false);
bool include_all = false);
}

View File

@ -10,3 +10,13 @@
-
2020-01-01 12:00:00 fact1 t1_val1 2020-01-01 12:00:00 fact1 t2_val2
2020-01-01 13:00:00 fact3 t1_val3 2020-01-01 12:00:00 fact1 t2_val2
-
2020-01-01 12:00:00 fact1 t1_val1 2019-01-01 12:00:00 fact4 t2_val2
2020-01-01 12:00:00 fact1 t1_val1 2020-01-01 12:00:00 fact1 t2_val2
2020-01-01 13:00:00 fact3 t1_val3 2019-01-01 12:00:00 fact4 t2_val2
2020-01-01 13:00:00 fact3 t1_val3 2020-01-01 12:00:00 fact1 t2_val2
-
2020-02-02 13:00:00 fact2 t1_val2 2020-02-05 13:00:00 fact2 t1_val2
-
fact1t1_val1 fact1t2_val2
fact2t1_val2 fact2t1_val2

View File

@ -1,8 +1,19 @@
DROP TABLE IF EXISTS t1;
DROP TABLE IF EXISTS t2;
CREATE TABLE t1 (time DateTime, foo String, dimension_1 String, dt Date MATERIALIZED toDate(time)) ENGINE = MergeTree() PARTITION BY toYYYYMM(dt) ORDER BY (dt, foo);
CREATE TABLE t2 (time DateTime, bar String, dimension_2 String, dt Date MATERIALIZED toDate(time)) ENGINE = MergeTree() PARTITION BY toYYYYMM(dt) ORDER BY (dt, bar);
CREATE TABLE t1 (
time DateTime, foo String, dimension_1 String,
dt Date MATERIALIZED toDate(time),
dt1 Date MATERIALIZED toDayOfYear(time),
aliascol1 ALIAS foo || dimension_1
) ENGINE = MergeTree() PARTITION BY toYYYYMM(dt) ORDER BY (dt, foo);
CREATE TABLE t2 (
time DateTime, bar String, dimension_2 String,
dt Date MATERIALIZED toDate(time),
dt2 Date MATERIALIZED toDayOfYear(time),
aliascol2 ALIAS bar || dimension_2
) ENGINE = MergeTree() PARTITION BY toYYYYMM(dt) ORDER BY (dt, bar);
INSERT INTO t1 VALUES ('2020-01-01 12:00:00', 'fact1', 't1_val1'), ('2020-02-02 13:00:00', 'fact2', 't1_val2'), ('2020-01-01 13:00:00', 'fact3', 't1_val3');
INSERT INTO t2 VALUES ('2020-01-01 12:00:00', 'fact1', 't2_val2'), ('2020-02-05 13:00:00', 'fact2', 't1_val2'), ('2019-01-01 12:00:00', 'fact4', 't2_val2');
@ -16,3 +27,12 @@ SELECT '-';
SELECT * FROM t1 ALL JOIN t2 ON t1.dt = t2.dt ORDER BY t1.time, t2.time;
SELECT '-';
SELECT * FROM t1 ALL JOIN t2 USING (dt) ORDER BY t1.time, t2.time;
SELECT '-';
SELECT * FROM t1 JOIN t2 ON t1.dt1 = t2.dt2 ORDER BY t1.time, t2.time;
SELECT '-';
SELECT * FROM t1 JOIN t2 ON t1.foo = t2.bar WHERE t2.aliascol2 == 'fact2t1_val2';
SELECT '-';
SELECT t1.aliascol1, t2.aliascol2 FROM t1 JOIN t2 ON t1.foo = t2.bar ORDER BY t1.time, t2.time;
-- SELECT '-';
-- SELECT * FROM t1 JOIN t2 ON t1.aliascol1 = t2.aliascol2 ORDER BY t1.time, t2.time;