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.snapshot; 019 020import java.io.IOException; 021import java.util.ArrayList; 022import java.util.HashMap; 023import java.util.List; 024import java.util.Map; 025import java.util.TreeMap; 026import org.apache.commons.lang3.NotImplementedException; 027import org.apache.hadoop.conf.Configuration; 028import org.apache.hadoop.fs.FileSystem; 029import org.apache.hadoop.fs.Path; 030import org.apache.hadoop.hbase.HRegionLocation; 031import org.apache.hadoop.hbase.ServerName; 032import org.apache.hadoop.hbase.TableName; 033import org.apache.hadoop.hbase.client.RegionInfo; 034import org.apache.hadoop.hbase.client.RegionInfoBuilder; 035import org.apache.hadoop.hbase.client.RegionLocator; 036import org.apache.hadoop.hbase.util.Bytes; 037import org.apache.yetus.audience.InterfaceAudience; 038 039import org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos; 040import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos; 041 042/** 043 * {@link RegionLocator} built using the most recent full backup's snapshot manifest for a given 044 * table. Useful for aligning any subsequent incremental backups along the same splits as the full 045 * backup 046 */ 047@InterfaceAudience.Private 048public final class SnapshotRegionLocator implements RegionLocator { 049 050 private static final String SNAPSHOT_MANIFEST_DIR_PREFIX = 051 "region.locator.snapshot.manifest.dir."; 052 053 private static final ServerName FAKE_SERVER_NAME = 054 ServerName.parseServerName("www.example.net,1234,1212121212"); 055 056 private final TableName tableName; 057 private final TreeMap<byte[], HRegionReplicas> regions; 058 059 private final List<HRegionLocation> rawLocations; 060 061 public static SnapshotRegionLocator create(Configuration conf, TableName table) 062 throws IOException { 063 Path workingDir = new Path(conf.get(getSnapshotManifestDirKey(table))); 064 FileSystem fs = workingDir.getFileSystem(conf); 065 SnapshotProtos.SnapshotDescription desc = 066 SnapshotDescriptionUtils.readSnapshotInfo(fs, workingDir); 067 SnapshotManifest manifest = SnapshotManifest.open(conf, fs, workingDir, desc); 068 069 TableName tableName = manifest.getTableDescriptor().getTableName(); 070 TreeMap<byte[], HRegionReplicas> replicas = new TreeMap<>(Bytes.BYTES_COMPARATOR); 071 List<HRegionLocation> rawLocations = new ArrayList<>(); 072 073 for (SnapshotProtos.SnapshotRegionManifest region : manifest.getRegionManifests()) { 074 HBaseProtos.RegionInfo ri = region.getRegionInfo(); 075 byte[] key = ri.getStartKey().toByteArray(); 076 077 SnapshotHRegionLocation location = toLocation(ri, tableName); 078 rawLocations.add(location); 079 HRegionReplicas hrr = replicas.get(key); 080 081 if (hrr == null) { 082 hrr = new HRegionReplicas(location); 083 } else { 084 hrr.addReplica(location); 085 } 086 087 replicas.put(key, hrr); 088 } 089 090 return new SnapshotRegionLocator(tableName, replicas, rawLocations); 091 } 092 093 private SnapshotRegionLocator(TableName tableName, TreeMap<byte[], HRegionReplicas> regions, 094 List<HRegionLocation> rawLocations) { 095 this.tableName = tableName; 096 this.regions = regions; 097 this.rawLocations = rawLocations; 098 } 099 100 @Override 101 public HRegionLocation getRegionLocation(byte[] row, int replicaId, boolean reload) 102 throws IOException { 103 return regions.floorEntry(row).getValue().getReplica(replicaId); 104 } 105 106 @Override 107 public List<HRegionLocation> getRegionLocations(byte[] row, boolean reload) throws IOException { 108 return List.of(getRegionLocation(row, reload)); 109 } 110 111 @Override 112 public void clearRegionLocationCache() { 113 114 } 115 116 @Override 117 public List<HRegionLocation> getAllRegionLocations() throws IOException { 118 return rawLocations; 119 } 120 121 @Override 122 public TableName getName() { 123 return tableName; 124 } 125 126 @Override 127 public void close() throws IOException { 128 129 } 130 131 public static boolean shouldUseSnapshotRegionLocator(Configuration conf, TableName table) { 132 return conf.get(getSnapshotManifestDirKey(table)) != null; 133 } 134 135 public static void setSnapshotManifestDir(Configuration conf, String dir, TableName table) { 136 conf.set(getSnapshotManifestDirKey(table), dir); 137 } 138 139 private static String getSnapshotManifestDirKey(TableName table) { 140 return SNAPSHOT_MANIFEST_DIR_PREFIX + table.getNameWithNamespaceInclAsString(); 141 } 142 143 private static SnapshotHRegionLocation toLocation(HBaseProtos.RegionInfo ri, 144 TableName tableName) { 145 RegionInfo region = RegionInfoBuilder.newBuilder(tableName) 146 .setStartKey(ri.getStartKey().toByteArray()).setEndKey(ri.getEndKey().toByteArray()) 147 .setRegionId(ri.getRegionId()).setReplicaId(ri.getReplicaId()).build(); 148 149 return new SnapshotHRegionLocation(region); 150 } 151 152 private static final class HRegionReplicas { 153 private final Map<Integer, SnapshotHRegionLocation> replicas = new HashMap<>(); 154 155 private HRegionReplicas(SnapshotHRegionLocation region) { 156 addReplica(region); 157 } 158 159 private void addReplica(SnapshotHRegionLocation replica) { 160 this.replicas.put(replica.getRegion().getReplicaId(), replica); 161 } 162 163 private HRegionLocation getReplica(int replicaId) { 164 return replicas.get(replicaId); 165 } 166 } 167 168 public static final class SnapshotHRegionLocation extends HRegionLocation { 169 170 public SnapshotHRegionLocation(RegionInfo regionInfo) { 171 super(regionInfo, FAKE_SERVER_NAME); 172 } 173 174 @Override 175 public ServerName getServerName() { 176 throw new NotImplementedException("SnapshotHRegionLocation doesn't have a server name"); 177 } 178 179 @Override 180 public String getHostname() { 181 throw new NotImplementedException("SnapshotHRegionLocation doesn't have a host name"); 182 } 183 184 @Override 185 public int getPort() { 186 throw new NotImplementedException("SnapshotHRegionLocation doesn't have a port"); 187 } 188 189 @Override 190 public int hashCode() { 191 return this.getRegion().hashCode(); 192 } 193 194 @Override 195 public boolean equals(Object o) { 196 return super.equals(o); 197 } 198 199 @Override 200 public int compareTo(HRegionLocation o) { 201 return this.getRegion().compareTo(o.getRegion()); 202 } 203 } 204}