Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

Enable connector to perform materialized view refresh itself #7960

Merged
Merged
Show file tree
Hide file tree
Changes from 1 commit
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Jump to
Jump to file
Failed to load files.
Diff view
Diff view
Expand Up @@ -19,6 +19,7 @@
import io.trino.spi.connector.SchemaTableName;

import java.util.Optional;
import java.util.concurrent.CompletableFuture;

public interface HiveMaterializedViewMetadata
{
Expand All @@ -29,4 +30,8 @@ public interface HiveMaterializedViewMetadata
Optional<ConnectorMaterializedViewDefinition> getMaterializedView(ConnectorSession session, SchemaTableName viewName);

MaterializedViewFreshness getMaterializedViewFreshness(ConnectorSession session, SchemaTableName name);

boolean delegateMaterializedViewRefreshToConnector(ConnectorSession session, SchemaTableName viewName);

CompletableFuture<?> refreshMaterializedView(ConnectorSession session, SchemaTableName viewName);
}
Expand Up @@ -131,6 +131,7 @@
import java.util.OptionalLong;
import java.util.Properties;
import java.util.Set;
import java.util.concurrent.CompletableFuture;
import java.util.function.Function;
import java.util.function.Predicate;
import java.util.function.Supplier;
Expand Down Expand Up @@ -3047,6 +3048,12 @@ public MaterializedViewFreshness getMaterializedViewFreshness(ConnectorSession s
return hiveMaterializedViewMetadata.getMaterializedViewFreshness(session, name);
}

@Override
public CompletableFuture<?> refreshMaterializedView(ConnectorSession session, SchemaTableName name)
{
return hiveMaterializedViewMetadata.refreshMaterializedView(session, name);
}

public static Optional<SchemaTableName> getSourceTableNameFromSystemTable(SchemaTableName tableName)
{
return Stream.of(SystemTableHandler.values())
Expand Down
Expand Up @@ -20,6 +20,7 @@
import io.trino.spi.connector.SchemaTableName;

import java.util.Optional;
import java.util.concurrent.CompletableFuture;

import static io.trino.spi.StandardErrorCode.NOT_FOUND;
import static io.trino.spi.StandardErrorCode.NOT_SUPPORTED;
Expand Down Expand Up @@ -50,4 +51,16 @@ public MaterializedViewFreshness getMaterializedViewFreshness(ConnectorSession s
{
throw new TrinoException(NOT_FOUND, "This connector does not support materialized views");
}

@Override
public boolean delegateMaterializedViewRefreshToConnector(ConnectorSession session, SchemaTableName viewName)
{
throw new TrinoException(NOT_SUPPORTED, "This connector does not support dropping materialized views");
Copy link
Member

Choose a reason for hiding this comment

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

nit: should be This connector does not support materialized views in both methods

Copy link
Member Author

Choose a reason for hiding this comment

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

created #8228

}

@Override
public CompletableFuture<?> refreshMaterializedView(ConnectorSession session, SchemaTableName viewName)
{
throw new TrinoException(NOT_SUPPORTED, "This connector does not support dropping materialized views");
}
}