001/**
002 * Copyright The Apache Software Foundation
003 *
004 * Licensed to the Apache Software Foundation (ASF) under one or more
005 * contributor license agreements. See the NOTICE file distributed with this
006 * work for additional information regarding copyright ownership. The ASF
007 * licenses this file to you under the Apache License, Version 2.0 (the
008 * "License"); you may not use this file except in compliance with the License.
009 * You may obtain a copy of the License at
010 *
011 * http://www.apache.org/licenses/LICENSE-2.0
012 *
013 * Unless required by applicable law or agreed to in writing, software
014 * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
015 * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
016 * License for the specific language governing permissions and limitations
017 * under the License.
018 */
019package org.apache.hadoop.hbase.io.hfile;
020
021import java.util.Iterator;
022
023import org.apache.yetus.audience.InterfaceAudience;
024import org.apache.hadoop.hbase.io.HeapSize;
025import org.apache.hadoop.hbase.io.hfile.BlockType.BlockCategory;
026import org.apache.hadoop.hbase.io.hfile.bucket.BucketCache;
027
028/**
029 * CombinedBlockCache is an abstraction layer that combines
030 * {@link FirstLevelBlockCache} and {@link BucketCache}. The smaller lruCache is used
031 * to cache bloom blocks and index blocks.  The larger Cache is used to
032 * cache data blocks. {@link #getBlock(BlockCacheKey, boolean, boolean, boolean)} reads
033 * first from the smaller l1Cache before looking for the block in the l2Cache.  Blocks evicted
034 * from l1Cache are put into the bucket cache.
035 * Metrics are the combined size and hits and misses of both caches.
036 */
037@InterfaceAudience.Private
038public class CombinedBlockCache implements ResizableBlockCache, HeapSize {
039  protected final FirstLevelBlockCache l1Cache;
040  protected final BlockCache l2Cache;
041  protected final CombinedCacheStats combinedCacheStats;
042
043  public CombinedBlockCache(FirstLevelBlockCache l1Cache, BlockCache l2Cache) {
044    this.l1Cache = l1Cache;
045    this.l2Cache = l2Cache;
046    this.combinedCacheStats = new CombinedCacheStats(l1Cache.getStats(),
047        l2Cache.getStats());
048  }
049
050  @Override
051  public long heapSize() {
052    long l2size = 0;
053    if (l2Cache instanceof HeapSize) {
054      l2size = ((HeapSize) l2Cache).heapSize();
055    }
056    return l1Cache.heapSize() + l2size;
057  }
058
059  @Override
060  public void cacheBlock(BlockCacheKey cacheKey, Cacheable buf, boolean inMemory) {
061    boolean metaBlock = buf.getBlockType().getCategory() != BlockCategory.DATA;
062    if (metaBlock) {
063      l1Cache.cacheBlock(cacheKey, buf, inMemory);
064    } else {
065      l2Cache.cacheBlock(cacheKey, buf, inMemory);
066    }
067  }
068
069  @Override
070  public void cacheBlock(BlockCacheKey cacheKey, Cacheable buf) {
071    cacheBlock(cacheKey, buf, false);
072  }
073
074  @Override
075  public Cacheable getBlock(BlockCacheKey cacheKey, boolean caching,
076      boolean repeat, boolean updateCacheMetrics) {
077    // TODO: is there a hole here, or just awkwardness since in the lruCache getBlock
078    // we end up calling l2Cache.getBlock.
079    // We are not in a position to exactly look at LRU cache or BC as BlockType may not be getting
080    // passed always.
081    return l1Cache.containsBlock(cacheKey)?
082        l1Cache.getBlock(cacheKey, caching, repeat, updateCacheMetrics):
083        l2Cache.getBlock(cacheKey, caching, repeat, updateCacheMetrics);
084  }
085
086  @Override
087  public boolean evictBlock(BlockCacheKey cacheKey) {
088    return l1Cache.evictBlock(cacheKey) || l2Cache.evictBlock(cacheKey);
089  }
090
091  @Override
092  public int evictBlocksByHfileName(String hfileName) {
093    return l1Cache.evictBlocksByHfileName(hfileName)
094        + l2Cache.evictBlocksByHfileName(hfileName);
095  }
096
097  @Override
098  public CacheStats getStats() {
099    return this.combinedCacheStats;
100  }
101
102  @Override
103  public void shutdown() {
104    l1Cache.shutdown();
105    l2Cache.shutdown();
106  }
107
108  @Override
109  public long size() {
110    return l1Cache.size() + l2Cache.size();
111  }
112
113  @Override
114  public long getMaxSize() {
115    return l1Cache.getMaxSize() + l2Cache.getMaxSize();
116  }
117
118  @Override
119  public long getCurrentDataSize() {
120    return l1Cache.getCurrentDataSize() + l2Cache.getCurrentDataSize();
121  }
122
123  @Override
124  public long getFreeSize() {
125    return l1Cache.getFreeSize() + l2Cache.getFreeSize();
126  }
127
128  @Override
129  public long getCurrentSize() {
130    return l1Cache.getCurrentSize() + l2Cache.getCurrentSize();
131  }
132
133  @Override
134  public long getBlockCount() {
135    return l1Cache.getBlockCount() + l2Cache.getBlockCount();
136  }
137
138  @Override
139  public long getDataBlockCount() {
140    return l1Cache.getDataBlockCount() + l2Cache.getDataBlockCount();
141  }
142
143  public static class CombinedCacheStats extends CacheStats {
144    private final CacheStats lruCacheStats;
145    private final CacheStats bucketCacheStats;
146
147    CombinedCacheStats(CacheStats lbcStats, CacheStats fcStats) {
148      super("CombinedBlockCache");
149      this.lruCacheStats = lbcStats;
150      this.bucketCacheStats = fcStats;
151    }
152
153    public CacheStats getLruCacheStats() {
154      return this.lruCacheStats;
155    }
156
157    public CacheStats getBucketCacheStats() {
158      return this.bucketCacheStats;
159    }
160
161    @Override
162    public long getDataMissCount() {
163      return lruCacheStats.getDataMissCount() + bucketCacheStats.getDataMissCount();
164    }
165
166    @Override
167    public long getLeafIndexMissCount() {
168      return lruCacheStats.getLeafIndexMissCount() + bucketCacheStats.getLeafIndexMissCount();
169    }
170
171    @Override
172    public long getBloomChunkMissCount() {
173      return lruCacheStats.getBloomChunkMissCount() + bucketCacheStats.getBloomChunkMissCount();
174    }
175
176    @Override
177    public long getMetaMissCount() {
178      return lruCacheStats.getMetaMissCount() + bucketCacheStats.getMetaMissCount();
179    }
180
181    @Override
182    public long getRootIndexMissCount() {
183      return lruCacheStats.getRootIndexMissCount() + bucketCacheStats.getRootIndexMissCount();
184    }
185
186    @Override
187    public long getIntermediateIndexMissCount() {
188      return lruCacheStats.getIntermediateIndexMissCount() +
189          bucketCacheStats.getIntermediateIndexMissCount();
190    }
191
192    @Override
193    public long getFileInfoMissCount() {
194      return lruCacheStats.getFileInfoMissCount() + bucketCacheStats.getFileInfoMissCount();
195    }
196
197    @Override
198    public long getGeneralBloomMetaMissCount() {
199      return lruCacheStats.getGeneralBloomMetaMissCount() +
200          bucketCacheStats.getGeneralBloomMetaMissCount();
201    }
202
203    @Override
204    public long getDeleteFamilyBloomMissCount() {
205      return lruCacheStats.getDeleteFamilyBloomMissCount() +
206          bucketCacheStats.getDeleteFamilyBloomMissCount();
207    }
208
209    @Override
210    public long getTrailerMissCount() {
211      return lruCacheStats.getTrailerMissCount() + bucketCacheStats.getTrailerMissCount();
212    }
213
214    @Override
215    public long getDataHitCount() {
216      return lruCacheStats.getDataHitCount() + bucketCacheStats.getDataHitCount();
217    }
218
219    @Override
220    public long getLeafIndexHitCount() {
221      return lruCacheStats.getLeafIndexHitCount() + bucketCacheStats.getLeafIndexHitCount();
222    }
223
224    @Override
225    public long getBloomChunkHitCount() {
226      return lruCacheStats.getBloomChunkHitCount() + bucketCacheStats.getBloomChunkHitCount();
227    }
228
229    @Override
230    public long getMetaHitCount() {
231      return lruCacheStats.getMetaHitCount() + bucketCacheStats.getMetaHitCount();
232    }
233
234    @Override
235    public long getRootIndexHitCount() {
236      return lruCacheStats.getRootIndexHitCount() + bucketCacheStats.getRootIndexHitCount();
237    }
238
239    @Override
240    public long getIntermediateIndexHitCount() {
241      return lruCacheStats.getIntermediateIndexHitCount() +
242          bucketCacheStats.getIntermediateIndexHitCount();
243    }
244
245    @Override
246    public long getFileInfoHitCount() {
247      return lruCacheStats.getFileInfoHitCount() + bucketCacheStats.getFileInfoHitCount();
248    }
249
250    @Override
251    public long getGeneralBloomMetaHitCount() {
252      return lruCacheStats.getGeneralBloomMetaHitCount() +
253          bucketCacheStats.getGeneralBloomMetaHitCount();
254    }
255
256    @Override
257    public long getDeleteFamilyBloomHitCount() {
258      return lruCacheStats.getDeleteFamilyBloomHitCount() +
259          bucketCacheStats.getDeleteFamilyBloomHitCount();
260    }
261
262    @Override
263    public long getTrailerHitCount() {
264      return lruCacheStats.getTrailerHitCount() + bucketCacheStats.getTrailerHitCount();
265    }
266
267    @Override
268    public long getRequestCount() {
269      return lruCacheStats.getRequestCount()
270          + bucketCacheStats.getRequestCount();
271    }
272
273    @Override
274    public long getRequestCachingCount() {
275      return lruCacheStats.getRequestCachingCount()
276          + bucketCacheStats.getRequestCachingCount();
277    }
278
279    @Override
280    public long getMissCount() {
281      return lruCacheStats.getMissCount() + bucketCacheStats.getMissCount();
282    }
283
284    @Override
285    public long getPrimaryMissCount() {
286      return lruCacheStats.getPrimaryMissCount() + bucketCacheStats.getPrimaryMissCount();
287    }
288
289    @Override
290    public long getMissCachingCount() {
291      return lruCacheStats.getMissCachingCount()
292          + bucketCacheStats.getMissCachingCount();
293    }
294
295    @Override
296    public long getHitCount() {
297      return lruCacheStats.getHitCount() + bucketCacheStats.getHitCount();
298    }
299
300    @Override
301    public long getPrimaryHitCount() {
302      return lruCacheStats.getPrimaryHitCount() + bucketCacheStats.getPrimaryHitCount();
303    }
304    @Override
305    public long getHitCachingCount() {
306      return lruCacheStats.getHitCachingCount()
307          + bucketCacheStats.getHitCachingCount();
308    }
309
310    @Override
311    public long getEvictionCount() {
312      return lruCacheStats.getEvictionCount()
313          + bucketCacheStats.getEvictionCount();
314    }
315
316    @Override
317    public long getEvictedCount() {
318      return lruCacheStats.getEvictedCount()
319          + bucketCacheStats.getEvictedCount();
320    }
321
322    @Override
323    public long getPrimaryEvictedCount() {
324      return lruCacheStats.getPrimaryEvictedCount()
325          + bucketCacheStats.getPrimaryEvictedCount();
326    }
327
328    @Override
329    public void rollMetricsPeriod() {
330      lruCacheStats.rollMetricsPeriod();
331      bucketCacheStats.rollMetricsPeriod();
332    }
333
334    @Override
335    public long getFailedInserts() {
336      return lruCacheStats.getFailedInserts() + bucketCacheStats.getFailedInserts();
337    }
338
339    @Override
340    public long getSumHitCountsPastNPeriods() {
341      return lruCacheStats.getSumHitCountsPastNPeriods()
342          + bucketCacheStats.getSumHitCountsPastNPeriods();
343    }
344
345    @Override
346    public long getSumRequestCountsPastNPeriods() {
347      return lruCacheStats.getSumRequestCountsPastNPeriods()
348          + bucketCacheStats.getSumRequestCountsPastNPeriods();
349    }
350
351    @Override
352    public long getSumHitCachingCountsPastNPeriods() {
353      return lruCacheStats.getSumHitCachingCountsPastNPeriods()
354          + bucketCacheStats.getSumHitCachingCountsPastNPeriods();
355    }
356
357    @Override
358    public long getSumRequestCachingCountsPastNPeriods() {
359      return lruCacheStats.getSumRequestCachingCountsPastNPeriods()
360          + bucketCacheStats.getSumRequestCachingCountsPastNPeriods();
361    }
362  }
363
364  @Override
365  public Iterator<CachedBlock> iterator() {
366    return new BlockCachesIterator(getBlockCaches());
367  }
368
369  @Override
370  public BlockCache[] getBlockCaches() {
371    return new BlockCache [] {this.l1Cache, this.l2Cache};
372  }
373
374  @Override
375  public void setMaxSize(long size) {
376    this.l1Cache.setMaxSize(size);
377  }
378
379  public int getRpcRefCount(BlockCacheKey cacheKey) {
380    return (this.l2Cache instanceof BucketCache)
381        ? ((BucketCache) this.l2Cache).getRpcRefCount(cacheKey)
382        : 0;
383  }
384
385  public FirstLevelBlockCache getFirstLevelCache() {
386    return l1Cache;
387  }
388}