This is an automated email from the ASF dual-hosted git repository.
roryqi pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/gravitino.git
The following commit(s) were added to refs/heads/main by this push:
new 19fac4fd92 [#10897] fix(iceberg): Fix the Trino failure to create the
remote catalog table (#10898)
19fac4fd92 is described below
commit 19fac4fd921dd759fcdb2432188ae9ddb4df2d4b
Author: roryqi <[email protected]>
AuthorDate: Thu Apr 30 15:50:50 2026 +0800
[#10897] fix(iceberg): Fix the Trino failure to create the remote catalog
table (#10898)
### What changes were proposed in this pull request?
Fix the Trino failure to create the remote catalog table
### Why are the changes needed?
Fix: #10897
### Does this PR introduce _any_ user-facing change?
No.
### How was this patch tested?
Added UT.
---
.../iceberg/service/CatalogWrapperForREST.java | 34 +++---
.../iceberg/service/TestCatalogWrapperForREST.java | 123 +++++++++++++++++++++
2 files changed, 140 insertions(+), 17 deletions(-)
diff --git
a/iceberg/iceberg-rest-server/src/main/java/org/apache/gravitino/iceberg/service/CatalogWrapperForREST.java
b/iceberg/iceberg-rest-server/src/main/java/org/apache/gravitino/iceberg/service/CatalogWrapperForREST.java
index 5baf43481d..18980ea857 100644
---
a/iceberg/iceberg-rest-server/src/main/java/org/apache/gravitino/iceberg/service/CatalogWrapperForREST.java
+++
b/iceberg/iceberg-rest-server/src/main/java/org/apache/gravitino/iceberg/service/CatalogWrapperForREST.java
@@ -704,32 +704,32 @@ public class CatalogWrapperForREST extends
IcebergCatalogWrapper {
properties.putAll(request.properties());
Map<String, String> config = Maps.newHashMap();
- String location;
+ Catalog.TableBuilder tableBuilder =
+ loadedCatalog
+ .buildTable(ident, request.schema())
+ .withPartitionSpec(request.spec())
+ .withSortOrder(request.writeOrder())
+ .withProperties(properties);
+
+ Table table;
if (request.location() != null) {
- location = request.location();
+ table =
tableBuilder.withLocation(request.location()).createTransaction().table();
} else {
- Table table =
- loadedCatalog
- .buildTable(ident, request.schema())
- .withPartitionSpec(request.spec())
- .withSortOrder(request.writeOrder())
- .withProperties(properties)
- .createTransaction()
- .table();
- Map<String, String> tableProperties =
retrieveFileIOProperties(table.io());
- config.putAll(
- MapUtils.getFilteredMap(
- tableProperties, key ->
catalogPropertiesToClientKeys.contains(key)));
-
config.putAll(CredentialPropertyUtils.filterCredentialProperties(tableProperties));
- location = table.location();
+ table = tableBuilder.createTransaction().table();
}
+ Map<String, String> tableProperties = retrieveFileIOProperties(table.io());
+ config.putAll(
+ MapUtils.getFilteredMap(
+ tableProperties, key ->
catalogPropertiesToClientKeys.contains(key)));
+
config.putAll(CredentialPropertyUtils.filterCredentialProperties(tableProperties));
+
TableMetadata metadata =
TableMetadata.newTableMetadata(
request.schema(),
request.spec() != null ? request.spec() :
PartitionSpec.unpartitioned(),
request.writeOrder() != null ? request.writeOrder() :
SortOrder.unsorted(),
- location,
+ table.location(),
properties);
return
LoadTableResponse.builder().withTableMetadata(metadata).addAllConfig(config).build();
diff --git
a/iceberg/iceberg-rest-server/src/test/java/org/apache/gravitino/iceberg/service/TestCatalogWrapperForREST.java
b/iceberg/iceberg-rest-server/src/test/java/org/apache/gravitino/iceberg/service/TestCatalogWrapperForREST.java
index 73c69f427a..1652548096 100644
---
a/iceberg/iceberg-rest-server/src/test/java/org/apache/gravitino/iceberg/service/TestCatalogWrapperForREST.java
+++
b/iceberg/iceberg-rest-server/src/test/java/org/apache/gravitino/iceberg/service/TestCatalogWrapperForREST.java
@@ -19,7 +19,11 @@
package org.apache.gravitino.iceberg.service;
+import static org.mockito.Mockito.any;
+import static org.mockito.Mockito.anyMap;
import static org.mockito.Mockito.mock;
+import static org.mockito.Mockito.never;
+import static org.mockito.Mockito.verify;
import static org.mockito.Mockito.when;
import com.google.common.collect.ImmutableMap;
@@ -27,8 +31,17 @@ import java.util.Map;
import java.util.concurrent.atomic.AtomicBoolean;
import org.apache.gravitino.catalog.lakehouse.iceberg.IcebergConstants;
import org.apache.gravitino.iceberg.common.IcebergConfig;
+import org.apache.iceberg.Schema;
+import org.apache.iceberg.Table;
+import org.apache.iceberg.Transaction;
import org.apache.iceberg.catalog.Catalog;
+import org.apache.iceberg.catalog.Namespace;
+import org.apache.iceberg.catalog.TableIdentifier;
+import org.apache.iceberg.io.FileIO;
import org.apache.iceberg.rest.RESTCatalog;
+import org.apache.iceberg.rest.requests.CreateTableRequest;
+import org.apache.iceberg.rest.responses.LoadTableResponse;
+import org.apache.iceberg.types.Types;
import org.junit.jupiter.api.Assertions;
import org.junit.jupiter.api.Test;
@@ -175,6 +188,102 @@ public class TestCatalogWrapperForREST {
}
}
+ @Test
+ void testStageTableCreateWithLocationIncludesFileIoProperties() throws
Exception {
+ RESTCatalog catalog = mock(RESTCatalog.class);
+ Catalog.TableBuilder tableBuilder = mock(Catalog.TableBuilder.class);
+ Transaction transaction = mock(Transaction.class);
+ Table table = mock(Table.class);
+ FileIO fileIO = mock(FileIO.class);
+ when(catalog.buildTable(any(TableIdentifier.class),
any())).thenReturn(tableBuilder);
+ when(tableBuilder.withPartitionSpec(any())).thenReturn(tableBuilder);
+ when(tableBuilder.withSortOrder(any())).thenReturn(tableBuilder);
+ when(tableBuilder.withProperties(anyMap())).thenReturn(tableBuilder);
+
when(tableBuilder.withLocation("s3://bucket/warehouse/table")).thenReturn(tableBuilder);
+ when(tableBuilder.createTransaction()).thenReturn(transaction);
+ when(transaction.table()).thenReturn(table);
+ when(table.io()).thenReturn(fileIO);
+ when(table.location()).thenReturn("s3://bucket/warehouse/table");
+ when(fileIO.properties())
+ .thenReturn(
+ ImmutableMap.of(
+ IcebergConstants.IO_IMPL,
+ "org.apache.iceberg.aws.s3.S3FileIO",
+ IcebergConstants.ICEBERG_S3_ENDPOINT,
+ "http://localhost:9000"));
+
+ IcebergConfig config =
+ new IcebergConfig(
+ ImmutableMap.of(
+ IcebergConstants.CATALOG_BACKEND,
+ "memory",
+ IcebergConstants.WAREHOUSE,
+ "/tmp/warehouse"));
+ CatalogWrapperForREST wrapper = new StaticCatalogWrapperForREST("test",
config, catalog);
+
+ Schema schema = new Schema(Types.NestedField.required(1, "id",
Types.IntegerType.get()));
+ CreateTableRequest request =
+ CreateTableRequest.builder()
+ .withName("tbl")
+ .withSchema(schema)
+ .withLocation("s3://bucket/warehouse/table")
+ .stageCreate()
+ .build();
+
+ LoadTableResponse response = wrapper.createTable(Namespace.of("db"),
request, false);
+
+ Assertions.assertEquals(
+ "org.apache.iceberg.aws.s3.S3FileIO",
response.config().get(IcebergConstants.IO_IMPL));
+ Assertions.assertEquals(
+ "http://localhost:9000",
response.config().get(IcebergConstants.ICEBERG_S3_ENDPOINT));
+ verify(tableBuilder).withLocation("s3://bucket/warehouse/table");
+ }
+
+ @Test
+ void testStageTableCreateWithNullLocationDoesNotCallWithLocation() {
+ RESTCatalog catalog = mock(RESTCatalog.class);
+ Catalog.TableBuilder tableBuilder = mock(Catalog.TableBuilder.class);
+ Transaction transaction = mock(Transaction.class);
+ Table table = mock(Table.class);
+ FileIO fileIO = mock(FileIO.class);
+ when(catalog.buildTable(any(TableIdentifier.class),
any())).thenReturn(tableBuilder);
+ when(tableBuilder.withPartitionSpec(any())).thenReturn(tableBuilder);
+ when(tableBuilder.withSortOrder(any())).thenReturn(tableBuilder);
+ when(tableBuilder.withProperties(anyMap())).thenReturn(tableBuilder);
+ when(tableBuilder.createTransaction()).thenReturn(transaction);
+ when(transaction.table()).thenReturn(table);
+ when(table.io()).thenReturn(fileIO);
+
when(table.location()).thenReturn("s3://bucket/warehouse/default-location");
+ when(fileIO.properties())
+ .thenReturn(
+ ImmutableMap.of(
+ IcebergConstants.IO_IMPL,
+ "org.apache.iceberg.aws.s3.S3FileIO",
+ IcebergConstants.ICEBERG_S3_ENDPOINT,
+ "http://localhost:9000"));
+
+ IcebergConfig config =
+ new IcebergConfig(
+ ImmutableMap.of(
+ IcebergConstants.CATALOG_BACKEND,
+ "memory",
+ IcebergConstants.WAREHOUSE,
+ "/tmp/warehouse"));
+ CatalogWrapperForREST wrapper = new StaticCatalogWrapperForREST("test",
config, catalog);
+
+ Schema schema = new Schema(Types.NestedField.required(1, "id",
Types.IntegerType.get()));
+ CreateTableRequest request =
+
CreateTableRequest.builder().withName("tbl").withSchema(schema).stageCreate().build();
+
+ LoadTableResponse response = wrapper.createTable(Namespace.of("db"),
request, false);
+
+ Assertions.assertEquals(
+ "org.apache.iceberg.aws.s3.S3FileIO",
response.config().get(IcebergConstants.IO_IMPL));
+ Assertions.assertEquals(
+ "http://localhost:9000",
response.config().get(IcebergConstants.ICEBERG_S3_ENDPOINT));
+ verify(tableBuilder, never()).withLocation(any());
+ }
+
private static class LazyCheckCatalogWrapperForREST extends
CatalogWrapperForREST {
LazyCheckCatalogWrapperForREST(String catalogName, IcebergConfig config) {
@@ -189,4 +298,18 @@ public class TestCatalogWrapperForREST {
return super.getCatalog();
}
}
+
+ private static class StaticCatalogWrapperForREST extends
CatalogWrapperForREST {
+ private final Catalog catalog;
+
+ StaticCatalogWrapperForREST(String catalogName, IcebergConfig config,
Catalog catalog) {
+ super(catalogName, config);
+ this.catalog = catalog;
+ }
+
+ @Override
+ public Catalog getCatalog() {
+ return catalog;
+ }
+ }
}