Skip to content

Commit

Permalink
Add stats for metastore partition cache
Browse files Browse the repository at this point in the history
  • Loading branch information
NikhilCollooru authored and highker committed Apr 6, 2022
1 parent 5589e6d commit 10211d4
Show file tree
Hide file tree
Showing 10 changed files with 173 additions and 15 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -40,7 +40,9 @@
import com.facebook.presto.hive.gcs.HiveGcsConfigurationInitializer;
import com.facebook.presto.hive.metastore.CachingHiveMetastore;
import com.facebook.presto.hive.metastore.ExtendedHiveMetastore;
import com.facebook.presto.hive.metastore.HiveMetastoreCacheStats;
import com.facebook.presto.hive.metastore.HivePartitionMutator;
import com.facebook.presto.hive.metastore.MetastoreCacheStats;
import com.facebook.presto.hive.metastore.MetastoreConfig;
import com.fasterxml.jackson.databind.DeserializationContext;
import com.fasterxml.jackson.databind.deser.std.FromStringDeserializer;
Expand Down Expand Up @@ -100,6 +102,8 @@ public void configure(Binder binder)
configBinder(binder).bindConfig(MetastoreConfig.class);
configBinder(binder).bindConfig(HiveClientConfig.class);
configBinder(binder).bindConfig(MetastoreClientConfig.class);
binder.bind(MetastoreCacheStats.class).to(HiveMetastoreCacheStats.class).in(Scopes.SINGLETON);
newExporter(binder).export(MetastoreCacheStats.class).as(generatedNameOf(MetastoreCacheStats.class, connectorId));
binder.bind(ExtendedHiveMetastore.class).to(CachingHiveMetastore.class).in(Scopes.SINGLETON);
binder.bind(HdfsConfiguration.class).annotatedWith(ForMetastoreHdfsEnvironment.class).to(HiveCachingHdfsConfiguration.class).in(Scopes.SINGLETON);

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -54,6 +54,7 @@
import static com.facebook.presto.hive.metastore.CachingHiveMetastore.MetastoreCacheScope.ALL;
import static com.facebook.presto.hive.metastore.HivePartitionName.hivePartitionName;
import static com.facebook.presto.hive.metastore.HiveTableName.hiveTableName;
import static com.facebook.presto.hive.metastore.NoopMetastoreCacheStats.NOOP_METASTORE_CACHE_STATS;
import static com.facebook.presto.hive.metastore.PartitionFilter.partitionFilter;
import static com.google.common.base.MoreObjects.toStringHelper;
import static com.google.common.base.Preconditions.checkArgument;
Expand All @@ -78,8 +79,6 @@
public class CachingHiveMetastore
implements ExtendedHiveMetastore
{
private static final String NO_IMPERSONATION_USER = "no-impersonation-caching-user";

public enum MetastoreCacheScope
{
ALL, PARTITION
Expand Down Expand Up @@ -108,6 +107,7 @@ public enum MetastoreCacheScope
public CachingHiveMetastore(
@ForCachingHiveMetastore ExtendedHiveMetastore delegate,
@ForCachingHiveMetastore ExecutorService executor,
MetastoreCacheStats metastoreCacheStats,
MetastoreClientConfig metastoreClientConfig)
{
this(
Expand All @@ -119,7 +119,8 @@ public CachingHiveMetastore(
metastoreClientConfig.getMetastoreCacheMaximumSize(),
metastoreClientConfig.isPartitionVersioningEnabled(),
metastoreClientConfig.getMetastoreCacheScope(),
metastoreClientConfig.getPartitionCacheValidationPercentage());
metastoreClientConfig.getPartitionCacheValidationPercentage(),
metastoreCacheStats);
}

public CachingHiveMetastore(
Expand All @@ -131,7 +132,8 @@ public CachingHiveMetastore(
long maximumSize,
boolean partitionVersioningEnabled,
MetastoreCacheScope metastoreCacheScope,
double partitionCacheValidationPercentage)
double partitionCacheValidationPercentage,
MetastoreCacheStats metastoreCacheStats)
{
this(
delegate,
Expand All @@ -142,7 +144,8 @@ public CachingHiveMetastore(
maximumSize,
partitionVersioningEnabled,
metastoreCacheScope,
partitionCacheValidationPercentage);
partitionCacheValidationPercentage,
metastoreCacheStats);
}

public static CachingHiveMetastore memoizeMetastore(ExtendedHiveMetastore delegate, boolean isMetastoreImpersonationEnabled, long maximumSize)
Expand All @@ -156,7 +159,8 @@ public static CachingHiveMetastore memoizeMetastore(ExtendedHiveMetastore delega
maximumSize,
false,
ALL,
0.0);
0.0,
NOOP_METASTORE_CACHE_STATS);
}

private CachingHiveMetastore(
Expand All @@ -168,7 +172,8 @@ private CachingHiveMetastore(
long maximumSize,
boolean partitionVersioningEnabled,
MetastoreCacheScope metastoreCacheScope,
double partitionCacheValidationPercentage)
double partitionCacheValidationPercentage,
MetastoreCacheStats metastoreCacheStats)
{
this.delegate = requireNonNull(delegate, "delegate is null");
requireNonNull(executor, "executor is null");
Expand Down Expand Up @@ -269,6 +274,7 @@ public Map<KeyAndContext<HivePartitionName>, Optional<Partition>> loadAll(Iterab
return loadPartitionsByNames(partitionNames);
}
}, executor));
metastoreCacheStats.setPartitionCache(partitionCache);

tablePrivilegesCache = newCacheBuilder(cacheExpiresAfterWriteMillis, cacheRefreshMills, cacheMaxSize)
.build(asyncReloading(CacheLoader.from(this::loadTablePrivileges), executor));
Expand Down Expand Up @@ -1034,7 +1040,6 @@ private static CacheBuilder<Object, Object> newCacheBuilder(OptionalLong expires
if (refreshMillis.isPresent() && (!expiresAfterWriteMillis.isPresent() || expiresAfterWriteMillis.getAsLong() > refreshMillis.getAsLong())) {
cacheBuilder = cacheBuilder.refreshAfterWrite(refreshMillis.getAsLong(), MILLISECONDS);
}
cacheBuilder = cacheBuilder.maximumSize(maximumSize);
return cacheBuilder;
return cacheBuilder.maximumSize(maximumSize).recordStats();
}
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,57 @@
/*
* Licensed 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 com.facebook.presto.hive.metastore;

import com.google.common.cache.LoadingCache;
import org.weakref.jmx.Managed;

public class HiveMetastoreCacheStats
implements MetastoreCacheStats
{
private LoadingCache<?, ?> partitionCache;

@Override
public void setPartitionCache(LoadingCache<?, ?> partitionCache)
{
this.partitionCache = partitionCache;
}

@Managed
@Override
public long getPartitionCacheHit()
{
return partitionCache.stats().hitCount();
}

@Managed
@Override
public long getPartitionCacheMiss()
{
return partitionCache.stats().missCount();
}

@Managed
@Override
public long getPartitionCacheEviction()
{
return partitionCache.stats().evictionCount();
}

@Managed
@Override
public long getPartitionCacheSize()
{
return partitionCache.size();
}
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,29 @@
/*
* Licensed 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 com.facebook.presto.hive.metastore;

import com.google.common.cache.LoadingCache;

public interface MetastoreCacheStats
{
void setPartitionCache(LoadingCache<?, ?> partitionCache);

long getPartitionCacheHit();

long getPartitionCacheMiss();

long getPartitionCacheEviction();

long getPartitionCacheSize();
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,46 @@
/*
* Licensed 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 com.facebook.presto.hive.metastore;

import com.google.common.cache.LoadingCache;

public class NoopMetastoreCacheStats
implements MetastoreCacheStats
{
public static final NoopMetastoreCacheStats NOOP_METASTORE_CACHE_STATS = new NoopMetastoreCacheStats();

public void setPartitionCache(LoadingCache<?, ?> partitionCache)
{
}

public long getPartitionCacheHit()
{
return 0;
}

public long getPartitionCacheMiss()
{
return 0;
}

public long getPartitionCacheEviction()
{
return 0;
}

public long getPartitionCacheSize()
{
return 0;
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -38,6 +38,7 @@
import java.util.function.Function;

import static com.facebook.airlift.concurrent.Threads.daemonThreadsNamed;
import static com.facebook.presto.hive.metastore.NoopMetastoreCacheStats.NOOP_METASTORE_CACHE_STATS;
import static com.facebook.presto.hive.metastore.Partition.Builder;
import static com.facebook.presto.hive.metastore.thrift.MockHiveMetastoreClient.BAD_DATABASE;
import static com.facebook.presto.hive.metastore.thrift.MockHiveMetastoreClient.PARTITION_VERSION;
Expand Down Expand Up @@ -84,7 +85,8 @@ public void setUp()
1000,
false,
MetastoreCacheScope.ALL,
0.0);
0.0,
NOOP_METASTORE_CACHE_STATS);
stats = thriftHiveMetastore.getStats();
}

Expand Down Expand Up @@ -223,7 +225,8 @@ public void testCachingWithPartitionVersioning()
1000,
true,
MetastoreCacheScope.PARTITION,
0.0);
0.0,
NOOP_METASTORE_CACHE_STATS);

assertEquals(mockClient.getAccessCount(), 0);
assertEquals(partitionCachingEnabledmetastore.getPartitionNamesByFilter(TEST_METASTORE_CONTEXT, TEST_DATABASE, TEST_TABLE, ImmutableMap.of()), EXPECTED_PARTITIONS);
Expand Down Expand Up @@ -270,7 +273,8 @@ private void assertInvalidateCache(MockPartitionMutator partitionMutator)
1000,
true,
MetastoreCacheScope.PARTITION,
0.0);
0.0,
NOOP_METASTORE_CACHE_STATS);

int clientAccessCount = 0;
for (int i = 0; i < 100; i++) {
Expand Down Expand Up @@ -305,7 +309,8 @@ public void testPartitionCacheValidation()
1000,
true,
MetastoreCacheScope.PARTITION,
100.0);
100.0,
NOOP_METASTORE_CACHE_STATS);

// Warmup the cache
partitionCacheVerificationEnabledMetastore.getPartitionsByNames(TEST_METASTORE_CONTEXT, TEST_DATABASE, TEST_TABLE, ImmutableList.of(TEST_PARTITION1, TEST_PARTITION2));
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -22,7 +22,9 @@
import com.facebook.presto.hive.cache.HiveCachingHdfsConfiguration;
import com.facebook.presto.hive.datasink.DataSinkFactory;
import com.facebook.presto.hive.datasink.OutputStreamDataSinkFactory;
import com.facebook.presto.hive.metastore.HiveMetastoreCacheStats;
import com.facebook.presto.hive.metastore.HivePartitionMutator;
import com.facebook.presto.hive.metastore.MetastoreCacheStats;
import com.facebook.presto.hive.orc.DwrfBatchPageSourceFactory;
import com.facebook.presto.hive.orc.DwrfSelectivePageSourceFactory;
import com.facebook.presto.hive.orc.OrcBatchPageSourceFactory;
Expand Down Expand Up @@ -205,6 +207,9 @@ public void configure(Binder binder)

configBinder(binder).bindConfig(ParquetFileWriterConfig.class);
fileWriterFactoryBinder.addBinding().to(ParquetFileWriterFactory.class).in(Scopes.SINGLETON);

binder.bind(MetastoreCacheStats.class).to(HiveMetastoreCacheStats.class).in(Scopes.SINGLETON);
newExporter(binder).export(MetastoreCacheStats.class).as(generatedNameOf(MetastoreCacheStats.class, connectorId));
binder.install(new MetastoreClientModule());

binder.bind(HiveEncryptionInformationProvider.class).in(Scopes.SINGLETON);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -272,6 +272,7 @@
import static com.facebook.presto.hive.metastore.MetastoreUtil.createDirectory;
import static com.facebook.presto.hive.metastore.MetastoreUtil.getMetastoreHeaders;
import static com.facebook.presto.hive.metastore.MetastoreUtil.toPartitionValues;
import static com.facebook.presto.hive.metastore.NoopMetastoreCacheStats.NOOP_METASTORE_CACHE_STATS;
import static com.facebook.presto.hive.metastore.PrestoTableType.MANAGED_TABLE;
import static com.facebook.presto.hive.metastore.StorageFormat.fromHiveStorageFormat;
import static com.facebook.presto.hive.rule.HiveFilterPushdown.pushdownFilter;
Expand Down Expand Up @@ -931,7 +932,8 @@ protected final void setup(String host, int port, String databaseName, String ti
10000,
false,
MetastoreCacheScope.ALL,
0.0);
0.0,
NOOP_METASTORE_CACHE_STATS);

setup(databaseName, hiveClientConfig, cacheConfig, metastoreClientConfig, metastore);
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -107,6 +107,7 @@
import static com.facebook.presto.hive.HiveTestUtils.getDefaultOrcFileWriterFactory;
import static com.facebook.presto.hive.HiveTestUtils.getTypes;
import static com.facebook.presto.hive.metastore.MetastoreOperationResult.EMPTY_RESULT;
import static com.facebook.presto.hive.metastore.NoopMetastoreCacheStats.NOOP_METASTORE_CACHE_STATS;
import static com.facebook.presto.spi.SplitContext.NON_CACHEABLE;
import static com.facebook.presto.spi.connector.ConnectorSplitManager.SplitSchedulingStrategy.UNGROUPED_SCHEDULING;
import static com.facebook.presto.testing.MaterializedResult.materializeSourceDataStream;
Expand Down Expand Up @@ -500,7 +501,7 @@ private static class TestingHiveMetastore

public TestingHiveMetastore(ExtendedHiveMetastore delegate, ExecutorService executor, MetastoreClientConfig metastoreClientConfig, Path basePath, HdfsEnvironment hdfsEnvironment)
{
super(delegate, executor, metastoreClientConfig);
super(delegate, executor, NOOP_METASTORE_CACHE_STATS, metastoreClientConfig);
this.basePath = basePath;
this.hdfsEnvironment = hdfsEnvironment;
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -40,7 +40,9 @@
import com.facebook.presto.hive.gcs.HiveGcsConfigurationInitializer;
import com.facebook.presto.hive.metastore.CachingHiveMetastore;
import com.facebook.presto.hive.metastore.ExtendedHiveMetastore;
import com.facebook.presto.hive.metastore.HiveMetastoreCacheStats;
import com.facebook.presto.hive.metastore.HivePartitionMutator;
import com.facebook.presto.hive.metastore.MetastoreCacheStats;
import com.facebook.presto.hive.metastore.MetastoreConfig;
import com.facebook.presto.iceberg.optimizer.IcebergPlanOptimizer;
import com.facebook.presto.orc.CachingStripeMetadataSource;
Expand Down Expand Up @@ -114,6 +116,8 @@ public void configure(Binder binder)
binder.bind(HdfsConfiguration.class).annotatedWith(ForMetastoreHdfsEnvironment.class).to(HiveCachingHdfsConfiguration.class).in(Scopes.SINGLETON);
binder.bind(HdfsConfiguration.class).annotatedWith(ForCachingFileSystem.class).to(HiveHdfsConfiguration.class).in(Scopes.SINGLETON);
binder.bind(PartitionMutator.class).to(HivePartitionMutator.class).in(Scopes.SINGLETON);
binder.bind(MetastoreCacheStats.class).to(HiveMetastoreCacheStats.class).in(Scopes.SINGLETON);
newExporter(binder).export(MetastoreCacheStats.class).as(generatedNameOf(MetastoreCacheStats.class, connectorId));
binder.bind(ExtendedHiveMetastore.class).to(CachingHiveMetastore.class).in(Scopes.SINGLETON);
binder.bind(MBeanServer.class).toInstance(new TestingMBeanServer());

Expand Down

0 comments on commit 10211d4

Please sign in to comment.