rdblue commented on a change in pull request #3543:
URL: https://github.com/apache/iceberg/pull/3543#discussion_r766972294
##########
File path: core/src/main/java/org/apache/iceberg/CachingCatalog.java
##########
@@ -30,23 +35,98 @@
import org.apache.iceberg.catalog.TableIdentifier;
import org.apache.iceberg.exceptions.AlreadyExistsException;
import org.apache.iceberg.relocated.com.google.common.collect.ImmutableList;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+/**
+ * Class that wraps an Iceberg Catalog to cache table's.
+ *
+ * When `expirationIntervalMillis` is non-zero, the cache will expire tables
after
+ * the given interval provided they have not been accessed via the catalog in
that time.
+ * Each lookup of the table via the catalog will restart the expiration time.
+ *
+ * If the duration is zero, the cache will retain entries forever, or until
+ * an action is taken on the cache to refresh the cache's table entry.
+ */
public class CachingCatalog implements Catalog {
+
+ private static final Logger LOG =
LoggerFactory.getLogger(CachingCatalog.class);
+ private static final RemovalListener<TableIdentifier, Table>
identLoggingRemovalListener =
+ (key, value, cause) -> LOG.debug("{} was evicted from the TableCache
with {} cause", key, cause);
+
+ // TODO - Update this to use the default expiration interval millis once
things are more complete.
+ // Don't want to break current tests.
public static Catalog wrap(Catalog catalog) {
- return wrap(catalog, true);
+ return wrap(catalog, 0);
+ }
+
+ public static Catalog wrap(Catalog catalog, long expirationIntervalMillis) {
+ return wrap(catalog, true, expirationIntervalMillis);
+ }
+
+ public static Catalog wrap(Catalog catalog, Duration expirationInterval) {
+ return wrap(catalog, true, expirationInterval.toMillis());
}
- public static Catalog wrap(Catalog catalog, boolean caseSensitive) {
- return new CachingCatalog(catalog, caseSensitive);
+ public static Catalog wrap(Catalog catalog, boolean caseSensitive, long
expirationIntervalMillis) {
+ return new CachingCatalog(catalog, caseSensitive,
expirationIntervalMillis);
}
- private final Cache<TableIdentifier, Table> tableCache =
Caffeine.newBuilder().softValues().build();
private final Catalog catalog;
private final boolean caseSensitive;
+ private final boolean expirationEnabled;
+ @SuppressWarnings("checkstyle:VisibilityModifier")
+ protected final long expirationIntervalMillis;
+ @SuppressWarnings("checkstyle:VisibilityModifier")
+ protected final Cache<TableIdentifier, Table> tableCache;
+
+ private CachingCatalog(Catalog catalog, boolean caseSensitive, long
expirationIntervalMillis) {
+ this(catalog, caseSensitive, expirationIntervalMillis,
Ticker.systemTicker());
+ }
- private CachingCatalog(Catalog catalog, boolean caseSensitive) {
+ @SuppressWarnings("checkstyle:VisibilityModifier")
+ protected CachingCatalog(Catalog catalog, boolean caseSensitive, long
expirationIntervalMillis, Ticker ticker) {
this.catalog = catalog;
this.caseSensitive = caseSensitive;
+ // Set negative values to zero to avoid creating a Duration with a
negative value
+ this.expirationIntervalMillis = expirationIntervalMillis <= 0 ? 0 :
expirationIntervalMillis;
+ this.expirationEnabled = expirationIntervalMillis > 0;
+ this.tableCache = createTableCache(ticker);
+ }
+
+ private Cache<TableIdentifier, Table> createTableCache(Ticker ticker) {
+ Caffeine<TableIdentifier, Table> cacheBuilder = Caffeine
+ .newBuilder()
+ .softValues()
+ .removalListener(identLoggingRemovalListener);
+
+ // Only setup ticker and background cleanup tasks from expiration if table
expiration is enabled.
+ if (expirationEnabled) {
+ return cacheBuilder
+ .writer(new CacheWriter<TableIdentifier, Table>() {
+ @Override
+ public void write(TableIdentifier tableIdentifier, Table table) {
+ }
Review comment:
Either way.
##########
File path: core/src/test/java/org/apache/iceberg/util/FakeTicker.java
##########
@@ -0,0 +1,61 @@
+/*
+ * 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.util;
+
+import com.github.benmanes.caffeine.cache.Ticker;
+import java.time.Duration;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicLong;
+
+/**
+ * A Ticker whose value can be advanced programmatically in test.
+ *
+ * This is modified from the Guava package
com.google.common.testing.FakeTicker,
Review comment:
If it's completely replaced at this point, then let's just remove the
comment about being from Guava.
##########
File path:
spark/v3.2/spark/src/main/java/org/apache/iceberg/spark/SparkCatalog.java
##########
@@ -383,14 +390,52 @@ public boolean dropNamespace(String[] namespace) throws
NoSuchNamespaceException
@Override
public final void initialize(String name, CaseInsensitiveStringMap options) {
- this.cacheEnabled =
Boolean.parseBoolean(options.getOrDefault("cache-enabled", "true"));
- Catalog catalog = buildIcebergCatalog(name, options);
+ this.cacheEnabled = PropertyUtil.propertyAsBoolean(options,
+ CatalogProperties.TABLE_CACHE_ENABLED,
CatalogProperties.TABLE_CACHE_ENABLED_DEFAULT);
+
+ // If the user disabled caching and did not set the
cache.expiration-interval-ms, we'll set it to zero for
+ // them on their behalf. If they disabled caching but explicitly set a
non-zero cache expiration
+ // interval, we will fail initialization as that's an invalid
configuration.
+ long defaultCacheExpirationInterval =
+ cacheEnabled ?
CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS_DEFAULT : 0L;
+
+ this.cacheExpirationIntervalMs = PropertyUtil.propertyAsLong(options,
+ CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS,
+ defaultCacheExpirationInterval);
+
+ // Normalize usage of -1 to 0 as we'll call Duration.ofMillis on this
value.
+ if (cacheExpirationIntervalMs < 0) {
+ this.cacheExpirationIntervalMs = 0;
+ }
+
+ Preconditions.checkArgument(cacheEnabled || cacheExpirationIntervalMs <=
0L,
+ "The catalog's table cache expiration interval must be set to zero via
the property %s if caching is disabled",
+ CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS);
+
+ // If the user didn't specify TABLE_CACHE_EXPIRATION_INTERVAL_MS, put it
into the SparkConf and the
+ // options map in case it's assumed to be elsewhere (such as cloning a
spark session and
+ // re-instantiating the catalog).
Review comment:
Let's not set options. I think that removes the need for this then.
##########
File path:
spark/v3.2/spark/src/main/java/org/apache/iceberg/spark/SparkCatalog.java
##########
@@ -383,14 +390,52 @@ public boolean dropNamespace(String[] namespace) throws
NoSuchNamespaceException
@Override
public final void initialize(String name, CaseInsensitiveStringMap options) {
- this.cacheEnabled =
Boolean.parseBoolean(options.getOrDefault("cache-enabled", "true"));
- Catalog catalog = buildIcebergCatalog(name, options);
+ this.cacheEnabled = PropertyUtil.propertyAsBoolean(options,
+ CatalogProperties.TABLE_CACHE_ENABLED,
CatalogProperties.TABLE_CACHE_ENABLED_DEFAULT);
+
+ // If the user disabled caching and did not set the
cache.expiration-interval-ms, we'll set it to zero for
+ // them on their behalf. If they disabled caching but explicitly set a
non-zero cache expiration
+ // interval, we will fail initialization as that's an invalid
configuration.
+ long defaultCacheExpirationInterval =
+ cacheEnabled ?
CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS_DEFAULT : 0L;
+
+ this.cacheExpirationIntervalMs = PropertyUtil.propertyAsLong(options,
+ CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS,
+ defaultCacheExpirationInterval);
+
+ // Normalize usage of -1 to 0 as we'll call Duration.ofMillis on this
value.
+ if (cacheExpirationIntervalMs < 0) {
Review comment:
I'm fine with that. Negative values turn off caching, just like 0 does
then?
##########
File path: core/src/main/java/org/apache/iceberg/CachingCatalog.java
##########
@@ -30,23 +35,98 @@
import org.apache.iceberg.catalog.TableIdentifier;
import org.apache.iceberg.exceptions.AlreadyExistsException;
import org.apache.iceberg.relocated.com.google.common.collect.ImmutableList;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+/**
+ * Class that wraps an Iceberg Catalog to cache table's.
+ *
+ * When `expirationIntervalMillis` is non-zero, the cache will expire tables
after
+ * the given interval provided they have not been accessed via the catalog in
that time.
+ * Each lookup of the table via the catalog will restart the expiration time.
+ *
+ * If the duration is zero, the cache will retain entries forever, or until
+ * an action is taken on the cache to refresh the cache's table entry.
+ */
public class CachingCatalog implements Catalog {
+
+ private static final Logger LOG =
LoggerFactory.getLogger(CachingCatalog.class);
+ private static final RemovalListener<TableIdentifier, Table>
identLoggingRemovalListener =
+ (key, value, cause) -> LOG.debug("{} was evicted from the TableCache
with {} cause", key, cause);
+
+ // TODO - Update this to use the default expiration interval millis once
things are more complete.
+ // Don't want to break current tests.
public static Catalog wrap(Catalog catalog) {
- return wrap(catalog, true);
+ return wrap(catalog, 0);
+ }
+
+ public static Catalog wrap(Catalog catalog, long expirationIntervalMillis) {
+ return wrap(catalog, true, expirationIntervalMillis);
+ }
+
+ public static Catalog wrap(Catalog catalog, Duration expirationInterval) {
+ return wrap(catalog, true, expirationInterval.toMillis());
}
- public static Catalog wrap(Catalog catalog, boolean caseSensitive) {
- return new CachingCatalog(catalog, caseSensitive);
+ public static Catalog wrap(Catalog catalog, boolean caseSensitive, long
expirationIntervalMillis) {
+ return new CachingCatalog(catalog, caseSensitive,
expirationIntervalMillis);
}
- private final Cache<TableIdentifier, Table> tableCache =
Caffeine.newBuilder().softValues().build();
private final Catalog catalog;
private final boolean caseSensitive;
+ private final boolean expirationEnabled;
+ @SuppressWarnings("checkstyle:VisibilityModifier")
+ protected final long expirationIntervalMillis;
+ @SuppressWarnings("checkstyle:VisibilityModifier")
+ protected final Cache<TableIdentifier, Table> tableCache;
+
+ private CachingCatalog(Catalog catalog, boolean caseSensitive, long
expirationIntervalMillis) {
+ this(catalog, caseSensitive, expirationIntervalMillis,
Ticker.systemTicker());
+ }
- private CachingCatalog(Catalog catalog, boolean caseSensitive) {
+ @SuppressWarnings("checkstyle:VisibilityModifier")
+ protected CachingCatalog(Catalog catalog, boolean caseSensitive, long
expirationIntervalMillis, Ticker ticker) {
this.catalog = catalog;
this.caseSensitive = caseSensitive;
+ // Set negative values to zero to avoid creating a Duration with a
negative value
+ this.expirationIntervalMillis = expirationIntervalMillis <= 0 ? 0 :
expirationIntervalMillis;
+ this.expirationEnabled = expirationIntervalMillis > 0;
Review comment:
We have to deal with `cache-enabled` because that is existing behavior.
But if we use the expiration interval this way, people don't actually have to
worry about it in practice. 0 -> don't cache, 1+ -> cache for a while, really
big -> don't expire cached tables. That is fairly simple.
##########
File path: core/src/main/java/org/apache/iceberg/CachingCatalog.java
##########
@@ -30,23 +35,98 @@
import org.apache.iceberg.catalog.TableIdentifier;
import org.apache.iceberg.exceptions.AlreadyExistsException;
import org.apache.iceberg.relocated.com.google.common.collect.ImmutableList;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+/**
+ * Class that wraps an Iceberg Catalog to cache table's.
+ *
+ * When `expirationIntervalMillis` is non-zero, the cache will expire tables
after
+ * the given interval provided they have not been accessed via the catalog in
that time.
+ * Each lookup of the table via the catalog will restart the expiration time.
+ *
+ * If the duration is zero, the cache will retain entries forever, or until
+ * an action is taken on the cache to refresh the cache's table entry.
+ */
public class CachingCatalog implements Catalog {
+
+ private static final Logger LOG =
LoggerFactory.getLogger(CachingCatalog.class);
+ private static final RemovalListener<TableIdentifier, Table>
identLoggingRemovalListener =
+ (key, value, cause) -> LOG.debug("{} was evicted from the TableCache
with {} cause", key, cause);
+
+ // TODO - Update this to use the default expiration interval millis once
things are more complete.
+ // Don't want to break current tests.
public static Catalog wrap(Catalog catalog) {
- return wrap(catalog, true);
+ return wrap(catalog, 0);
+ }
+
+ public static Catalog wrap(Catalog catalog, long expirationIntervalMillis) {
+ return wrap(catalog, true, expirationIntervalMillis);
+ }
+
+ public static Catalog wrap(Catalog catalog, Duration expirationInterval) {
+ return wrap(catalog, true, expirationInterval.toMillis());
}
- public static Catalog wrap(Catalog catalog, boolean caseSensitive) {
- return new CachingCatalog(catalog, caseSensitive);
+ public static Catalog wrap(Catalog catalog, boolean caseSensitive, long
expirationIntervalMillis) {
+ return new CachingCatalog(catalog, caseSensitive,
expirationIntervalMillis);
}
- private final Cache<TableIdentifier, Table> tableCache =
Caffeine.newBuilder().softValues().build();
private final Catalog catalog;
private final boolean caseSensitive;
+ private final boolean expirationEnabled;
+ @SuppressWarnings("checkstyle:VisibilityModifier")
+ protected final long expirationIntervalMillis;
+ @SuppressWarnings("checkstyle:VisibilityModifier")
+ protected final Cache<TableIdentifier, Table> tableCache;
+
+ private CachingCatalog(Catalog catalog, boolean caseSensitive, long
expirationIntervalMillis) {
+ this(catalog, caseSensitive, expirationIntervalMillis,
Ticker.systemTicker());
+ }
- private CachingCatalog(Catalog catalog, boolean caseSensitive) {
+ @SuppressWarnings("checkstyle:VisibilityModifier")
+ protected CachingCatalog(Catalog catalog, boolean caseSensitive, long
expirationIntervalMillis, Ticker ticker) {
this.catalog = catalog;
this.caseSensitive = caseSensitive;
+ // Set negative values to zero to avoid creating a Duration with a
negative value
+ this.expirationIntervalMillis = expirationIntervalMillis <= 0 ? 0 :
expirationIntervalMillis;
+ this.expirationEnabled = expirationIntervalMillis > 0;
Review comment:
Yeah, if you set the cache expiration to immediately then we can skip
caching. `cache-enable=true` is the default so it's fine.
##########
File path:
spark/v3.2/spark/src/main/java/org/apache/iceberg/spark/SparkCatalog.java
##########
@@ -383,14 +390,31 @@ public boolean dropNamespace(String[] namespace) throws
NoSuchNamespaceException
@Override
public final void initialize(String name, CaseInsensitiveStringMap options) {
- this.cacheEnabled =
Boolean.parseBoolean(options.getOrDefault("cache-enabled", "true"));
+ this.cacheEnabled = PropertyUtil.propertyAsBoolean(options,
+ CatalogProperties.TABLE_CACHE_ENABLED,
CatalogProperties.TABLE_CACHE_ENABLED_DEFAULT);
+
+ // If the user disabled caching, we turn off the cache-expiration for them
+ long defaultCacheExpirationInterval = cacheEnabled ?
+ CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS_DEFAULT :
+ CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS_OFF;
Review comment:
Indentation is off here.
##########
File path:
spark/v3.2/spark/src/main/java/org/apache/iceberg/spark/SparkCatalog.java
##########
@@ -383,14 +390,31 @@ public boolean dropNamespace(String[] namespace) throws
NoSuchNamespaceException
@Override
public final void initialize(String name, CaseInsensitiveStringMap options) {
- this.cacheEnabled =
Boolean.parseBoolean(options.getOrDefault("cache-enabled", "true"));
+ this.cacheEnabled = PropertyUtil.propertyAsBoolean(options,
+ CatalogProperties.TABLE_CACHE_ENABLED,
CatalogProperties.TABLE_CACHE_ENABLED_DEFAULT);
+
+ // If the user disabled caching, we turn off the cache-expiration for them
+ long defaultCacheExpirationInterval = cacheEnabled ?
+ CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS_DEFAULT :
+ CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS_OFF;
+
+ this.cacheExpirationIntervalMs = PropertyUtil.propertyAsLong(options,
+ CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS,
+ defaultCacheExpirationInterval);
Review comment:
I think this should use
`CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS_DEFAULT` instead of
setting the default based on `cacheEnabled`. If the cache is disabled, then the
expiration interval is ignored. There is no need for extra logic to set it to
some value when we will ignore it.
##########
File path: core/src/main/java/org/apache/iceberg/CachingCatalog.java
##########
@@ -29,24 +34,103 @@
import org.apache.iceberg.catalog.Namespace;
import org.apache.iceberg.catalog.TableIdentifier;
import org.apache.iceberg.exceptions.AlreadyExistsException;
+import org.apache.iceberg.relocated.com.google.common.base.Preconditions;
import org.apache.iceberg.relocated.com.google.common.collect.ImmutableList;
-
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * Class that wraps an Iceberg Catalog to cache tables.
+ * <p>
+ * When {@code expirationIntervalMillis} is positive, the cache will expire
tables after
+ * the given interval provided they have not been accessed via the catalog in
that time.
+ * Each lookup of the table via the catalog will restart the expiration time.
+ * <p>
+ * If the duration is negative / -1, cache-entry expiration is disabled.
+ * <p>
+ * If the duration is zero, caching should be turned off. If a value of zero
gets passed to
+ * this {@link Catalog} wrapper, that is a bug.
+ */
public class CachingCatalog implements Catalog {
+
+ private static final Logger LOG =
LoggerFactory.getLogger(CachingCatalog.class);
+ private static final RemovalListener<TableIdentifier, Table>
identLoggingRemovalListener =
+ (key, value, cause) -> LOG.debug("Evicted {} from the table cache ({})",
key, cause);
+
public static Catalog wrap(Catalog catalog) {
- return wrap(catalog, true);
+ return wrap(catalog,
CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS_OFF);
+ }
+
+ public static Catalog wrap(Catalog catalog, long expirationIntervalMillis) {
+ return wrap(catalog, true, expirationIntervalMillis);
}
- public static Catalog wrap(Catalog catalog, boolean caseSensitive) {
- return new CachingCatalog(catalog, caseSensitive);
+ public static Catalog wrap(Catalog catalog, Duration expirationInterval) {
Review comment:
I don't see much value in this overload. This supports setting
milliseconds directly and the only use of this is
`Duration.ofMillis(cacheExpirationIntervalMs)` to pass in here.
##########
File path:
spark/v3.2/spark/src/test/java/org/apache/iceberg/spark/source/TestSparkCatalogCacheExpiration.java
##########
@@ -0,0 +1,134 @@
+/*
+ * 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.spark.source;
+
+import java.util.Map;
+import org.apache.iceberg.CachingCatalog;
+import org.apache.iceberg.CatalogProperties;
+import org.apache.iceberg.catalog.Catalog;
+import org.apache.iceberg.relocated.com.google.common.collect.ImmutableMap;
+import org.apache.iceberg.spark.SparkCatalog;
+import org.apache.iceberg.spark.SparkCatalogTestBase;
+import org.apache.spark.sql.connector.catalog.TableCatalog;
+import org.assertj.core.api.Assertions;
+import org.junit.Test;
+import org.junit.runners.Parameterized;
+
+public class TestSparkCatalogCacheExpiration extends SparkCatalogTestBase {
Review comment:
I don't think that this should inherit from `SparkCatalogTestBase`. We
don't need to set up 4 different Spark catalogs for this. Can you use
`SparkTestBaseWithCatalog` instead?
##########
File path:
spark/v3.2/spark/src/main/java/org/apache/iceberg/spark/SparkCatalog.java
##########
@@ -383,14 +390,31 @@ public boolean dropNamespace(String[] namespace) throws
NoSuchNamespaceException
@Override
public final void initialize(String name, CaseInsensitiveStringMap options) {
- this.cacheEnabled =
Boolean.parseBoolean(options.getOrDefault("cache-enabled", "true"));
+ this.cacheEnabled = PropertyUtil.propertyAsBoolean(options,
+ CatalogProperties.TABLE_CACHE_ENABLED,
CatalogProperties.TABLE_CACHE_ENABLED_DEFAULT);
+
+ // If the user disabled caching, we turn off the cache-expiration for them
+ long defaultCacheExpirationInterval = cacheEnabled ?
+ CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS_DEFAULT :
+ CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS_OFF;
Review comment:
Looks like this isn't needed anyway.
##########
File path:
spark/v3.2/spark/src/main/java/org/apache/iceberg/spark/SparkCatalog.java
##########
@@ -83,7 +89,8 @@
private String catalogName = null;
private Catalog icebergCatalog = null;
- private boolean cacheEnabled = true;
+ private boolean cacheEnabled = CatalogProperties.TABLE_CACHE_ENABLED_DEFAULT;
+ private long cacheExpirationIntervalMs =
CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS_DEFAULT;
Review comment:
Do these need to be fields? Can we remove them and just make them local
within `initialize`?
##########
File path: core/src/main/java/org/apache/iceberg/CachingCatalog.java
##########
@@ -29,24 +34,99 @@
import org.apache.iceberg.catalog.Namespace;
import org.apache.iceberg.catalog.TableIdentifier;
import org.apache.iceberg.exceptions.AlreadyExistsException;
+import org.apache.iceberg.relocated.com.google.common.base.Preconditions;
import org.apache.iceberg.relocated.com.google.common.collect.ImmutableList;
-
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * Class that wraps an Iceberg Catalog to cache tables.
+ * <p>
+ * When {@code expirationIntervalMillis} is positive, the cache will expire
tables after
+ * the given interval provided they have not been accessed via the catalog in
that time.
+ * Each lookup of the table via the catalog will restart the expiration time.
+ * <p>
+ * If the duration is negative / -1, cache-entry expiration is disabled.
+ * <p>
+ * If the duration is zero, caching should be turned off. If a value of zero
gets passed to
+ * this {@link Catalog} wrapper, that is a bug.
+ */
public class CachingCatalog implements Catalog {
+
+ private static final Logger LOG =
LoggerFactory.getLogger(CachingCatalog.class);
+ private static final RemovalListener<TableIdentifier, Table>
identLoggingRemovalListener =
+ (key, value, cause) -> LOG.debug("Evicted {} from the table cache ({})",
key, cause);
+
public static Catalog wrap(Catalog catalog) {
- return wrap(catalog, true);
+ return wrap(catalog,
CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS_OFF);
Review comment:
I think this should default to the default expiration interval.
##########
File path: core/src/main/java/org/apache/iceberg/CachingCatalog.java
##########
@@ -29,24 +34,99 @@
import org.apache.iceberg.catalog.Namespace;
import org.apache.iceberg.catalog.TableIdentifier;
import org.apache.iceberg.exceptions.AlreadyExistsException;
+import org.apache.iceberg.relocated.com.google.common.base.Preconditions;
import org.apache.iceberg.relocated.com.google.common.collect.ImmutableList;
-
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * Class that wraps an Iceberg Catalog to cache tables.
+ * <p>
+ * When {@code expirationIntervalMillis} is positive, the cache will expire
tables after
+ * the given interval provided they have not been accessed via the catalog in
that time.
+ * Each lookup of the table via the catalog will restart the expiration time.
+ * <p>
+ * If the duration is negative / -1, cache-entry expiration is disabled.
+ * <p>
+ * If the duration is zero, caching should be turned off. If a value of zero
gets passed to
+ * this {@link Catalog} wrapper, that is a bug.
+ */
public class CachingCatalog implements Catalog {
+
+ private static final Logger LOG =
LoggerFactory.getLogger(CachingCatalog.class);
+ private static final RemovalListener<TableIdentifier, Table>
identLoggingRemovalListener =
+ (key, value, cause) -> LOG.debug("Evicted {} from the table cache ({})",
key, cause);
+
public static Catalog wrap(Catalog catalog) {
- return wrap(catalog, true);
+ return wrap(catalog,
CatalogProperties.TABLE_CACHE_EXPIRATION_INTERVAL_MS_OFF);
Review comment:
Actually, if this isn't called by Iceberg then you're right: we should
match the existing behavior.
--
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
To unsubscribe, e-mail: [email protected]
For queries about this service, please contact Infrastructure at:
[email protected]
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]