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 @@ -33,6 +33,7 @@
import org.apache.flink.table.catalog.CatalogDatabase;
import org.apache.flink.table.catalog.CatalogDatabaseImpl;
import org.apache.flink.table.catalog.CatalogFunction;
import org.apache.flink.table.catalog.CatalogMaterializedTable;
import org.apache.flink.table.catalog.CatalogPartition;
import org.apache.flink.table.catalog.CatalogPartitionSpec;
import org.apache.flink.table.catalog.CatalogTable;
Expand Down Expand Up @@ -426,6 +427,11 @@ public void createTable(ObjectPath tablePath, CatalogBaseTable table, boolean ig
+ "create table without 'connector'='iceberg' related properties in an iceberg table.");
}

if (table instanceof CatalogMaterializedTable) {
throw new UnsupportedOperationException(
"Materialized tables are not supported by Iceberg's Flink catalog.");
}

Preconditions.checkArgument(table instanceof ResolvedCatalogTable, "table should be resolved");
createIcebergTable(tablePath, (ResolvedCatalogTable) table, ignoreIfExists);
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -33,8 +33,10 @@
import org.apache.flink.table.api.Schema.UnresolvedPrimaryKey;
import org.apache.flink.table.api.TableException;
import org.apache.flink.table.api.ValidationException;
import org.apache.flink.table.catalog.CatalogMaterializedTable;
import org.apache.flink.table.catalog.CatalogTable;
import org.apache.flink.table.catalog.CommonCatalogOptions;
import org.apache.flink.table.catalog.IntervalFreshness;
import org.apache.flink.table.catalog.ObjectPath;
import org.apache.flink.table.catalog.exceptions.TableNotExistException;
import org.apache.iceberg.BaseTable;
Expand Down Expand Up @@ -706,6 +708,31 @@ private void validateTableFiles(Table tbl, DataFile... expectedFiles) {
assertThat(actualFilePaths).as("Files should match").isEqualTo(expectedFilePaths);
}

@TestTemplate
public void testCreateMaterializedTableIsUnsupported() {
CatalogMaterializedTable materializedTable =
CatalogMaterializedTable.newBuilder()
.schema(
org.apache.flink.table.api.Schema.newBuilder()
.column("id", DataTypes.BIGINT())
.build())
.definitionQuery("SELECT id FROM tl")
.freshness(IntervalFreshness.ofMinute("5"))
.logicalRefreshMode(CatalogMaterializedTable.LogicalRefreshMode.AUTOMATIC)
.refreshMode(CatalogMaterializedTable.RefreshMode.CONTINUOUS)
.refreshStatus(CatalogMaterializedTable.RefreshStatus.INITIALIZING)
.build();

assertThatThrownBy(
() ->
getTableEnv()
.getCatalog(catalogName)
.get()
.createTable(new ObjectPath(DATABASE, "mt_table"), materializedTable, false))
.isInstanceOf(UnsupportedOperationException.class)
.hasMessage("Materialized tables are not supported by Iceberg's Flink catalog.");
}

private Table table(String name) {
return validationCatalog.loadTable(TableIdentifier.of(icebergNamespace, name));
}
Expand Down
Loading