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.regionserver;
019
020import java.io.IOException;
021import java.util.ArrayList;
022import java.util.Collection;
023import java.util.Collections;
024import java.util.List;
025import java.util.Map;
026import java.util.OptionalDouble;
027import java.util.concurrent.ConcurrentHashMap;
028import java.util.concurrent.ScheduledExecutorService;
029import java.util.concurrent.TimeUnit;
030import java.util.stream.Collectors;
031import org.apache.commons.lang3.StringUtils;
032import org.apache.hadoop.hbase.CompatibilitySingletonFactory;
033import org.apache.hadoop.hbase.HConstants;
034import org.apache.hadoop.hbase.HDFSBlocksDistribution;
035import org.apache.hadoop.hbase.HRegionInfo;
036import org.apache.hadoop.hbase.ServerName;
037import org.apache.hadoop.hbase.io.ByteBuffAllocator;
038import org.apache.hadoop.hbase.io.FSDataInputStreamWrapper;
039import org.apache.hadoop.hbase.io.asyncfs.monitor.ExcludeDatanodeManager;
040import org.apache.hadoop.hbase.io.hfile.BlockCache;
041import org.apache.hadoop.hbase.io.hfile.CacheStats;
042import org.apache.hadoop.hbase.io.hfile.CombinedBlockCache;
043import org.apache.hadoop.hbase.mob.MobFileCache;
044import org.apache.hadoop.hbase.regionserver.wal.MetricsWALSource;
045import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
046import org.apache.hadoop.hbase.util.FSUtils;
047import org.apache.hadoop.hbase.wal.WALProvider;
048import org.apache.hadoop.hbase.zookeeper.ZKWatcher;
049import org.apache.hadoop.hdfs.DFSHedgedReadMetrics;
050import org.apache.hadoop.metrics2.MetricsExecutor;
051import org.apache.yetus.audience.InterfaceAudience;
052import org.slf4j.Logger;
053import org.slf4j.LoggerFactory;
054
055/**
056 * Impl for exposing HRegionServer Information through Hadoop's metrics 2 system.
057 */
058@InterfaceAudience.Private
059class MetricsRegionServerWrapperImpl implements MetricsRegionServerWrapper {
060
061  private static final Logger LOG = LoggerFactory.getLogger(MetricsRegionServerWrapperImpl.class);
062
063  private final HRegionServer regionServer;
064  private final MetricsWALSource metricsWALSource;
065  private final ByteBuffAllocator allocator;
066
067  private BlockCache blockCache;
068  private BlockCache l1Cache = null;
069  private BlockCache l2Cache = null;
070  private MobFileCache mobFileCache;
071  private CacheStats cacheStats;
072  private CacheStats l1Stats = null;
073  private CacheStats l2Stats = null;
074  private volatile long numWALFiles = 0;
075  private volatile long walFileSize = 0;
076  private volatile long mobFileCacheAccessCount = 0;
077  private volatile long mobFileCacheMissCount = 0;
078  private volatile double mobFileCacheHitRatio = 0;
079  private volatile long mobFileCacheEvictedCount = 0;
080  private volatile long mobFileCacheCount = 0;
081
082  private volatile RegionMetricAggregate aggregate = new RegionMetricAggregate(null);
083
084  protected final Map<String, ArrayList<Long>> requestsCountCache =
085    new ConcurrentHashMap<String, ArrayList<Long>>();
086
087  private ScheduledExecutorService executor;
088  private Runnable runnable;
089  private long period;
090
091  /**
092   * Can be null if not on hdfs.
093   */
094  private DFSHedgedReadMetrics dfsHedgedReadMetrics;
095
096  private final ExcludeDatanodeManager excludeDatanodeManager;
097
098  public MetricsRegionServerWrapperImpl(final HRegionServer regionServer) {
099    this.regionServer = regionServer;
100    initBlockCache();
101    initMobFileCache();
102    this.excludeDatanodeManager = this.regionServer.getWalFactory().getExcludeDatanodeManager();
103
104    this.period = regionServer.getConfiguration().getLong(HConstants.REGIONSERVER_METRICS_PERIOD,
105      HConstants.DEFAULT_REGIONSERVER_METRICS_PERIOD);
106
107    this.executor = CompatibilitySingletonFactory.getInstance(MetricsExecutor.class).getExecutor();
108    this.runnable = new RegionServerMetricsWrapperRunnable();
109    this.executor.scheduleWithFixedDelay(this.runnable, this.period, this.period,
110      TimeUnit.MILLISECONDS);
111    this.metricsWALSource = CompatibilitySingletonFactory.getInstance(MetricsWALSource.class);
112    this.allocator = regionServer.getRpcServer().getByteBuffAllocator();
113
114    try {
115      this.dfsHedgedReadMetrics = FSUtils.getDFSHedgedReadMetrics(regionServer.getConfiguration());
116    } catch (IOException e) {
117      LOG.warn("Failed to get hedged metrics", e);
118    }
119    if (LOG.isInfoEnabled()) {
120      LOG.info("Computing regionserver metrics every " + this.period + " milliseconds");
121    }
122  }
123
124  private void initBlockCache() {
125    this.blockCache = this.regionServer.getBlockCache().orElse(null);
126    this.cacheStats = this.blockCache != null ? this.blockCache.getStats() : null;
127    if (this.cacheStats != null) {
128      if (this.cacheStats instanceof CombinedBlockCache.CombinedCacheStats) {
129        l1Stats = ((CombinedBlockCache.CombinedCacheStats) this.cacheStats).getLruCacheStats();
130        l2Stats = ((CombinedBlockCache.CombinedCacheStats) this.cacheStats).getBucketCacheStats();
131      } else {
132        l1Stats = this.cacheStats;
133      }
134    }
135    if (this.blockCache != null) {
136      if (this.blockCache instanceof CombinedBlockCache) {
137        l1Cache = ((CombinedBlockCache) this.blockCache).getFirstLevelCache();
138        l2Cache = ((CombinedBlockCache) this.blockCache).getSecondLevelCache();
139      } else {
140        l1Cache = this.blockCache;
141      }
142    }
143  }
144
145  /**
146   * Initializes the mob file cache.
147   */
148  private void initMobFileCache() {
149    this.mobFileCache = this.regionServer.getMobFileCache().orElse(null);
150  }
151
152  @Override
153  public String getClusterId() {
154    return regionServer.getClusterId();
155  }
156
157  @Override
158  public long getStartCode() {
159    return regionServer.getStartcode();
160  }
161
162  @Override
163  public String getZookeeperQuorum() {
164    ZKWatcher zk = regionServer.getZooKeeper();
165    if (zk == null) {
166      return "";
167    }
168    return zk.getQuorum();
169  }
170
171  @Override
172  public String getCoprocessors() {
173    String[] coprocessors = regionServer.getRegionServerCoprocessors();
174    if (coprocessors == null || coprocessors.length == 0) {
175      return "";
176    }
177    return StringUtils.join(coprocessors, ", ");
178  }
179
180  @Override
181  public String getServerName() {
182    ServerName serverName = regionServer.getServerName();
183    if (serverName == null) {
184      return "";
185    }
186    return serverName.getServerName();
187  }
188
189  @Override
190  public long getNumOnlineRegions() {
191    Collection<HRegion> onlineRegionsLocalContext = regionServer.getOnlineRegionsLocalContext();
192    if (onlineRegionsLocalContext == null) {
193      return 0;
194    }
195    return onlineRegionsLocalContext.size();
196  }
197
198  @Override
199  public long getTotalRequestCount() {
200    return regionServer.rpcServices.requestCount.sum();
201  }
202
203  @Override
204  public long getTotalRowActionRequestCount() {
205    return aggregate.readRequestsCount + aggregate.writeRequestsCount;
206  }
207
208  @Override
209  public int getSplitQueueSize() {
210    final CompactSplit compactSplit = regionServer.getCompactSplitThread();
211    return compactSplit == null ? 0 : compactSplit.getSplitQueueSize();
212  }
213
214  @Override
215  public int getCompactionQueueSize() {
216    final CompactSplit compactSplit = regionServer.getCompactSplitThread();
217    return compactSplit == null ? 0 : compactSplit.getCompactionQueueSize();
218  }
219
220  @Override
221  public int getSmallCompactionQueueSize() {
222    final CompactSplit compactSplit = regionServer.getCompactSplitThread();
223    return compactSplit == null ? 0 : compactSplit.getSmallCompactionQueueSize();
224  }
225
226  @Override
227  public int getLargeCompactionQueueSize() {
228    final CompactSplit compactSplit = regionServer.getCompactSplitThread();
229    return compactSplit == null ? 0 : compactSplit.getLargeCompactionQueueSize();
230  }
231
232  @Override
233  public int getFlushQueueSize() {
234    // If there is no flusher there should be no queue.
235    if (this.regionServer.getMemStoreFlusher() == null) {
236      return 0;
237    }
238    return this.regionServer.getMemStoreFlusher().getFlushQueueSize();
239  }
240
241  @Override
242  public long getBlockCacheCount() {
243    return this.blockCache != null ? this.blockCache.getBlockCount() : 0L;
244  }
245
246  @Override
247  public long getBlockCacheDataBlockCount() {
248    return this.blockCache != null ? this.blockCache.getDataBlockCount() : 0L;
249  }
250
251  @Override
252  public long getMemStoreLimit() {
253    return this.regionServer.getRegionServerAccounting().getGlobalMemStoreLimit();
254  }
255
256  @Override
257  public long getOnHeapMemStoreLimit() {
258    return this.regionServer.getRegionServerAccounting().getGlobalOnHeapMemStoreLimit();
259  }
260
261  @Override
262  public long getOffHeapMemStoreLimit() {
263    return this.regionServer.getRegionServerAccounting().getGlobalOffHeapMemStoreLimit();
264  }
265
266  @Override
267  public long getBlockCacheSize() {
268    return this.blockCache != null ? this.blockCache.getCurrentSize() : 0L;
269  }
270
271  @Override
272  public long getBlockCacheFreeSize() {
273    return this.blockCache != null ? this.blockCache.getFreeSize() : 0L;
274  }
275
276  @Override
277  public long getBlockCacheHitCount() {
278    return this.cacheStats != null ? this.cacheStats.getHitCount() : 0L;
279  }
280
281  @Override
282  public long getBlockCachePrimaryHitCount() {
283    return this.cacheStats != null ? this.cacheStats.getPrimaryHitCount() : 0L;
284  }
285
286  @Override
287  public long getBlockCacheHitCachingCount() {
288    return this.cacheStats != null ? this.cacheStats.getHitCachingCount() : 0L;
289  }
290
291  @Override
292  public long getBlockCacheMissCount() {
293    return this.cacheStats != null ? this.cacheStats.getMissCount() : 0L;
294  }
295
296  @Override
297  public long getBlockCachePrimaryMissCount() {
298    return this.cacheStats != null ? this.cacheStats.getPrimaryMissCount() : 0L;
299  }
300
301  @Override
302  public long getBlockCacheMissCachingCount() {
303    return this.cacheStats != null ? this.cacheStats.getMissCachingCount() : 0L;
304  }
305
306  @Override
307  public long getBlockCacheEvictedCount() {
308    return this.cacheStats != null ? this.cacheStats.getEvictedCount() : 0L;
309  }
310
311  @Override
312  public long getBlockCachePrimaryEvictedCount() {
313    return this.cacheStats != null ? this.cacheStats.getPrimaryEvictedCount() : 0L;
314  }
315
316  @Override
317  public double getBlockCacheHitPercent() {
318    double ratio = this.cacheStats != null ? this.cacheStats.getHitRatio() : 0.0;
319    if (Double.isNaN(ratio)) {
320      ratio = 0;
321    }
322    return (ratio * 100);
323  }
324
325  @Override
326  public double getBlockCacheHitCachingPercent() {
327    double ratio = this.cacheStats != null ? this.cacheStats.getHitCachingRatio() : 0.0;
328    if (Double.isNaN(ratio)) {
329      ratio = 0;
330    }
331    return (ratio * 100);
332  }
333
334  @Override
335  public long getBlockCacheFailedInsertions() {
336    return this.cacheStats != null ? this.cacheStats.getFailedInserts() : 0L;
337  }
338
339  public long getL1CacheSize() {
340    return this.l1Cache != null ? this.l1Cache.getCurrentSize() : 0L;
341  }
342
343  public long getL1CacheFreeSize() {
344    return this.l1Cache != null ? this.l1Cache.getFreeSize() : 0L;
345  }
346
347  public long getL1CacheCount() {
348    return this.l1Cache != null ? this.l1Cache.getBlockCount() : 0L;
349  }
350
351  public long getL1CacheEvictedCount() {
352    return this.l1Stats != null ? this.l1Stats.getEvictedCount() : 0L;
353  }
354
355  public long getL2CacheSize() {
356    return this.l2Cache != null ? this.l2Cache.getCurrentSize() : 0L;
357  }
358
359  public long getL2CacheFreeSize() {
360    return this.l2Cache != null ? this.l2Cache.getFreeSize() : 0L;
361  }
362
363  public long getL2CacheCount() {
364    return this.l2Cache != null ? this.l2Cache.getBlockCount() : 0L;
365  }
366
367  public long getL2CacheEvictedCount() {
368    return this.l2Stats != null ? this.l2Stats.getEvictedCount() : 0L;
369  }
370
371  @Override
372  public long getL1CacheHitCount() {
373    return this.l1Stats != null ? this.l1Stats.getHitCount() : 0L;
374  }
375
376  @Override
377  public long getL1CacheMissCount() {
378    return this.l1Stats != null ? this.l1Stats.getMissCount() : 0L;
379  }
380
381  @Override
382  public double getL1CacheHitRatio() {
383    return this.l1Stats != null ? this.l1Stats.getHitRatio() : 0.0;
384  }
385
386  @Override
387  public double getL1CacheMissRatio() {
388    return this.l1Stats != null ? this.l1Stats.getMissRatio() : 0.0;
389  }
390
391  @Override
392  public long getL2CacheHitCount() {
393    return this.l2Stats != null ? this.l2Stats.getHitCount() : 0L;
394  }
395
396  @Override
397  public long getL2CacheMissCount() {
398    return this.l2Stats != null ? this.l2Stats.getMissCount() : 0L;
399  }
400
401  @Override
402  public double getL2CacheHitRatio() {
403    return this.l2Stats != null ? this.l2Stats.getHitRatio() : 0.0;
404  }
405
406  @Override
407  public double getL2CacheMissRatio() {
408    return this.l2Stats != null ? this.l2Stats.getMissRatio() : 0.0;
409  }
410
411  @Override
412  public void forceRecompute() {
413    this.runnable.run();
414  }
415
416  @Override
417  public long getNumStores() {
418    return aggregate.numStores;
419  }
420
421  @Override
422  public long getNumWALFiles() {
423    return numWALFiles;
424  }
425
426  @Override
427  public long getWALFileSize() {
428    return walFileSize;
429  }
430
431  @Override
432  public List<String> getWALExcludeDNs() {
433    if (excludeDatanodeManager == null) {
434      return Collections.emptyList();
435    }
436    return excludeDatanodeManager.getExcludeDNs().entrySet().stream()
437      .map(e -> e.getKey().toString() + ", " + e.getValue()).collect(Collectors.toList());
438  }
439
440  @Override
441  public long getNumWALSlowAppend() {
442    return metricsWALSource.getSlowAppendCount();
443  }
444
445  @Override
446  public long getNumStoreFiles() {
447    return aggregate.numStoreFiles;
448  }
449
450  @Override
451  public long getMaxStoreFiles() {
452    return aggregate.maxStoreFileCount;
453  }
454
455  @Override
456  public long getMaxStoreFileAge() {
457    return aggregate.maxStoreFileAge;
458  }
459
460  @Override
461  public long getMinStoreFileAge() {
462    return aggregate.minStoreFileAge;
463  }
464
465  @Override
466  public long getAvgStoreFileAge() {
467    return aggregate.avgStoreFileAge;
468  }
469
470  @Override
471  public long getNumReferenceFiles() {
472    return aggregate.numReferenceFiles;
473  }
474
475  @Override
476  public long getMemStoreSize() {
477    return aggregate.memstoreSize;
478  }
479
480  @Override
481  public long getOnHeapMemStoreSize() {
482    return aggregate.onHeapMemstoreSize;
483  }
484
485  @Override
486  public long getOffHeapMemStoreSize() {
487    return aggregate.offHeapMemstoreSize;
488  }
489
490  @Override
491  public long getStoreFileSize() {
492    return aggregate.storeFileSize;
493  }
494
495  @Override
496  public double getRequestsPerSecond() {
497    return aggregate.requestsPerSecond;
498  }
499
500  @Override
501  public long getReadRequestsCount() {
502    return aggregate.readRequestsCount;
503  }
504
505  @Override
506  public double getReadRequestsRatePerSecond() {
507    return aggregate.readRequestsRatePerSecond;
508  }
509
510  @Override
511  public long getFilteredReadRequestsCount() {
512    return aggregate.filteredReadRequestsCount;
513  }
514
515  @Override
516  public long getWriteRequestsCount() {
517    return aggregate.writeRequestsCount;
518  }
519
520  @Override
521  public double getWriteRequestsRatePerSecond() {
522    return aggregate.writeRequestsRatePerSecond;
523  }
524
525  @Override
526  public long getRpcGetRequestsCount() {
527    return regionServer.rpcServices.rpcGetRequestCount.sum();
528  }
529
530  @Override
531  public long getRpcScanRequestsCount() {
532    return regionServer.rpcServices.rpcScanRequestCount.sum();
533  }
534
535  @Override
536  public long getRpcFullScanRequestsCount() {
537    return regionServer.rpcServices.rpcFullScanRequestCount.sum();
538  }
539
540  @Override
541  public long getRpcMultiRequestsCount() {
542    return regionServer.rpcServices.rpcMultiRequestCount.sum();
543  }
544
545  @Override
546  public long getRpcMutateRequestsCount() {
547    return regionServer.rpcServices.rpcMutateRequestCount.sum();
548  }
549
550  @Override
551  public long getCheckAndMutateChecksFailed() {
552    return aggregate.checkAndMutateChecksFailed;
553  }
554
555  @Override
556  public long getCheckAndMutateChecksPassed() {
557    return aggregate.checkAndMutateChecksPassed;
558  }
559
560  @Override
561  public long getStoreFileIndexSize() {
562    return aggregate.storefileIndexSize;
563  }
564
565  @Override
566  public long getTotalStaticIndexSize() {
567    return aggregate.totalStaticIndexSize;
568  }
569
570  @Override
571  public long getTotalStaticBloomSize() {
572    return aggregate.totalStaticBloomSize;
573  }
574
575  @Override
576  public long getBloomFilterRequestsCount() {
577    return aggregate.bloomFilterRequestsCount;
578  }
579
580  @Override
581  public long getBloomFilterNegativeResultsCount() {
582    return aggregate.bloomFilterNegativeResultsCount;
583  }
584
585  @Override
586  public long getBloomFilterEligibleRequestsCount() {
587    return aggregate.bloomFilterEligibleRequestsCount;
588  }
589
590  @Override
591  public long getNumMutationsWithoutWAL() {
592    return aggregate.numMutationsWithoutWAL;
593  }
594
595  @Override
596  public long getDataInMemoryWithoutWAL() {
597    return aggregate.dataInMemoryWithoutWAL;
598  }
599
600  @Override
601  public double getPercentFileLocal() {
602    return aggregate.percentFileLocal;
603  }
604
605  @Override
606  public double getPercentFileLocalSecondaryRegions() {
607    return aggregate.percentFileLocalSecondaryRegions;
608  }
609
610  @Override
611  public long getUpdatesBlockedTime() {
612    if (this.regionServer.getMemStoreFlusher() == null) {
613      return 0;
614    }
615    return this.regionServer.getMemStoreFlusher().getUpdatesBlockedMsHighWater().sum();
616  }
617
618  @Override
619  public long getFlushedCellsCount() {
620    return aggregate.flushedCellsCount;
621  }
622
623  @Override
624  public long getCompactedCellsCount() {
625    return aggregate.compactedCellsCount;
626  }
627
628  @Override
629  public long getMajorCompactedCellsCount() {
630    return aggregate.majorCompactedCellsCount;
631  }
632
633  @Override
634  public long getFlushedCellsSize() {
635    return aggregate.flushedCellsSize;
636  }
637
638  @Override
639  public long getCompactedCellsSize() {
640    return aggregate.compactedCellsSize;
641  }
642
643  @Override
644  public long getMajorCompactedCellsSize() {
645    return aggregate.majorCompactedCellsSize;
646  }
647
648  @Override
649  public long getCellsCountCompactedFromMob() {
650    return aggregate.cellsCountCompactedFromMob;
651  }
652
653  @Override
654  public long getCellsCountCompactedToMob() {
655    return aggregate.cellsCountCompactedToMob;
656  }
657
658  @Override
659  public long getCellsSizeCompactedFromMob() {
660    return aggregate.cellsSizeCompactedFromMob;
661  }
662
663  @Override
664  public long getCellsSizeCompactedToMob() {
665    return aggregate.cellsSizeCompactedToMob;
666  }
667
668  @Override
669  public long getMobFlushCount() {
670    return aggregate.mobFlushCount;
671  }
672
673  @Override
674  public long getMobFlushedCellsCount() {
675    return aggregate.mobFlushedCellsCount;
676  }
677
678  @Override
679  public long getMobFlushedCellsSize() {
680    return aggregate.mobFlushedCellsSize;
681  }
682
683  @Override
684  public long getMobScanCellsCount() {
685    return aggregate.mobScanCellsCount;
686  }
687
688  @Override
689  public long getMobScanCellsSize() {
690    return aggregate.mobScanCellsSize;
691  }
692
693  @Override
694  public long getMobFileCacheAccessCount() {
695    return mobFileCacheAccessCount;
696  }
697
698  @Override
699  public long getMobFileCacheMissCount() {
700    return mobFileCacheMissCount;
701  }
702
703  @Override
704  public long getMobFileCacheCount() {
705    return mobFileCacheCount;
706  }
707
708  @Override
709  public long getMobFileCacheEvictedCount() {
710    return mobFileCacheEvictedCount;
711  }
712
713  @Override
714  public double getMobFileCacheHitPercent() {
715    return mobFileCacheHitRatio * 100;
716  }
717
718  @Override
719  public int getActiveScanners() {
720    return regionServer.getRSRpcServices().getScannersCount();
721  }
722
723  private static final class RegionMetricAggregate {
724    private long numStores = 0;
725    private long numStoreFiles = 0;
726    private long memstoreSize = 0;
727    private long onHeapMemstoreSize = 0;
728    private long offHeapMemstoreSize = 0;
729    private long storeFileSize = 0;
730    private long maxStoreFileCount = 0;
731    private long maxStoreFileAge = 0;
732    private long minStoreFileAge = Long.MAX_VALUE;
733    private long avgStoreFileAge = 0;
734    private long numReferenceFiles = 0;
735
736    private double requestsPerSecond = 0.0;
737    private long readRequestsCount = 0;
738    private double readRequestsRatePerSecond = 0;
739    private long filteredReadRequestsCount = 0;
740    private long writeRequestsCount = 0;
741    private double writeRequestsRatePerSecond = 0;
742    private long checkAndMutateChecksFailed = 0;
743    private long checkAndMutateChecksPassed = 0;
744    private long storefileIndexSize = 0;
745    private long totalStaticIndexSize = 0;
746    private long totalStaticBloomSize = 0;
747    private long bloomFilterRequestsCount = 0;
748    private long bloomFilterNegativeResultsCount = 0;
749    private long bloomFilterEligibleRequestsCount = 0;
750    private long numMutationsWithoutWAL = 0;
751    private long dataInMemoryWithoutWAL = 0;
752    private double percentFileLocal = 0;
753    private double percentFileLocalSecondaryRegions = 0;
754    private long flushedCellsCount = 0;
755    private long compactedCellsCount = 0;
756    private long majorCompactedCellsCount = 0;
757    private long flushedCellsSize = 0;
758    private long compactedCellsSize = 0;
759    private long majorCompactedCellsSize = 0;
760    private long cellsCountCompactedToMob = 0;
761    private long cellsCountCompactedFromMob = 0;
762    private long cellsSizeCompactedToMob = 0;
763    private long cellsSizeCompactedFromMob = 0;
764    private long mobFlushCount = 0;
765    private long mobFlushedCellsCount = 0;
766    private long mobFlushedCellsSize = 0;
767    private long mobScanCellsCount = 0;
768    private long mobScanCellsSize = 0;
769    private long blockedRequestsCount = 0L;
770    private long averageRegionSize = 0L;
771    private long totalReadRequestsDelta = 0;
772    private long totalWriteRequestsDelta = 0;
773
774    private RegionMetricAggregate(RegionMetricAggregate other) {
775      if (other != null) {
776        requestsPerSecond = other.requestsPerSecond;
777        readRequestsRatePerSecond = other.readRequestsRatePerSecond;
778        writeRequestsRatePerSecond = other.writeRequestsRatePerSecond;
779      }
780    }
781
782    private void aggregate(HRegionServer regionServer,
783      Map<String, ArrayList<Long>> requestsCountCache) {
784      HDFSBlocksDistribution hdfsBlocksDistribution = new HDFSBlocksDistribution();
785      HDFSBlocksDistribution hdfsBlocksDistributionSecondaryRegions = new HDFSBlocksDistribution();
786
787      long avgAgeNumerator = 0;
788      long numHFiles = 0;
789      int regionCount = 0;
790
791      for (HRegion r : regionServer.getOnlineRegionsLocalContext()) {
792        Deltas deltas = calculateReadWriteDeltas(r, requestsCountCache);
793        totalReadRequestsDelta += deltas.readRequestsCountDelta;
794        totalWriteRequestsDelta += deltas.writeRequestsCountDelta;
795
796        numMutationsWithoutWAL += r.getNumMutationsWithoutWAL();
797        dataInMemoryWithoutWAL += r.getDataInMemoryWithoutWAL();
798        readRequestsCount += r.getReadRequestsCount();
799        filteredReadRequestsCount += r.getFilteredReadRequestsCount();
800        writeRequestsCount += r.getWriteRequestsCount();
801        checkAndMutateChecksFailed += r.getCheckAndMutateChecksFailed();
802        checkAndMutateChecksPassed += r.getCheckAndMutateChecksPassed();
803        blockedRequestsCount += r.getBlockedRequestsCount();
804
805        StoreFileStats storeFileStats = aggregateStores(r.getStores());
806        numHFiles += storeFileStats.numHFiles;
807        avgAgeNumerator += storeFileStats.avgAgeNumerator;
808
809        HDFSBlocksDistribution distro = r.getHDFSBlocksDistribution();
810        hdfsBlocksDistribution.add(distro);
811        if (r.getRegionInfo().getReplicaId() != HRegionInfo.DEFAULT_REPLICA_ID) {
812          hdfsBlocksDistributionSecondaryRegions.add(distro);
813        }
814
815        regionCount++;
816      }
817
818      float localityIndex =
819        hdfsBlocksDistribution.getBlockLocalityIndex(regionServer.getServerName().getHostname());
820      percentFileLocal = Double.isNaN(localityIndex) ? 0 : (localityIndex * 100);
821
822      float localityIndexSecondaryRegions = hdfsBlocksDistributionSecondaryRegions
823        .getBlockLocalityIndex(regionServer.getServerName().getHostname());
824      percentFileLocalSecondaryRegions =
825        Double.isNaN(localityIndexSecondaryRegions) ? 0 : (localityIndexSecondaryRegions * 100);
826
827      if (regionCount > 0) {
828        averageRegionSize = (memstoreSize + storeFileSize) / regionCount;
829      }
830
831      // if there were no store files, we'll never have updated this with Math.min
832      // so set it to 0, which is a better value to display in case of no storefiles
833      if (minStoreFileAge == Long.MAX_VALUE) {
834        this.minStoreFileAge = 0;
835      }
836
837      if (numHFiles != 0) {
838        avgStoreFileAge = avgAgeNumerator / numHFiles;
839      }
840    }
841
842    private static final class Deltas {
843      private final long readRequestsCountDelta;
844      private final long writeRequestsCountDelta;
845
846      private Deltas(long readRequestsCountDelta, long writeRequestsCountDelta) {
847        this.readRequestsCountDelta = readRequestsCountDelta;
848        this.writeRequestsCountDelta = writeRequestsCountDelta;
849      }
850    }
851
852    private Deltas calculateReadWriteDeltas(HRegion r,
853      Map<String, ArrayList<Long>> requestsCountCache) {
854      String encodedRegionName = r.getRegionInfo().getEncodedName();
855      long currentReadRequestsCount = r.getReadRequestsCount();
856      long currentWriteRequestsCount = r.getWriteRequestsCount();
857      if (requestsCountCache.containsKey(encodedRegionName)) {
858        long lastReadRequestsCount = requestsCountCache.get(encodedRegionName).get(0);
859        long lastWriteRequestsCount = requestsCountCache.get(encodedRegionName).get(1);
860
861        // Update cache for our next comparison
862        requestsCountCache.get(encodedRegionName).set(0, currentReadRequestsCount);
863        requestsCountCache.get(encodedRegionName).set(1, currentWriteRequestsCount);
864
865        long readRequestsDelta = currentReadRequestsCount - lastReadRequestsCount;
866        long writeRequestsDelta = currentWriteRequestsCount - lastWriteRequestsCount;
867        return new Deltas(readRequestsDelta, writeRequestsDelta);
868      } else {
869        // List[0] -> readRequestCount
870        // List[1] -> writeRequestCount
871        ArrayList<Long> requests = new ArrayList<Long>(2);
872        requests.add(currentReadRequestsCount);
873        requests.add(currentWriteRequestsCount);
874        requestsCountCache.put(encodedRegionName, requests);
875        return new Deltas(currentReadRequestsCount, currentWriteRequestsCount);
876      }
877    }
878
879    public void updateRates(long timeSinceLastRun, long expectedPeriod) {
880      requestsPerSecond =
881        (totalReadRequestsDelta + totalWriteRequestsDelta) / (timeSinceLastRun / 1000.0);
882
883      double readRequestsRatePerMilliSecond = (double) totalReadRequestsDelta / expectedPeriod;
884      double writeRequestsRatePerMilliSecond = (double) totalWriteRequestsDelta / expectedPeriod;
885
886      readRequestsRatePerSecond = readRequestsRatePerMilliSecond * 1000.0;
887      writeRequestsRatePerSecond = writeRequestsRatePerMilliSecond * 1000.0;
888    }
889
890    private static final class StoreFileStats {
891      private final long numHFiles;
892      private final long avgAgeNumerator;
893
894      private StoreFileStats(long numHFiles, long avgAgeNumerator) {
895        this.numHFiles = numHFiles;
896        this.avgAgeNumerator = avgAgeNumerator;
897      }
898    }
899
900    private StoreFileStats aggregateStores(List<HStore> stores) {
901      numStores += stores.size();
902      long numHFiles = 0;
903      long avgAgeNumerator = 0;
904      for (Store store : stores) {
905        numStoreFiles += store.getStorefilesCount();
906        memstoreSize += store.getMemStoreSize().getDataSize();
907        onHeapMemstoreSize += store.getMemStoreSize().getHeapSize();
908        offHeapMemstoreSize += store.getMemStoreSize().getOffHeapSize();
909        storeFileSize += store.getStorefilesSize();
910        maxStoreFileCount = Math.max(maxStoreFileCount, store.getStorefilesCount());
911
912        maxStoreFileAge =
913          Math.max(store.getMaxStoreFileAge().orElse(maxStoreFileAge), maxStoreFileAge);
914        minStoreFileAge =
915          Math.min(store.getMinStoreFileAge().orElse(minStoreFileAge), minStoreFileAge);
916
917        long storeHFiles = store.getNumHFiles();
918        numHFiles += storeHFiles;
919        numReferenceFiles += store.getNumReferenceFiles();
920
921        OptionalDouble storeAvgStoreFileAge = store.getAvgStoreFileAge();
922        if (storeAvgStoreFileAge.isPresent()) {
923          avgAgeNumerator =
924            (long) (avgAgeNumerator + storeAvgStoreFileAge.getAsDouble() * storeHFiles);
925        }
926
927        storefileIndexSize += store.getStorefilesRootLevelIndexSize();
928        totalStaticBloomSize += store.getTotalStaticBloomSize();
929        totalStaticIndexSize += store.getTotalStaticIndexSize();
930        bloomFilterRequestsCount += store.getBloomFilterRequestsCount();
931        bloomFilterNegativeResultsCount += store.getBloomFilterNegativeResultsCount();
932        bloomFilterEligibleRequestsCount += store.getBloomFilterEligibleRequestsCount();
933        flushedCellsCount += store.getFlushedCellsCount();
934        compactedCellsCount += store.getCompactedCellsCount();
935        majorCompactedCellsCount += store.getMajorCompactedCellsCount();
936        flushedCellsSize += store.getFlushedCellsSize();
937        compactedCellsSize += store.getCompactedCellsSize();
938        majorCompactedCellsSize += store.getMajorCompactedCellsSize();
939        if (store instanceof HMobStore) {
940          HMobStore mobStore = (HMobStore) store;
941          cellsCountCompactedToMob += mobStore.getCellsCountCompactedToMob();
942          cellsCountCompactedFromMob += mobStore.getCellsCountCompactedFromMob();
943          cellsSizeCompactedToMob += mobStore.getCellsSizeCompactedToMob();
944          cellsSizeCompactedFromMob += mobStore.getCellsSizeCompactedFromMob();
945          mobFlushCount += mobStore.getMobFlushCount();
946          mobFlushedCellsCount += mobStore.getMobFlushedCellsCount();
947          mobFlushedCellsSize += mobStore.getMobFlushedCellsSize();
948          mobScanCellsCount += mobStore.getMobScanCellsCount();
949          mobScanCellsSize += mobStore.getMobScanCellsSize();
950        }
951      }
952
953      return new StoreFileStats(numHFiles, avgAgeNumerator);
954    }
955
956  }
957
958  /**
959   * This is the runnable that will be executed on the executor every PERIOD number of seconds It
960   * will take metrics/numbers from all of the regions and use them to compute point in time
961   * metrics.
962   */
963  public class RegionServerMetricsWrapperRunnable implements Runnable {
964
965    private long lastRan = 0;
966
967    @Override
968    synchronized public void run() {
969      try {
970        RegionMetricAggregate newVal = new RegionMetricAggregate(aggregate);
971        newVal.aggregate(regionServer, requestsCountCache);
972
973        // Compute the number of requests per second
974        long currentTime = EnvironmentEdgeManager.currentTime();
975
976        // assume that it took PERIOD seconds to start the executor.
977        // this is a guess but it's a pretty good one.
978        if (lastRan == 0) {
979          lastRan = currentTime - period;
980        }
981
982        long timeSinceLastRun = currentTime - lastRan;
983        // If we've time traveled keep the last requests per second.
984        if (timeSinceLastRun > 0) {
985          newVal.updateRates(timeSinceLastRun, period);
986        }
987
988        aggregate = newVal;
989
990        final WALProvider provider = regionServer.getWalFactory().getWALProvider();
991        final WALProvider metaProvider = regionServer.getWalFactory().getMetaWALProvider();
992        numWALFiles = (provider == null ? 0 : provider.getNumLogFiles())
993          + (metaProvider == null ? 0 : metaProvider.getNumLogFiles());
994        walFileSize = (provider == null ? 0 : provider.getLogFileSize())
995          + (metaProvider == null ? 0 : metaProvider.getLogFileSize());
996
997        mobFileCacheAccessCount = mobFileCache != null ? mobFileCache.getAccessCount() : 0L;
998        mobFileCacheMissCount = mobFileCache != null ? mobFileCache.getMissCount() : 0L;
999        mobFileCacheHitRatio = mobFileCache != null ? mobFileCache.getHitRatio() : 0.0;
1000        if (Double.isNaN(mobFileCacheHitRatio)) {
1001          mobFileCacheHitRatio = 0.0;
1002        }
1003        mobFileCacheEvictedCount = mobFileCache != null ? mobFileCache.getEvictedFileCount() : 0L;
1004        mobFileCacheCount = mobFileCache != null ? mobFileCache.getCacheSize() : 0;
1005
1006        lastRan = currentTime;
1007      } catch (Throwable e) {
1008        LOG.warn("Caught exception! Will suppress and retry.", e);
1009      }
1010    }
1011  }
1012
1013  @Override
1014  public long getHedgedReadOps() {
1015    return this.dfsHedgedReadMetrics == null ? 0 : this.dfsHedgedReadMetrics.getHedgedReadOps();
1016  }
1017
1018  @Override
1019  public long getHedgedReadWins() {
1020    return this.dfsHedgedReadMetrics == null ? 0 : this.dfsHedgedReadMetrics.getHedgedReadWins();
1021  }
1022
1023  @Override
1024  public long getHedgedReadOpsInCurThread() {
1025    return this.dfsHedgedReadMetrics == null
1026      ? 0
1027      : this.dfsHedgedReadMetrics.getHedgedReadOpsInCurThread();
1028  }
1029
1030  @Override
1031  public long getTotalBytesRead() {
1032    return FSDataInputStreamWrapper.getTotalBytesRead();
1033  }
1034
1035  @Override
1036  public long getLocalBytesRead() {
1037    return FSDataInputStreamWrapper.getLocalBytesRead();
1038  }
1039
1040  @Override
1041  public long getShortCircuitBytesRead() {
1042    return FSDataInputStreamWrapper.getShortCircuitBytesRead();
1043  }
1044
1045  @Override
1046  public long getZeroCopyBytesRead() {
1047    return FSDataInputStreamWrapper.getZeroCopyBytesRead();
1048  }
1049
1050  @Override
1051  public long getBlockedRequestsCount() {
1052    return aggregate.blockedRequestsCount;
1053  }
1054
1055  @Override
1056  public long getAverageRegionSize() {
1057    return aggregate.averageRegionSize;
1058  }
1059
1060  @Override
1061  public long getDataMissCount() {
1062    return this.cacheStats != null ? this.cacheStats.getDataMissCount() : 0L;
1063  }
1064
1065  @Override
1066  public long getLeafIndexMissCount() {
1067    return this.cacheStats != null ? this.cacheStats.getLeafIndexMissCount() : 0L;
1068  }
1069
1070  @Override
1071  public long getBloomChunkMissCount() {
1072    return this.cacheStats != null ? this.cacheStats.getBloomChunkMissCount() : 0L;
1073  }
1074
1075  @Override
1076  public long getMetaMissCount() {
1077    return this.cacheStats != null ? this.cacheStats.getMetaMissCount() : 0L;
1078  }
1079
1080  @Override
1081  public long getRootIndexMissCount() {
1082    return this.cacheStats != null ? this.cacheStats.getRootIndexMissCount() : 0L;
1083  }
1084
1085  @Override
1086  public long getIntermediateIndexMissCount() {
1087    return this.cacheStats != null ? this.cacheStats.getIntermediateIndexMissCount() : 0L;
1088  }
1089
1090  @Override
1091  public long getFileInfoMissCount() {
1092    return this.cacheStats != null ? this.cacheStats.getFileInfoMissCount() : 0L;
1093  }
1094
1095  @Override
1096  public long getGeneralBloomMetaMissCount() {
1097    return this.cacheStats != null ? this.cacheStats.getGeneralBloomMetaMissCount() : 0L;
1098  }
1099
1100  @Override
1101  public long getDeleteFamilyBloomMissCount() {
1102    return this.cacheStats != null ? this.cacheStats.getDeleteFamilyBloomMissCount() : 0L;
1103  }
1104
1105  @Override
1106  public long getTrailerMissCount() {
1107    return this.cacheStats != null ? this.cacheStats.getTrailerMissCount() : 0L;
1108  }
1109
1110  @Override
1111  public long getDataHitCount() {
1112    return this.cacheStats != null ? this.cacheStats.getDataHitCount() : 0L;
1113  }
1114
1115  @Override
1116  public long getLeafIndexHitCount() {
1117    return this.cacheStats != null ? this.cacheStats.getLeafIndexHitCount() : 0L;
1118  }
1119
1120  @Override
1121  public long getBloomChunkHitCount() {
1122    return this.cacheStats != null ? this.cacheStats.getBloomChunkHitCount() : 0L;
1123  }
1124
1125  @Override
1126  public long getMetaHitCount() {
1127    return this.cacheStats != null ? this.cacheStats.getMetaHitCount() : 0L;
1128  }
1129
1130  @Override
1131  public long getRootIndexHitCount() {
1132    return this.cacheStats != null ? this.cacheStats.getRootIndexHitCount() : 0L;
1133  }
1134
1135  @Override
1136  public long getIntermediateIndexHitCount() {
1137    return this.cacheStats != null ? this.cacheStats.getIntermediateIndexHitCount() : 0L;
1138  }
1139
1140  @Override
1141  public long getFileInfoHitCount() {
1142    return this.cacheStats != null ? this.cacheStats.getFileInfoHitCount() : 0L;
1143  }
1144
1145  @Override
1146  public long getGeneralBloomMetaHitCount() {
1147    return this.cacheStats != null ? this.cacheStats.getGeneralBloomMetaHitCount() : 0L;
1148  }
1149
1150  @Override
1151  public long getDeleteFamilyBloomHitCount() {
1152    return this.cacheStats != null ? this.cacheStats.getDeleteFamilyBloomHitCount() : 0L;
1153  }
1154
1155  @Override
1156  public long getTrailerHitCount() {
1157    return this.cacheStats != null ? this.cacheStats.getTrailerHitCount() : 0L;
1158  }
1159
1160  @Override
1161  public long getByteBuffAllocatorHeapAllocationBytes() {
1162    return ByteBuffAllocator.getHeapAllocationBytes(allocator, ByteBuffAllocator.HEAP);
1163  }
1164
1165  @Override
1166  public long getByteBuffAllocatorPoolAllocationBytes() {
1167    return this.allocator.getPoolAllocationBytes();
1168  }
1169
1170  @Override
1171  public double getByteBuffAllocatorHeapAllocRatio() {
1172    return ByteBuffAllocator.getHeapAllocationRatio(allocator, ByteBuffAllocator.HEAP);
1173  }
1174
1175  @Override
1176  public long getByteBuffAllocatorTotalBufferCount() {
1177    return this.allocator.getTotalBufferCount();
1178  }
1179
1180  @Override
1181  public long getByteBuffAllocatorUsedBufferCount() {
1182    return this.allocator.getUsedBufferCount();
1183  }
1184
1185  // Visible for testing
1186  long getPeriod() {
1187    return period;
1188  }
1189}