You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@phoenix.apache.org by ka...@apache.org on 2019/05/15 22:16:36 UTC

[phoenix] branch master updated: PHOENIX-5231 Configurable Stats Cache

This is an automated email from the ASF dual-hosted git repository.

karanmehta93 pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/phoenix.git


The following commit(s) were added to refs/heads/master by this push:
     new 7b0246e  PHOENIX-5231 Configurable Stats Cache
7b0246e is described below

commit 7b0246e4f031eabab12a4176052de5ac7bcbe1f3
Author: Daniel <da...@salesforce.com>
AuthorDate: Fri Apr 5 19:00:23 2019 -0700

    PHOENIX-5231 Configurable Stats Cache
---
 .../phoenix/end2end/ConfigurableCacheIT.java       | 150 ++++++++++++
 .../PhoenixNonRetryableRuntimeException.java       |  34 +++
 .../phoenix/query/ConnectionQueryServicesImpl.java |  17 +-
 .../query/ConnectionlessQueryServicesImpl.java     |  18 +-
 .../query/DefaultGuidePostsCacheFactory.java       |  45 ++++
 .../org/apache/phoenix/query/EmptyStatsLoader.java |  35 +++
 .../org/apache/phoenix/query/GuidePostsCache.java  | 255 +--------------------
 .../phoenix/query/GuidePostsCacheFactory.java      |  46 ++++
 .../apache/phoenix/query/GuidePostsCacheImpl.java  | 148 ++++++++++++
 .../phoenix/query/GuidePostsCacheProvider.java     |  79 +++++++
 .../phoenix/query/GuidePostsCacheWrapper.java      |  74 ++++++
 .../phoenix/query/ITGuidePostsCacheFactory.java    |  52 +++++
 .../phoenix/query/PhoenixStatsCacheLoader.java     |   2 +-
 .../org/apache/phoenix/query/QueryServices.java    |   5 +
 .../apache/phoenix/query/QueryServicesOptions.java |   2 +
 .../org/apache/phoenix/query/StatsLoaderImpl.java  | 104 +++++++++
 ...org.apache.phoenix.query.GuidePostsCacheFactory |  17 ++
 .../phoenix/query/GuidePostsCacheProviderTest.java | 122 ++++++++++
 .../phoenix/query/GuidePostsCacheWrapperTest.java  | 106 +++++++++
 .../phoenix/query/PhoenixStatsCacheLoaderTest.java |   2 +-
 .../PhoenixStatsCacheRemovalListenerTest.java      |   2 +-
 ...org.apache.phoenix.query.GuidePostsCacheFactory |  19 ++
 22 files changed, 1073 insertions(+), 261 deletions(-)

diff --git a/phoenix-core/src/it/java/org/apache/phoenix/end2end/ConfigurableCacheIT.java b/phoenix-core/src/it/java/org/apache/phoenix/end2end/ConfigurableCacheIT.java
new file mode 100644
index 0000000..4043052
--- /dev/null
+++ b/phoenix-core/src/it/java/org/apache/phoenix/end2end/ConfigurableCacheIT.java
@@ -0,0 +1,150 @@
+/*
+ * 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.phoenix.end2end;
+
+import static org.apache.phoenix.util.TestUtil.TEST_PROPERTIES;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.fail;
+
+import java.sql.Connection;
+import java.sql.DriverManager;
+import java.util.Properties;
+
+import org.apache.phoenix.query.ITGuidePostsCacheFactory;
+import org.apache.phoenix.query.QueryServices;
+import org.apache.phoenix.util.PhoenixRuntime;
+import org.apache.phoenix.util.PropertiesUtil;
+import org.junit.BeforeClass;
+import org.junit.Test;
+
+/**
+ * This tests that the configured client statistics cache is used during execution.  These tests
+ * use a class ITGuidePostsCacheFactory which is for testing only that keeps track of the number
+ * of cache instances generated.
+ */
+public class ConfigurableCacheIT extends ParallelStatsEnabledIT {
+
+    static String table;
+
+    @BeforeClass
+    public static void initTables() throws Exception {
+        table = generateUniqueName();
+        // Use phoenix test driver for setup
+        try (Connection conn = DriverManager.getConnection(getUrl())) {
+            conn.createStatement()
+                    .execute("CREATE TABLE " + table
+                            + " (k INTEGER PRIMARY KEY, c1.a bigint, c2.b bigint)"
+                            + " GUIDE_POSTS_WIDTH=20");
+            conn.createStatement().execute("upsert into " + table + " values (100,1,3)");
+            conn.createStatement().execute("upsert into " + table + " values (101,2,4)");
+            conn.createStatement().execute("upsert into " + table + " values (102,2,4)");
+            conn.createStatement().execute("upsert into " + table + " values (103,2,4)");
+            conn.createStatement().execute("upsert into " + table + " values (104,2,4)");
+            conn.createStatement().execute("upsert into " + table + " values (105,2,4)");
+            conn.createStatement().execute("upsert into " + table + " values (106,2,4)");
+            conn.createStatement().execute("upsert into " + table + " values (107,2,4)");
+            conn.createStatement().execute("upsert into " + table + " values (108,2,4)");
+            conn.createStatement().execute("upsert into " + table + " values (109,2,4)");
+            conn.commit();
+            conn.createStatement().execute("UPDATE STATISTICS " + table);
+            conn.commit();
+        }
+        ;
+    }
+
+    private Connection getCacheFactory(String principal, String cacheFactoryString)
+            throws Exception {
+
+        String url = getUrl();
+        url = url.replace(";" + PhoenixRuntime.PHOENIX_TEST_DRIVER_URL_PARAM, "");
+
+        // As there is a map of connections in the phoenix driver need to differentiate the url to
+        // pick different QueryServices
+        url = url + PhoenixRuntime.JDBC_PROTOCOL_SEPARATOR + principal;
+
+        // Load defaults from QueryServicesTestImpl
+        Properties props = PropertiesUtil.deepCopy(TEST_PROPERTIES);
+
+        // Parameterized URL
+        props.put(QueryServices.GUIDE_POSTS_CACHE_FACTORY_CLASS, cacheFactoryString);
+
+        // Stats Connection Props
+        props.put(QueryServices.STATS_GUIDEPOST_WIDTH_BYTES_ATTRIB, Long.toString(20));
+        props.put(QueryServices.STATS_UPDATE_FREQ_MS_ATTRIB, Long.toString(5l));
+        props.put(QueryServices.MAX_SERVER_METADATA_CACHE_TIME_TO_LIVE_MS_ATTRIB, Long.toString(5));
+        props.put(QueryServices.USE_STATS_FOR_PARALLELIZATION, Boolean.toString(true));
+
+        Connection conn = DriverManager.getConnection(url, props);
+        return conn;
+    }
+
+    /**
+     * Test that if we don't specify the cacheFactory we won't increase the count of test.
+     * @throws Exception
+     */
+    @Test
+    public void testWithDefaults() throws Exception {
+        int initialCount = ITGuidePostsCacheFactory.getCount();
+        try (Connection conn = DriverManager.getConnection(getUrl())) {
+            conn.createStatement().executeQuery("SELECT * FROM " + table);
+        }
+        assertEquals(initialCount, ITGuidePostsCacheFactory.getCount());
+    }
+
+    /**
+     * Tests that with a single ConnectionInfo we will not create more than one.
+     * @throws Exception
+     */
+    @Test
+    public void testWithSingle() throws Exception {
+        int initialCount = ITGuidePostsCacheFactory.getCount();
+
+        try (Connection conn =
+                getCacheFactory("User1", ITGuidePostsCacheFactory.class.getTypeName())) {
+            conn.createStatement().executeQuery("SELECT * FROM " + table);
+        }
+        try (Connection conn =
+                getCacheFactory("User1", ITGuidePostsCacheFactory.class.getTypeName())) {
+            conn.createStatement().executeQuery("SELECT * FROM " + table);
+        }
+        assertEquals(initialCount + 1, ITGuidePostsCacheFactory.getCount());
+    }
+
+    /**
+     * Tests with 2 ConnectionInfo's
+     * @throws Exception
+     */
+    @Test
+    public void testWithMultiple() throws Exception {
+        int initialCount = ITGuidePostsCacheFactory.getCount();
+        try (Connection conn =
+                getCacheFactory("User4", ITGuidePostsCacheFactory.class.getTypeName())) {
+            conn.createStatement().executeQuery("SELECT * FROM " + table);
+        }
+        try (Connection conn =
+                getCacheFactory("User6", ITGuidePostsCacheFactory.class.getTypeName())) {
+            conn.createStatement().executeQuery("SELECT * FROM " + table);
+        }
+        assertEquals(initialCount + 2, ITGuidePostsCacheFactory.getCount());
+    }
+
+    /**
+     * Tests that non-existent cacheFactory fails with exception
+     * @throws Exception
+     */
+    @Test(expected = Exception.class)
+    public void testBadCache() throws Exception {
+        try (Connection conn =
+                getCacheFactory("User8", "org.notreal.class")) {
+        }
+        fail();
+    }
+}
diff --git a/phoenix-core/src/main/java/org/apache/phoenix/exception/PhoenixNonRetryableRuntimeException.java b/phoenix-core/src/main/java/org/apache/phoenix/exception/PhoenixNonRetryableRuntimeException.java
new file mode 100644
index 0000000..89f7c06
--- /dev/null
+++ b/phoenix-core/src/main/java/org/apache/phoenix/exception/PhoenixNonRetryableRuntimeException.java
@@ -0,0 +1,34 @@
+/**
+ * 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.phoenix.exception;
+
+public class PhoenixNonRetryableRuntimeException extends RuntimeException {
+    public PhoenixNonRetryableRuntimeException() { }
+
+    public PhoenixNonRetryableRuntimeException(String msg) {
+        super(msg);
+    }
+
+    public PhoenixNonRetryableRuntimeException(String msg, Throwable throwable) {
+        super(msg, throwable);
+    }
+
+    public PhoenixNonRetryableRuntimeException(Throwable throwable) {
+        super(throwable);
+    }
+}
diff --git a/phoenix-core/src/main/java/org/apache/phoenix/query/ConnectionQueryServicesImpl.java b/phoenix-core/src/main/java/org/apache/phoenix/query/ConnectionQueryServicesImpl.java
index e8ad734..c5d47dd 100644
--- a/phoenix-core/src/main/java/org/apache/phoenix/query/ConnectionQueryServicesImpl.java
+++ b/phoenix-core/src/main/java/org/apache/phoenix/query/ConnectionQueryServicesImpl.java
@@ -291,7 +291,9 @@ public class ConnectionQueryServicesImpl extends DelegateQueryServices implement
     private static final Logger logger = LoggerFactory.getLogger(ConnectionQueryServicesImpl.class);
     private static final int INITIAL_CHILD_SERVICES_CAPACITY = 100;
     private static final int DEFAULT_OUT_OF_ORDER_MUTATIONS_WAIT_TIME_MS = 1000;
-    private static final int TTL_FOR_MUTEX = 15 * 60; // 15min 
+    private static final int TTL_FOR_MUTEX = 15 * 60; // 15min
+    private final GuidePostsCacheProvider
+            GUIDE_POSTS_CACHE_PROVIDER = new GuidePostsCacheProvider();
     protected final Configuration config;
     protected final ConnectionInfo connectionInfo;
     // Copy of config.getProps(), but read-only to prevent synchronization that we
@@ -300,7 +302,7 @@ public class ConnectionQueryServicesImpl extends DelegateQueryServices implement
     private final String userName;
     private final User user;
     private final ConcurrentHashMap<ImmutableBytesWritable,ConnectionQueryServices> childServices;
-    private final GuidePostsCache tableStatsCache;
+    private final GuidePostsCacheWrapper tableStatsCache;
 
     // Cache the latest meta data here for future connections
     // writes guarded by "latestMetaDataLock"
@@ -423,8 +425,11 @@ public class ConnectionQueryServicesImpl extends DelegateQueryServices implement
             list.add(queue);
         }
         connectionQueues = ImmutableList.copyOf(list);
+
         // A little bit of a smell to leak `this` here, but should not be a problem
-        this.tableStatsCache = new GuidePostsCache(this, config);
+        this.tableStatsCache = GUIDE_POSTS_CACHE_PROVIDER.getGuidePostsCache(props.get(GUIDE_POSTS_CACHE_FACTORY_CLASS,
+                QueryServicesOptions.DEFAULT_GUIDE_POSTS_CACHE_FACTORY_CLASS), this, config);
+
         this.isAutoUpgradeEnabled = config.getBoolean(AUTO_UPGRADE_ENABLED, QueryServicesOptions.DEFAULT_AUTO_UPGRADE_ENABLED);
         this.maxConnectionsAllowed = config.getInt(QueryServices.CLIENT_CONNECTION_MAX_ALLOWED_CONNECTIONS,
             QueryServicesOptions.DEFAULT_CLIENT_CONNECTION_MAX_ALLOWED_CONNECTIONS);
@@ -1797,10 +1802,12 @@ public class ConnectionQueryServicesImpl extends DelegateQueryServices implement
         return result;
     }
 
-    private void invalidateTableStats(final List<byte[]> tableNamesToDelete) {
+    private void invalidateTableStats(final List<byte[]> tableNamesToDelete) throws SQLException {
         if (tableNamesToDelete != null) {
             for (byte[] tableName : tableNamesToDelete) {
-                tableStatsCache.invalidateAll(tableName);
+                TableName tn = TableName.valueOf(tableName);
+                TableDescriptor htableDesc = this.getTableDescriptor(tableName);
+                tableStatsCache.invalidateAll(htableDesc);
             }
         }
     }
diff --git a/phoenix-core/src/main/java/org/apache/phoenix/query/ConnectionlessQueryServicesImpl.java b/phoenix-core/src/main/java/org/apache/phoenix/query/ConnectionlessQueryServicesImpl.java
index ce107b0..cfde99c 100644
--- a/phoenix-core/src/main/java/org/apache/phoenix/query/ConnectionlessQueryServicesImpl.java
+++ b/phoenix-core/src/main/java/org/apache/phoenix/query/ConnectionlessQueryServicesImpl.java
@@ -29,6 +29,7 @@ import java.util.Map.Entry;
 import java.util.Objects;
 import java.util.Properties;
 import java.util.Set;
+import java.util.concurrent.ExecutionException;
 
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.hbase.HConstants;
@@ -109,7 +110,8 @@ import com.google.common.collect.Maps;
  */
 public class ConnectionlessQueryServicesImpl extends DelegateQueryServices implements ConnectionQueryServices  {
     private static ServerName SERVER_NAME = ServerName.parseServerName(HConstants.LOCALHOST + Addressing.HOSTNAME_PORT_SEPARATOR + HConstants.DEFAULT_ZOOKEPER_CLIENT_PORT);
-    
+    private static final GuidePostsCacheProvider
+            GUIDE_POSTS_CACHE_PROVIDER = new GuidePostsCacheProvider();
     private final ReadOnlyProps props;
     private PMetaData metaData;
     private final Map<SequenceKey, SequenceInfo> sequenceMap = Maps.newHashMap();
@@ -118,7 +120,7 @@ public class ConnectionlessQueryServicesImpl extends DelegateQueryServices imple
     private volatile boolean initialized;
     private volatile SQLException initializationException;
     private final Map<String, List<HRegionLocation>> tableSplits = Maps.newHashMap();
-    private final GuidePostsCache guidePostsCache;
+    private final GuidePostsCacheWrapper guidePostsCache;
     private final Configuration config;
 
     private User user;
@@ -147,10 +149,13 @@ public class ConnectionlessQueryServicesImpl extends DelegateQueryServices imple
         // Without making a copy of the configuration we cons up, we lose some of our properties
         // on the server side during testing.
         this.config = HBaseFactoryProvider.getConfigurationFactory().getConfiguration(config);
-        this.guidePostsCache = new GuidePostsCache(this, config);
+
         // set replication required parameter
         ConfigUtil.setReplicationConfigIfAbsent(this.config);
         this.props = new ReadOnlyProps(this.config.iterator());
+
+        this.guidePostsCache = GUIDE_POSTS_CACHE_PROVIDER.getGuidePostsCache(props.get(GUIDE_POSTS_CACHE_FACTORY_CLASS,
+                QueryServicesOptions.DEFAULT_GUIDE_POSTS_CACHE_FACTORY_CLASS), null, config);
     }
 
     private PMetaData newEmptyMetaData() {
@@ -600,7 +605,12 @@ public class ConnectionlessQueryServicesImpl extends DelegateQueryServices imple
 
     @Override
     public GuidePostsInfo getTableStats(GuidePostsKey key) {
-        GuidePostsInfo info = guidePostsCache.getCache().getIfPresent(key);
+        GuidePostsInfo info = null;
+        try {
+            info = guidePostsCache.get(key);
+        } catch(ExecutionException e){
+            return GuidePostsInfo.NO_GUIDEPOST;
+        }
         if (null == info) {
           return GuidePostsInfo.NO_GUIDEPOST;
         }
diff --git a/phoenix-core/src/main/java/org/apache/phoenix/query/DefaultGuidePostsCacheFactory.java b/phoenix-core/src/main/java/org/apache/phoenix/query/DefaultGuidePostsCacheFactory.java
new file mode 100644
index 0000000..4f50036
--- /dev/null
+++ b/phoenix-core/src/main/java/org/apache/phoenix/query/DefaultGuidePostsCacheFactory.java
@@ -0,0 +1,45 @@
+/**
+ * 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.phoenix.query;
+
+import static org.apache.phoenix.query.QueryServices.STATS_COLLECTION_ENABLED;
+import static org.apache.phoenix.query.QueryServicesOptions.DEFAULT_STATS_COLLECTION_ENABLED;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.phoenix.util.ReadOnlyProps;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import com.google.common.base.Preconditions;
+
+public class DefaultGuidePostsCacheFactory implements GuidePostsCacheFactory {
+
+    private static final Logger LOGGER = LoggerFactory.getLogger(DefaultGuidePostsCacheFactory.class);
+
+    @Override
+    public PhoenixStatsLoader getPhoenixStatsLoader(ConnectionQueryServices queryServices, ReadOnlyProps readOnlyProps,
+            Configuration config) {
+        Preconditions.checkNotNull(config);
+
+        final boolean isStatsEnabled = config.getBoolean(STATS_COLLECTION_ENABLED, DEFAULT_STATS_COLLECTION_ENABLED);
+        if (queryServices == null || !isStatsEnabled) {
+            LOGGER.info("Using EmptyStatsLoader from DefaultGuidePostsCacheFactory");
+            return new EmptyStatsLoader();
+        }
+        return new StatsLoaderImpl(queryServices);
+    }
+
+    @Override
+    public GuidePostsCache getGuidePostsCache(PhoenixStatsLoader phoenixStatsLoader, Configuration config) {
+        LOGGER.debug("DefaultGuidePostsCacheFactory guide post cache construction.");
+        PhoenixStatsCacheLoader cacheLoader = new PhoenixStatsCacheLoader(phoenixStatsLoader, config);
+        return new GuidePostsCacheImpl(cacheLoader, config);
+    }
+}
diff --git a/phoenix-core/src/main/java/org/apache/phoenix/query/EmptyStatsLoader.java b/phoenix-core/src/main/java/org/apache/phoenix/query/EmptyStatsLoader.java
new file mode 100644
index 0000000..76024a3
--- /dev/null
+++ b/phoenix-core/src/main/java/org/apache/phoenix/query/EmptyStatsLoader.java
@@ -0,0 +1,35 @@
+/**
+ * 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.phoenix.query;
+
+import org.apache.phoenix.schema.stats.GuidePostsInfo;
+import org.apache.phoenix.schema.stats.GuidePostsKey;
+
+/**
+ * {@link PhoenixStatsLoader} implementation for the Stats Loader.
+ * Empty stats loader if stats are disabled
+ */
+class EmptyStatsLoader implements PhoenixStatsLoader {
+    @Override
+    public boolean needsLoad() {
+        return false;
+    }
+
+    @Override
+    public GuidePostsInfo loadStats(GuidePostsKey statsKey) throws Exception {
+        return GuidePostsInfo.NO_GUIDEPOST;
+    }
+
+    @Override
+    public GuidePostsInfo loadStats(GuidePostsKey statsKey, GuidePostsInfo prevGuidepostInfo) throws Exception {
+        return GuidePostsInfo.NO_GUIDEPOST;
+    }
+}
diff --git a/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCache.java b/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCache.java
index b24a1e3..6679a75 100644
--- a/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCache.java
+++ b/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCache.java
@@ -16,263 +16,20 @@
  */
 package org.apache.phoenix.query;
 
-import static org.apache.phoenix.query.QueryServices.STATS_COLLECTION_ENABLED;
-import static org.apache.phoenix.query.QueryServicesOptions.DEFAULT_STATS_COLLECTION_ENABLED;
-
-import java.io.IOException;
-import java.util.List;
-import java.util.Objects;
-import java.util.concurrent.*;
-
-import org.apache.hadoop.conf.Configuration;
-import org.apache.hadoop.hbase.HConstants;
-import org.apache.hadoop.hbase.TableName;
-import org.apache.hadoop.hbase.TableNotFoundException;
-import org.apache.hadoop.hbase.client.Table;
 import org.apache.hadoop.hbase.client.TableDescriptor;
-import org.apache.hadoop.hbase.util.Bytes;
-import org.apache.phoenix.jdbc.PhoenixDatabaseMetaData;
-import org.apache.phoenix.schema.PColumnFamily;
 import org.apache.phoenix.schema.PTable;
 import org.apache.phoenix.schema.stats.GuidePostsInfo;
 import org.apache.phoenix.schema.stats.GuidePostsKey;
-import org.apache.phoenix.schema.stats.StatisticsUtil;
-import org.apache.phoenix.util.SchemaUtil;
-import org.slf4j.Logger;
-import org.slf4j.LoggerFactory;
-
-import com.google.common.cache.CacheBuilder;
-import com.google.common.cache.LoadingCache;
-import com.google.common.cache.RemovalCause;
-import com.google.common.cache.RemovalListener;
-import com.google.common.cache.RemovalNotification;
-import com.google.common.cache.Weigher;
-
-/**
- * "Client-side" cache for storing {@link GuidePostsInfo} for a column family. Intended to decouple
- * Phoenix from a specific version of Guava's cache.
- */
-public class GuidePostsCache {
-    private static final Logger logger = LoggerFactory.getLogger(GuidePostsCache.class);
-
-    private final ConnectionQueryServices queryServices;
-    private final LoadingCache<GuidePostsKey, GuidePostsInfo> cache;
-
-    public GuidePostsCache(ConnectionQueryServices queryServices, Configuration config) {
-        this.queryServices = Objects.requireNonNull(queryServices);
-
-        // Number of millis to expire cache values after write
-        final long statsUpdateFrequency = config.getLong(
-                QueryServices.STATS_UPDATE_FREQ_MS_ATTRIB,
-                QueryServicesOptions.DEFAULT_STATS_UPDATE_FREQ_MS);
-
-        // Maximum total weight (size in bytes) of stats entries
-        final long maxTableStatsCacheSize = config.getLong(
-                QueryServices.STATS_MAX_CACHE_SIZE,
-                QueryServicesOptions.DEFAULT_STATS_MAX_CACHE_SIZE);
-
-		final boolean isStatsEnabled = config.getBoolean(STATS_COLLECTION_ENABLED, DEFAULT_STATS_COLLECTION_ENABLED);
-
-        PhoenixStatsCacheLoader cacheLoader = new PhoenixStatsCacheLoader(
-                isStatsEnabled ? new StatsLoaderImpl() : new EmptyStatsLoader(), config);
-
-        cache = CacheBuilder.newBuilder()
-                // Refresh entries a given amount of time after they were written
-                .refreshAfterWrite(statsUpdateFrequency, TimeUnit.MILLISECONDS)
-                // Maximum total weight (size in bytes) of stats entries
-                .maximumWeight(maxTableStatsCacheSize)
-                // Defer actual size to the PTableStats.getEstimatedSize()
-                .weigher(new Weigher<GuidePostsKey, GuidePostsInfo>() {
-                    @Override public int weigh(GuidePostsKey key, GuidePostsInfo info) {
-                        return info.getEstimatedSize();
-                    }
-                })
-                // Log removals at TRACE for debugging
-                .removalListener(new PhoenixStatsCacheRemovalListener())
-                // Automatically load the cache when entries need to be refreshed
-                .build(cacheLoader);
-    }
-
-    /**
-     * {@link PhoenixStatsLoader} implementation for the Stats Loader.
-     */
-    protected class StatsLoaderImpl implements PhoenixStatsLoader {
-        @Override
-        public boolean needsLoad() {
-            // For now, whenever it's called, we try to load stats from stats table
-            // no matter it has been updated or not.
-            // Here are the possible optimizations we can do here:
-            // 1. Load stats from the stats table only when the stats get updated on the server side.
-            // 2. Support different refresh cycle for different tables.
-            return true;
-        }
-
-        @Override
-        public GuidePostsInfo loadStats(GuidePostsKey statsKey) throws Exception {
-            return loadStats(statsKey, GuidePostsInfo.NO_GUIDEPOST);
-        }
-
-        @Override
-        public GuidePostsInfo loadStats(GuidePostsKey statsKey, GuidePostsInfo prevGuidepostInfo) throws Exception {
-            assert(prevGuidepostInfo != null);
-
-            TableName tableName = SchemaUtil.getPhysicalName(
-                    PhoenixDatabaseMetaData.SYSTEM_STATS_NAME_BYTES,
-                    queryServices.getProps());
-            Table statsHTable = queryServices.getTable(tableName.getName());
-
-            try {
-                GuidePostsInfo guidePostsInfo = StatisticsUtil.readStatistics(statsHTable, statsKey,
-                        HConstants.LATEST_TIMESTAMP);
-                traceStatsUpdate(statsKey, guidePostsInfo);
-                return guidePostsInfo;
-            } catch (TableNotFoundException e) {
-                // On a fresh install, stats might not yet be created, don't warn about this.
-                logger.debug("Unable to locate Phoenix stats table: " + tableName.toString(), e);
-                return prevGuidepostInfo;
-            } catch (IOException e) {
-                logger.warn("Unable to read from stats table: " + tableName.toString(), e);
-                return prevGuidepostInfo;
-            } finally {
-                try {
-                    statsHTable.close();
-                } catch (IOException e) {
-                    // Log, but continue. We have our stats anyway now.
-                    logger.warn("Unable to close stats table: " + tableName.toString(), e);
-                }
-            }
-        }
-
-        /**
-         * Logs a trace message for newly inserted entries to the stats cache.
-         */
-        void traceStatsUpdate(GuidePostsKey key, GuidePostsInfo info) {
-            if (logger.isTraceEnabled()) {
-                logger.trace("Updating local TableStats cache (id={}) for {}, size={}bytes",
-                        new Object[] {Objects.hashCode(GuidePostsCache.this), key, info.getEstimatedSize()});
-            }
-        }
-    }
-
-    /**
-     * {@link PhoenixStatsLoader} implementation for the Stats Loader.
-     * Empty stats loader if stats are disabled
-     */
-	protected class EmptyStatsLoader implements PhoenixStatsLoader {
-        @Override
-        public boolean needsLoad() {
-            return false;
-        }
-
-        @Override
-        public GuidePostsInfo loadStats(GuidePostsKey statsKey) throws Exception {
-            return GuidePostsInfo.NO_GUIDEPOST;
-        }
-
-        @Override
-        public GuidePostsInfo loadStats(GuidePostsKey statsKey, GuidePostsInfo prevGuidepostInfo) throws Exception {
-            return GuidePostsInfo.NO_GUIDEPOST;
-        }
-	}
-
-    /**
-     * Returns the underlying cache. Try to use the provided methods instead of accessing the cache
-     * directly.
-     */
-    LoadingCache<GuidePostsKey, GuidePostsInfo> getCache() {
-        return cache;
-    }
+import java.util.concurrent.ExecutionException;
 
-    /**
-     * Returns the PTableStats for the given <code>tableName</code, using the provided
-     * <code>valueLoader</code> if no such mapping exists.
-     *
-     * @see com.google.common.cache.LoadingCache#get(Object)
-     */
-    public GuidePostsInfo get(GuidePostsKey key) throws ExecutionException {
-        return getCache().get(key);
-    }
 
-    /**
-     * Cache the given <code>stats</code> to the cache for the given <code>tableName</code>.
-     *
-     * @see com.google.common.cache.Cache#put(Object, Object)
-     */
-    public void put(GuidePostsKey key, GuidePostsInfo info) {
-        getCache().put(Objects.requireNonNull(key), Objects.requireNonNull(info));
-    }
+public interface GuidePostsCache {
+    GuidePostsInfo get(GuidePostsKey key) throws ExecutionException;
 
-    /**
-     * Removes the mapping for <code>tableName</code> if it exists.
-     *
-     * @see com.google.common.cache.Cache#invalidate(Object)
-     */
-    public void invalidate(GuidePostsKey key) {
-        getCache().invalidate(Objects.requireNonNull(key));
-    }
+    void put(GuidePostsKey key, GuidePostsInfo info);
 
-    /**
-     * Removes all mappings from the cache.
-     *
-     * @see com.google.common.cache.Cache#invalidateAll()
-     */
-    public void invalidateAll() {
-        getCache().invalidateAll();
-    }
-    
-    /**
-     * Removes all mappings where the {@link org.apache.phoenix.schema.stats.GuidePostsKey#getPhysicalName()}
-     * equals physicalName. Because all keys in the map must be iterated, this method should be avoided.
-     * @param physicalName
-     */
-    public void invalidateAll(byte[] physicalName) {
-        for (GuidePostsKey key : getCache().asMap().keySet()) {
-            if (Bytes.compareTo(key.getPhysicalName(), physicalName) == 0) {
-                invalidate(key);
-            }
-        }
-    }
-    
-    public void invalidateAll(TableDescriptor htableDesc) {
-        byte[] tableName = htableDesc.getTableName().getName();
-        for (byte[] fam : htableDesc.getColumnFamilyNames()) {
-            invalidate(new GuidePostsKey(tableName, fam));
-        }
-    }
-    
-    public void invalidateAll(PTable table) {
-        byte[] physicalName = table.getPhysicalName().getBytes();
-        List<PColumnFamily> families = table.getColumnFamilies();
-        if (families.isEmpty()) {
-            invalidate(new GuidePostsKey(physicalName, SchemaUtil.getEmptyColumnFamily(table)));
-        } else {
-            for (PColumnFamily family : families) {
-                invalidate(new GuidePostsKey(physicalName, family.getName().getBytes()));
-            }
-        }
-    }
+    void invalidate(GuidePostsKey key);
 
-    /**
-     * A {@link RemovalListener} implementation to track evictions from the table stats cache.
-     */
-    static class PhoenixStatsCacheRemovalListener implements
-            RemovalListener<GuidePostsKey, GuidePostsInfo> {
-        @Override
-        public void onRemoval(RemovalNotification<GuidePostsKey, GuidePostsInfo> notification) {
-            if (logger.isTraceEnabled()) {
-                final RemovalCause cause = notification.getCause();
-                if (wasEvicted(cause)) {
-                    GuidePostsKey key = notification.getKey();
-                    logger.trace("Cached stats for {} with size={}bytes was evicted due to cause={}",
-                            new Object[] {key, notification.getValue().getEstimatedSize(),
-                                    cause});
-                }
-            }
-        }
+    void invalidateAll();
 
-        boolean wasEvicted(RemovalCause cause) {
-            // This is actually a method on RemovalCause but isn't exposed
-            return RemovalCause.EXPLICIT != cause && RemovalCause.REPLACED != cause;
-        }
-    }
 }
diff --git a/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCacheFactory.java b/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCacheFactory.java
new file mode 100644
index 0000000..7d0cbaa
--- /dev/null
+++ b/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCacheFactory.java
@@ -0,0 +1,46 @@
+/*
+ * 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.phoenix.query;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.phoenix.util.ReadOnlyProps;
+
+/**
+ * Interface for configurable GuidePostsCache interface construction
+ * Class is meant to be defined in the ConnectionQueryServices property
+ * Implementations must provide a default constructor
+ */
+public interface GuidePostsCacheFactory {
+
+    /**
+     * Interface for a PhoenixStatsLoader
+     * @param clientConnectionQueryServices current client connectionQueryServices note not
+     *                                      necessary to use this connection
+     * @param readOnlyProps properties from HBase configuration
+     * @param config a Configuration for the current Phoenix/Hbase
+     * @return PhoenixStatsLoader interface
+     */
+    PhoenixStatsLoader getPhoenixStatsLoader(ConnectionQueryServices clientConnectionQueryServices, ReadOnlyProps readOnlyProps, Configuration config);
+
+    /**
+     * @param phoenixStatsLoader The passed in stats loader will come from getPhoenixStatsLoader
+     * @param config a Configuration for the current Phoenix/Hbase
+     * @return GuidePostsCache interface
+     */
+    GuidePostsCache getGuidePostsCache(PhoenixStatsLoader phoenixStatsLoader, Configuration config);
+}
diff --git a/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCacheImpl.java b/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCacheImpl.java
new file mode 100644
index 0000000..24b695c
--- /dev/null
+++ b/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCacheImpl.java
@@ -0,0 +1,148 @@
+/*
+ * 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.phoenix.query;
+
+import java.util.Objects;
+import java.util.concurrent.ExecutionException;
+import java.util.concurrent.TimeUnit;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.phoenix.schema.stats.GuidePostsInfo;
+import org.apache.phoenix.schema.stats.GuidePostsKey;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import com.google.common.base.Preconditions;
+import com.google.common.cache.CacheBuilder;
+import com.google.common.cache.LoadingCache;
+import com.google.common.cache.RemovalCause;
+import com.google.common.cache.RemovalListener;
+import com.google.common.cache.RemovalNotification;
+import com.google.common.cache.Weigher;
+
+/**
+ * "Client-side" cache for storing {@link GuidePostsInfo} for a column family. Intended to decouple
+ * Phoenix from a specific version of Guava's cache.
+ */
+public class GuidePostsCacheImpl implements GuidePostsCache {
+    private static final Logger logger = LoggerFactory.getLogger(GuidePostsCacheImpl.class);
+
+    private final LoadingCache<GuidePostsKey, GuidePostsInfo> cache;
+
+    public GuidePostsCacheImpl(PhoenixStatsCacheLoader cacheLoader, Configuration config) {
+        Preconditions.checkNotNull(cacheLoader);
+
+        // Number of millis to expire cache values after write
+        final long statsUpdateFrequency = config.getLong(
+                QueryServices.STATS_UPDATE_FREQ_MS_ATTRIB,
+                QueryServicesOptions.DEFAULT_STATS_UPDATE_FREQ_MS);
+
+        // Maximum total weight (size in bytes) of stats entries
+        final long maxTableStatsCacheSize = config.getLong(
+                QueryServices.STATS_MAX_CACHE_SIZE,
+                QueryServicesOptions.DEFAULT_STATS_MAX_CACHE_SIZE);
+
+        cache = CacheBuilder.newBuilder()
+                // Refresh entries a given amount of time after they were written
+                .refreshAfterWrite(statsUpdateFrequency, TimeUnit.MILLISECONDS)
+                // Maximum total weight (size in bytes) of stats entries
+                .maximumWeight(maxTableStatsCacheSize)
+                // Defer actual size to the PTableStats.getEstimatedSize()
+                .weigher(new Weigher<GuidePostsKey, GuidePostsInfo>() {
+                    @Override public int weigh(GuidePostsKey key, GuidePostsInfo info) {
+                        return info.getEstimatedSize();
+                    }
+                })
+                // Log removals at TRACE for debugging
+                .removalListener(new PhoenixStatsCacheRemovalListener())
+                // Automatically load the cache when entries need to be refreshed
+                .build(cacheLoader);
+    }
+
+    /**
+     * Returns the underlying cache. Try to use the provided methods instead of accessing the cache
+     * directly.
+     */
+    LoadingCache<GuidePostsKey, GuidePostsInfo> getCache() {
+        return cache;
+    }
+
+    /**
+     * Returns the PTableStats for the given <code>tableName</code, using the provided
+     * <code>valueLoader</code> if no such mapping exists.
+     *
+     * @see com.google.common.cache.LoadingCache#get(Object)
+     */
+    @Override
+    public GuidePostsInfo get(GuidePostsKey key) throws ExecutionException {
+        return getCache().get(key);
+    }
+
+    /**
+     * Cache the given <code>stats</code> to the cache for the given <code>tableName</code>.
+     *
+     * @see com.google.common.cache.Cache#put(Object, Object)
+     */
+    @Override
+    public void put(GuidePostsKey key, GuidePostsInfo info) {
+        getCache().put(Objects.requireNonNull(key), Objects.requireNonNull(info));
+    }
+
+    /**
+     * Removes the mapping for <code>tableName</code> if it exists.
+     *
+     * @see com.google.common.cache.Cache#invalidate(Object)
+     */
+    @Override
+    public void invalidate(GuidePostsKey key) {
+        getCache().invalidate(Objects.requireNonNull(key));
+    }
+
+    /**
+     * Removes all mappings from the cache.
+     *
+     * @see com.google.common.cache.Cache#invalidateAll()
+     */
+    @Override
+    public void invalidateAll() {
+        getCache().invalidateAll();
+    }
+
+    /**
+     * A {@link RemovalListener} implementation to track evictions from the table stats cache.
+     */
+    static class PhoenixStatsCacheRemovalListener implements
+            RemovalListener<GuidePostsKey, GuidePostsInfo> {
+        @Override
+        public void onRemoval(RemovalNotification<GuidePostsKey, GuidePostsInfo> notification) {
+            if (logger.isTraceEnabled()) {
+                final RemovalCause cause = notification.getCause();
+                if (wasEvicted(cause)) {
+                    GuidePostsKey key = notification.getKey();
+                    logger.trace("Cached stats for {} with size={}bytes was evicted due to cause={}",
+                            new Object[] {key, notification.getValue().getEstimatedSize(),
+                                    cause});
+                }
+            }
+        }
+
+        boolean wasEvicted(RemovalCause cause) {
+            // This is actually a method on RemovalCause but isn't exposed
+            return RemovalCause.EXPLICIT != cause && RemovalCause.REPLACED != cause;
+        }
+    }
+}
\ No newline at end of file
diff --git a/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCacheProvider.java b/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCacheProvider.java
new file mode 100644
index 0000000..cd66cfe
--- /dev/null
+++ b/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCacheProvider.java
@@ -0,0 +1,79 @@
+/**
+ * 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.phoenix.query;
+
+import java.util.List;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.phoenix.exception.PhoenixNonRetryableRuntimeException;
+import org.apache.phoenix.util.InstanceResolver;
+import org.apache.phoenix.util.ReadOnlyProps;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import com.google.common.annotations.VisibleForTesting;
+import com.google.common.base.Preconditions;
+
+public class GuidePostsCacheProvider {
+
+    private static final Logger LOGGER = LoggerFactory.getLogger(GuidePostsCacheProvider.class);
+
+    GuidePostsCacheFactory guidePostsCacheFactory = null;
+
+    @VisibleForTesting
+    GuidePostsCacheFactory loadAndGetGuidePostsCacheFactory(String classString) {
+        Preconditions.checkNotNull(classString);
+        if (guidePostsCacheFactory == null) {
+            try {
+
+                Class clazz = ClassLoader.getSystemClassLoader().loadClass(classString);
+                if (!GuidePostsCacheFactory.class.isAssignableFrom(clazz)) {
+                    String msg = String.format(
+                            "Could not load/instantiate class %s is not an instance of GuidePostsCacheFactory",
+                            classString);
+                    LOGGER.error(msg);
+                    throw new PhoenixNonRetryableRuntimeException(msg);
+                }
+
+                List<GuidePostsCacheFactory> factoryList = InstanceResolver.get(GuidePostsCacheFactory.class, null);
+                for (GuidePostsCacheFactory factory : factoryList) {
+                    if (clazz.isInstance(factory)) {
+                        guidePostsCacheFactory = factory;
+                        LOGGER.info(String.format("Sucessfully loaded class for GuidePostsCacheFactor of type: %s",
+                                classString));
+                        break;
+                    }
+                }
+                if (guidePostsCacheFactory == null) {
+                    String msg = String.format("Could not load/instantiate class %s", classString);
+                    LOGGER.error(msg);
+                    throw new PhoenixNonRetryableRuntimeException(msg);
+                }
+            } catch (ClassNotFoundException e) {
+                LOGGER.error(String.format("Could not load/instantiate class %s", classString), e);
+                throw new PhoenixNonRetryableRuntimeException(e);
+            }
+        }
+        return guidePostsCacheFactory;
+    }
+
+    public GuidePostsCacheWrapper getGuidePostsCache(String classStr, ConnectionQueryServices queryServices,
+            Configuration config) {
+        ReadOnlyProps props = null;
+        if (queryServices != null) {
+            props = queryServices.getProps();
+        }
+        GuidePostsCacheFactory guidePostCacheFactory = loadAndGetGuidePostsCacheFactory(classStr);
+        PhoenixStatsLoader phoenixStatsLoader = guidePostsCacheFactory.getPhoenixStatsLoader(queryServices, props,
+                config);
+        GuidePostsCache guidePostsCache = guidePostCacheFactory.getGuidePostsCache(phoenixStatsLoader, config);
+        return new GuidePostsCacheWrapper(guidePostsCache);
+    }
+}
diff --git a/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCacheWrapper.java b/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCacheWrapper.java
new file mode 100644
index 0000000..0d66f22
--- /dev/null
+++ b/phoenix-core/src/main/java/org/apache/phoenix/query/GuidePostsCacheWrapper.java
@@ -0,0 +1,74 @@
+/*
+ * 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.phoenix.query;
+
+import com.google.common.base.Preconditions;
+import org.apache.hadoop.hbase.client.TableDescriptor;
+import org.apache.phoenix.schema.PColumnFamily;
+import org.apache.phoenix.schema.PTable;
+import org.apache.phoenix.schema.stats.GuidePostsInfo;
+import org.apache.phoenix.schema.stats.GuidePostsKey;
+import org.apache.phoenix.util.SchemaUtil;
+
+import java.util.List;
+import java.util.concurrent.ExecutionException;
+
+public class GuidePostsCacheWrapper {
+
+    private final GuidePostsCache guidePostsCache;
+
+    GuidePostsCacheWrapper(GuidePostsCache guidePostsCache){
+        this.guidePostsCache = guidePostsCache;
+    }
+
+    GuidePostsInfo get(GuidePostsKey key) throws ExecutionException {
+        return guidePostsCache.get(key);
+    }
+
+    void put(GuidePostsKey key, GuidePostsInfo info){
+        guidePostsCache.put(key,info);
+    }
+
+    void invalidate(GuidePostsKey key){
+        guidePostsCache.invalidate(key);
+    }
+
+    void invalidateAll(){
+        guidePostsCache.invalidateAll();
+    }
+
+    public void invalidateAll(TableDescriptor htableDesc) {
+        Preconditions.checkNotNull(htableDesc);
+        byte[] tableName = htableDesc.getTableName().getName();
+        for (byte[] fam : htableDesc.getColumnFamilyNames()) {
+            invalidate(new GuidePostsKey(tableName, fam));
+        }
+    }
+
+    public void invalidateAll(PTable table) {
+        Preconditions.checkNotNull(table);
+        byte[] physicalName = table.getPhysicalName().getBytes();
+        List<PColumnFamily> families = table.getColumnFamilies();
+        if (families.isEmpty()) {
+            invalidate(new GuidePostsKey(physicalName, SchemaUtil.getEmptyColumnFamily(table)));
+        } else {
+            for (PColumnFamily family : families) {
+                invalidate(new GuidePostsKey(physicalName, family.getName().getBytes()));
+            }
+        }
+    }
+}
diff --git a/phoenix-core/src/main/java/org/apache/phoenix/query/ITGuidePostsCacheFactory.java b/phoenix-core/src/main/java/org/apache/phoenix/query/ITGuidePostsCacheFactory.java
new file mode 100644
index 0000000..a22a650
--- /dev/null
+++ b/phoenix-core/src/main/java/org/apache/phoenix/query/ITGuidePostsCacheFactory.java
@@ -0,0 +1,52 @@
+/**
+ * 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.phoenix.query;
+
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.atomic.AtomicInteger;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.phoenix.util.ReadOnlyProps;
+
+/**
+ * Test Class Only used to verify in e2e tests
+ */
+public class ITGuidePostsCacheFactory implements GuidePostsCacheFactory {
+    public static ConcurrentHashMap<Integer, DefaultGuidePostsCacheFactory> map =
+            new ConcurrentHashMap<>();
+    private static AtomicInteger count = new AtomicInteger();
+    private Integer key;
+
+    public ITGuidePostsCacheFactory() {
+        key = count.getAndIncrement();
+        map.put(key, new DefaultGuidePostsCacheFactory());
+    }
+
+    public static int getCount() {
+        return count.get();
+    }
+
+    public static ConcurrentHashMap<Integer, DefaultGuidePostsCacheFactory> getMap(){
+        return map;
+    }
+
+    @Override
+    public PhoenixStatsLoader getPhoenixStatsLoader(ConnectionQueryServices queryServices,
+            ReadOnlyProps readOnlyProps, Configuration config) {
+        return map.get(key).getPhoenixStatsLoader(queryServices, readOnlyProps, config);
+    }
+
+    @Override
+    public GuidePostsCache getGuidePostsCache(PhoenixStatsLoader phoenixStatsLoader,
+            Configuration config) {
+        return map.get(key).getGuidePostsCache(phoenixStatsLoader, config);
+    }
+}
diff --git a/phoenix-core/src/main/java/org/apache/phoenix/query/PhoenixStatsCacheLoader.java b/phoenix-core/src/main/java/org/apache/phoenix/query/PhoenixStatsCacheLoader.java
index 8caa934..8a6baf2 100644
--- a/phoenix-core/src/main/java/org/apache/phoenix/query/PhoenixStatsCacheLoader.java
+++ b/phoenix-core/src/main/java/org/apache/phoenix/query/PhoenixStatsCacheLoader.java
@@ -32,7 +32,7 @@ import java.util.concurrent.ExecutorService;
 import java.util.concurrent.Executors;
 
 /**
- * {@link CacheLoader} implementation for the Phoenix Table Stats cache.
+ * {@link CacheLoader} asynchronous implementation for the Phoenix Table Stats cache.
  */
 public class PhoenixStatsCacheLoader extends CacheLoader<GuidePostsKey, GuidePostsInfo> {
     private static final Logger logger = LoggerFactory.getLogger(PhoenixStatsCacheLoader.class);
diff --git a/phoenix-core/src/main/java/org/apache/phoenix/query/QueryServices.java b/phoenix-core/src/main/java/org/apache/phoenix/query/QueryServices.java
index 6742cb3..904ed56 100644
--- a/phoenix-core/src/main/java/org/apache/phoenix/query/QueryServices.java
+++ b/phoenix-core/src/main/java/org/apache/phoenix/query/QueryServices.java
@@ -367,6 +367,11 @@ public interface QueryServices extends SQLCloseable {
     public static final String ALLOW_SPLITTABLE_SYSTEM_CATALOG_ROLLBACK =
             "phoenix.allow.system.catalog.rollback";
 
+    // Phoenix parameter used to indicate what implementation is used for providing the client
+    // stats guide post cache.
+    // QueryServicesOptions.DEFAULT_GUIDE_POSTS_CACHE_FACTORY_CLASS is used if this is not provided
+    public static final String GUIDE_POSTS_CACHE_FACTORY_CLASS = "phoenix.guide.posts.cache.factory.class";
+
     /**
      * Get executor service used for parallel scans
      */
diff --git a/phoenix-core/src/main/java/org/apache/phoenix/query/QueryServicesOptions.java b/phoenix-core/src/main/java/org/apache/phoenix/query/QueryServicesOptions.java
index b7fc119..5b0e010 100644
--- a/phoenix-core/src/main/java/org/apache/phoenix/query/QueryServicesOptions.java
+++ b/phoenix-core/src/main/java/org/apache/phoenix/query/QueryServicesOptions.java
@@ -387,6 +387,8 @@ public class QueryServicesOptions {
 
 	public static final boolean DEFAULT_SYSTEM_CATALOG_SPLITTABLE = true;
 
+    public static final String DEFAULT_GUIDE_POSTS_CACHE_FACTORY_CLASS = "org.apache.phoenix.query.DefaultGuidePostsCacheFactory";
+
     private final Configuration config;
 
     private QueryServicesOptions(Configuration config) {
diff --git a/phoenix-core/src/main/java/org/apache/phoenix/query/StatsLoaderImpl.java b/phoenix-core/src/main/java/org/apache/phoenix/query/StatsLoaderImpl.java
new file mode 100644
index 0000000..bc90d32
--- /dev/null
+++ b/phoenix-core/src/main/java/org/apache/phoenix/query/StatsLoaderImpl.java
@@ -0,0 +1,104 @@
+/*
+ * 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.phoenix.query;
+
+import org.apache.hadoop.hbase.HConstants;
+import org.apache.hadoop.hbase.TableName;
+import org.apache.hadoop.hbase.TableNotFoundException;
+import org.apache.hadoop.hbase.client.Table;
+import org.apache.phoenix.jdbc.PhoenixDatabaseMetaData;
+import org.apache.phoenix.schema.stats.GuidePostsInfo;
+import org.apache.phoenix.schema.stats.GuidePostsKey;
+import org.apache.phoenix.schema.stats.StatisticsUtil;
+import org.apache.phoenix.util.SchemaUtil;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.io.IOException;
+import java.util.Objects;
+
+/**
+ * {@link PhoenixStatsLoader} implementation for the Stats Loader.
+ */
+class StatsLoaderImpl implements PhoenixStatsLoader {
+
+    private static final Logger LOGGER = LoggerFactory.getLogger(StatsLoaderImpl.class);
+
+    private final ConnectionQueryServices queryServices;
+
+    public StatsLoaderImpl(ConnectionQueryServices queryServices){
+        this.queryServices = queryServices;
+    }
+
+    @Override
+    public boolean needsLoad() {
+        // For now, whenever it's called, we try to load stats from stats table
+        // no matter it has been updated or not.
+        // Here are the possible optimizations we can do here:
+        // 1. Load stats from the stats table only when the stats get updated on the server side.
+        // 2. Support different refresh cycle for different tables.
+        return true;
+    }
+
+    @Override
+    public GuidePostsInfo loadStats(GuidePostsKey statsKey) throws Exception {
+        return loadStats(statsKey, GuidePostsInfo.NO_GUIDEPOST);
+    }
+
+    @Override
+    public GuidePostsInfo loadStats(GuidePostsKey statsKey, GuidePostsInfo prevGuidepostInfo) throws Exception {
+        assert(prevGuidepostInfo != null);
+
+        TableName tableName = SchemaUtil.getPhysicalName(
+                PhoenixDatabaseMetaData.SYSTEM_STATS_NAME_BYTES,
+                queryServices.getProps());
+        Table statsHTable = queryServices.getTable(tableName.getName());
+
+        try {
+            GuidePostsInfo guidePostsInfo = StatisticsUtil.readStatistics(statsHTable, statsKey,
+                    HConstants.LATEST_TIMESTAMP);
+            traceStatsUpdate(statsKey, guidePostsInfo);
+            return guidePostsInfo;
+        } catch (TableNotFoundException e) {
+            // On a fresh install, stats might not yet be created, don't warn about this.
+            LOGGER.debug("Unable to locate Phoenix stats table: " + tableName.toString(), e);
+            return prevGuidepostInfo;
+        } catch (IOException e) {
+            LOGGER.warn("Unable to read from stats table: " + tableName.toString(), e);
+            return prevGuidepostInfo;
+        } finally {
+            try {
+                statsHTable.close();
+            } catch (IOException e) {
+                // Log, but continue. We have our stats anyway now.
+                LOGGER.warn("Unable to close stats table: " + tableName.toString(), e);
+            }
+        }
+    }
+
+    /**
+     * Logs a trace message for newly inserted entries to the stats cache.
+     */
+    void traceStatsUpdate(GuidePostsKey key, GuidePostsInfo info) {
+        if (LOGGER.isTraceEnabled()) {
+            LOGGER.trace("Updating local TableStats cache (id={}) for {}, size={}bytes",
+                    new Object[] { Objects.hashCode(this), key, info.getEstimatedSize()});
+        }
+    }
+}
diff --git a/phoenix-core/src/main/resources/META-INF/services/org.apache.phoenix.query.GuidePostsCacheFactory b/phoenix-core/src/main/resources/META-INF/services/org.apache.phoenix.query.GuidePostsCacheFactory
new file mode 100644
index 0000000..a3a40a9
--- /dev/null
+++ b/phoenix-core/src/main/resources/META-INF/services/org.apache.phoenix.query.GuidePostsCacheFactory
@@ -0,0 +1,17 @@
+# 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.
+org.apache.phoenix.query.DefaultGuidePostsCacheFactory
+org.apache.phoenix.query.ITGuidePostsCacheFactory
\ No newline at end of file
diff --git a/phoenix-core/src/test/java/org/apache/phoenix/query/GuidePostsCacheProviderTest.java b/phoenix-core/src/test/java/org/apache/phoenix/query/GuidePostsCacheProviderTest.java
new file mode 100644
index 0000000..f3c1e27
--- /dev/null
+++ b/phoenix-core/src/test/java/org/apache/phoenix/query/GuidePostsCacheProviderTest.java
@@ -0,0 +1,122 @@
+/**
+ * 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.phoenix.query;
+
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertTrue;
+
+import java.util.ServiceLoader;
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.ConcurrentMap;
+import java.util.concurrent.CountDownLatch;
+import java.util.concurrent.ExecutorService;
+import java.util.concurrent.Executors;
+import java.util.concurrent.TimeUnit;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.phoenix.exception.PhoenixNonRetryableRuntimeException;
+import org.apache.phoenix.util.InstanceResolver;
+import org.apache.phoenix.util.ReadOnlyProps;
+import org.junit.Before;
+import org.junit.Test;
+import org.mockito.Mockito;
+
+public class GuidePostsCacheProviderTest {
+
+    static GuidePostsCache testCache = null;
+    static PhoenixStatsLoader phoenixStatsLoader = null;
+
+    public static class TestGuidePostsCacheFactory implements  GuidePostsCacheFactory {
+
+        public static volatile int count=0;
+
+        public TestGuidePostsCacheFactory() {
+            count++;
+        }
+
+        @Override public PhoenixStatsLoader getPhoenixStatsLoader(
+                ConnectionQueryServices clientConnectionQueryServices, ReadOnlyProps readOnlyProps,
+                Configuration config) {
+            return phoenixStatsLoader;
+        }
+
+        @Override
+        public GuidePostsCache getGuidePostsCache(PhoenixStatsLoader phoenixStatsLoader,
+                Configuration config) {
+            return testCache;
+        }
+    }
+
+    private GuidePostsCacheProvider helper;
+
+    @Before public void init(){
+        TestGuidePostsCacheFactory.count = 0;
+        helper = new GuidePostsCacheProvider();
+    }
+
+
+    @Test(expected = java.lang.NullPointerException.class)
+    public void loadAndGetGuidePostsCacheFactoryNullStringFailure(){
+            helper.loadAndGetGuidePostsCacheFactory(null);
+    }
+
+    @Test(expected = PhoenixNonRetryableRuntimeException.class)
+    public void loadAndGetGuidePostsCacheFactoryBadStringFailure(){
+        helper.loadAndGetGuidePostsCacheFactory("not a class");
+    }
+
+    @Test(expected = PhoenixNonRetryableRuntimeException.class)
+    public void loadAndGetGuidePostsCacheFactoryNonImplementingClassFailure(){
+        helper.loadAndGetGuidePostsCacheFactory(Object.class.getTypeName());
+    }
+
+    @Test
+    public void loadAndGetGuidePostsCacheFactoryTestFactory(){
+        GuidePostsCacheFactory factory = helper.loadAndGetGuidePostsCacheFactory(
+                TestGuidePostsCacheFactory.class.getTypeName());
+        assertTrue(factory instanceof TestGuidePostsCacheFactory);
+    }
+
+
+    @Test
+    public void getSingletonSimpleTest(){
+        GuidePostsCacheFactory factory1 = helper.loadAndGetGuidePostsCacheFactory(
+                TestGuidePostsCacheFactory.class.getTypeName());
+        assertTrue(factory1 instanceof TestGuidePostsCacheFactory);
+
+        GuidePostsCacheFactory factory2 = helper.loadAndGetGuidePostsCacheFactory(
+                TestGuidePostsCacheFactory.class.getTypeName());
+        assertTrue(factory2 instanceof TestGuidePostsCacheFactory);
+
+        assertEquals(factory1,factory2);
+        assertEquals(1,TestGuidePostsCacheFactory.count);
+    }
+
+    @Test
+    public void getGuidePostsCacheWrapper(){
+        testCache = Mockito.mock(GuidePostsCache.class);
+        ConnectionQueryServices mockQueryServices = Mockito.mock(ConnectionQueryServices.class);
+        Configuration mockConfiguration = Mockito.mock(Configuration.class);
+        GuidePostsCacheWrapper
+                value =
+                helper.getGuidePostsCache(TestGuidePostsCacheFactory.class.getTypeName(),
+                        mockQueryServices, mockConfiguration);
+        value.invalidateAll();
+        Mockito.verify(testCache,Mockito.atLeastOnce()).invalidateAll();
+    }
+}
diff --git a/phoenix-core/src/test/java/org/apache/phoenix/query/GuidePostsCacheWrapperTest.java b/phoenix-core/src/test/java/org/apache/phoenix/query/GuidePostsCacheWrapperTest.java
new file mode 100644
index 0000000..2de3de4
--- /dev/null
+++ b/phoenix-core/src/test/java/org/apache/phoenix/query/GuidePostsCacheWrapperTest.java
@@ -0,0 +1,106 @@
+/**
+ * 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.phoenix.query;
+
+import com.google.common.collect.Lists;
+import org.apache.hadoop.hbase.TableName;
+import org.apache.hadoop.hbase.client.TableDescriptor;
+import org.apache.hadoop.hbase.util.Bytes;
+import org.apache.phoenix.schema.PColumnFamily;
+import org.apache.phoenix.schema.PName;
+import org.apache.phoenix.schema.PTable;
+import org.apache.phoenix.schema.stats.GuidePostsKey;
+import org.junit.Before;
+import org.junit.Test;
+import org.mockito.Mock;
+import org.mockito.Mockito;
+import org.mockito.MockitoAnnotations;
+
+import java.util.HashSet;
+import java.util.List;
+import java.util.Set;
+
+public class GuidePostsCacheWrapperTest {
+
+    @Mock
+    GuidePostsCache cache;
+
+    GuidePostsCacheWrapper wrapper;
+
+    byte[] table = org.apache.hadoop.hbase.util.Bytes.toBytes("tableName");
+    byte[] columnFamily1 = Bytes.toBytesBinary("cf1");
+    byte[] columnFamily2 = Bytes.toBytesBinary("cf2");
+
+    @Before
+    public void init() {
+        MockitoAnnotations.initMocks(this);
+
+        wrapper = new GuidePostsCacheWrapper(cache);
+    }
+
+    @Test
+    public void invalidateAllTableDescriptor() {
+        Set<byte[]> cfSet = new HashSet<>();
+        cfSet.add(columnFamily1);
+        cfSet.add(columnFamily2);
+
+
+
+        TableDescriptor tableDesc = Mockito.mock(TableDescriptor.class);
+        TableName tableName = TableName.valueOf(table);
+
+        Mockito.when(tableDesc.getColumnFamilyNames()).thenReturn(cfSet);
+        Mockito.when(tableDesc.getTableName()).thenReturn(tableName);
+
+        wrapper.invalidateAll(tableDesc);
+        Mockito.verify(cache,Mockito.times(1)).invalidate(new GuidePostsKey(table,columnFamily1));
+        Mockito.verify(cache,Mockito.times(1)).invalidate(new GuidePostsKey(table,columnFamily2));
+    }
+
+    @Test
+    public void invalidateAllPTable(){
+        PTable ptable = Mockito.mock(PTable.class);
+        PName pname = Mockito.mock(PName.class);
+        PName pnamecf1 = Mockito.mock(PName.class);
+        PName pnamecf2 = Mockito.mock(PName.class);
+
+        Mockito.when(ptable.getPhysicalName()).thenReturn(pname);
+        Mockito.when(pname.getBytes()).thenReturn(table);
+
+        PColumnFamily cf1 = Mockito.mock(PColumnFamily.class);
+        PColumnFamily cf2 = Mockito.mock(PColumnFamily.class);
+        Mockito.when(cf1.getName()).thenReturn(pnamecf1);
+        Mockito.when(cf2.getName()).thenReturn(pnamecf2);
+        Mockito.when(pnamecf1.getBytes()).thenReturn(columnFamily1);
+        Mockito.when(pnamecf2.getBytes()).thenReturn(columnFamily2);
+
+        List<PColumnFamily> cfList = Lists.newArrayList(cf1,cf2);
+        Mockito.when(ptable.getColumnFamilies()).thenReturn(cfList);
+
+        wrapper.invalidateAll(ptable);
+
+        Mockito.verify(cache,Mockito.times(1)).invalidate(new GuidePostsKey(table,columnFamily1));
+        Mockito.verify(cache,Mockito.times(1)).invalidate(new GuidePostsKey(table,columnFamily2));
+    }
+
+    @Test(expected = NullPointerException.class)
+    public void invalidateAllTableDescriptorNull() {
+        TableDescriptor tableDesc = null;
+        wrapper.invalidateAll(tableDesc);
+    }
+
+    @Test(expected = NullPointerException.class)
+    public void invalidateAllPTableNull(){
+        PTable ptable = null;
+        wrapper.invalidateAll(ptable);
+    }
+
+}
diff --git a/phoenix-core/src/test/java/org/apache/phoenix/query/PhoenixStatsCacheLoaderTest.java b/phoenix-core/src/test/java/org/apache/phoenix/query/PhoenixStatsCacheLoaderTest.java
index e9c6d40..7ab9bb7 100644
--- a/phoenix-core/src/test/java/org/apache/phoenix/query/PhoenixStatsCacheLoaderTest.java
+++ b/phoenix-core/src/test/java/org/apache/phoenix/query/PhoenixStatsCacheLoaderTest.java
@@ -121,7 +121,7 @@ public class PhoenixStatsCacheLoaderTest {
                     }
                 })
                 // Log removals at TRACE for debugging
-                .removalListener(new GuidePostsCache.PhoenixStatsCacheRemovalListener())
+                .removalListener(new GuidePostsCacheImpl.PhoenixStatsCacheRemovalListener())
                 // Automatically load the cache when entries are missing
                 .build(new PhoenixStatsCacheLoader(new TestStatsLoaderImpl(
                         firstTimeRefreshedSignal, secondTimeRefreshedSignal), config));
diff --git a/phoenix-core/src/test/java/org/apache/phoenix/query/PhoenixStatsCacheRemovalListenerTest.java b/phoenix-core/src/test/java/org/apache/phoenix/query/PhoenixStatsCacheRemovalListenerTest.java
index 7f84219..8bc77a9 100644
--- a/phoenix-core/src/test/java/org/apache/phoenix/query/PhoenixStatsCacheRemovalListenerTest.java
+++ b/phoenix-core/src/test/java/org/apache/phoenix/query/PhoenixStatsCacheRemovalListenerTest.java
@@ -19,7 +19,7 @@ package org.apache.phoenix.query;
 import static org.junit.Assert.assertFalse;
 import static org.junit.Assert.assertTrue;
 
-import org.apache.phoenix.query.GuidePostsCache.PhoenixStatsCacheRemovalListener;
+import org.apache.phoenix.query.GuidePostsCacheImpl.PhoenixStatsCacheRemovalListener;
 import org.junit.Test;
 
 import com.google.common.cache.RemovalCause;
diff --git a/phoenix-core/src/test/resources/META-INF/services/org.apache.phoenix.query.GuidePostsCacheFactory b/phoenix-core/src/test/resources/META-INF/services/org.apache.phoenix.query.GuidePostsCacheFactory
new file mode 100644
index 0000000..95a2438
--- /dev/null
+++ b/phoenix-core/src/test/resources/META-INF/services/org.apache.phoenix.query.GuidePostsCacheFactory
@@ -0,0 +1,19 @@
+# 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.
+org.apache.phoenix.query.DefaultGuidePostsCacheFactory
+org.apache.phoenix.query.ITGuidePostsCacheFactory
+# Test Implementations
+org.apache.phoenix.query.GuidePostsCacheProviderTest$TestGuidePostsCacheFactory
\ No newline at end of file