Skip to content
Open
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension


Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -106,6 +106,7 @@ statement
| CREATE (OR REPLACE)? MATERIALIZED VIEW
(IF NOT EXISTS)? qualifiedName
(GRACE PERIOD interval)?
(WHEN STALE (INLINE | FAIL))?
(COMMENT string)?
(WITH properties)? AS rootQuery #createMaterializedView
| CREATE (OR REPLACE)? VIEW qualifiedName
Expand Down Expand Up @@ -1035,10 +1036,10 @@ nonReserved
| CALL | CALLED | CASCADE | CATALOG | CATALOGS | COLUMN | COLUMNS | COMMENT | COMMIT | COMMITTED | CONDITIONAL | COPARTITION | CORRESPONDING | COUNT | CURRENT
| DATA | DATE | DAY | DECLARE | DEFAULT | DEFINE | DEFINER | DENY | DESC | DESCRIPTOR | DETERMINISTIC | DISTRIBUTED | DO | DOUBLE
| ELSEIF | EMPTY | ENCODING | ERROR | EXCLUDING | EXECUTE | EXPLAIN
| FAST | FETCH | FILTER | FINAL | FIRST | FOLLOWING | FORMAT | FORWARD | FUNCTION | FUNCTIONS
| FAIL | FAST | FETCH | FILTER | FINAL | FIRST | FOLLOWING | FORMAT | FORWARD | FUNCTION | FUNCTIONS
| GRACE | GRANT | GRANTED | GRANTS | GRAPHVIZ | GROUPS
| HOUR
| IF | IGNORE | IMMEDIATE | INCLUDING | INITIAL | INPUT | INTERVAL | INVOKER | IO | ITERATE | ISOLATION
| IF | IGNORE | IMMEDIATE | INCLUDING | INITIAL | INLINE | INPUT | INTERVAL | INVOKER | IO | ITERATE | ISOLATION
| JSON
| KEEP | KEY | KEYS
| LANGUAGE | LAST | LATERAL | LEADING | LEAVE | LEVEL | LIMIT | LOCAL | LOGICAL | LOOP
Expand All @@ -1049,7 +1050,7 @@ nonReserved
| QUOTES
| RANGE | READ | REFRESH | RENAME | REPEAT | REPEATABLE | REPLACE | RESET | RESPECT | RESTRICT | RETURN | RETURNING | RETURNS | REVOKE | ROLE | ROLES | ROLLBACK | ROW | ROWS | RUNNING
| SCALAR | SCHEMA | SCHEMAS | SECOND | SECURITY | SEEK | SERIALIZABLE | SESSION | SET | SETS
| SHOW | SOME | START | STATS | SUBSET | SUBSTRING | SYSTEM
| SHOW | SOME | STALE | START | STATS | SUBSET | SUBSTRING | SYSTEM
| TABLES | TABLESAMPLE | TEXT | TEXT_STRING | TIES | TIME | TIMESTAMP | TO | TRAILING | TRANSACTION | TRUNCATE | TRY_CAST | TYPE
| UNBOUNDED | UNCOMMITTED | UNCONDITIONAL | UNIQUE | UNKNOWN | UNMATCHED | UNTIL | UPDATE | USE | USER | UTF16 | UTF32 | UTF8
| VALIDATE | VALUE | VERBOSE | VERSION | VIEW
Expand Down Expand Up @@ -1141,6 +1142,7 @@ EXECUTE: 'EXECUTE';
EXISTS: 'EXISTS';
EXPLAIN: 'EXPLAIN';
EXTRACT: 'EXTRACT';
FAIL: 'FAIL';
FALSE: 'FALSE';
FAST: 'FAST';
FETCH: 'FETCH';
Expand Down Expand Up @@ -1171,6 +1173,7 @@ IMMEDIATE: 'IMMEDIATE';
IN: 'IN';
INCLUDING: 'INCLUDING';
INITIAL: 'INITIAL';
INLINE: 'INLINE';
INNER: 'INNER';
INPUT: 'INPUT';
INSERT: 'INSERT';
Expand Down Expand Up @@ -1301,6 +1304,7 @@ SET: 'SET';
SETS: 'SETS';
SHOW: 'SHOW';
SOME: 'SOME';
STALE: 'STALE';
START: 'START';
STATS: 'STATS';
SUBSET: 'SUBSET';
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -109,6 +109,7 @@ public void test()
"EXISTS",
"EXPLAIN",
"EXTRACT",
"FAIL",
"FALSE",
"FAST",
"FETCH",
Expand Down Expand Up @@ -139,6 +140,7 @@ public void test()
"IN",
"INCLUDING",
"INITIAL",
"INLINE",
"INNER",
"INPUT",
"INSERT",
Expand Down Expand Up @@ -270,6 +272,7 @@ public void test()
"SHOW",
"SKIP",
"SOME",
"STALE",
"START",
"STATS",
"STRING",
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -27,6 +27,7 @@
import io.trino.metadata.ViewColumn;
import io.trino.security.AccessControl;
import io.trino.spi.TrinoException;
import io.trino.spi.connector.ConnectorMaterializedViewDefinition.WhenStaleBehavior;
import io.trino.spi.type.Type;
import io.trino.sql.PlannerContext;
import io.trino.sql.analyzer.Analysis;
Expand Down Expand Up @@ -55,6 +56,7 @@
import static io.trino.spi.StandardErrorCode.NOT_SUPPORTED;
import static io.trino.spi.StandardErrorCode.TYPE_MISMATCH;
import static io.trino.spi.connector.ConnectorCapabilities.MATERIALIZED_VIEW_GRACE_PERIOD;
import static io.trino.spi.connector.ConnectorCapabilities.MATERIALIZED_VIEW_WHEN_STALE_BEHAVIOR;
import static io.trino.sql.SqlFormatterUtil.getFormattedSql;
import static io.trino.sql.analyzer.ConstantEvaluator.evaluateConstant;
import static io.trino.sql.analyzer.SemanticExceptions.semanticException;
Expand Down Expand Up @@ -156,12 +158,21 @@ Analysis executeInternal(
return Duration.ofMillis(milliseconds);
});

Optional<WhenStaleBehavior> whenStale = statement.getWhenStaleBehavior()
.map(_ -> {
if (!plannerContext.getMetadata().getConnectorCapabilities(session, catalogHandle).contains(MATERIALIZED_VIEW_WHEN_STALE_BEHAVIOR)) {
throw semanticException(NOT_SUPPORTED, statement, "Catalog '%s' does not support WHEN STALE", catalogName);
}
Comment on lines +163 to +165
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Should catalogs need to handle anything in order to support it ? If it is some sort of MV property (like security mode in case of views) - It should be supported for all connectors supporting MVs right ? Or we could fail during the creation or alter phase ?

Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

I don't understand why we need a connector capability for this.
What's the default behavior of WHEN STALE going to be?

Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

oh, i get it -- storing the clause requires a capability.

throw semanticException(NOT_SUPPORTED, statement, "WHEN STALE is not supported yet");
});

MaterializedViewDefinition definition = new MaterializedViewDefinition(
sql,
session.getCatalog(),
session.getSchema(),
columns,
gracePeriod,
whenStale,
statement.getComment(),
session.getIdentity(),
session.getPath().getPath().stream()
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -16,6 +16,7 @@
import io.trino.spi.connector.CatalogSchemaName;
import io.trino.spi.connector.CatalogSchemaTableName;
import io.trino.spi.connector.ConnectorMaterializedViewDefinition;
import io.trino.spi.connector.ConnectorMaterializedViewDefinition.WhenStaleBehavior;
import io.trino.spi.security.Identity;

import java.time.Duration;
Expand All @@ -31,6 +32,7 @@ public class MaterializedViewDefinition
extends ViewDefinition
{
private final Optional<Duration> gracePeriod;
private final Optional<WhenStaleBehavior> whenStaleBehavior;
private final Optional<CatalogSchemaTableName> storageTable;

public MaterializedViewDefinition(
Expand All @@ -39,6 +41,7 @@ public MaterializedViewDefinition(
Optional<String> schema,
List<ViewColumn> columns,
Optional<Duration> gracePeriod,
Optional<WhenStaleBehavior> whenStaleBehavior,
Optional<String> comment,
Identity owner,
List<CatalogSchemaName> path,
Expand All @@ -47,6 +50,7 @@ public MaterializedViewDefinition(
super(originalSql, catalog, schema, columns, comment, Optional.of(owner), path);
this.gracePeriod = requireNonNull(gracePeriod, "gracePeriod is null");
checkArgument(gracePeriod.isEmpty() || !gracePeriod.get().isNegative(), "gracePeriod cannot be negative: %s", gracePeriod);
this.whenStaleBehavior = requireNonNull(whenStaleBehavior, "whenStaleBehavior is null");
this.storageTable = requireNonNull(storageTable, "storageTable is null");
}

Expand All @@ -55,6 +59,11 @@ public Optional<Duration> getGracePeriod()
return gracePeriod;
}

public Optional<WhenStaleBehavior> getWhenStaleBehavior()
{
return whenStaleBehavior;
}

public Optional<CatalogSchemaTableName> getStorageTable()
{
return storageTable;
Expand All @@ -71,6 +80,7 @@ public ConnectorMaterializedViewDefinition toConnectorMaterializedViewDefinition
.map(column -> new ConnectorMaterializedViewDefinition.Column(column.name(), column.type(), column.comment()))
.collect(toImmutableList()),
getGracePeriod(),
whenStaleBehavior,
getComment(),
getRunAsIdentity().map(Identity::getUser),
getPath());
Expand All @@ -85,6 +95,7 @@ public String toString()
.add("schema", getSchema().orElse(null))
.add("columns", getColumns())
.add("gracePeriod", gracePeriod.orElse(null))
.add("whenStaleBehavior", whenStaleBehavior.orElse(null))
.add("comment", getComment().orElse(null))
.add("runAsIdentity", getRunAsIdentity())
.add("path", getPath())
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -1827,6 +1827,7 @@ private static MaterializedViewDefinition createMaterializedViewDefinition(Conne
.map(column -> new ViewColumn(column.getName(), column.getType(), Optional.empty()))
.collect(toImmutableList()),
view.getGracePeriod(),
view.getWhenStaleBehavior(),
view.getComment(),
runAsIdentity,
view.getPath(),
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -562,6 +562,7 @@ private Query showCreateMaterializedView(ShowCreate node)
false,
false,
Optional.empty(), // TODO support GRACE PERIOD
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

did we miss doing this?

Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Yeah, it looks like someone (6d5215c) forgot about it 😀.

Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

do you want to create a separate PR to have this addressed?

Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Sure!

Optional.empty(), // TODO support WHEN STALE
propertyNodes,
viewDefinition.get().getComment())).trim();
return singleValueQuery("Create Materialized View", sql);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -205,6 +205,7 @@ protected MaterializedViewDefinition someMaterializedView(String sql, List<ViewC
columns,
Optional.empty(),
Optional.empty(),
Optional.empty(),
Identity.ofUser("owner"),
ImmutableList.of(),
Optional.empty());
Expand Down Expand Up @@ -591,6 +592,7 @@ public void setMaterializedViewColumnComment(Session session, QualifiedObjectNam
.map(currentViewColumn -> columnName.equals(currentViewColumn.name()) ? new ViewColumn(currentViewColumn.name(), currentViewColumn.type(), comment) : currentViewColumn)
.collect(toImmutableList()),
view.getGracePeriod(),
view.getWhenStaleBehavior(),
view.getComment(),
view.getRunAsIdentity().get(),
view.getPath(),
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -127,6 +127,7 @@ void testCreateMaterializedViewIfNotExists()
false,
true,
Optional.empty(),
Optional.empty(),
ImmutableList.of(),
Optional.empty());

Expand All @@ -147,6 +148,7 @@ void testCreateMaterializedViewWithExistingView()
false,
false,
Optional.empty(),
Optional.empty(),
ImmutableList.of(),
Optional.empty());

Expand All @@ -169,6 +171,7 @@ void testCreateMaterializedViewWithInvalidProperty()
false,
true,
Optional.empty(),
Optional.empty(),
ImmutableList.of(new Property(new NodeLocation(1, 88), new Identifier("baz"), new StringLiteral("abc"))),
Optional.empty());

Expand All @@ -191,6 +194,7 @@ void testCreateMaterializedViewWithDefaultProperties()
false,
true,
Optional.empty(),
Optional.empty(),
ImmutableList.of(
new Property(new Identifier("foo")), // set foo to DEFAULT
new Property(new Identifier("bar"))), // set bar to DEFAULT
Expand All @@ -217,6 +221,7 @@ public void testCreateDenyPermission()
false,
true,
Optional.empty(),
Optional.empty(),
ImmutableList.of(),
Optional.empty());
queryRunner.getAccessControl().deny(privilege("test_mv_deny", CREATE_MATERIALIZED_VIEW));
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -7889,6 +7889,7 @@ public void setup()
Optional.of("s1"),
ImmutableList.of(new ViewColumn("a", BIGINT.getTypeId(), Optional.empty())),
Optional.of(Duration.ZERO),
Optional.empty(),
Optional.of("comment"),
Identity.ofUser("user"),
ImmutableList.of(),
Expand Down Expand Up @@ -8019,6 +8020,7 @@ public void setup()
ImmutableList.of(new ViewColumn("a", BIGINT.getTypeId(), Optional.empty())),
Optional.of(Duration.ZERO),
Optional.empty(),
Optional.empty(),
Identity.ofUser("some user"),
ImmutableList.of(),
Optional.of(new CatalogSchemaTableName(TPCH_CATALOG, "s1", "t1"))),
Expand Down Expand Up @@ -8073,6 +8075,7 @@ public void setup()
ImmutableList.of(new ViewColumn("a", BIGINT.getTypeId(), Optional.empty()), new ViewColumn("b", BIGINT.getTypeId(), Optional.empty())),
Optional.empty(),
Optional.empty(),
Optional.empty(),
Identity.ofUser("some user"),
ImmutableList.of(),
// t3 has a, b column and hidden column x
Expand All @@ -8093,6 +8096,7 @@ public void setup()
ImmutableList.of(new ViewColumn("a", BIGINT.getTypeId(), Optional.empty())),
Optional.empty(),
Optional.empty(),
Optional.empty(),
Identity.ofUser("some user"),
ImmutableList.of(),
Optional.of(new CatalogSchemaTableName(TPCH_CATALOG, "s1", "t2"))),
Expand All @@ -8112,6 +8116,7 @@ public void setup()
ImmutableList.of(new ViewColumn("a", BIGINT.getTypeId(), Optional.empty()), new ViewColumn("c", BIGINT.getTypeId(), Optional.empty())),
Optional.empty(),
Optional.empty(),
Optional.empty(),
Identity.ofUser("some user"),
ImmutableList.of(),
Optional.of(new CatalogSchemaTableName(TPCH_CATALOG, "s1", "t2"))),
Expand All @@ -8131,6 +8136,7 @@ public void setup()
ImmutableList.of(new ViewColumn("a", BIGINT.getTypeId(), Optional.empty()), new ViewColumn("b", RowType.anonymousRow(TINYINT).getTypeId(), Optional.empty())),
Optional.empty(),
Optional.empty(),
Optional.empty(),
Identity.ofUser("some user"),
ImmutableList.of(),
Optional.of(new CatalogSchemaTableName(TPCH_CATALOG, "s1", "t2"))),
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -186,6 +186,7 @@ protected PlanTester createPlanTester()
ImmutableList.of(new ViewColumn("a", BIGINT.getTypeId(), Optional.empty()), new ViewColumn("b", BIGINT.getTypeId(), Optional.empty())),
Optional.of(STALE_MV_STALENESS.plusHours(1)),
Optional.empty(),
Optional.empty(),
Identity.ofUser("some user"),
ImmutableList.of(),
Optional.of(new CatalogSchemaTableName(TEST_CATALOG_NAME, SCHEMA, "storage_table")));
Expand Down Expand Up @@ -220,6 +221,7 @@ protected PlanTester createPlanTester()
ImmutableList.of(new ViewColumn("a", BIGINT.getTypeId(), Optional.empty()), new ViewColumn("b", BIGINT.getTypeId(), Optional.empty())),
Optional.empty(),
Optional.empty(),
Optional.empty(),
Identity.ofUser("some user"),
ImmutableList.of(),
Optional.of(new CatalogSchemaTableName(TEST_CATALOG_NAME, SCHEMA, "storage_table_with_casts")));
Expand Down Expand Up @@ -254,6 +256,7 @@ protected PlanTester createPlanTester()
ImmutableList.of(new ViewColumn("id", BIGINT.getTypeId(), Optional.empty()), new ViewColumn("ts", timestampWithTimezone3.getTypeId(), Optional.empty())),
Optional.empty(),
Optional.empty(),
Optional.empty(),
Identity.ofUser("some user"),
ImmutableList.of(),
Optional.of(new CatalogSchemaTableName(TEST_CATALOG_NAME, SCHEMA, "timestamp_test_storage")));
Expand Down Expand Up @@ -285,6 +288,7 @@ private void createMaterializedView(String materializedViewName, String query)
ImmutableList.of(new ViewColumn("a", BIGINT.getTypeId(), Optional.empty()), new ViewColumn("b", BIGINT.getTypeId(), Optional.empty())),
Optional.of(STALE_MV_STALENESS.plusHours(1)),
Optional.empty(),
Optional.empty(),
Identity.ofUser("some user"),
ImmutableList.of(),
Optional.of(new CatalogSchemaTableName(TEST_CATALOG_NAME, SCHEMA, "storage_table")));
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -133,6 +133,7 @@ public TestColumnMask()
new ConnectorMaterializedViewDefinition.Column("comment", VarcharType.createVarcharType(152).getTypeId(), Optional.empty())),
Optional.of(Duration.ZERO),
Optional.empty(),
Optional.empty(),
Optional.of(VIEW_OWNER),
ImmutableList.of());

Expand All @@ -148,6 +149,7 @@ public TestColumnMask()
new ConnectorMaterializedViewDefinition.Column("comment", VarcharType.createVarcharType(152).getTypeId(), Optional.empty())),
Optional.of(Duration.ZERO),
Optional.empty(),
Optional.empty(),
Optional.of(VIEW_OWNER),
ImmutableList.of());

Expand All @@ -163,6 +165,7 @@ public TestColumnMask()
new ConnectorMaterializedViewDefinition.Column("comment", VarcharType.createVarcharType(152).getTypeId(), Optional.empty())),
Optional.of(Duration.ZERO),
Optional.empty(),
Optional.empty(),
Optional.of(VIEW_OWNER),
ImmutableList.of());

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -61,6 +61,7 @@ private static ConnectorMaterializedViewDefinition someMaterializedView()
ImmutableList.of(new Column("test", BIGINT.getTypeId(), Optional.empty())),
Optional.of(Duration.ZERO),
Optional.empty(),
Optional.empty(),
Optional.of("owner"),
ImmutableList.of());
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -1262,6 +1262,8 @@ protected Void visitCreateMaterializedView(CreateMaterializedView node, Integer
builder.append(formatName(node.getName()));
node.getGracePeriod().ifPresent(interval ->
builder.append("\nGRACE PERIOD ").append(formatExpression(interval)));
node.getWhenStaleBehavior().ifPresent(whenStale ->
builder.append("\nWHEN STALE ").append(whenStale.name()));
node.getComment().ifPresent(comment -> builder
.append("\nCOMMENT ")
.append(formatStringLiteral(comment)));
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -57,6 +57,7 @@
import io.trino.sql.tree.CreateCatalog;
import io.trino.sql.tree.CreateFunction;
import io.trino.sql.tree.CreateMaterializedView;
import io.trino.sql.tree.CreateMaterializedView.WhenStaleBehavior;
import io.trino.sql.tree.CreateRole;
import io.trino.sql.tree.CreateSchema;
import io.trino.sql.tree.CreateTable;
Expand Down Expand Up @@ -613,6 +614,14 @@ public Node visitCreateMaterializedView(SqlBaseParser.CreateMaterializedViewCont
gracePeriod = Optional.of((IntervalLiteral) visit(context.interval()));
}

Optional<WhenStaleBehavior> whenStale = Optional.empty();
if (context.INLINE() != null) {
whenStale = Optional.of(WhenStaleBehavior.INLINE);
}
else if (context.FAIL() != null) {
whenStale = Optional.of(WhenStaleBehavior.FAIL);
}

Optional<String> comment = Optional.empty();
if (context.COMMENT() != null) {
comment = Optional.of(visitString(context.string()).getValue());
Expand All @@ -630,6 +639,7 @@ public Node visitCreateMaterializedView(SqlBaseParser.CreateMaterializedViewCont
context.REPLACE() != null,
context.EXISTS() != null,
gracePeriod,
whenStale,
properties,
comment);
}
Expand Down
Loading
Loading