Skip to content
Merged
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
13 changes: 13 additions & 0 deletions api/src/main/java/org/apache/iceberg/catalog/Catalog.java
Original file line number Diff line number Diff line change
Expand Up @@ -325,6 +325,19 @@ default TableBuilder buildTable(TableIdentifier identifier, Schema schema) {
throw new UnsupportedOperationException(this.getClass().getName() + " does not implement buildTable");
}

/**
* Initialize a catalog given a custom name and a map of catalog properties.
* <p>
* A custom Catalog implementation must have a no-arg constructor.
* A compute engine like Spark or Flink will first initialize the catalog without any arguments,
* and then call this method to complete catalog initialization with properties passed into the engine.
*
* @param name a custom name for the catalog
* @param properties catalog properties
*/
default void initialize(String name, Map<String, String> properties) {
}

/**
* A builder used to create valid {@link Table tables} or start create/replace {@link Transaction transactions}.
* <p>
Expand Down
28 changes: 28 additions & 0 deletions core/src/main/java/org/apache/iceberg/CatalogProperties.java
Original file line number Diff line number Diff line change
@@ -0,0 +1,28 @@
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing,
* software distributed under the License is distributed on an
* "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
* KIND, either express or implied. See the License for the
* specific language governing permissions and limitations
* under the License.
*/

package org.apache.iceberg;

public class CatalogProperties {

private CatalogProperties() {
}

public static final String CATALOG_IMPL = "catalog-impl";
}
51 changes: 51 additions & 0 deletions core/src/main/java/org/apache/iceberg/CatalogUtil.java
Original file line number Diff line number Diff line change
Expand Up @@ -22,9 +22,14 @@
import java.io.IOException;
import java.util.Map;
import java.util.Set;
import org.apache.hadoop.conf.Configurable;
import org.apache.hadoop.conf.Configuration;
import org.apache.iceberg.catalog.Catalog;
import org.apache.iceberg.common.DynConstructors;
import org.apache.iceberg.exceptions.RuntimeIOException;
import org.apache.iceberg.io.FileIO;
import org.apache.iceberg.relocated.com.google.common.base.Joiner;
import org.apache.iceberg.relocated.com.google.common.base.Preconditions;
import org.apache.iceberg.relocated.com.google.common.collect.Iterables;
import org.apache.iceberg.relocated.com.google.common.collect.MapMaker;
import org.apache.iceberg.relocated.com.google.common.collect.Sets;
Expand Down Expand Up @@ -117,4 +122,50 @@ private static void deleteFiles(FileIO io, Set<ManifestFile> allManifests) {
}
});
}

/**
* Load a custom catalog implementation.
* <p>
* The catalog must have a no-arg constructor.
* If the class implements {@link Configurable},
* a Hadoop config will be passed using {@link Configurable#setConf(Configuration)}.
* {@link Catalog#initialize(String catalogName, Map options)} is called to complete the initialization.
*
* @param impl catalog implementation full class name
* @param catalogName catalog name
* @param properties catalog properties
* @param hadoopConf hadoop configuration if needed
* @return initialized catalog object
* @throws IllegalArgumentException if no-arg constructor not found or error during initialization
*/
public static Catalog loadCatalog(
String impl,
String catalogName,
Map<String, String> properties,
Configuration hadoopConf) {
Preconditions.checkNotNull(impl, "Cannot initialize custom Catalog, impl class name is null");
DynConstructors.Ctor<Catalog> ctor;
try {
ctor = DynConstructors.builder(Catalog.class).impl(impl).buildChecked();
} catch (NoSuchMethodException e) {
throw new IllegalArgumentException(String.format(
"Cannot initialize Catalog, missing no-arg constructor: %s", impl), e);
}

Catalog catalog;
try {
catalog = ctor.newInstance();

} catch (ClassCastException e) {
throw new IllegalArgumentException(
String.format("Cannot initialize Catalog, %s does not implement Catalog.", impl), e);
}

if (catalog instanceof Configurable) {
((Configurable) catalog).setConf(hadoopConf);
}

catalog.initialize(catalogName, properties);
return catalog;
}
}
216 changes: 216 additions & 0 deletions core/src/test/java/org/apache/iceberg/TestCatalogUtil.java
Original file line number Diff line number Diff line change
@@ -0,0 +1,216 @@
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing,
* software distributed under the License is distributed on an
* "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
* KIND, either express or implied. See the License for the
* specific language governing permissions and limitations
* under the License.
*/

package org.apache.iceberg;

import java.util.HashMap;
import java.util.List;
import java.util.Map;
import org.apache.hadoop.conf.Configurable;
import org.apache.hadoop.conf.Configuration;
import org.apache.iceberg.catalog.Catalog;
import org.apache.iceberg.catalog.Namespace;
import org.apache.iceberg.catalog.TableIdentifier;
import org.junit.Assert;
import org.junit.Test;

public class TestCatalogUtil {

@Test
public void loadCustomCatalog() {
Map<String, String> options = new HashMap<>();
options.put("key", "val");
Configuration hadoopConf = new Configuration();
String name = "custom";
Catalog catalog = CatalogUtil.loadCatalog(TestCatalog.class.getName(), name, options, hadoopConf);
Assert.assertTrue(catalog instanceof TestCatalog);
Assert.assertEquals(name, ((TestCatalog) catalog).catalogName);
Assert.assertEquals(options, ((TestCatalog) catalog).flinkOptions);
}

@Test
public void loadCustomCatalog_withHadoopConfig() {
Map<String, String> options = new HashMap<>();
options.put("key", "val");
Configuration hadoopConf = new Configuration();
hadoopConf.set("key", "val");
String name = "custom";
Catalog catalog = CatalogUtil.loadCatalog(TestCatalogConfigurable.class.getName(), name, options, hadoopConf);
Assert.assertTrue(catalog instanceof TestCatalogConfigurable);
Assert.assertEquals(name, ((TestCatalogConfigurable) catalog).catalogName);
Assert.assertEquals(options, ((TestCatalogConfigurable) catalog).flinkOptions);
Assert.assertEquals(hadoopConf, ((TestCatalogConfigurable) catalog).configuration);
}

@Test
public void loadCustomCatalog_NoArgConstructorNotFound() {
Map<String, String> options = new HashMap<>();
options.put("key", "val");
Configuration hadoopConf = new Configuration();
String name = "custom";
AssertHelpers.assertThrows("must have no-arg constructor",
IllegalArgumentException.class,
"missing no-arg constructor",
() -> CatalogUtil.loadCatalog(TestCatalogBadConstructor.class.getName(), name, options, hadoopConf));
}

@Test
public void loadCustomCatalog_NotImplementCatalog() {
Map<String, String> options = new HashMap<>();
options.put("key", "val");
Configuration hadoopConf = new Configuration();
String name = "custom";

AssertHelpers.assertThrows("must implement catalog",
IllegalArgumentException.class,
"does not implement Catalog",
() -> CatalogUtil.loadCatalog(TestCatalogNoInterface.class.getName(), name, options, hadoopConf));
}

public static class TestCatalog extends BaseMetastoreCatalog {

private String catalogName;
private Map<String, String> flinkOptions;

public TestCatalog() {
}

@Override
public void initialize(String name, Map<String, String> properties) {
this.catalogName = name;
this.flinkOptions = properties;
}

@Override
protected TableOperations newTableOps(TableIdentifier tableIdentifier) {
return null;
}

@Override
protected String defaultWarehouseLocation(TableIdentifier tableIdentifier) {
return null;
}

@Override
public List<TableIdentifier> listTables(Namespace namespace) {
return null;
}

@Override
public boolean dropTable(TableIdentifier identifier, boolean purge) {
return false;
}

@Override
public void renameTable(TableIdentifier from, TableIdentifier to) {

}
}

public static class TestCatalogConfigurable extends BaseMetastoreCatalog implements Configurable {

private String catalogName;
private Map<String, String> flinkOptions;
private Configuration configuration;

public TestCatalogConfigurable() {
}

@Override
public void initialize(String name, Map<String, String> properties) {
this.catalogName = name;
this.flinkOptions = properties;
}

@Override
public void setConf(Configuration conf) {
this.configuration = conf;
}

@Override
public Configuration getConf() {
return configuration;
}

@Override
protected TableOperations newTableOps(TableIdentifier tableIdentifier) {
return null;
}

@Override
protected String defaultWarehouseLocation(TableIdentifier tableIdentifier) {
return null;
}

@Override
public List<TableIdentifier> listTables(Namespace namespace) {
return null;
}

@Override
public boolean dropTable(TableIdentifier identifier, boolean purge) {
return false;
}

@Override
public void renameTable(TableIdentifier from, TableIdentifier to) {

}
}

public static class TestCatalogBadConstructor extends BaseMetastoreCatalog {

public TestCatalogBadConstructor(String arg) {
}

@Override
protected TableOperations newTableOps(TableIdentifier tableIdentifier) {
return null;
}

@Override
protected String defaultWarehouseLocation(TableIdentifier tableIdentifier) {
return null;
}

@Override
public List<TableIdentifier> listTables(Namespace namespace) {
return null;
}

@Override
public boolean dropTable(TableIdentifier identifier, boolean purge) {
return false;
}

@Override
public void renameTable(TableIdentifier from, TableIdentifier to) {

}

@Override
public void initialize(String name, Map<String, String> properties) {
}
}

public static class TestCatalogNoInterface {
public TestCatalogNoInterface() {
}
}
}
41 changes: 41 additions & 0 deletions flink/src/main/java/org/apache/iceberg/flink/CatalogLoader.java
Original file line number Diff line number Diff line change
Expand Up @@ -20,12 +20,16 @@
package org.apache.iceberg.flink;

import java.io.Serializable;
import java.util.Map;
import org.apache.hadoop.conf.Configuration;
import org.apache.iceberg.CatalogUtil;
import org.apache.iceberg.catalog.Catalog;
import org.apache.iceberg.hadoop.HadoopCatalog;
import org.apache.iceberg.hadoop.SerializableConfiguration;
import org.apache.iceberg.hive.HiveCatalog;
import org.apache.iceberg.relocated.com.google.common.base.MoreObjects;
import org.apache.iceberg.relocated.com.google.common.base.Preconditions;
import org.apache.iceberg.relocated.com.google.common.collect.Maps;

/**
* Serializable loader to load an Iceberg {@link Catalog}.
Expand All @@ -49,6 +53,10 @@ static CatalogLoader hive(String name, Configuration hadoopConf, String uri, Str
return new HiveCatalogLoader(name, hadoopConf, uri, warehouse, clientPoolSize);
}

static CatalogLoader custom(String name, Map<String, String> properties, Configuration hadoopConf, String impl) {
return new CustomCatalogLoader(name, properties, hadoopConf, impl);
}

class HadoopCatalogLoader implements CatalogLoader {
private final String catalogName;
private final SerializableConfiguration hadoopConf;
Expand Down Expand Up @@ -105,4 +113,37 @@ public String toString() {
.toString();
}
}

class CustomCatalogLoader implements CatalogLoader {

private final SerializableConfiguration hadoopConf;
private final Map<String, String> properties;
private final String name;
private final String impl;

private CustomCatalogLoader(
String name,
Map<String, String> properties,
Configuration conf,
String impl) {
this.hadoopConf = new SerializableConfiguration(conf);
this.properties = Maps.newHashMap(properties); // wrap into a hashmap for serialization
this.name = name;
this.impl = Preconditions.checkNotNull(impl, "Cannot initialize custom Catalog, impl class name is null");
}

@Override
public Catalog loadCatalog() {
return CatalogUtil.loadCatalog(impl, name, properties, hadoopConf.get());
}

@Override
public String toString() {
return MoreObjects.toStringHelper(this)
.add("name", name)
.add("impl", impl)
.toString();
}
}

}
Loading