org.apache.hadoop.hbase.quotas.FileArchiverNotifierImpl Maven / Gradle / Ivy
Go to download
Show more of this group Show more artifacts with this name
Show all versions of hbase-server Show documentation
Show all versions of hbase-server Show documentation
Server functionality for HBase
/*
* Licensed to the Apache Software Foundation (ASF) under one or more
* contributor license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright ownership.
* The ASF licenses this file to you under the Apache License, Version 2.0
* (the "License"); you may not use this file except in compliance with
* the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hbase.quotas;
import java.io.IOException;
import java.util.ArrayList;
import java.util.Collection;
import java.util.Collections;
import java.util.HashMap;
import java.util.HashSet;
import java.util.Iterator;
import java.util.List;
import java.util.Map;
import java.util.Map.Entry;
import java.util.Objects;
import java.util.Set;
import java.util.concurrent.locks.ReentrantReadWriteLock;
import java.util.concurrent.locks.ReentrantReadWriteLock.ReadLock;
import java.util.concurrent.locks.ReentrantReadWriteLock.WriteLock;
import java.util.function.Predicate;
import java.util.stream.Collectors;
import org.apache.commons.lang3.builder.HashCodeBuilder;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.fs.FileStatus;
import org.apache.hadoop.fs.FileSystem;
import org.apache.hadoop.fs.Path;
import org.apache.hadoop.hbase.TableName;
import org.apache.hadoop.hbase.client.Connection;
import org.apache.hadoop.hbase.client.Get;
import org.apache.hadoop.hbase.client.Put;
import org.apache.hadoop.hbase.client.Result;
import org.apache.hadoop.hbase.client.Table;
import org.apache.hadoop.hbase.snapshot.SnapshotDescriptionUtils;
import org.apache.hadoop.hbase.snapshot.SnapshotManifest;
import org.apache.hadoop.hbase.util.FSUtils;
import org.apache.hadoop.hbase.util.HFileArchiveUtil;
import org.apache.hadoop.util.StringUtils;
import org.apache.yetus.audience.InterfaceAudience;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import org.apache.hbase.thirdparty.com.google.common.collect.HashMultimap;
import org.apache.hbase.thirdparty.com.google.common.collect.Multimap;
import org.apache.hbase.thirdparty.com.google.protobuf.InvalidProtocolBufferException;
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription;
import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotRegionManifest;
import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotRegionManifest.FamilyFiles;
import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotRegionManifest.StoreFile;
/**
* Tracks file archiving and updates the hbase quota table.
*/
@InterfaceAudience.Private
public class FileArchiverNotifierImpl implements FileArchiverNotifier {
private static final Logger LOG = LoggerFactory.getLogger(FileArchiverNotifierImpl.class);
private final Connection conn;
private final Configuration conf;
private final FileSystem fs;
private final TableName tn;
private final ReadLock readLock;
private final WriteLock writeLock;
private volatile long lastFullCompute = Long.MIN_VALUE;
private List currentSnapshots = Collections.emptyList();
private static final Map NAMESPACE_LOCKS = new HashMap<>();
/**
* An Exception thrown when SnapshotSize updates to hbase:quota fail to be written.
*/
@InterfaceAudience.Private
public static class QuotaSnapshotSizeSerializationException extends IOException {
private static final long serialVersionUID = 1L;
public QuotaSnapshotSizeSerializationException(String msg) {
super(msg);
}
}
public FileArchiverNotifierImpl(
Connection conn, Configuration conf, FileSystem fs, TableName tn) {
this.conn = conn;
this.conf = conf;
this.fs = fs;
this.tn = tn;
ReentrantReadWriteLock lock = new ReentrantReadWriteLock();
readLock = lock.readLock();
writeLock = lock.writeLock();
}
static synchronized Object getLockForNamespace(String namespace) {
return NAMESPACE_LOCKS.computeIfAbsent(namespace, (ns) -> new Object());
}
/**
* Returns a strictly-increasing measure of time extracted by {@link System#nanoTime()}.
*/
long getLastFullCompute() {
return lastFullCompute;
}
@Override
public void addArchivedFiles(Set> fileSizes) throws IOException {
long start = System.nanoTime();
readLock.lock();
try {
// We want to catch the case where we got an archival request, but there was a full
// re-computation in progress that was blocking us. Most likely, the full computation is going
// to already include the changes we were going to make.
//
// Same as "start < lastFullCompute" but avoiding numeric overflow per the
// System.nanoTime() javadoc
if (lastFullCompute != Long.MIN_VALUE && start - lastFullCompute < 0) {
if (LOG.isTraceEnabled()) {
LOG.trace("A full computation was performed after this request was received."
+ " Ignoring requested updates: " + fileSizes);
}
return;
}
if (LOG.isTraceEnabled()) {
LOG.trace("currentSnapshots: " + currentSnapshots + " fileSize: "+ fileSizes);
}
// Write increment to quota table for the correct snapshot. Only do this if we have snapshots
// and some files that were archived.
if (!currentSnapshots.isEmpty() && !fileSizes.isEmpty()) {
// We get back the files which no snapshot referenced (the files which will be deleted soon)
groupArchivedFiledBySnapshotAndRecordSize(currentSnapshots, fileSizes);
}
} finally {
readLock.unlock();
}
}
/**
* For each file in the map, this updates the first snapshot (lexicographic snapshot name) that
* references this file. The result of this computation is serialized to the quota table.
*
* @param snapshots A collection of HBase snapshots to group the files into
* @param fileSizes A map of file names to their sizes
*/
void groupArchivedFiledBySnapshotAndRecordSize(
List snapshots, Set> fileSizes) throws IOException {
// Make a copy as we'll modify it.
final Map filesToUpdate = new HashMap<>(fileSizes.size());
for (Entry entry : fileSizes) {
filesToUpdate.put(entry.getKey(), entry.getValue());
}
// Track the change in size to each snapshot
final Map snapshotSizeChanges = new HashMap<>();
for (String snapshot : snapshots) {
// For each file in `filesToUpdate`, check if `snapshot` refers to it.
// If `snapshot` does, remove it from `filesToUpdate` and add it to `snapshotSizeChanges`.
bucketFilesToSnapshot(snapshot, filesToUpdate, snapshotSizeChanges);
if (filesToUpdate.isEmpty()) {
// If we have no more files recently archived, we have nothing more to check
break;
}
}
// We have computed changes to the snapshot size, we need to record them.
if (!snapshotSizeChanges.isEmpty()) {
if (LOG.isTraceEnabled()) {
LOG.trace("Writing snapshot size changes for: " + snapshotSizeChanges);
}
persistSnapshotSizeChanges(snapshotSizeChanges);
}
}
/**
* For the given snapshot, find all files which this {@code snapshotName} references. After a file
* is found to be referenced by the snapshot, it is removed from {@code filesToUpdate} and
* {@code snapshotSizeChanges} is updated in concert.
*
* @param snapshotName The snapshot to check
* @param filesToUpdate A mapping of archived files to their size
* @param snapshotSizeChanges A mapping of snapshots and their change in size
*/
void bucketFilesToSnapshot(
String snapshotName, Map filesToUpdate, Map snapshotSizeChanges)
throws IOException {
// A quick check to avoid doing work if the caller unnecessarily invoked this method.
if (filesToUpdate.isEmpty()) {
return;
}
Path snapshotDir = SnapshotDescriptionUtils.getCompletedSnapshotDir(
snapshotName, FSUtils.getRootDir(conf));
SnapshotDescription sd = SnapshotDescriptionUtils.readSnapshotInfo(fs, snapshotDir);
SnapshotManifest manifest = SnapshotManifest.open(conf, fs, snapshotDir, sd);
// For each region referenced by the snapshot
for (SnapshotRegionManifest rm : manifest.getRegionManifests()) {
// For each column family in this region
for (FamilyFiles ff : rm.getFamilyFilesList()) {
// And each store file in that family
for (StoreFile sf : ff.getStoreFilesList()) {
Long valueOrNull = filesToUpdate.remove(sf.getName());
if (valueOrNull != null) {
// This storefile was recently archived, we should update this snapshot with its size
snapshotSizeChanges.merge(snapshotName, valueOrNull, Long::sum);
}
// Short-circuit, if we have no more files that were archived, we don't need to iterate
// over the rest of the snapshot.
if (filesToUpdate.isEmpty()) {
return;
}
}
}
}
}
/**
* Reads the current size for each snapshot to update, generates a new update based on that value,
* and then writes the new update.
*
* @param snapshotSizeChanges A map of snapshot name to size change
*/
void persistSnapshotSizeChanges(Map snapshotSizeChanges) throws IOException {
try (Table quotaTable = conn.getTable(QuotaTableUtil.QUOTA_TABLE_NAME)) {
// Create a list (with a more typical ordering implied)
final List> snapshotSizeEntries = new ArrayList<>(
snapshotSizeChanges.entrySet());
// Create the Gets for each snapshot we need to update
final List snapshotSizeGets = snapshotSizeEntries.stream()
.map((e) -> QuotaTableUtil.makeGetForSnapshotSize(tn, e.getKey()))
.collect(Collectors.toList());
final Iterator> iterator = snapshotSizeEntries.iterator();
// A List to store each Put we'll create from the Get's we retrieve
final List updates = new ArrayList<>(snapshotSizeEntries.size());
// TODO Push this down to the RegionServer with a coprocessor:
//
// We would really like to piggy-back on the row-lock already being grabbed
// to handle the update of the row in the quota table. However, because the value
// is a serialized protobuf, the standard Increment API doesn't work for us. With a CP, we
// can just send the size deltas to the RS and atomically update the serialized PB object
// while relying on the row-lock for synchronization.
//
// Synchronizing on the namespace string is a "minor smell" but passable as this is
// only invoked via a single caller (the active Master). Using the namespace name lets us
// have some parallelism without worry of on caller seeing stale data from the quota table.
synchronized (getLockForNamespace(tn.getNamespaceAsString())) {
final Result[] existingSnapshotSizes = quotaTable.get(snapshotSizeGets);
long totalSizeChange = 0;
// Read the current size values (if they exist) to generate the new value
for (Result result : existingSnapshotSizes) {
Entry entry = iterator.next();
String snapshot = entry.getKey();
Long size = entry.getValue();
// Track the total size change for the namespace this table belongs in
totalSizeChange += size;
// Get the size of the previous value (or zero)
long previousSize = getSnapshotSizeFromResult(result);
// Create an update. A file was archived from the table, so the table's size goes
// down, but the snapshot's size goes up.
updates.add(QuotaTableUtil.createPutForSnapshotSize(tn, snapshot, previousSize + size));
}
// Create an update for the summation of all snapshots in the namespace
if (totalSizeChange != 0) {
long previousSize = getPreviousNamespaceSnapshotSize(
quotaTable, tn.getNamespaceAsString());
updates.add(QuotaTableUtil.createPutForNamespaceSnapshotSize(
tn.getNamespaceAsString(), previousSize + totalSizeChange));
}
// Send all of the quota table updates in one batch.
List