cgivre commented on code in PR #3023:
URL: https://github.com/apache/drill/pull/3023#discussion_r2765321125


##########
exec/java-exec/src/main/java/org/apache/drill/exec/cache/CustomCacheManager.java:
##########
@@ -0,0 +1,152 @@
+/*
+ * 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.drill.exec.cache;
+
+import java.util.concurrent.TimeUnit;
+
+import org.apache.calcite.rel.RelNode;
+import org.apache.drill.common.config.DrillConfig;
+import org.apache.drill.exec.physical.PhysicalPlan;
+import org.apache.drill.exec.planner.sql.handlers.DefaultSqlHandler.CacheKey;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import com.github.benmanes.caffeine.cache.Cache;
+import com.github.benmanes.caffeine.cache.Caffeine;
+
+/**
+ * Manages caches for query plans and transform results to improve query 
planning performance.
+ * Uses Caffeine cache with configurable TTL and maximum size.
+ */
+public class CustomCacheManager {
+  private static final Logger logger = 
LoggerFactory.getLogger(CustomCacheManager.class);
+
+  private static volatile Cache<String, PhysicalPlan> queryCache;
+  private static volatile Cache<CacheKey, RelNode> transformCache;
+  private static volatile boolean initialized = false;
+
+  private static final int DEFAULT_MAX_ENTRIES = 100;
+  private static final int DEFAULT_TTL_MINUTES = 300;
+
+  private CustomCacheManager() {
+    // Utility class
+  }
+
+  /**
+   * Lazily initializes the caches if not already initialized.
+   * Uses double-checked locking for thread safety.
+   */
+  private static void ensureInitialized() {
+    if (!initialized) {
+      synchronized (CustomCacheManager.class) {
+        if (!initialized) {
+          initializeCaches();
+          initialized = true;
+        }
+      }
+    }
+  }
+
+  private static void initializeCaches() {
+    DrillConfig config = DrillConfig.create();
+
+    int queryMaxEntries = getConfigInt(config, 
"planner.query.cache.max_entries_amount", DEFAULT_MAX_ENTRIES);
+    int queryTtlMinutes = getConfigInt(config, 
"planner.query.cache.plan_cache_ttl_minutes", DEFAULT_TTL_MINUTES);
+    int transformMaxEntries = getConfigInt(config, 
"planner.transform.cache.max_entries_amount", DEFAULT_MAX_ENTRIES);
+    int transformTtlMinutes = getConfigInt(config, 
"planner.transform.cache.plan_cache_ttl_minutes", DEFAULT_TTL_MINUTES);
+
+    queryCache = Caffeine.newBuilder()
+        .maximumSize(queryMaxEntries)
+        .expireAfterWrite(queryTtlMinutes, TimeUnit.MINUTES)
+        .recordStats()
+        .build();
+
+    transformCache = Caffeine.newBuilder()
+        .maximumSize(transformMaxEntries)
+        .expireAfterWrite(transformTtlMinutes, TimeUnit.MINUTES)
+        .recordStats()
+        .build();
+
+    logger.debug("Query plan cache initialized with maxEntries={}, 
ttlMinutes={}", queryMaxEntries, queryTtlMinutes);
+    logger.debug("Transform cache initialized with maxEntries={}, 
ttlMinutes={}", transformMaxEntries, transformTtlMinutes);
+  }
+
+  private static int getConfigInt(DrillConfig config, String path, int 
defaultValue) {
+    if (config.hasPath(path)) {
+      int value = config.getInt(path);
+      logger.debug("Config {}: {}", path, value);
+      return value;
+    }
+    logger.debug("Config {} not found, using default: {}", path, defaultValue);
+    return defaultValue;
+  }
+
+  public static PhysicalPlan getQueryPlan(String sql) {
+    ensureInitialized();
+    return queryCache.getIfPresent(sql);
+  }
+
+  public static void putQueryPlan(String sql, PhysicalPlan plan) {
+    ensureInitialized();
+    queryCache.put(sql, plan);
+  }
+
+  public static RelNode getTransformedPlan(CacheKey key) {
+    ensureInitialized();
+    return transformCache.getIfPresent(key);
+  }
+
+  public static void putTransformedPlan(CacheKey key, RelNode plan) {
+    ensureInitialized();
+    transformCache.put(key, plan);
+  }
+
+  public static void logCacheStats() {
+    ensureInitialized();

Review Comment:
   These stats would be really useful to expose in the UI.  However, I'd 
suggest adding that in another PR.  Let's get this merged first!



##########
exec/java-exec/src/main/java/org/apache/drill/exec/work/foreman/Foreman.java:
##########
@@ -481,6 +485,7 @@ private void runFragment(List<PlanFragment> fragmentsList) 
throws ExecutionSetup
    * Moves query to RUNNING state.
    */
   private void startQueryProcessing() {
+    logger.info("Starting query processing");

Review Comment:
   Please make debug.   Drill already emits a lot of logs. 



##########
exec/java-exec/src/main/java/org/apache/drill/exec/server/options/SystemOptionManager.java:
##########
@@ -82,6 +82,10 @@ public static CaseInsensitiveMap<OptionDefinition> 
createDefaultOptionDefinition
     // here.
     @SuppressWarnings("deprecation")
     final OptionDefinition[] definitions = new OptionDefinition[]{
+      new OptionDefinition(PlannerSettings.PLAN_CACHE),
+      // new OptionDefinition(PlannerSettings.PLAN_CACHE_TTL),

Review Comment:
   Please remove commented out lines.



##########
exec/java-exec/src/main/java/org/apache/drill/exec/planner/sql/handlers/DefaultSqlHandler.java:
##########
@@ -404,11 +460,15 @@ protected RelNode transform(PlannerType plannerType, 
PlannerPhase phase, RelNode
               .getName());
       output = program.run(planner, input, toTraits,
           ImmutableList.of(), ImmutableList.of());
-
       break;
     }
     }
 
+    if (planCacheEnabled) {
+      CustomCacheManager.putTransformedPlan(key, output);
+      logger.debug("Cached transform result for phase: {}", phase);
+    }

Review Comment:
   Do you think it would be possible to include a flag in the results that 
would indicate that the query plan was retrieved from cache?



##########
exec/java-exec/src/main/java/org/apache/drill/exec/work/foreman/Foreman.java:
##########
@@ -269,9 +271,11 @@ public void run() {
         final String sql = queryRequest.getPlan();
         // log query id, username and query text before starting any real 
work. Also, put
         // them together such that it is easy to search based on query id
+        long start = new Date().getTime();
         logger.info("Query text for query with id {} issued by {}: {}", 
queryIdString,
             queryContext.getQueryUserName(), sql);
         runSQL(sql);
+        logger.info("RunSQL is executed within {}", new Date().getTime() - 
start);

Review Comment:
   Please change to debug. 



-- 
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]

Reply via email to