diff --git a/sdks/java/extensions/sql/datacatalog/build.gradle b/sdks/java/extensions/sql/datacatalog/build.gradle index 1e1c41ffd8c9..5068c8bde80a 100644 --- a/sdks/java/extensions/sql/datacatalog/build.gradle +++ b/sdks/java/extensions/sql/datacatalog/build.gradle @@ -20,7 +20,7 @@ import groovy.json.JsonOutput plugins { id 'org.apache.beam.module' } -applyJavaNature(enableChecker:false,automaticModuleName: 'org.apache.beam.sdk.extensions.sql.datacatalog') +applyJavaNature(automaticModuleName: 'org.apache.beam.sdk.extensions.sql.datacatalog') dependencies { compile(library.java.google_cloud_datacatalog_v1beta1) { diff --git a/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/DataCatalogTableProvider.java b/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/DataCatalogTableProvider.java index 0166fb643570..f9195a38770c 100644 --- a/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/DataCatalogTableProvider.java +++ b/sdks/java/extensions/sql/datacatalog/src/main/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/DataCatalogTableProvider.java @@ -101,7 +101,7 @@ public Map getTables() { } @Override - public @Nullable Table getTableByFullName(TableName fullTableName) { + public Table getTableByFullName(TableName fullTableName) { ImmutableList allNameParts = ImmutableList.builder() @@ -116,10 +116,14 @@ public Map getTables() { @Override public BeamSqlTable buildBeamSqlTable(Table table) { - return DELEGATE_PROVIDERS.get(table.getType()).buildBeamSqlTable(table); + TableProvider tableProvider = DELEGATE_PROVIDERS.get(table.getType()); + if (tableProvider == null) { + throw new RuntimeException("TableProvider is null"); + } + return tableProvider.buildBeamSqlTable(table); } - private @Nullable Table loadTable(String tableName) { + private Table loadTable(String tableName) { if (!tableCache.containsKey(tableName)) { tableCache.put(tableName, loadTableFromDC(tableName)); } diff --git a/sdks/java/extensions/sql/datacatalog/src/test/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/DataCatalogBigQueryIT.java b/sdks/java/extensions/sql/datacatalog/src/test/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/DataCatalogBigQueryIT.java index 24c5fac58859..b86414db3209 100644 --- a/sdks/java/extensions/sql/datacatalog/src/test/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/DataCatalogBigQueryIT.java +++ b/sdks/java/extensions/sql/datacatalog/src/test/java/org/apache/beam/sdk/extensions/sql/meta/provider/datacatalog/DataCatalogBigQueryIT.java @@ -66,9 +66,11 @@ public static Iterable dialects() { }); } + @SuppressWarnings("initialization.fields.uninitialized") @Parameterized.Parameter(0) public String dialectName; + @SuppressWarnings("initialization.fields.uninitialized") @Parameterized.Parameter(1) public Class queryPlanner; @@ -95,7 +97,7 @@ public void testRead() throws Exception { readPipeline .getOptions() .as(BeamSqlPipelineOptions.class) - .setPlannerName(queryPlanner.getCanonicalName()); + .setPlannerName(queryPlanner.getName()); try (DataCatalogTableProvider tableProvider = DataCatalogTableProvider.create( diff --git a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamCalciteSchema.java b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamCalciteSchema.java index e8a1f5f8b9e9..7e2af88d53ab 100644 --- a/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamCalciteSchema.java +++ b/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamCalciteSchema.java @@ -31,6 +31,7 @@ import org.apache.beam.vendor.calcite.v1_20_0.org.apache.calcite.schema.SchemaPlus; import org.apache.beam.vendor.calcite.v1_20_0.org.apache.calcite.schema.SchemaVersion; import org.apache.beam.vendor.calcite.v1_20_0.org.apache.calcite.schema.Schemas; +import org.checkerframework.checker.nullness.qual.Nullable; /** Adapter from {@link TableProvider} to {@link Schema}. */ public class BeamCalciteSchema implements Schema { @@ -89,7 +90,7 @@ public Set getTableNames() { } @Override - public RelProtoDataType getType(String name) { + public @Nullable RelProtoDataType getType(String name) { return null; }