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.util.Map;
021import java.util.Map.Entry;
022import java.util.concurrent.atomic.AtomicBoolean;
023import org.apache.hadoop.hbase.metrics.Interns;
024import org.apache.hadoop.metrics2.MetricsRecordBuilder;
025import org.apache.hadoop.metrics2.lib.DynamicMetricsRegistry;
026import org.apache.hadoop.metrics2.lib.MutableFastCounter;
027import org.apache.yetus.audience.InterfaceAudience;
028import org.slf4j.Logger;
029import org.slf4j.LoggerFactory;
030
031@InterfaceAudience.Private
032public class MetricsRegionSourceImpl implements MetricsRegionSource {
033
034  private static final Logger LOG = LoggerFactory.getLogger(MetricsRegionSourceImpl.class);
035
036  private static final String _STORE = "_store_";
037
038  private AtomicBoolean closed = new AtomicBoolean(false);
039
040  // Non-final so that we can null out the wrapper
041  // This is just paranoia. We really really don't want to
042  // leak a whole region by way of keeping the
043  // regionWrapper around too long.
044  private MetricsRegionWrapper regionWrapper;
045
046  private final MetricsRegionAggregateSourceImpl agg;
047  private final DynamicMetricsRegistry registry;
048
049  private final String regionNamePrefix;
050  private final String regionNamePrefix1;
051  private final String regionNamePrefix2;
052  private final String regionPutKey;
053  private final String regionDeleteKey;
054  private final String regionGetKey;
055  private final String regionIncrementKey;
056  private final String regionAppendKey;
057  private final String regionScanKey;
058
059  /*
060   * Implementation note: Do not put histograms per region. With hundreds of regions in a server
061   * histograms allocate too many counters. See HBASE-17016.
062   */
063  private final MutableFastCounter regionPut;
064  private final MutableFastCounter regionDelete;
065  private final MutableFastCounter regionIncrement;
066  private final MutableFastCounter regionAppend;
067  private final MutableFastCounter regionGet;
068  private final MutableFastCounter regionScan;
069
070  private final int hashCode;
071
072  public MetricsRegionSourceImpl(MetricsRegionWrapper regionWrapper,
073    MetricsRegionAggregateSourceImpl aggregate) {
074    this.regionWrapper = regionWrapper;
075    agg = aggregate;
076    hashCode = regionWrapper.getRegionHashCode();
077    agg.register(this);
078
079    LOG.debug("Creating new MetricsRegionSourceImpl for table " + regionWrapper.getTableName() + " "
080      + regionWrapper.getRegionName());
081
082    registry = agg.getMetricsRegistry();
083
084    regionNamePrefix1 = "Namespace_" + regionWrapper.getNamespace() + "_table_"
085      + regionWrapper.getTableName() + "_region_" + regionWrapper.getRegionName();
086    regionNamePrefix2 = "_metric_";
087    regionNamePrefix = regionNamePrefix1 + regionNamePrefix2;
088
089    String suffix = "Count";
090
091    regionPutKey = regionNamePrefix + MetricsRegionServerSource.PUT_KEY + suffix;
092    regionPut = registry.getCounter(regionPutKey, 0L);
093
094    regionDeleteKey = regionNamePrefix + MetricsRegionServerSource.DELETE_KEY + suffix;
095    regionDelete = registry.getCounter(regionDeleteKey, 0L);
096
097    regionIncrementKey = regionNamePrefix + MetricsRegionServerSource.INCREMENT_KEY + suffix;
098    regionIncrement = registry.getCounter(regionIncrementKey, 0L);
099
100    regionAppendKey = regionNamePrefix + MetricsRegionServerSource.APPEND_KEY + suffix;
101    regionAppend = registry.getCounter(regionAppendKey, 0L);
102
103    regionGetKey = regionNamePrefix + MetricsRegionServerSource.GET_KEY + suffix;
104    regionGet = registry.getCounter(regionGetKey, 0L);
105
106    regionScanKey = regionNamePrefix + MetricsRegionServerSource.SCAN_KEY + suffix;
107    regionScan = registry.getCounter(regionScanKey, 0L);
108  }
109
110  @Override
111  public void close() {
112    boolean wasClosed = closed.getAndSet(true);
113
114    // Has someone else already closed this for us?
115    if (wasClosed) {
116      return;
117    }
118
119    // Before removing the metrics remove this region from the aggregate region bean.
120    // This should mean that it's unlikely that snapshot and close happen at the same time.
121    agg.deregister(this);
122
123    // While it's un-likely that snapshot and close happen at the same time it's still possible.
124    // So grab the lock to ensure that all calls to snapshot are done before we remove the metrics
125    synchronized (this) {
126      if (LOG.isTraceEnabled()) {
127        LOG.trace("Removing region Metrics: " + regionWrapper.getRegionName());
128      }
129
130      registry.removeMetric(regionPutKey);
131      registry.removeMetric(regionDeleteKey);
132      registry.removeMetric(regionIncrementKey);
133      registry.removeMetric(regionAppendKey);
134      registry.removeMetric(regionGetKey);
135      registry.removeMetric(regionScanKey);
136
137      regionWrapper = null;
138    }
139  }
140
141  @Override
142  public void updatePut() {
143    regionPut.incr();
144  }
145
146  @Override
147  public void updateDelete() {
148    regionDelete.incr();
149  }
150
151  @Override
152  public void updateGet() {
153    regionGet.incr();
154  }
155
156  @Override
157  public void updateScan() {
158    regionScan.incr();
159  }
160
161  @Override
162  public void updateIncrement() {
163    regionIncrement.incr();
164  }
165
166  @Override
167  public void updateAppend() {
168    regionAppend.incr();
169  }
170
171  @Override
172  public MetricsRegionAggregateSource getAggregateSource() {
173    return agg;
174  }
175
176  @Override
177  public int compareTo(MetricsRegionSource source) {
178    if (!(source instanceof MetricsRegionSourceImpl)) {
179      return -1;
180    }
181    return Long.compare(hashCode, ((MetricsRegionSourceImpl) source).hashCode);
182  }
183
184  void snapshot(MetricsRecordBuilder mrb, boolean ignored) {
185
186    // If there is a close that started be double extra sure
187    // that we're not getting any locks and not putting data
188    // into the metrics that should be removed. So early out
189    // before even getting the lock.
190    if (closed.get()) {
191      return;
192    }
193
194    // Grab the read
195    // This ensures that removes of the metrics
196    // can't happen while we are putting them back in.
197    synchronized (this) {
198
199      // It's possible that a close happened between checking
200      // the closed variable and getting the lock.
201      if (closed.get()) {
202        return;
203      }
204
205      mrb.addGauge(Interns.info(regionNamePrefix + MetricsRegionServerSource.STORE_COUNT,
206        MetricsRegionServerSource.STORE_COUNT_DESC), this.regionWrapper.getNumStores());
207      mrb.addGauge(Interns.info(regionNamePrefix + MetricsRegionServerSource.STOREFILE_COUNT,
208        MetricsRegionServerSource.STOREFILE_COUNT_DESC), this.regionWrapper.getNumStoreFiles());
209      mrb.addGauge(Interns.info(regionNamePrefix + MetricsRegionServerSource.STORE_REF_COUNT,
210        MetricsRegionServerSource.STORE_REF_COUNT), this.regionWrapper.getStoreRefCount());
211      mrb.addGauge(
212        Interns.info(
213          regionNamePrefix + MetricsRegionServerSource.MAX_COMPACTED_STORE_FILE_REF_COUNT,
214          MetricsRegionServerSource.MAX_COMPACTED_STORE_FILE_REF_COUNT),
215        this.regionWrapper.getMaxCompactedStoreFileRefCount());
216      mrb.addGauge(Interns.info(regionNamePrefix + MetricsRegionServerSource.MEMSTORE_SIZE,
217        MetricsRegionServerSource.MEMSTORE_SIZE_DESC), this.regionWrapper.getMemStoreSize());
218      mrb.addGauge(
219        Interns.info(regionNamePrefix + MetricsRegionServerSource.MAX_STORE_FILE_AGE,
220          MetricsRegionServerSource.MAX_STORE_FILE_AGE_DESC),
221        this.regionWrapper.getMaxStoreFileAge());
222      mrb.addGauge(
223        Interns.info(regionNamePrefix + MetricsRegionServerSource.MIN_STORE_FILE_AGE,
224          MetricsRegionServerSource.MIN_STORE_FILE_AGE_DESC),
225        this.regionWrapper.getMinStoreFileAge());
226      mrb.addGauge(
227        Interns.info(regionNamePrefix + MetricsRegionServerSource.AVG_STORE_FILE_AGE,
228          MetricsRegionServerSource.AVG_STORE_FILE_AGE_DESC),
229        this.regionWrapper.getAvgStoreFileAge());
230      mrb.addGauge(
231        Interns.info(regionNamePrefix + MetricsRegionServerSource.NUM_REFERENCE_FILES,
232          MetricsRegionServerSource.NUM_REFERENCE_FILES_DESC),
233        this.regionWrapper.getNumReferenceFiles());
234      mrb.addGauge(Interns.info(regionNamePrefix + MetricsRegionServerSource.STOREFILE_SIZE,
235        MetricsRegionServerSource.STOREFILE_SIZE_DESC), this.regionWrapper.getStoreFileSize());
236      mrb.addGauge(
237        Interns.info(regionNamePrefix + MetricsRegionServerSource.CURRENT_REGION_CACHE_RATIO,
238          MetricsRegionServerSource.CURRENT_REGION_CACHE_RATIO_DESC),
239        this.regionWrapper.getCurrentRegionCacheRatio());
240      mrb.addCounter(
241        Interns.info(regionNamePrefix + MetricsRegionSource.COMPACTIONS_COMPLETED_COUNT,
242          MetricsRegionSource.COMPACTIONS_COMPLETED_DESC),
243        this.regionWrapper.getNumCompactionsCompleted());
244      mrb.addCounter(
245        Interns.info(regionNamePrefix + MetricsRegionSource.COMPACTIONS_FAILED_COUNT,
246          MetricsRegionSource.COMPACTIONS_FAILED_DESC),
247        this.regionWrapper.getNumCompactionsFailed());
248      mrb.addCounter(
249        Interns.info(regionNamePrefix + MetricsRegionSource.LAST_MAJOR_COMPACTION_AGE,
250          MetricsRegionSource.LAST_MAJOR_COMPACTION_DESC),
251        this.regionWrapper.getLastMajorCompactionAge());
252      mrb.addCounter(Interns.info(regionNamePrefix + MetricsRegionSource.NUM_BYTES_COMPACTED_COUNT,
253        MetricsRegionSource.NUM_BYTES_COMPACTED_DESC), this.regionWrapper.getNumBytesCompacted());
254      mrb.addCounter(Interns.info(regionNamePrefix + MetricsRegionSource.NUM_FILES_COMPACTED_COUNT,
255        MetricsRegionSource.NUM_FILES_COMPACTED_DESC), this.regionWrapper.getNumFilesCompacted());
256      mrb.addCounter(
257        Interns.info(regionNamePrefix + MetricsRegionServerSource.READ_REQUEST_COUNT,
258          MetricsRegionServerSource.READ_REQUEST_COUNT_DESC),
259        this.regionWrapper.getReadRequestCount());
260      mrb.addCounter(Interns.info(regionNamePrefix + MetricsRegionServerSource.CP_REQUEST_COUNT,
261        MetricsRegionServerSource.CP_REQUEST_COUNT_DESC), this.regionWrapper.getCpRequestCount());
262      mrb.addCounter(
263        Interns.info(regionNamePrefix + MetricsRegionServerSource.FILTERED_READ_REQUEST_COUNT,
264          MetricsRegionServerSource.FILTERED_READ_REQUEST_COUNT_DESC),
265        this.regionWrapper.getFilteredReadRequestCount());
266      mrb.addCounter(
267        Interns.info(regionNamePrefix + MetricsRegionServerSource.WRITE_REQUEST_COUNT,
268          MetricsRegionServerSource.WRITE_REQUEST_COUNT_DESC),
269        this.regionWrapper.getWriteRequestCount());
270      mrb.addCounter(Interns.info(regionNamePrefix + MetricsRegionSource.REPLICA_ID,
271        MetricsRegionSource.REPLICA_ID_DESC), this.regionWrapper.getReplicaId());
272      mrb.addCounter(
273        Interns.info(regionNamePrefix + MetricsRegionSource.COMPACTIONS_QUEUED_COUNT,
274          MetricsRegionSource.COMPACTIONS_QUEUED_DESC),
275        this.regionWrapper.getNumCompactionsQueued());
276      mrb.addCounter(Interns.info(regionNamePrefix + MetricsRegionSource.FLUSHES_QUEUED_COUNT,
277        MetricsRegionSource.FLUSHES_QUEUED_DESC), this.regionWrapper.getNumFlushesQueued());
278      mrb.addCounter(
279        Interns.info(regionNamePrefix + MetricsRegionSource.MAX_COMPACTION_QUEUE_SIZE,
280          MetricsRegionSource.MAX_COMPACTION_QUEUE_DESC),
281        this.regionWrapper.getMaxCompactionQueueSize());
282      mrb.addCounter(Interns.info(regionNamePrefix + MetricsRegionSource.MAX_FLUSH_QUEUE_SIZE,
283        MetricsRegionSource.MAX_FLUSH_QUEUE_DESC), this.regionWrapper.getMaxFlushQueueSize());
284      addCounter(mrb, this.regionWrapper.getMemstoreOnlyRowReadsCount(),
285        MetricsRegionSource.ROW_READS_ONLY_ON_MEMSTORE,
286        MetricsRegionSource.ROW_READS_ONLY_ON_MEMSTORE_DESC);
287      addCounter(mrb, this.regionWrapper.getMixedRowReadsCount(),
288        MetricsRegionSource.MIXED_ROW_READS, MetricsRegionSource.MIXED_ROW_READS_ON_STORE_DESC);
289    }
290  }
291
292  private void addCounter(MetricsRecordBuilder mrb, Map<String, Long> metricMap, String metricName,
293    String metricDesc) {
294    if (metricMap != null) {
295      for (Entry<String, Long> entry : metricMap.entrySet()) {
296        // append 'store' and its name to the metric
297        mrb.addCounter(Interns.info(
298          this.regionNamePrefix1 + _STORE + entry.getKey() + this.regionNamePrefix2 + metricName,
299          metricDesc), entry.getValue());
300      }
301    }
302  }
303
304  @Override
305  public int hashCode() {
306    return hashCode;
307  }
308
309  @Override
310  public boolean equals(Object obj) {
311    return obj == this
312      || (obj instanceof MetricsRegionSourceImpl && compareTo((MetricsRegionSourceImpl) obj) == 0);
313  }
314}