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.List;
024import java.util.Map;
025import java.util.OptionalDouble;
026import java.util.OptionalLong;
027import java.util.concurrent.ConcurrentHashMap;
028import java.util.concurrent.ScheduledExecutorService;
029import java.util.concurrent.TimeUnit;
030import org.apache.commons.lang3.StringUtils;
031import org.apache.hadoop.hbase.CompatibilitySingletonFactory;
032import org.apache.hadoop.hbase.HConstants;
033import org.apache.hadoop.hbase.HDFSBlocksDistribution;
034import org.apache.hadoop.hbase.ServerName;
035import org.apache.hadoop.hbase.client.RegionInfo;
036import org.apache.hadoop.hbase.io.hfile.BlockCache;
037import org.apache.hadoop.hbase.io.hfile.CacheConfig;
038import org.apache.hadoop.hbase.io.hfile.CacheStats;
039import org.apache.hadoop.hbase.mob.MobCacheConfig;
040import org.apache.hadoop.hbase.mob.MobFileCache;
041import org.apache.hadoop.hbase.regionserver.wal.MetricsWALSource;
042import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
043import org.apache.hadoop.hbase.util.FSUtils;
044import org.apache.hadoop.hbase.wal.WALProvider;
045import org.apache.hadoop.hbase.zookeeper.ZKWatcher;
046import org.apache.hadoop.hdfs.DFSHedgedReadMetrics;
047import org.apache.hadoop.metrics2.MetricsExecutor;
048import org.apache.yetus.audience.InterfaceAudience;
049import org.slf4j.Logger;
050import org.slf4j.LoggerFactory;
051
052/**
053 * Impl for exposing HRegionServer Information through Hadoop's metrics 2 system.
054 */
055@InterfaceAudience.Private
056class MetricsRegionServerWrapperImpl
057    implements MetricsRegionServerWrapper {
058
059  private static final Logger LOG = LoggerFactory.getLogger(MetricsRegionServerWrapperImpl.class);
060
061  private final HRegionServer regionServer;
062  private final MetricsWALSource metricsWALSource;
063
064  private BlockCache blockCache;
065  private MobFileCache mobFileCache;
066
067  private volatile long numStores = 0;
068  private volatile long numWALFiles = 0;
069  private volatile long walFileSize = 0;
070  private volatile long numStoreFiles = 0;
071  private volatile long memstoreSize = 0;
072  private volatile long storeFileSize = 0;
073  private volatile long maxStoreFileAge = 0;
074  private volatile long minStoreFileAge = 0;
075  private volatile long avgStoreFileAge = 0;
076  private volatile long numReferenceFiles = 0;
077  private volatile double requestsPerSecond = 0.0;
078  private volatile long readRequestsCount = 0;
079  private volatile long filteredReadRequestsCount = 0;
080  private volatile long writeRequestsCount = 0;
081  private volatile long checkAndMutateChecksFailed = 0;
082  private volatile long checkAndMutateChecksPassed = 0;
083  private volatile long storefileIndexSize = 0;
084  private volatile long totalStaticIndexSize = 0;
085  private volatile long totalStaticBloomSize = 0;
086  private volatile long numMutationsWithoutWAL = 0;
087  private volatile long dataInMemoryWithoutWAL = 0;
088  private volatile double percentFileLocal = 0;
089  private volatile double percentFileLocalSecondaryRegions = 0;
090  private volatile long flushedCellsCount = 0;
091  private volatile long compactedCellsCount = 0;
092  private volatile long majorCompactedCellsCount = 0;
093  private volatile long flushedCellsSize = 0;
094  private volatile long compactedCellsSize = 0;
095  private volatile long majorCompactedCellsSize = 0;
096  private volatile long cellsCountCompactedToMob = 0;
097  private volatile long cellsCountCompactedFromMob = 0;
098  private volatile long cellsSizeCompactedToMob = 0;
099  private volatile long cellsSizeCompactedFromMob = 0;
100  private volatile long mobFlushCount = 0;
101  private volatile long mobFlushedCellsCount = 0;
102  private volatile long mobFlushedCellsSize = 0;
103  private volatile long mobScanCellsCount = 0;
104  private volatile long mobScanCellsSize = 0;
105  private volatile long mobFileCacheAccessCount = 0;
106  private volatile long mobFileCacheMissCount = 0;
107  private volatile double mobFileCacheHitRatio = 0;
108  private volatile long mobFileCacheEvictedCount = 0;
109  private volatile long mobFileCacheCount = 0;
110  private volatile long blockedRequestsCount = 0L;
111  private volatile long averageRegionSize = 0L;
112  protected final Map<String, ArrayList<Long>>
113      requestsCountCache = new ConcurrentHashMap<String, ArrayList<Long>>();
114
115  private CacheStats cacheStats;
116  private CacheStats l1Stats = null;
117  private CacheStats l2Stats = null;
118  private ScheduledExecutorService executor;
119  private Runnable runnable;
120  private long period;
121
122  /**
123   * Can be null if not on hdfs.
124   */
125  private DFSHedgedReadMetrics dfsHedgedReadMetrics;
126
127  public MetricsRegionServerWrapperImpl(final HRegionServer regionServer) {
128    this.regionServer = regionServer;
129    initBlockCache();
130    initMobFileCache();
131
132    this.period =
133        regionServer.conf.getLong(HConstants.REGIONSERVER_METRICS_PERIOD,
134          HConstants.DEFAULT_REGIONSERVER_METRICS_PERIOD);
135
136    this.executor = CompatibilitySingletonFactory.getInstance(MetricsExecutor.class).getExecutor();
137    this.runnable = new RegionServerMetricsWrapperRunnable();
138    this.executor.scheduleWithFixedDelay(this.runnable, this.period, this.period,
139      TimeUnit.MILLISECONDS);
140    this.metricsWALSource = CompatibilitySingletonFactory.getInstance(MetricsWALSource.class);
141
142    try {
143      this.dfsHedgedReadMetrics = FSUtils.getDFSHedgedReadMetrics(regionServer.getConfiguration());
144    } catch (IOException e) {
145      LOG.warn("Failed to get hedged metrics", e);
146    }
147    if (LOG.isInfoEnabled()) {
148      LOG.info("Computing regionserver metrics every " + this.period + " milliseconds");
149    }
150  }
151
152  /**
153   * It's possible that due to threading the block cache could not be initialized
154   * yet (testing multiple region servers in one jvm).  So we need to try and initialize
155   * the blockCache and cacheStats reference multiple times until we succeed.
156   */
157  private synchronized  void initBlockCache() {
158    CacheConfig cacheConfig = this.regionServer.cacheConfig;
159    if (cacheConfig != null) {
160      l1Stats = cacheConfig.getOnHeapCacheStats();
161      l2Stats = cacheConfig.getL2CacheStats();
162      if (this.blockCache == null) {
163        this.blockCache = cacheConfig.getBlockCache();
164      }
165    }
166
167    if (this.blockCache != null && this.cacheStats == null) {
168      this.cacheStats = blockCache.getStats();
169    }
170  }
171
172  /**
173   * Initializes the mob file cache.
174   */
175  private synchronized void initMobFileCache() {
176    MobCacheConfig mobCacheConfig = this.regionServer.mobCacheConfig;
177    if (mobCacheConfig != null && this.mobFileCache == null) {
178      this.mobFileCache = mobCacheConfig.getMobFileCache();
179    }
180  }
181
182  @Override
183  public String getClusterId() {
184    return regionServer.getClusterId();
185  }
186
187  @Override
188  public long getStartCode() {
189    return regionServer.getStartcode();
190  }
191
192  @Override
193  public String getZookeeperQuorum() {
194    ZKWatcher zk = regionServer.getZooKeeper();
195    if (zk == null) {
196      return "";
197    }
198    return zk.getQuorum();
199  }
200
201  @Override
202  public String getCoprocessors() {
203    String[] coprocessors = regionServer.getRegionServerCoprocessors();
204    if (coprocessors == null || coprocessors.length == 0) {
205      return "";
206    }
207    return StringUtils.join(coprocessors, ", ");
208  }
209
210  @Override
211  public String getServerName() {
212    ServerName serverName = regionServer.getServerName();
213    if (serverName == null) {
214      return "";
215    }
216    return serverName.getServerName();
217  }
218
219  @Override
220  public long getNumOnlineRegions() {
221    Collection<HRegion> onlineRegionsLocalContext = regionServer.getOnlineRegionsLocalContext();
222    if (onlineRegionsLocalContext == null) {
223      return 0;
224    }
225    return onlineRegionsLocalContext.size();
226  }
227
228  @Override
229  public long getTotalRequestCount() {
230    return regionServer.rpcServices.requestCount.sum();
231  }
232
233  @Override
234  public long getTotalRowActionRequestCount() {
235    return readRequestsCount + writeRequestsCount;
236  }
237
238  @Override
239  public int getSplitQueueSize() {
240    if (this.regionServer.compactSplitThread == null) {
241      return 0;
242    }
243    return this.regionServer.compactSplitThread.getSplitQueueSize();
244  }
245
246  @Override
247  public int getCompactionQueueSize() {
248    //The thread could be zero.  if so assume there is no queue.
249    if (this.regionServer.compactSplitThread == null) {
250      return 0;
251    }
252    return this.regionServer.compactSplitThread.getCompactionQueueSize();
253  }
254
255  @Override
256  public int getSmallCompactionQueueSize() {
257    //The thread could be zero.  if so assume there is no queue.
258    if (this.regionServer.compactSplitThread == null) {
259      return 0;
260    }
261    return this.regionServer.compactSplitThread.getSmallCompactionQueueSize();
262  }
263
264  @Override
265  public int getLargeCompactionQueueSize() {
266    //The thread could be zero.  if so assume there is no queue.
267    if (this.regionServer.compactSplitThread == null) {
268      return 0;
269    }
270    return this.regionServer.compactSplitThread.getLargeCompactionQueueSize();
271  }
272
273  @Override
274  public int getFlushQueueSize() {
275    //If there is no flusher there should be no queue.
276    if (this.regionServer.cacheFlusher == null) {
277      return 0;
278    }
279    return this.regionServer.cacheFlusher.getFlushQueueSize();
280  }
281
282  @Override
283  public long getBlockCacheCount() {
284    return this.blockCache != null ? this.blockCache.getBlockCount() : 0L;
285  }
286
287  @Override
288  public long getMemStoreLimit() {
289    return this.regionServer.getRegionServerAccounting().getGlobalMemStoreLimit();
290  }
291
292  @Override
293  public long getBlockCacheSize() {
294    return this.blockCache != null ? this.blockCache.getCurrentSize() : 0L;
295  }
296
297  @Override
298  public long getBlockCacheFreeSize() {
299    return this.blockCache != null ? this.blockCache.getFreeSize() : 0L;
300  }
301
302  @Override
303  public long getBlockCacheHitCount() {
304    return this.cacheStats != null ? this.cacheStats.getHitCount() : 0L;
305  }
306
307  @Override
308  public long getBlockCachePrimaryHitCount() {
309    return this.cacheStats != null ? this.cacheStats.getPrimaryHitCount() : 0L;
310  }
311
312  @Override
313  public long getBlockCacheMissCount() {
314    return this.cacheStats != null ? this.cacheStats.getMissCount() : 0L;
315  }
316
317  @Override
318  public long getBlockCachePrimaryMissCount() {
319    return this.cacheStats != null ? this.cacheStats.getPrimaryMissCount() : 0L;
320  }
321
322  @Override
323  public long getBlockCacheEvictedCount() {
324    return this.cacheStats != null ? this.cacheStats.getEvictedCount() : 0L;
325  }
326
327  @Override
328  public long getBlockCachePrimaryEvictedCount() {
329    return this.cacheStats != null ? this.cacheStats.getPrimaryEvictedCount() : 0L;
330  }
331
332  @Override
333  public double getBlockCacheHitPercent() {
334    double ratio = this.cacheStats != null ? this.cacheStats.getHitRatio() : 0.0;
335    if (Double.isNaN(ratio)) {
336      ratio = 0;
337    }
338    return (ratio * 100);
339  }
340
341  @Override
342  public double getBlockCacheHitCachingPercent() {
343    double ratio = this.cacheStats != null ? this.cacheStats.getHitCachingRatio() : 0.0;
344    if (Double.isNaN(ratio)) {
345      ratio = 0;
346    }
347    return (ratio * 100);
348  }
349
350  @Override
351  public long getBlockCacheFailedInsertions() {
352    return this.cacheStats != null ? this.cacheStats.getFailedInserts() : 0L;
353  }
354
355  @Override
356  public long getL1CacheHitCount() {
357    return this.l1Stats != null ? this.l1Stats.getHitCount() : 0L;
358  }
359
360  @Override
361  public long getL1CacheMissCount() {
362    return this.l1Stats != null ? this.l1Stats.getMissCount() : 0L;
363  }
364
365  @Override
366  public double getL1CacheHitRatio() {
367    return this.l1Stats != null ? this.l1Stats.getHitRatio() : 0.0;
368  }
369
370  @Override
371  public double getL1CacheMissRatio() {
372    return this.l1Stats != null ? this.l1Stats.getMissRatio() : 0.0;
373  }
374
375  @Override
376  public long getL2CacheHitCount() {
377    return this.l2Stats != null ? this.l2Stats.getHitCount() : 0L;
378  }
379
380  @Override
381  public long getL2CacheMissCount() {
382    return this.l2Stats != null ? this.l2Stats.getMissCount() : 0L;
383  }
384
385  @Override
386  public double getL2CacheHitRatio() {
387    return this.l2Stats != null ? this.l2Stats.getHitRatio() : 0.0;
388  }
389
390  @Override
391  public double getL2CacheMissRatio() {
392    return this.l2Stats != null ? this.l2Stats.getMissRatio() : 0.0;
393  }
394
395  @Override public void forceRecompute() {
396    this.runnable.run();
397  }
398
399  @Override
400  public long getNumStores() {
401    return numStores;
402  }
403
404  @Override
405  public long getNumWALFiles() {
406    return numWALFiles;
407  }
408
409  @Override
410  public long getWALFileSize() {
411    return walFileSize;
412  }
413
414  @Override
415  public long getNumWALSlowAppend() {
416    return metricsWALSource.getSlowAppendCount();
417  }
418
419  @Override
420  public long getNumStoreFiles() {
421    return numStoreFiles;
422  }
423
424  @Override
425  public long getMaxStoreFileAge() {
426    return maxStoreFileAge;
427  }
428
429  @Override
430  public long getMinStoreFileAge() {
431    return minStoreFileAge;
432  }
433
434  @Override
435  public long getAvgStoreFileAge() {
436    return avgStoreFileAge;
437  }
438
439  @Override
440  public long getNumReferenceFiles() {
441    return numReferenceFiles;
442  }
443
444  @Override
445  public long getMemStoreSize() {
446    return memstoreSize;
447  }
448
449  @Override
450  public long getStoreFileSize() {
451    return storeFileSize;
452  }
453
454  @Override public double getRequestsPerSecond() {
455    return requestsPerSecond;
456  }
457
458  @Override
459  public long getReadRequestsCount() {
460    return readRequestsCount;
461  }
462
463  @Override
464  public long getFilteredReadRequestsCount() {
465    return filteredReadRequestsCount;
466  }
467
468  @Override
469  public long getWriteRequestsCount() {
470    return writeRequestsCount;
471  }
472
473  @Override
474  public long getRpcGetRequestsCount() {
475    return regionServer.rpcServices.rpcGetRequestCount.sum();
476  }
477
478  @Override
479  public long getRpcScanRequestsCount() {
480    return regionServer.rpcServices.rpcScanRequestCount.sum();
481  }
482
483  @Override
484  public long getRpcMultiRequestsCount() {
485    return regionServer.rpcServices.rpcMultiRequestCount.sum();
486  }
487
488  @Override
489  public long getRpcMutateRequestsCount() {
490    return regionServer.rpcServices.rpcMutateRequestCount.sum();
491  }
492
493  @Override
494  public long getCheckAndMutateChecksFailed() {
495    return checkAndMutateChecksFailed;
496  }
497
498  @Override
499  public long getCheckAndMutateChecksPassed() {
500    return checkAndMutateChecksPassed;
501  }
502
503  @Override
504  public long getStoreFileIndexSize() {
505    return storefileIndexSize;
506  }
507
508  @Override
509  public long getTotalStaticIndexSize() {
510    return totalStaticIndexSize;
511  }
512
513  @Override
514  public long getTotalStaticBloomSize() {
515    return totalStaticBloomSize;
516  }
517
518  @Override
519  public long getNumMutationsWithoutWAL() {
520    return numMutationsWithoutWAL;
521  }
522
523  @Override
524  public long getDataInMemoryWithoutWAL() {
525    return dataInMemoryWithoutWAL;
526  }
527
528  @Override
529  public double getPercentFileLocal() {
530    return percentFileLocal;
531  }
532
533  @Override
534  public double getPercentFileLocalSecondaryRegions() {
535    return percentFileLocalSecondaryRegions;
536  }
537
538  @Override
539  public long getUpdatesBlockedTime() {
540    if (this.regionServer.cacheFlusher == null) {
541      return 0;
542    }
543    return this.regionServer.cacheFlusher.getUpdatesBlockedMsHighWater().sum();
544  }
545
546  @Override
547  public long getFlushedCellsCount() {
548    return flushedCellsCount;
549  }
550
551  @Override
552  public long getCompactedCellsCount() {
553    return compactedCellsCount;
554  }
555
556  @Override
557  public long getMajorCompactedCellsCount() {
558    return majorCompactedCellsCount;
559  }
560
561  @Override
562  public long getFlushedCellsSize() {
563    return flushedCellsSize;
564  }
565
566  @Override
567  public long getCompactedCellsSize() {
568    return compactedCellsSize;
569  }
570
571  @Override
572  public long getMajorCompactedCellsSize() {
573    return majorCompactedCellsSize;
574  }
575
576  @Override
577  public long getCellsCountCompactedFromMob() {
578    return cellsCountCompactedFromMob;
579  }
580
581  @Override
582  public long getCellsCountCompactedToMob() {
583    return cellsCountCompactedToMob;
584  }
585
586  @Override
587  public long getCellsSizeCompactedFromMob() {
588    return cellsSizeCompactedFromMob;
589  }
590
591  @Override
592  public long getCellsSizeCompactedToMob() {
593    return cellsSizeCompactedToMob;
594  }
595
596  @Override
597  public long getMobFlushCount() {
598    return mobFlushCount;
599  }
600
601  @Override
602  public long getMobFlushedCellsCount() {
603    return mobFlushedCellsCount;
604  }
605
606  @Override
607  public long getMobFlushedCellsSize() {
608    return mobFlushedCellsSize;
609  }
610
611  @Override
612  public long getMobScanCellsCount() {
613    return mobScanCellsCount;
614  }
615
616  @Override
617  public long getMobScanCellsSize() {
618    return mobScanCellsSize;
619  }
620
621  @Override
622  public long getMobFileCacheAccessCount() {
623    return mobFileCacheAccessCount;
624  }
625
626  @Override
627  public long getMobFileCacheMissCount() {
628    return mobFileCacheMissCount;
629  }
630
631  @Override
632  public long getMobFileCacheCount() {
633    return mobFileCacheCount;
634  }
635
636  @Override
637  public long getMobFileCacheEvictedCount() {
638    return mobFileCacheEvictedCount;
639  }
640
641  @Override
642  public double getMobFileCacheHitPercent() {
643    return mobFileCacheHitRatio * 100;
644  }
645
646  /**
647   * This is the runnable that will be executed on the executor every PERIOD number of seconds
648   * It will take metrics/numbers from all of the regions and use them to compute point in
649   * time metrics.
650   */
651  public class RegionServerMetricsWrapperRunnable implements Runnable {
652
653    private long lastRan = 0;
654
655    @Override
656    synchronized public void run() {
657      try {
658        initBlockCache();
659        initMobFileCache();
660
661        HDFSBlocksDistribution hdfsBlocksDistribution =
662            new HDFSBlocksDistribution();
663        HDFSBlocksDistribution hdfsBlocksDistributionSecondaryRegions =
664            new HDFSBlocksDistribution();
665
666        long tempNumStores = 0, tempNumStoreFiles = 0, tempMemstoreSize = 0, tempStoreFileSize = 0;
667        long tempMaxStoreFileAge = 0, tempNumReferenceFiles = 0;
668        long avgAgeNumerator = 0, numHFiles = 0;
669        long tempMinStoreFileAge = Long.MAX_VALUE;
670        long tempReadRequestsCount = 0, tempFilteredReadRequestsCount = 0,
671          tempWriteRequestsCount = 0;
672        long tempCheckAndMutateChecksFailed = 0;
673        long tempCheckAndMutateChecksPassed = 0;
674        long tempStorefileIndexSize = 0;
675        long tempTotalStaticIndexSize = 0;
676        long tempTotalStaticBloomSize = 0;
677        long tempNumMutationsWithoutWAL = 0;
678        long tempDataInMemoryWithoutWAL = 0;
679        double tempPercentFileLocal = 0;
680        double tempPercentFileLocalSecondaryRegions = 0;
681        long tempFlushedCellsCount = 0;
682        long tempCompactedCellsCount = 0;
683        long tempMajorCompactedCellsCount = 0;
684        long tempFlushedCellsSize = 0;
685        long tempCompactedCellsSize = 0;
686        long tempMajorCompactedCellsSize = 0;
687        long tempCellsCountCompactedToMob = 0;
688        long tempCellsCountCompactedFromMob = 0;
689        long tempCellsSizeCompactedToMob = 0;
690        long tempCellsSizeCompactedFromMob = 0;
691        long tempMobFlushCount = 0;
692        long tempMobFlushedCellsCount = 0;
693        long tempMobFlushedCellsSize = 0;
694        long tempMobScanCellsCount = 0;
695        long tempMobScanCellsSize = 0;
696        long tempBlockedRequestsCount = 0;
697        int regionCount = 0;
698
699        long currentReadRequestsCount = 0;
700        long currentWriteRequestsCount = 0;
701        long lastReadRequestsCount = 0;
702        long lastWriteRequestsCount = 0;
703        long readRequestsDelta = 0;
704        long writeRequestsDelta = 0;
705        long totalReadRequestsDelta = 0;
706        long totalWriteRequestsDelta = 0;
707        String encodedRegionName;
708        for (HRegion r : regionServer.getOnlineRegionsLocalContext()) {
709          encodedRegionName = r.getRegionInfo().getEncodedName();
710          currentReadRequestsCount = r.getReadRequestsCount();
711          currentWriteRequestsCount = r.getWriteRequestsCount();
712          if (requestsCountCache.containsKey(encodedRegionName)) {
713            lastReadRequestsCount = requestsCountCache.get(encodedRegionName).get(0);
714            lastWriteRequestsCount = requestsCountCache.get(encodedRegionName).get(1);
715            readRequestsDelta = currentReadRequestsCount - lastReadRequestsCount;
716            writeRequestsDelta = currentWriteRequestsCount - lastWriteRequestsCount;
717            totalReadRequestsDelta += readRequestsDelta;
718            totalWriteRequestsDelta += writeRequestsDelta;
719            //Update cache for our next comparision
720            requestsCountCache.get(encodedRegionName).set(0,currentReadRequestsCount);
721            requestsCountCache.get(encodedRegionName).set(1,currentWriteRequestsCount);
722          } else {
723            // List[0] -> readRequestCount
724            // List[1] -> writeRequestCount
725            ArrayList<Long> requests = new ArrayList<Long>(2);
726            requests.add(currentReadRequestsCount);
727            requests.add(currentWriteRequestsCount);
728            requestsCountCache.put(encodedRegionName, requests);
729            totalReadRequestsDelta += currentReadRequestsCount;
730            totalWriteRequestsDelta += currentWriteRequestsCount;
731          }
732          tempNumMutationsWithoutWAL += r.getNumMutationsWithoutWAL();
733          tempDataInMemoryWithoutWAL += r.getDataInMemoryWithoutWAL();
734          tempReadRequestsCount += r.getReadRequestsCount();
735          tempFilteredReadRequestsCount += r.getFilteredReadRequestsCount();
736          tempWriteRequestsCount += r.getWriteRequestsCount();
737          tempCheckAndMutateChecksFailed += r.getCheckAndMutateChecksFailed();
738          tempCheckAndMutateChecksPassed += r.getCheckAndMutateChecksPassed();
739          tempBlockedRequestsCount += r.getBlockedRequestsCount();
740          List<? extends Store> storeList = r.getStores();
741          tempNumStores += storeList.size();
742          for (Store store : storeList) {
743            tempNumStoreFiles += store.getStorefilesCount();
744            tempMemstoreSize += store.getMemStoreSize().getDataSize();
745            tempStoreFileSize += store.getStorefilesSize();
746
747            OptionalLong storeMaxStoreFileAge = store.getMaxStoreFileAge();
748            if (storeMaxStoreFileAge.isPresent() &&
749                storeMaxStoreFileAge.getAsLong() > tempMaxStoreFileAge) {
750              tempMaxStoreFileAge = storeMaxStoreFileAge.getAsLong();
751            }
752
753            OptionalLong storeMinStoreFileAge = store.getMinStoreFileAge();
754            if (storeMinStoreFileAge.isPresent() &&
755                storeMinStoreFileAge.getAsLong() < tempMinStoreFileAge) {
756              tempMinStoreFileAge = storeMinStoreFileAge.getAsLong();
757            }
758
759            long storeHFiles = store.getNumHFiles();
760            numHFiles += storeHFiles;
761            tempNumReferenceFiles += store.getNumReferenceFiles();
762
763            OptionalDouble storeAvgStoreFileAge = store.getAvgStoreFileAge();
764            if (storeAvgStoreFileAge.isPresent()) {
765              avgAgeNumerator =
766                  (long) (avgAgeNumerator + storeAvgStoreFileAge.getAsDouble() * storeHFiles);
767            }
768
769            tempStorefileIndexSize += store.getStorefilesRootLevelIndexSize();
770            tempTotalStaticBloomSize += store.getTotalStaticBloomSize();
771            tempTotalStaticIndexSize += store.getTotalStaticIndexSize();
772            tempFlushedCellsCount += store.getFlushedCellsCount();
773            tempCompactedCellsCount += store.getCompactedCellsCount();
774            tempMajorCompactedCellsCount += store.getMajorCompactedCellsCount();
775            tempFlushedCellsSize += store.getFlushedCellsSize();
776            tempCompactedCellsSize += store.getCompactedCellsSize();
777            tempMajorCompactedCellsSize += store.getMajorCompactedCellsSize();
778            if (store instanceof HMobStore) {
779              HMobStore mobStore = (HMobStore) store;
780              tempCellsCountCompactedToMob += mobStore.getCellsCountCompactedToMob();
781              tempCellsCountCompactedFromMob += mobStore.getCellsCountCompactedFromMob();
782              tempCellsSizeCompactedToMob += mobStore.getCellsSizeCompactedToMob();
783              tempCellsSizeCompactedFromMob += mobStore.getCellsSizeCompactedFromMob();
784              tempMobFlushCount += mobStore.getMobFlushCount();
785              tempMobFlushedCellsCount += mobStore.getMobFlushedCellsCount();
786              tempMobFlushedCellsSize += mobStore.getMobFlushedCellsSize();
787              tempMobScanCellsCount += mobStore.getMobScanCellsCount();
788              tempMobScanCellsSize += mobStore.getMobScanCellsSize();
789            }
790          }
791
792          HDFSBlocksDistribution distro = r.getHDFSBlocksDistribution();
793          hdfsBlocksDistribution.add(distro);
794          if (r.getRegionInfo().getReplicaId() != RegionInfo.DEFAULT_REPLICA_ID) {
795            hdfsBlocksDistributionSecondaryRegions.add(distro);
796          }
797          regionCount++;
798        }
799        float localityIndex = hdfsBlocksDistribution.getBlockLocalityIndex(
800            regionServer.getServerName().getHostname());
801        tempPercentFileLocal = Double.isNaN(tempBlockedRequestsCount) ? 0 : (localityIndex * 100);
802
803        float localityIndexSecondaryRegions = hdfsBlocksDistributionSecondaryRegions
804            .getBlockLocalityIndex(regionServer.getServerName().getHostname());
805        tempPercentFileLocalSecondaryRegions = Double.
806            isNaN(localityIndexSecondaryRegions) ? 0 : (localityIndexSecondaryRegions * 100);
807
808        // Compute the number of requests per second
809        long currentTime = EnvironmentEdgeManager.currentTime();
810
811        // assume that it took PERIOD seconds to start the executor.
812        // this is a guess but it's a pretty good one.
813        if (lastRan == 0) {
814          lastRan = currentTime - period;
815        }
816        // If we've time traveled keep the last requests per second.
817        if ((currentTime - lastRan) > 0) {
818          requestsPerSecond = (totalReadRequestsDelta + totalWriteRequestsDelta) /
819              ((currentTime - lastRan) / 1000.0);
820        }
821        lastRan = currentTime;
822
823        WALProvider provider = regionServer.walFactory.getWALProvider();
824        WALProvider metaProvider = regionServer.walFactory.getMetaWALProvider();
825        numWALFiles = (provider == null ? 0 : provider.getNumLogFiles()) +
826            (metaProvider == null ? 0 : metaProvider.getNumLogFiles());
827        walFileSize = (provider == null ? 0 : provider.getLogFileSize()) +
828            (provider == null ? 0 : provider.getLogFileSize());
829        // Copy over computed values so that no thread sees half computed values.
830        numStores = tempNumStores;
831        numStoreFiles = tempNumStoreFiles;
832        memstoreSize = tempMemstoreSize;
833        storeFileSize = tempStoreFileSize;
834        maxStoreFileAge = tempMaxStoreFileAge;
835        if (regionCount > 0) {
836          averageRegionSize = (memstoreSize + storeFileSize) / regionCount;
837        }
838        if (tempMinStoreFileAge != Long.MAX_VALUE) {
839          minStoreFileAge = tempMinStoreFileAge;
840        }
841
842        if (numHFiles != 0) {
843          avgStoreFileAge = avgAgeNumerator / numHFiles;
844        }
845
846        numReferenceFiles= tempNumReferenceFiles;
847        readRequestsCount = tempReadRequestsCount;
848        filteredReadRequestsCount = tempFilteredReadRequestsCount;
849        writeRequestsCount = tempWriteRequestsCount;
850        checkAndMutateChecksFailed = tempCheckAndMutateChecksFailed;
851        checkAndMutateChecksPassed = tempCheckAndMutateChecksPassed;
852        storefileIndexSize = tempStorefileIndexSize;
853        totalStaticIndexSize = tempTotalStaticIndexSize;
854        totalStaticBloomSize = tempTotalStaticBloomSize;
855        numMutationsWithoutWAL = tempNumMutationsWithoutWAL;
856        dataInMemoryWithoutWAL = tempDataInMemoryWithoutWAL;
857        percentFileLocal = tempPercentFileLocal;
858        percentFileLocalSecondaryRegions = tempPercentFileLocalSecondaryRegions;
859        flushedCellsCount = tempFlushedCellsCount;
860        compactedCellsCount = tempCompactedCellsCount;
861        majorCompactedCellsCount = tempMajorCompactedCellsCount;
862        flushedCellsSize = tempFlushedCellsSize;
863        compactedCellsSize = tempCompactedCellsSize;
864        majorCompactedCellsSize = tempMajorCompactedCellsSize;
865        cellsCountCompactedToMob = tempCellsCountCompactedToMob;
866        cellsCountCompactedFromMob = tempCellsCountCompactedFromMob;
867        cellsSizeCompactedToMob = tempCellsSizeCompactedToMob;
868        cellsSizeCompactedFromMob = tempCellsSizeCompactedFromMob;
869        mobFlushCount = tempMobFlushCount;
870        mobFlushedCellsCount = tempMobFlushedCellsCount;
871        mobFlushedCellsSize = tempMobFlushedCellsSize;
872        mobScanCellsCount = tempMobScanCellsCount;
873        mobScanCellsSize = tempMobScanCellsSize;
874        mobFileCacheAccessCount = mobFileCache != null ? mobFileCache.getAccessCount() : 0L;
875        mobFileCacheMissCount = mobFileCache != null ? mobFileCache.getMissCount() : 0L;
876        mobFileCacheHitRatio = mobFileCache != null ? mobFileCache.getHitRatio() : 0.0;
877        if (Double.isNaN(mobFileCacheHitRatio)) {
878          mobFileCacheHitRatio = 0.0;
879        }
880        mobFileCacheEvictedCount = mobFileCache != null ? mobFileCache.getEvictedFileCount() : 0L;
881        mobFileCacheCount = mobFileCache != null ? mobFileCache.getCacheSize() : 0;
882        blockedRequestsCount = tempBlockedRequestsCount;
883      } catch (Throwable e) {
884        LOG.warn("Caught exception! Will suppress and retry.", e);
885      }
886    }
887  }
888
889  @Override
890  public long getHedgedReadOps() {
891    return this.dfsHedgedReadMetrics == null? 0: this.dfsHedgedReadMetrics.getHedgedReadOps();
892  }
893
894  @Override
895  public long getHedgedReadWins() {
896    return this.dfsHedgedReadMetrics == null? 0: this.dfsHedgedReadMetrics.getHedgedReadWins();
897  }
898
899  @Override
900  public long getBlockedRequestsCount() {
901    return blockedRequestsCount;
902  }
903
904  @Override
905  public long getAverageRegionSize() {
906    return averageRegionSize;
907  }
908
909  @Override
910  public long getDataMissCount() {
911    return this.cacheStats != null ? this.cacheStats.getDataMissCount() : 0L;
912  }
913
914  @Override
915  public long getLeafIndexMissCount() {
916    return this.cacheStats != null ? this.cacheStats.getLeafIndexMissCount() : 0L;
917  }
918
919  @Override
920  public long getBloomChunkMissCount() {
921    return this.cacheStats != null ? this.cacheStats.getBloomChunkMissCount() : 0L;
922  }
923
924  @Override
925  public long getMetaMissCount() {
926    return this.cacheStats != null ? this.cacheStats.getMetaMissCount() : 0L;
927  }
928
929  @Override
930  public long getRootIndexMissCount() {
931    return this.cacheStats != null ? this.cacheStats.getRootIndexMissCount() : 0L;
932  }
933
934  @Override
935  public long getIntermediateIndexMissCount() {
936    return this.cacheStats != null ? this.cacheStats.getIntermediateIndexMissCount() : 0L;
937  }
938
939  @Override
940  public long getFileInfoMissCount() {
941    return this.cacheStats != null ? this.cacheStats.getFileInfoMissCount() : 0L;
942  }
943
944  @Override
945  public long getGeneralBloomMetaMissCount() {
946    return this.cacheStats != null ? this.cacheStats.getGeneralBloomMetaMissCount() : 0L;
947  }
948
949  @Override
950  public long getDeleteFamilyBloomMissCount() {
951    return this.cacheStats != null ? this.cacheStats.getDeleteFamilyBloomMissCount() : 0L;
952  }
953
954  @Override
955  public long getTrailerMissCount() {
956    return this.cacheStats != null ? this.cacheStats.getTrailerMissCount() : 0L;
957  }
958
959  @Override
960  public long getDataHitCount() {
961    return this.cacheStats != null ? this.cacheStats.getDataHitCount() : 0L;
962  }
963
964  @Override
965  public long getLeafIndexHitCount() {
966    return this.cacheStats != null ? this.cacheStats.getLeafIndexHitCount() : 0L;
967  }
968
969  @Override
970  public long getBloomChunkHitCount() {
971    return this.cacheStats != null ? this.cacheStats.getBloomChunkHitCount() : 0L;
972  }
973
974  @Override
975  public long getMetaHitCount() {
976    return this.cacheStats != null ? this.cacheStats.getMetaHitCount() : 0L;
977  }
978
979  @Override
980  public long getRootIndexHitCount() {
981    return this.cacheStats != null ? this.cacheStats.getRootIndexHitCount() : 0L;
982  }
983
984  @Override
985  public long getIntermediateIndexHitCount() {
986    return this.cacheStats != null ? this.cacheStats.getIntermediateIndexHitCount() : 0L;
987  }
988
989  @Override
990  public long getFileInfoHitCount() {
991    return this.cacheStats != null ? this.cacheStats.getFileInfoHitCount() : 0L;
992  }
993
994  @Override
995  public long getGeneralBloomMetaHitCount() {
996    return this.cacheStats != null ? this.cacheStats.getGeneralBloomMetaHitCount() : 0L;
997  }
998
999  @Override
1000  public long getDeleteFamilyBloomHitCount() {
1001    return this.cacheStats != null ? this.cacheStats.getDeleteFamilyBloomHitCount() : 0L;
1002  }
1003
1004  @Override
1005  public long getTrailerHitCount() {
1006    return this.cacheStats != null ? this.cacheStats.getTrailerHitCount() : 0L;
1007  }
1008}