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.io.hfile;
019
020import java.util.Iterator;
021import java.util.Map;
022import java.util.Optional;
023import org.apache.commons.lang3.mutable.Mutable;
024import org.apache.commons.lang3.mutable.MutableBoolean;
025import org.apache.hadoop.fs.Path;
026import org.apache.hadoop.hbase.io.HeapSize;
027import org.apache.hadoop.hbase.io.hfile.bucket.BucketCache;
028import org.apache.hadoop.hbase.util.Pair;
029import org.apache.yetus.audience.InterfaceAudience;
030import org.slf4j.Logger;
031import org.slf4j.LoggerFactory;
032
033/**
034 * CombinedBlockCache is an abstraction layer that combines {@link FirstLevelBlockCache} and
035 * {@link BucketCache}. The smaller lruCache is used to cache bloom blocks and index blocks. The
036 * larger Cache is used to cache data blocks.
037 * {@link #getBlock(BlockCacheKey, boolean, boolean, boolean)} reads first from the smaller l1Cache
038 * before looking for the block in the l2Cache. Blocks evicted from l1Cache are put into the bucket
039 * cache. Metrics are the combined size and hits and misses of both caches.
040 */
041@InterfaceAudience.Private
042public class CombinedBlockCache implements ResizableBlockCache, HeapSize {
043  protected final FirstLevelBlockCache l1Cache;
044  protected final BlockCache l2Cache;
045  protected final CombinedCacheStats combinedCacheStats;
046
047  private static final Logger LOG = LoggerFactory.getLogger(CombinedBlockCache.class);
048
049  public CombinedBlockCache(FirstLevelBlockCache l1Cache, BlockCache l2Cache) {
050    this.l1Cache = l1Cache;
051    this.l2Cache = l2Cache;
052    this.combinedCacheStats = new CombinedCacheStats(l1Cache.getStats(), l2Cache.getStats());
053  }
054
055  @Override
056  public long heapSize() {
057    long l2size = 0;
058    if (l2Cache instanceof HeapSize) {
059      l2size = ((HeapSize) l2Cache).heapSize();
060    }
061    return l1Cache.heapSize() + l2size;
062  }
063
064  @Override
065  public void cacheBlock(BlockCacheKey cacheKey, Cacheable buf, boolean inMemory) {
066    cacheBlock(cacheKey, buf, inMemory, false);
067  }
068
069  @Override
070  public void cacheBlock(BlockCacheKey cacheKey, Cacheable buf, boolean inMemory,
071    boolean waitWhenCache) {
072    boolean metaBlock = isMetaBlock(buf.getBlockType());
073    if (metaBlock) {
074      l1Cache.cacheBlock(cacheKey, buf, inMemory);
075    } else {
076      l2Cache.cacheBlock(cacheKey, buf, inMemory, waitWhenCache);
077    }
078  }
079
080  @Override
081  public void cacheBlock(BlockCacheKey cacheKey, Cacheable buf) {
082    cacheBlock(cacheKey, buf, false);
083  }
084
085  @Override
086  public Cacheable getBlock(BlockCacheKey cacheKey, boolean caching, boolean repeat,
087    boolean updateCacheMetrics) {
088    Cacheable block = null;
089    // We don't know the block type. We should try to get it on one of the caches only,
090    // but not both otherwise we'll over compute on misses. Here we check if the key is on L1,
091    // if so, call getBlock on L1 and that will compute the hit. Otherwise, we'll try to get it from
092    // L2 and whatever happens, we'll update the stats there.
093    boolean existInL1 = l1Cache.containsBlock(cacheKey);
094    // if we know it's in L1, just delegate call to l1 and return it
095    if (existInL1) {
096      block = l1Cache.getBlock(cacheKey, caching, repeat, false);
097    } else {
098      block = l2Cache.getBlock(cacheKey, caching, repeat, false);
099    }
100    if (updateCacheMetrics) {
101      boolean metaBlock = isMetaBlock(cacheKey.getBlockType());
102      if (metaBlock) {
103        if (!existInL1 && block != null) {
104          LOG.warn("Cache key {} had block type {}, but was found in L2 cache.", cacheKey,
105            cacheKey.getBlockType());
106          updateBlockMetrics(block, cacheKey, l2Cache, caching);
107        } else {
108          updateBlockMetrics(block, cacheKey, l1Cache, caching);
109        }
110      } else {
111        if (existInL1) {
112          LOG.warn("Cache key {} had block type {}, but was found in L1 cache.", cacheKey,
113            cacheKey.getBlockType());
114          updateBlockMetrics(block, cacheKey, l1Cache, caching);
115        } else {
116          updateBlockMetrics(block, cacheKey, l2Cache, caching);
117        }
118      }
119    }
120    return block;
121  }
122
123  private void updateBlockMetrics(Cacheable block, BlockCacheKey key, BlockCache cache,
124    boolean caching) {
125    if (block == null) {
126      cache.getStats().miss(caching, key.isPrimary(), key.getBlockType());
127    } else {
128      cache.getStats().hit(caching, key.isPrimary(), key.getBlockType());
129
130    }
131  }
132
133  private Cacheable getBlockWithType(BlockCacheKey cacheKey, boolean caching, boolean repeat,
134    boolean updateCacheMetrics) {
135    boolean metaBlock = isMetaBlock(cacheKey.getBlockType());
136    if (metaBlock) {
137      return l1Cache.getBlock(cacheKey, caching, repeat, updateCacheMetrics);
138    } else {
139      return l2Cache.getBlock(cacheKey, caching, repeat, updateCacheMetrics);
140    }
141  }
142
143  @Override
144  public boolean evictBlock(BlockCacheKey cacheKey) {
145    return l1Cache.evictBlock(cacheKey) || l2Cache.evictBlock(cacheKey);
146  }
147
148  @Override
149  public int evictBlocksByHfileName(String hfileName) {
150    return l1Cache.evictBlocksByHfileName(hfileName) + l2Cache.evictBlocksByHfileName(hfileName);
151  }
152
153  @Override
154  public CacheStats getStats() {
155    return this.combinedCacheStats;
156  }
157
158  @Override
159  public void shutdown() {
160    l1Cache.shutdown();
161    l2Cache.shutdown();
162  }
163
164  @Override
165  public long size() {
166    return l1Cache.size() + l2Cache.size();
167  }
168
169  @Override
170  public long getMaxSize() {
171    return l1Cache.getMaxSize() + l2Cache.getMaxSize();
172  }
173
174  @Override
175  public long getCurrentDataSize() {
176    return l1Cache.getCurrentDataSize() + l2Cache.getCurrentDataSize();
177  }
178
179  @Override
180  public long getFreeSize() {
181    return l1Cache.getFreeSize() + l2Cache.getFreeSize();
182  }
183
184  @Override
185  public long getCurrentSize() {
186    return l1Cache.getCurrentSize() + l2Cache.getCurrentSize();
187  }
188
189  @Override
190  public long getBlockCount() {
191    return l1Cache.getBlockCount() + l2Cache.getBlockCount();
192  }
193
194  @Override
195  public long getDataBlockCount() {
196    return l1Cache.getDataBlockCount() + l2Cache.getDataBlockCount();
197  }
198
199  public static class CombinedCacheStats extends CacheStats {
200    private final CacheStats lruCacheStats;
201    private final CacheStats bucketCacheStats;
202
203    CombinedCacheStats(CacheStats lbcStats, CacheStats fcStats) {
204      super("CombinedBlockCache");
205      this.lruCacheStats = lbcStats;
206      this.bucketCacheStats = fcStats;
207    }
208
209    public CacheStats getLruCacheStats() {
210      return this.lruCacheStats;
211    }
212
213    public CacheStats getBucketCacheStats() {
214      return this.bucketCacheStats;
215    }
216
217    @Override
218    public long getDataMissCount() {
219      return lruCacheStats.getDataMissCount() + bucketCacheStats.getDataMissCount();
220    }
221
222    @Override
223    public long getLeafIndexMissCount() {
224      return lruCacheStats.getLeafIndexMissCount() + bucketCacheStats.getLeafIndexMissCount();
225    }
226
227    @Override
228    public long getBloomChunkMissCount() {
229      return lruCacheStats.getBloomChunkMissCount() + bucketCacheStats.getBloomChunkMissCount();
230    }
231
232    @Override
233    public long getMetaMissCount() {
234      return lruCacheStats.getMetaMissCount() + bucketCacheStats.getMetaMissCount();
235    }
236
237    @Override
238    public long getRootIndexMissCount() {
239      return lruCacheStats.getRootIndexMissCount() + bucketCacheStats.getRootIndexMissCount();
240    }
241
242    @Override
243    public long getIntermediateIndexMissCount() {
244      return lruCacheStats.getIntermediateIndexMissCount()
245        + bucketCacheStats.getIntermediateIndexMissCount();
246    }
247
248    @Override
249    public long getFileInfoMissCount() {
250      return lruCacheStats.getFileInfoMissCount() + bucketCacheStats.getFileInfoMissCount();
251    }
252
253    @Override
254    public long getGeneralBloomMetaMissCount() {
255      return lruCacheStats.getGeneralBloomMetaMissCount()
256        + bucketCacheStats.getGeneralBloomMetaMissCount();
257    }
258
259    @Override
260    public long getDeleteFamilyBloomMissCount() {
261      return lruCacheStats.getDeleteFamilyBloomMissCount()
262        + bucketCacheStats.getDeleteFamilyBloomMissCount();
263    }
264
265    @Override
266    public long getTrailerMissCount() {
267      return lruCacheStats.getTrailerMissCount() + bucketCacheStats.getTrailerMissCount();
268    }
269
270    @Override
271    public long getDataHitCount() {
272      return lruCacheStats.getDataHitCount() + bucketCacheStats.getDataHitCount();
273    }
274
275    @Override
276    public long getLeafIndexHitCount() {
277      return lruCacheStats.getLeafIndexHitCount() + bucketCacheStats.getLeafIndexHitCount();
278    }
279
280    @Override
281    public long getBloomChunkHitCount() {
282      return lruCacheStats.getBloomChunkHitCount() + bucketCacheStats.getBloomChunkHitCount();
283    }
284
285    @Override
286    public long getMetaHitCount() {
287      return lruCacheStats.getMetaHitCount() + bucketCacheStats.getMetaHitCount();
288    }
289
290    @Override
291    public long getRootIndexHitCount() {
292      return lruCacheStats.getRootIndexHitCount() + bucketCacheStats.getRootIndexHitCount();
293    }
294
295    @Override
296    public long getIntermediateIndexHitCount() {
297      return lruCacheStats.getIntermediateIndexHitCount()
298        + bucketCacheStats.getIntermediateIndexHitCount();
299    }
300
301    @Override
302    public long getFileInfoHitCount() {
303      return lruCacheStats.getFileInfoHitCount() + bucketCacheStats.getFileInfoHitCount();
304    }
305
306    @Override
307    public long getGeneralBloomMetaHitCount() {
308      return lruCacheStats.getGeneralBloomMetaHitCount()
309        + bucketCacheStats.getGeneralBloomMetaHitCount();
310    }
311
312    @Override
313    public long getDeleteFamilyBloomHitCount() {
314      return lruCacheStats.getDeleteFamilyBloomHitCount()
315        + bucketCacheStats.getDeleteFamilyBloomHitCount();
316    }
317
318    @Override
319    public long getTrailerHitCount() {
320      return lruCacheStats.getTrailerHitCount() + bucketCacheStats.getTrailerHitCount();
321    }
322
323    @Override
324    public long getRequestCount() {
325      return lruCacheStats.getRequestCount() + bucketCacheStats.getRequestCount();
326    }
327
328    @Override
329    public long getRequestCachingCount() {
330      return lruCacheStats.getRequestCachingCount() + bucketCacheStats.getRequestCachingCount();
331    }
332
333    @Override
334    public long getMissCount() {
335      return lruCacheStats.getMissCount() + bucketCacheStats.getMissCount();
336    }
337
338    @Override
339    public long getPrimaryMissCount() {
340      return lruCacheStats.getPrimaryMissCount() + bucketCacheStats.getPrimaryMissCount();
341    }
342
343    @Override
344    public long getMissCachingCount() {
345      return lruCacheStats.getMissCachingCount() + bucketCacheStats.getMissCachingCount();
346    }
347
348    @Override
349    public long getHitCount() {
350      return lruCacheStats.getHitCount() + bucketCacheStats.getHitCount();
351    }
352
353    @Override
354    public long getPrimaryHitCount() {
355      return lruCacheStats.getPrimaryHitCount() + bucketCacheStats.getPrimaryHitCount();
356    }
357
358    @Override
359    public long getHitCachingCount() {
360      return lruCacheStats.getHitCachingCount() + bucketCacheStats.getHitCachingCount();
361    }
362
363    @Override
364    public long getEvictionCount() {
365      return lruCacheStats.getEvictionCount() + bucketCacheStats.getEvictionCount();
366    }
367
368    @Override
369    public long getEvictedCount() {
370      return lruCacheStats.getEvictedCount() + bucketCacheStats.getEvictedCount();
371    }
372
373    @Override
374    public long getPrimaryEvictedCount() {
375      return lruCacheStats.getPrimaryEvictedCount() + bucketCacheStats.getPrimaryEvictedCount();
376    }
377
378    @Override
379    public void rollMetricsPeriod() {
380      lruCacheStats.rollMetricsPeriod();
381      bucketCacheStats.rollMetricsPeriod();
382    }
383
384    @Override
385    public long getFailedInserts() {
386      return lruCacheStats.getFailedInserts() + bucketCacheStats.getFailedInserts();
387    }
388
389    @Override
390    public long getSumHitCountsPastNPeriods() {
391      return lruCacheStats.getSumHitCountsPastNPeriods()
392        + bucketCacheStats.getSumHitCountsPastNPeriods();
393    }
394
395    @Override
396    public long getSumRequestCountsPastNPeriods() {
397      return lruCacheStats.getSumRequestCountsPastNPeriods()
398        + bucketCacheStats.getSumRequestCountsPastNPeriods();
399    }
400
401    @Override
402    public long getSumHitCachingCountsPastNPeriods() {
403      return lruCacheStats.getSumHitCachingCountsPastNPeriods()
404        + bucketCacheStats.getSumHitCachingCountsPastNPeriods();
405    }
406
407    @Override
408    public long getSumRequestCachingCountsPastNPeriods() {
409      return lruCacheStats.getSumRequestCachingCountsPastNPeriods()
410        + bucketCacheStats.getSumRequestCachingCountsPastNPeriods();
411    }
412  }
413
414  @Override
415  public Iterator<CachedBlock> iterator() {
416    return new BlockCachesIterator(getBlockCaches());
417  }
418
419  @Override
420  public BlockCache[] getBlockCaches() {
421    return new BlockCache[] { this.l1Cache, this.l2Cache };
422  }
423
424  /**
425   * Returns the list of fully cached files
426   */
427  @Override
428  public Optional<Map<String, Pair<String, Long>>> getFullyCachedFiles() {
429    return this.l2Cache.getFullyCachedFiles();
430  }
431
432  @Override
433  public void setMaxSize(long size) {
434    this.l1Cache.setMaxSize(size);
435  }
436
437  public int getRpcRefCount(BlockCacheKey cacheKey) {
438    return (this.l2Cache instanceof BucketCache)
439      ? ((BucketCache) this.l2Cache).getRpcRefCount(cacheKey)
440      : 0;
441  }
442
443  public FirstLevelBlockCache getFirstLevelCache() {
444    return l1Cache;
445  }
446
447  public BlockCache getSecondLevelCache() {
448    return l2Cache;
449  }
450
451  @Override
452  public void notifyFileCachingCompleted(Path fileName, int totalBlockCount, int dataBlockCount,
453    long size) {
454    l1Cache.getBlockCount();
455    l1Cache.notifyFileCachingCompleted(fileName, totalBlockCount, dataBlockCount, size);
456    l2Cache.notifyFileCachingCompleted(fileName, totalBlockCount, dataBlockCount, size);
457
458  }
459
460  @Override
461  public void notifyFileBlockEvicted(String fileName) {
462    l1Cache.notifyFileBlockEvicted(fileName);
463    l1Cache.notifyFileBlockEvicted(fileName);
464  }
465
466  @Override
467  public Optional<Boolean> blockFitsIntoTheCache(HFileBlock block) {
468    if (isMetaBlock(block.getBlockType())) {
469      return l1Cache.blockFitsIntoTheCache(block);
470    } else {
471      return l2Cache.blockFitsIntoTheCache(block);
472    }
473  }
474
475  @Override
476  public Optional<Boolean> shouldCacheFile(String fileName) {
477    Optional<Boolean> l1Result = l1Cache.shouldCacheFile(fileName);
478    Optional<Boolean> l2Result = l2Cache.shouldCacheFile(fileName);
479    final Mutable<Boolean> combinedResult = new MutableBoolean(true);
480    l1Result.ifPresent(b -> combinedResult.setValue(b && combinedResult.getValue()));
481    l2Result.ifPresent(b -> combinedResult.setValue(b && combinedResult.getValue()));
482    return Optional.of(combinedResult.getValue());
483  }
484
485  @Override
486  public Optional<Boolean> isAlreadyCached(BlockCacheKey key) {
487    boolean result =
488      l1Cache.isAlreadyCached(key).orElseGet(() -> l2Cache.isAlreadyCached(key).orElse(false));
489    return Optional.of(result);
490  }
491
492  @Override
493  public Optional<Integer> getBlockSize(BlockCacheKey key) {
494    Optional<Integer> l1Result = l1Cache.getBlockSize(key);
495    return l1Result.isPresent() ? l1Result : l2Cache.getBlockSize(key);
496  }
497
498}