You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@iotdb.apache.org by GitBox <gi...@apache.org> on 2019/07/18 07:33:05 UTC

[GitHub] [incubator-iotdb] jt2594838 commented on a change in pull request #262: [IOTDB-144]meta data cache for query

jt2594838 commented on a change in pull request #262: [IOTDB-144]meta data cache for query
URL: https://github.com/apache/incubator-iotdb/pull/262#discussion_r304757979
 
 

 ##########
 File path: iotdb/src/main/java/org/apache/iotdb/db/engine/cache/DeviceMetaDataCache.java
 ##########
 @@ -19,77 +19,144 @@
 package org.apache.iotdb.db.engine.cache;
 
 import java.io.IOException;
-import java.util.LinkedHashMap;
+import java.util.ArrayList;
+import java.util.HashSet;
+import java.util.List;
 import java.util.Map;
-import java.util.Objects;
+import java.util.Set;
 import java.util.concurrent.atomic.AtomicLong;
+import org.apache.iotdb.db.conf.IoTDBConfig;
+import org.apache.iotdb.db.conf.IoTDBDescriptor;
+import org.apache.iotdb.db.engine.StorageEngine;
+import org.apache.iotdb.tsfile.file.metadata.ChunkMetaData;
 import org.apache.iotdb.tsfile.file.metadata.TsDeviceMetadata;
 import org.apache.iotdb.tsfile.file.metadata.TsFileMetaData;
+import org.apache.iotdb.tsfile.read.common.Path;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
 /**
- * This class is used to cache <code>DeviceMetaDataCache</code> of tsfile in IoTDB.
+ * This class is used to cache <code>List<ChunkMetaData></code> of tsfile in IoTDB. The caching
+ * strategy is LRU.
  */
 public class DeviceMetaDataCache {
 
   private static final Logger logger = LoggerFactory.getLogger(DeviceMetaDataCache.class);
+  private static final IoTDBConfig config = IoTDBDescriptor.getInstance().getConfig();
 
-  private static final int CACHE_SIZE = 100;
+  private static StorageEngine storageEngine = StorageEngine.getInstance();
+
+  private static final long MEMORY_THRESHOLD_IN_B = (long) (0.3 * config
+      .getAllocateMemoryForRead());
   /**
-   * key: the file path + deviceId.
+   * key: file path dot deviceId dot sensorId.
+   * <p>
+   * value: chunkMetaData list of one timeseries in the file.
    */
-  private LinkedHashMap<String, TsDeviceMetadata> lruCache;
+  private LruLinkedHashMap<String, List<ChunkMetaData>> lruCache;
 
   private AtomicLong cacheHintNum = new AtomicLong();
   private AtomicLong cacheRequestNum = new AtomicLong();
 
-  private DeviceMetaDataCache(int cacheSize) {
-    lruCache = new LruLinkedHashMap(cacheSize, true);
+  /**
+   * approximate estimate of chunkMetaData size
+   */
+  private long chunkMetaDataSize = 0;
+
+  private DeviceMetaDataCache(long memoryThreshold) {
+    lruCache = new LruLinkedHashMap<String, List<ChunkMetaData>>(memoryThreshold, true) {
+      @Override
+      protected long calEntrySize(String key, List<ChunkMetaData> value) {
+        if (chunkMetaDataSize == 0 && !value.isEmpty()) {
+          chunkMetaDataSize = RamUsageEstimator.sizeOf(value.get(0));
+        }
+        return value.size() * chunkMetaDataSize + key.length();
+      }
+    };
   }
 
   public static DeviceMetaDataCache getInstance() {
     return RowGroupBlockMetaDataCacheSingleton.INSTANCE;
   }
 
   /**
-   * get {@link TsDeviceMetadata}. THREAD SAFE.
+   * get {@link ChunkMetaData}. THREAD SAFE.
    */
-  public TsDeviceMetadata get(String filePath, String deviceId, TsFileMetaData fileMetaData)
+  public List<ChunkMetaData> get(String filePath, Path seriesPath)
       throws IOException {
-    // The key(the tsfile path and deviceId) for the lruCache
+    StringBuilder builder = new StringBuilder(filePath).append(".").append(seriesPath.getDevice());
+    String devicePathStr = builder.toString();
+    String key = builder.append(".").append(seriesPath.getMeasurement()).toString();
+    Object devicePathObject = devicePathStr.intern();
 
-    String jointPath = filePath + deviceId;
-    Object jointPathObject = jointPath.intern();
     synchronized (lruCache) {
       cacheRequestNum.incrementAndGet();
-      if (lruCache.containsKey(jointPath)) {
+      if (lruCache.containsKey(key)) {
         cacheHintNum.incrementAndGet();
         if (logger.isDebugEnabled()) {
           logger.debug(
               "Cache hint: the number of requests for cache is {}, "
                   + "the number of hints for cache is {}",
               cacheRequestNum.get(), cacheHintNum.get());
         }
-        return lruCache.get(jointPath);
+        return new ArrayList<>(lruCache.get(key));
       }
     }
-    synchronized (jointPathObject) {
+    synchronized (devicePathObject) {
       synchronized (lruCache) {
-        if (lruCache.containsKey(jointPath)) {
-          return lruCache.get(jointPath);
+        if (lruCache.containsKey(key)) {
+          cacheHintNum.incrementAndGet();
+          return new ArrayList<>(lruCache.get(key));
         }
       }
       if (logger.isDebugEnabled()) {
         logger.debug("Cache didn't hint: the number of requests for cache is {}",
             cacheRequestNum.get());
       }
+      TsFileMetaData fileMetaData = TsFileMetaDataCache.getInstance().get(filePath);
       TsDeviceMetadata blockMetaData = TsFileMetadataUtils
-          .getTsRowGroupBlockMetaData(filePath, deviceId,
+          .getTsRowGroupBlockMetaData(filePath, seriesPath.getDevice(),
               fileMetaData);
+      Map<Path, List<ChunkMetaData>> chunkMetaData = TsFileMetadataUtils
+          .getChunkMetaDataList(calHotSensorSet(seriesPath), blockMetaData);
       synchronized (lruCache) {
-        lruCache.put(jointPath, blockMetaData);
-        return lruCache.get(jointPath);
+        chunkMetaData.forEach((path, chunkMetaDataList) -> {
+          String k = devicePathStr + "." + path.getMeasurement();
+          if (!lruCache.containsKey(k)) {
+            lruCache.put(k, chunkMetaDataList);
+          }
+        });
+        if (chunkMetaData.containsKey(seriesPath)) {
+          return new ArrayList<>(chunkMetaData.get(seriesPath));
+        }
+        return new ArrayList<>();
+      }
+    }
+  }
+
+  /**
+   * calculate the most frequently query sensors set.
+   *
+   * @param seriesPath the series to be queried in a query statements.
+   */
+  private Set<String> calHotSensorSet(Path seriesPath) throws IOException {
+    double usedMemProportion = lruCache.getUsedMemoryProportion();
+
+    if (usedMemProportion < 0.6) {
+      return new HashSet<>();
+    } else {
+      double hotSensorProportion;
+      if (usedMemProportion < 0.8) {
+        hotSensorProportion = 0.1;
+      } else {
+        hotSensorProportion = 0.05;
 
 Review comment:
   I wonder how these numbers come out.

----------------------------------------------------------------
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.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services