001/* 002 * Licensed to the Apache Software Foundation (ASF) under one 003 * or more contributor license agreements. See the NOTICE file 004 * distributed with this work for additional information 005 * regarding copyright ownership. The ASF licenses this file 006 * to you under the Apache License, Version 2.0 (the 007 * "License"); you may not use this file except in compliance 008 * with the License. You may obtain a copy of the License at 009 * 010 * http://www.apache.org/licenses/LICENSE-2.0 011 * 012 * Unless required by applicable law or agreed to in writing, software 013 * distributed under the License is distributed on an "AS IS" BASIS, 014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. 015 * See the License for the specific language governing permissions and 016 * limitations under the License. 017 */ 018package org.apache.hadoop.hbase.mob; 019 020import java.io.IOException; 021import java.util.ArrayList; 022import java.util.Collections; 023import java.util.List; 024import java.util.Map; 025import java.util.concurrent.ConcurrentHashMap; 026import java.util.concurrent.Executors; 027import java.util.concurrent.ScheduledExecutorService; 028import java.util.concurrent.TimeUnit; 029import java.util.concurrent.atomic.AtomicLong; 030import java.util.concurrent.atomic.LongAdder; 031import java.util.concurrent.locks.ReentrantLock; 032import org.apache.hadoop.conf.Configuration; 033import org.apache.hadoop.fs.FileSystem; 034import org.apache.hadoop.fs.Path; 035import org.apache.hadoop.hbase.io.hfile.CacheConfig; 036import org.apache.hadoop.hbase.util.IdLock; 037import org.apache.yetus.audience.InterfaceAudience; 038import org.slf4j.Logger; 039import org.slf4j.LoggerFactory; 040 041import org.apache.hbase.thirdparty.com.google.common.hash.Hashing; 042import org.apache.hbase.thirdparty.com.google.common.util.concurrent.ThreadFactoryBuilder; 043 044/** 045 * The cache for mob files. This cache doesn't cache the mob file blocks. It only caches the 046 * references of mob files. We are doing this to avoid opening and closing mob files all the time. 047 * We just keep references open. 048 */ 049@InterfaceAudience.Private 050public class MobFileCache { 051 052 private static final Logger LOG = LoggerFactory.getLogger(MobFileCache.class); 053 054 /* 055 * Eviction and statistics thread. Periodically run to print the statistics and evict the lru 056 * cached mob files when the count of the cached files is larger than the threshold. 057 */ 058 static class EvictionThread extends Thread { 059 MobFileCache lru; 060 061 public EvictionThread(MobFileCache lru) { 062 super("MobFileCache.EvictionThread"); 063 setDaemon(true); 064 this.lru = lru; 065 } 066 067 @Override 068 public void run() { 069 lru.evict(); 070 } 071 } 072 073 // a ConcurrentHashMap, accesses to this map are synchronized. 074 private Map<String, CachedMobFile> map = null; 075 // caches access count 076 private final AtomicLong count = new AtomicLong(0); 077 private long lastAccess = 0; 078 private final LongAdder miss = new LongAdder(); 079 private long lastMiss = 0; 080 private final LongAdder evictedFileCount = new LongAdder(); 081 private long lastEvictedFileCount = 0; 082 083 // a lock to sync the evict to guarantee the eviction occurs in sequence. 084 // the method evictFile is not sync by this lock, the ConcurrentHashMap does the sync there. 085 private final ReentrantLock evictionLock = new ReentrantLock(true); 086 087 // stripes lock on each mob file based on its hash. Sync the openFile/closeFile operations. 088 private final IdLock keyLock = new IdLock(); 089 090 private final ScheduledExecutorService scheduleThreadPool = Executors.newScheduledThreadPool(1, 091 new ThreadFactoryBuilder().setNameFormat("MobFileCache #%d").setDaemon(true).build()); 092 private final Configuration conf; 093 094 // the count of the cached references to mob files 095 private final int mobFileMaxCacheSize; 096 private final boolean isCacheEnabled; 097 private float evictRemainRatio; 098 099 public MobFileCache(Configuration conf) { 100 this.conf = conf; 101 this.mobFileMaxCacheSize = 102 conf.getInt(MobConstants.MOB_FILE_CACHE_SIZE_KEY, MobConstants.DEFAULT_MOB_FILE_CACHE_SIZE); 103 isCacheEnabled = (mobFileMaxCacheSize > 0); 104 map = new ConcurrentHashMap<>(mobFileMaxCacheSize); 105 if (isCacheEnabled) { 106 long period = conf.getLong(MobConstants.MOB_CACHE_EVICT_PERIOD, 107 MobConstants.DEFAULT_MOB_CACHE_EVICT_PERIOD); // in seconds 108 evictRemainRatio = conf.getFloat(MobConstants.MOB_CACHE_EVICT_REMAIN_RATIO, 109 MobConstants.DEFAULT_EVICT_REMAIN_RATIO); 110 if (evictRemainRatio < 0.0) { 111 evictRemainRatio = 0.0f; 112 LOG.warn(MobConstants.MOB_CACHE_EVICT_REMAIN_RATIO + " is less than 0.0, 0.0 is used."); 113 } else if (evictRemainRatio > 1.0) { 114 evictRemainRatio = 1.0f; 115 LOG.warn(MobConstants.MOB_CACHE_EVICT_REMAIN_RATIO + " is larger than 1.0, 1.0 is used."); 116 } 117 this.scheduleThreadPool.scheduleAtFixedRate(new EvictionThread(this), period, period, 118 TimeUnit.SECONDS); 119 120 if (LOG.isDebugEnabled()) { 121 LOG.debug("MobFileCache enabled with cacheSize=" + mobFileMaxCacheSize + ", evictPeriods=" 122 + period + "sec, evictRemainRatio=" + evictRemainRatio); 123 } 124 } else { 125 LOG.info("MobFileCache disabled"); 126 } 127 } 128 129 /** 130 * Evicts the lru cached mob files when the count of the cached files is larger than the 131 * threshold. 132 */ 133 public void evict() { 134 if (isCacheEnabled) { 135 // Ensure only one eviction at a time 136 if (!evictionLock.tryLock()) { 137 return; 138 } 139 printStatistics(); 140 List<CachedMobFile> evictedFiles = new ArrayList<>(); 141 try { 142 if (map.size() <= mobFileMaxCacheSize) { 143 return; 144 } 145 List<CachedMobFile> files = new ArrayList<>(map.values()); 146 Collections.sort(files); 147 int start = (int) (mobFileMaxCacheSize * evictRemainRatio); 148 if (start >= 0) { 149 for (int i = start; i < files.size(); i++) { 150 String name = files.get(i).getFileName(); 151 CachedMobFile evictedFile = map.remove(name); 152 if (evictedFile != null) { 153 evictedFiles.add(evictedFile); 154 } 155 } 156 } 157 } finally { 158 evictionLock.unlock(); 159 } 160 // EvictionLock is released. Close the evicted files one by one. 161 // The closes are sync in the closeFile method. 162 for (CachedMobFile evictedFile : evictedFiles) { 163 closeFile(evictedFile); 164 } 165 evictedFileCount.add(evictedFiles.size()); 166 } 167 } 168 169 /** 170 * Evicts the cached file by the name. 171 * @param fileName The name of a cached file. 172 */ 173 public void evictFile(String fileName) { 174 if (isCacheEnabled) { 175 IdLock.Entry lockEntry = null; 176 try { 177 // obtains the lock to close the cached file. 178 lockEntry = keyLock.getLockEntry(hashFileName(fileName)); 179 CachedMobFile evictedFile = map.remove(fileName); 180 if (evictedFile != null) { 181 evictedFile.close(); 182 evictedFileCount.increment(); 183 } 184 } catch (IOException e) { 185 LOG.error("Failed to evict the file " + fileName, e); 186 } finally { 187 if (lockEntry != null) { 188 keyLock.releaseLockEntry(lockEntry); 189 } 190 } 191 } 192 } 193 194 /** 195 * Opens a mob file. 196 * @param fs The current file system. 197 * @param path The file path. 198 * @param cacheConf The current MobCacheConfig 199 * @return A opened mob file. 200 */ 201 public MobFile openFile(FileSystem fs, Path path, CacheConfig cacheConf) throws IOException { 202 if (!isCacheEnabled) { 203 MobFile mobFile = MobFile.create(fs, path, conf, cacheConf); 204 mobFile.open(); 205 return mobFile; 206 } else { 207 String fileName = path.getName(); 208 CachedMobFile cached = map.get(fileName); 209 IdLock.Entry lockEntry = keyLock.getLockEntry(hashFileName(fileName)); 210 try { 211 if (cached == null) { 212 cached = map.get(fileName); 213 if (cached == null) { 214 if (map.size() > mobFileMaxCacheSize) { 215 evict(); 216 } 217 cached = CachedMobFile.create(fs, path, conf, cacheConf); 218 cached.open(); 219 map.put(fileName, cached); 220 miss.increment(); 221 } 222 } 223 cached.open(); 224 cached.access(count.incrementAndGet()); 225 } finally { 226 keyLock.releaseLockEntry(lockEntry); 227 } 228 return cached; 229 } 230 } 231 232 /** 233 * Closes a mob file. 234 * @param file The mob file that needs to be closed. 235 */ 236 public void closeFile(MobFile file) { 237 IdLock.Entry lockEntry = null; 238 try { 239 if (!isCacheEnabled) { 240 file.close(); 241 } else { 242 lockEntry = keyLock.getLockEntry(hashFileName(file.getFileName())); 243 file.close(); 244 } 245 } catch (IOException e) { 246 LOG.error("MobFileCache, Exception happen during close " + file.getFileName(), e); 247 } finally { 248 if (lockEntry != null) { 249 keyLock.releaseLockEntry(lockEntry); 250 } 251 } 252 } 253 254 public void shutdown() { 255 this.scheduleThreadPool.shutdown(); 256 for (int i = 0; i < 100; i++) { 257 if (!this.scheduleThreadPool.isShutdown()) { 258 try { 259 Thread.sleep(10); 260 } catch (InterruptedException e) { 261 LOG.warn("Interrupted while sleeping"); 262 Thread.currentThread().interrupt(); 263 break; 264 } 265 } 266 } 267 268 if (!this.scheduleThreadPool.isShutdown()) { 269 List<Runnable> runnables = this.scheduleThreadPool.shutdownNow(); 270 LOG.debug("Still running " + runnables); 271 } 272 } 273 274 /** 275 * Gets the count of cached mob files. 276 * @return The count of the cached mob files. 277 */ 278 public int getCacheSize() { 279 return map == null ? 0 : map.size(); 280 } 281 282 /** 283 * Gets the count of accesses to the mob file cache. 284 * @return The count of accesses to the mob file cache. 285 */ 286 public long getAccessCount() { 287 return count.get(); 288 } 289 290 /** 291 * Gets the count of misses to the mob file cache. 292 * @return The count of misses to the mob file cache. 293 */ 294 public long getMissCount() { 295 return miss.sum(); 296 } 297 298 /** 299 * Gets the number of items evicted from the mob file cache. 300 * @return The number of items evicted from the mob file cache. 301 */ 302 public long getEvictedFileCount() { 303 return evictedFileCount.sum(); 304 } 305 306 /** 307 * Gets the hit ratio to the mob file cache. 308 * @return The hit ratio to the mob file cache. 309 */ 310 public double getHitRatio() { 311 return count.get() == 0 ? 0 : ((float) (count.get() - miss.sum())) / (float) count.get(); 312 } 313 314 /** 315 * Prints the statistics. 316 */ 317 public void printStatistics() { 318 long access = count.get() - lastAccess; 319 long missed = miss.sum() - lastMiss; 320 long evicted = evictedFileCount.sum() - lastEvictedFileCount; 321 int hitRatio = access == 0 ? 0 : (int) (((float) (access - missed)) / (float) access * 100); 322 LOG.info("MobFileCache Statistics, access: " + access + ", miss: " + missed + ", hit: " 323 + (access - missed) + ", hit ratio: " + hitRatio + "%, evicted files: " + evicted); 324 lastAccess += access; 325 lastMiss += missed; 326 lastEvictedFileCount += evicted; 327 } 328 329 /** 330 * Use murmurhash to reduce the conflicts of hashed file names. We should notice that the hash 331 * conflicts may bring deadlocks, when opening mob files with evicting some other files, as 332 * described in HBASE-28047. 333 */ 334 private long hashFileName(String fileName) { 335 return Hashing.murmur3_128().hashString(fileName, java.nio.charset.StandardCharsets.UTF_8) 336 .asLong(); 337 } 338}