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 @@ -292,9 +292,6 @@ public Map<String, Object> loadNamespaceMetadata(ConnectorSession session, Strin
if (database.locationUri() != null) {
metadata.put(LOCATION_PROPERTY, database.locationUri());
}
if (database.parameters() != null) {
metadata.putAll(database.parameters());
}
return metadata.buildOrThrow();
}
catch (EntityNotFoundException e) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -59,6 +59,7 @@

import static com.google.common.base.Throwables.throwIfUnchecked;
import static com.google.common.collect.ImmutableList.toImmutableList;
import static com.google.common.collect.ImmutableMap.toImmutableMap;
import static io.trino.cache.CacheUtils.uncheckedCacheGet;
import static io.trino.filesystem.Locations.appendPath;
import static io.trino.plugin.iceberg.IcebergSchemaProperties.LOCATION_PROPERTY;
Expand Down Expand Up @@ -125,7 +126,9 @@ public void dropNamespace(ConnectorSession session, String namespace)
public Map<String, Object> loadNamespaceMetadata(ConnectorSession session, String namespace)
{
try {
return ImmutableMap.copyOf(nessieClient.loadNamespaceMetadata(Namespace.of(namespace)));
return nessieClient.loadNamespaceMetadata(Namespace.of(namespace)).entrySet().stream()
.filter(metadata -> metadata.getKey().equals(LOCATION_PROPERTY))
Copy link
Contributor

Choose a reason for hiding this comment

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

can we have a method that says "isSupportedProperty" ?

.collect(toImmutableMap(Map.Entry::getKey, Map.Entry::getValue));
}
catch (NoSuchNamespaceException e) {
throw new SchemaNotFoundException(namespace);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -26,7 +26,6 @@
import io.trino.cache.EvictableCacheBuilder;
import io.trino.metastore.TableInfo;
import io.trino.plugin.iceberg.ColumnIdentity;
import io.trino.plugin.iceberg.IcebergSchemaProperties;
import io.trino.plugin.iceberg.IcebergUtil;
import io.trino.plugin.iceberg.catalog.TrinoCatalog;
import io.trino.plugin.iceberg.catalog.rest.IcebergRestCatalogConfig.SessionType;
Expand Down Expand Up @@ -85,11 +84,13 @@
import java.util.stream.Stream;

import static com.google.common.collect.ImmutableList.toImmutableList;
import static com.google.common.collect.ImmutableMap.toImmutableMap;
import static io.trino.cache.CacheUtils.uncheckedCacheGet;
import static io.trino.filesystem.Locations.appendPath;
import static io.trino.metastore.Table.TABLE_COMMENT;
import static io.trino.plugin.iceberg.IcebergErrorCode.ICEBERG_CATALOG_ERROR;
import static io.trino.plugin.iceberg.IcebergErrorCode.ICEBERG_UNSUPPORTED_VIEW_DIALECT;
import static io.trino.plugin.iceberg.IcebergSchemaProperties.LOCATION_PROPERTY;
import static io.trino.plugin.iceberg.IcebergUtil.quotedTableName;
import static io.trino.plugin.iceberg.catalog.AbstractTrinoCatalog.ICEBERG_VIEW_RUN_AS_OWNER;
import static io.trino.spi.StandardErrorCode.NOT_SUPPORTED;
Expand Down Expand Up @@ -221,7 +222,9 @@ public Map<String, Object> loadNamespaceMetadata(ConnectorSession session, Strin
{
try {
// Return immutable metadata as direct modifications will not be reflected on the namespace
return ImmutableMap.copyOf(restSessionCatalog.loadNamespaceMetadata(convert(session), toRemoteNamespace(session, toNamespace(namespace))));
return restSessionCatalog.loadNamespaceMetadata(convert(session), toRemoteNamespace(session, toNamespace(namespace))).entrySet().stream()
.filter(property -> property.getKey().equals(LOCATION_PROPERTY))
.collect(toImmutableMap(Map.Entry::getKey, Map.Entry::getValue));
}
catch (NoSuchNamespaceException e) {
throw new SchemaNotFoundException(namespace);
Expand Down Expand Up @@ -576,7 +579,7 @@ public String defaultTableLocation(ConnectorSession session, SchemaTableName sch
String tableName = createLocationForTable(schemaTableName.getTableName());

Map<String, Object> properties = loadNamespaceMetadata(session, schemaTableName.getSchemaName());
String databaseLocation = (String) properties.get(IcebergSchemaProperties.LOCATION_PROPERTY);
String databaseLocation = (String) properties.get(LOCATION_PROPERTY);
if (databaseLocation == null) {
// Iceberg REST catalog doesn't require location property.
// S3 Tables doesn't return the property.
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -167,6 +167,40 @@ public void testNonLowercaseNamespace()
}
}

@Test
public void testSchemaWithInvalidProperties()
throws Exception
{
String namespace = "test_schema_invalid_properties" + randomNameSuffix();

TrinoCatalog catalog = createTrinoCatalog(false);
createNamespaceWithProperties(catalog, namespace, ImmutableMap.of("invalid_property", "test-value"));
try {
ConnectorMetadata icebergMetadata = new IcebergMetadata(
PLANNER_CONTEXT.getTypeManager(),
jsonCodec(CommitTaskData.class),
catalog,
(_, _) -> {
throw new UnsupportedOperationException();
},
TABLE_STATISTICS_READER,
new TableStatisticsWriter(new NodeVersion("test-version")),
Optional.empty(),
false,
_ -> false,
newDirectExecutorService(),
directExecutor(),
newDirectExecutorService(),
newDirectExecutorService());

assertThat(icebergMetadata.getSchemaProperties(SESSION, namespace))
.doesNotContainKey("invalid_property");
}
finally {
catalog.dropNamespace(SESSION, namespace);
}
}

@Test
public void testCreateTable()
throws Exception
Expand Down Expand Up @@ -536,6 +570,8 @@ public void testListTables()
}
}

protected abstract void createNamespaceWithProperties(TrinoCatalog catalog, String namespace, Map<String, String> properties);

protected void createMaterializedView(
ConnectorSession session,
TrinoCatalog catalog,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,6 +19,7 @@
import io.trino.filesystem.Location;
import io.trino.filesystem.TrinoFileSystemFactory;
import io.trino.filesystem.local.LocalFileSystemFactory;
import io.trino.metastore.Database;
import io.trino.metastore.HiveMetastore;
import io.trino.metastore.cache.CachingHiveMetastore;
import io.trino.plugin.hive.TrinoViewHiveMetastore;
Expand All @@ -43,6 +44,7 @@
import java.io.IOException;
import java.nio.file.Files;
import java.nio.file.Path;
import java.util.Map;
import java.util.Optional;

import static com.google.common.io.MoreFiles.deleteRecursively;
Expand Down Expand Up @@ -89,6 +91,17 @@ public void tearDown()
deleteRecursively(tempDir, ALLOW_INSECURE);
}

@Override
protected void createNamespaceWithProperties(TrinoCatalog catalog, String namespace, Map<String, String> properties)
{
metastore.createDatabase(Database.builder()
.setDatabaseName(namespace)
.setOwnerName(Optional.of("test"))
.setOwnerType(Optional.of(PrincipalType.USER))
.setParameters(properties)
.build());
}

@Override
protected TrinoCatalog createTrinoCatalog(boolean useUniqueTableLocations)
{
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -74,6 +74,15 @@ protected TrinoCatalog createTrinoCatalog(boolean useUniqueTableLocations)
return createGlueTrinoCatalog(useUniqueTableLocations, false);
}

@Override
protected void createNamespaceWithProperties(TrinoCatalog catalog, String namespace, Map<String, String> properties)
{
try (GlueClient glueClient = GlueClient.create()) {
glueClient.createDatabase(database -> database
.databaseInput(input -> input.name(namespace).parameters(properties)));
}
}

private TrinoCatalog createGlueTrinoCatalog(boolean useUniqueTableLocations, boolean useSystemSecurity)
{
GlueClient glueClient = GlueClient.create();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -24,6 +24,7 @@
import io.trino.filesystem.s3.S3FileSystemConfig;
import io.trino.filesystem.s3.S3FileSystemFactory;
import io.trino.filesystem.s3.S3FileSystemStats;
import io.trino.metastore.Database;
import io.trino.metastore.Table;
import io.trino.metastore.TableInfo;
import io.trino.metastore.cache.CachingHiveMetastore;
Expand Down Expand Up @@ -67,6 +68,7 @@
import static io.trino.metastore.cache.CachingHiveMetastore.createPerTransactionCache;
import static io.trino.plugin.hive.TestingThriftHiveMetastoreBuilder.testingThriftHiveMetastoreBuilder;
import static io.trino.plugin.hive.containers.HiveHadoop.HIVE3_IMAGE;
import static io.trino.plugin.hive.metastore.thrift.ThriftMetastoreUtil.toMetastoreApiDatabase;
import static io.trino.plugin.iceberg.IcebergFileFormat.PARQUET;
import static io.trino.plugin.iceberg.IcebergTableProperties.FILE_FORMAT_PROPERTY;
import static io.trino.plugin.iceberg.IcebergTableProperties.FORMAT_VERSION_PROPERTY;
Expand Down Expand Up @@ -119,6 +121,20 @@ public void tearDown()
closer.close();
}

@Override
protected void createNamespaceWithProperties(TrinoCatalog catalog, String namespace, Map<String, String> properties)
{
ThriftMetastore thriftMetastore = testingThriftHiveMetastoreBuilder()
.metastoreClient(dataLake.getHiveMetastoreEndpoint())
.build(closer::register);
thriftMetastore.createDatabase(toMetastoreApiDatabase(Database.builder()
.setDatabaseName(namespace)
.setOwnerName(Optional.of("test"))
.setOwnerType(Optional.of(PrincipalType.USER))
.setParameters(properties)
.build()));
}

@Override
protected TrinoCatalog createTrinoCatalog(boolean useUniqueTableLocations)
{
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -29,6 +29,7 @@
import io.trino.spi.security.PrincipalType;
import io.trino.spi.security.TrinoPrincipal;
import io.trino.spi.type.TestingTypeManager;
import org.apache.iceberg.catalog.Namespace;
import org.apache.iceberg.nessie.NessieIcebergClient;
import org.junit.jupiter.api.AfterAll;
import org.junit.jupiter.api.BeforeAll;
Expand Down Expand Up @@ -84,6 +85,18 @@ public void teardownServer()
}
}

@Override
protected void createNamespaceWithProperties(TrinoCatalog catalog, String namespace, Map<String, String> properties)
{
IcebergNessieCatalogConfig icebergNessieCatalogConfig = new IcebergNessieCatalogConfig()
.setServerUri(URI.create(nessieContainer.getRestApiUri()));
NessieApiV2 nessieApi = NessieClientBuilder.createClientBuilderFromSystemSettings()
.withUri(nessieContainer.getRestApiUri())
.build(NessieApiV2.class);
NessieIcebergClient nessieClient = new NessieIcebergClient(nessieApi, icebergNessieCatalogConfig.getDefaultReferenceName(), null, ImmutableMap.of());
nessieClient.createNamespace(Namespace.of(namespace), properties);
}

@Override
protected TrinoCatalog createTrinoCatalog(boolean useUniqueTableLocations)
{
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -38,6 +38,7 @@
import java.util.Map;
import java.util.Optional;

import static com.google.common.collect.ImmutableMap.toImmutableMap;
import static com.google.common.util.concurrent.MoreExecutors.directExecutor;
import static com.google.common.util.concurrent.MoreExecutors.newDirectExecutorService;
import static io.airlift.json.JsonCodec.jsonCodec;
Expand All @@ -62,6 +63,17 @@ protected TrinoCatalog createTrinoCatalog(boolean useUniqueTableLocations)
return createTrinoRestCatalog(useUniqueTableLocations, ImmutableMap.of());
}

@Override
protected void createNamespaceWithProperties(TrinoCatalog catalog, String namespace, Map<String, String> properties)
{
catalog.createNamespace(
SESSION,
namespace,
properties.entrySet().stream()
.collect(toImmutableMap(Map.Entry::getKey, Map.Entry::getValue)),
new TrinoPrincipal(PrincipalType.USER, SESSION.getUser()));
}

private static TrinoRestCatalog createTrinoRestCatalog(boolean useUniqueTableLocations, Map<String, String> properties)
throws IOException
{
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -48,6 +48,7 @@
import org.apache.iceberg.expressions.Expressions;
import org.apache.iceberg.jdbc.JdbcClientPool;
import org.apache.iceberg.types.Types;
import org.junit.jupiter.api.Assumptions;
import org.junit.jupiter.api.BeforeAll;
import org.junit.jupiter.api.Test;

Expand Down Expand Up @@ -148,6 +149,12 @@ private static void executeOnSnowflake(TestingSnowflakeServer server, String sql
server.execute(SNOWFLAKE_TEST_SCHEMA, sql);
}

@Override
protected void createNamespaceWithProperties(TrinoCatalog catalog, String namespace, Map<String, String> namespaceProperties)
{
Assumptions.abort("Snowflake catalog does not support creating namespaces");
}

@Override
protected TrinoCatalog createTrinoCatalog(boolean useUniqueTableLocations)
{
Expand Down