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 */ 018 019package org.apache.hadoop.hbase.snapshot; 020 021import java.io.IOException; 022import java.io.InterruptedIOException; 023import java.util.ArrayList; 024import java.util.List; 025import java.util.concurrent.Callable; 026import java.util.concurrent.ExecutionException; 027import java.util.concurrent.Executor; 028import java.util.concurrent.ExecutorCompletionService; 029 030import org.apache.hadoop.conf.Configuration; 031import org.apache.hadoop.fs.FSDataInputStream; 032import org.apache.hadoop.fs.FSDataOutputStream; 033import org.apache.hadoop.fs.FileStatus; 034import org.apache.hadoop.fs.FileSystem; 035import org.apache.hadoop.fs.Path; 036import org.apache.hadoop.fs.PathFilter; 037import org.apache.hadoop.hbase.client.RegionInfo; 038import org.apache.hadoop.hbase.regionserver.StoreFileInfo; 039import org.apache.hadoop.hbase.util.FSUtils; 040import org.apache.yetus.audience.InterfaceAudience; 041import org.slf4j.Logger; 042import org.slf4j.LoggerFactory; 043import org.apache.hbase.thirdparty.com.google.protobuf.CodedInputStream; 044import org.apache.hbase.thirdparty.com.google.protobuf.InvalidProtocolBufferException; 045import org.apache.hbase.thirdparty.com.google.protobuf.UnsafeByteOperations; 046import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil; 047import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription; 048import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotRegionManifest; 049 050/** 051 * DO NOT USE DIRECTLY. USE {@link SnapshotManifest}. 052 * 053 * Snapshot v2 layout format 054 * - Single Manifest file containing all the information of regions 055 * - In the online-snapshot case each region will write a "region manifest" 056 * /snapshotName/manifest.regionName 057 */ 058@InterfaceAudience.Private 059public final class SnapshotManifestV2 { 060 private static final Logger LOG = LoggerFactory.getLogger(SnapshotManifestV2.class); 061 062 public static final int DESCRIPTOR_VERSION = 2; 063 064 public static final String SNAPSHOT_MANIFEST_PREFIX = "region-manifest."; 065 066 private SnapshotManifestV2() {} 067 068 static class ManifestBuilder implements SnapshotManifest.RegionVisitor< 069 SnapshotRegionManifest.Builder, SnapshotRegionManifest.FamilyFiles.Builder> { 070 private final Configuration conf; 071 private final Path snapshotDir; 072 private final FileSystem rootFs; 073 074 public ManifestBuilder(final Configuration conf, final FileSystem rootFs, 075 final Path snapshotDir) { 076 this.snapshotDir = snapshotDir; 077 this.conf = conf; 078 this.rootFs = rootFs; 079 } 080 081 @Override 082 public SnapshotRegionManifest.Builder regionOpen(final RegionInfo regionInfo) { 083 SnapshotRegionManifest.Builder manifest = SnapshotRegionManifest.newBuilder(); 084 manifest.setRegionInfo(ProtobufUtil.toRegionInfo(regionInfo)); 085 return manifest; 086 } 087 088 @Override 089 public void regionClose(final SnapshotRegionManifest.Builder region) throws IOException { 090 // we should ensure the snapshot dir exist, maybe it has been deleted by master 091 // see HBASE-16464 092 FileSystem workingDirFs = snapshotDir.getFileSystem(this.conf); 093 if (workingDirFs.exists(snapshotDir)) { 094 SnapshotRegionManifest manifest = region.build(); 095 FSDataOutputStream stream = workingDirFs.create( 096 getRegionManifestPath(snapshotDir, manifest)); 097 try { 098 manifest.writeTo(stream); 099 } finally { 100 stream.close(); 101 } 102 } else { 103 LOG.warn("can't write manifest without parent dir, maybe it has been deleted by master?"); 104 } 105 } 106 107 @Override 108 public SnapshotRegionManifest.FamilyFiles.Builder familyOpen( 109 final SnapshotRegionManifest.Builder region, final byte[] familyName) { 110 SnapshotRegionManifest.FamilyFiles.Builder family = 111 SnapshotRegionManifest.FamilyFiles.newBuilder(); 112 family.setFamilyName(UnsafeByteOperations.unsafeWrap(familyName)); 113 return family; 114 } 115 116 @Override 117 public void familyClose(final SnapshotRegionManifest.Builder region, 118 final SnapshotRegionManifest.FamilyFiles.Builder family) { 119 region.addFamilyFiles(family.build()); 120 } 121 122 @Override 123 public void storeFile(final SnapshotRegionManifest.Builder region, 124 final SnapshotRegionManifest.FamilyFiles.Builder family, final StoreFileInfo storeFile) 125 throws IOException { 126 SnapshotRegionManifest.StoreFile.Builder sfManifest = 127 SnapshotRegionManifest.StoreFile.newBuilder(); 128 sfManifest.setName(storeFile.getPath().getName()); 129 if (storeFile.isReference()) { 130 sfManifest.setReference(storeFile.getReference().convert()); 131 } 132 if (!storeFile.isReference() && !storeFile.isLink()) { 133 sfManifest.setFileSize(storeFile.getSize()); 134 } else { 135 sfManifest.setFileSize(storeFile.getReferencedFileStatus(rootFs).getLen()); 136 } 137 family.addStoreFiles(sfManifest.build()); 138 } 139 } 140 141 static List<SnapshotRegionManifest> loadRegionManifests(final Configuration conf, 142 final Executor executor, final FileSystem fs, final Path snapshotDir, 143 final SnapshotDescription desc, final int manifestSizeLimit) throws IOException { 144 FileStatus[] manifestFiles = FSUtils.listStatus(fs, snapshotDir, new PathFilter() { 145 @Override 146 public boolean accept(Path path) { 147 return path.getName().startsWith(SNAPSHOT_MANIFEST_PREFIX); 148 } 149 }); 150 151 if (manifestFiles == null || manifestFiles.length == 0) return null; 152 153 final ExecutorCompletionService<SnapshotRegionManifest> completionService = 154 new ExecutorCompletionService<>(executor); 155 for (final FileStatus st: manifestFiles) { 156 completionService.submit(new Callable<SnapshotRegionManifest>() { 157 @Override 158 public SnapshotRegionManifest call() throws IOException { 159 FSDataInputStream stream = fs.open(st.getPath()); 160 CodedInputStream cin = CodedInputStream.newInstance(stream); 161 cin.setSizeLimit(manifestSizeLimit); 162 163 try { 164 return SnapshotRegionManifest.parseFrom(cin); 165 } finally { 166 stream.close(); 167 } 168 } 169 }); 170 } 171 172 ArrayList<SnapshotRegionManifest> regionsManifest = new ArrayList<>(manifestFiles.length); 173 try { 174 for (int i = 0; i < manifestFiles.length; ++i) { 175 regionsManifest.add(completionService.take().get()); 176 } 177 } catch (InterruptedException e) { 178 throw new InterruptedIOException(e.getMessage()); 179 } catch (ExecutionException e) { 180 Throwable t = e.getCause(); 181 182 if(t instanceof InvalidProtocolBufferException) { 183 throw (InvalidProtocolBufferException)t; 184 } else { 185 IOException ex = new IOException("ExecutionException"); 186 ex.initCause(e.getCause()); 187 throw ex; 188 } 189 } 190 return regionsManifest; 191 } 192 193 static void deleteRegionManifest(final FileSystem fs, final Path snapshotDir, 194 final SnapshotRegionManifest manifest) throws IOException { 195 fs.delete(getRegionManifestPath(snapshotDir, manifest), true); 196 } 197 198 private static Path getRegionManifestPath(final Path snapshotDir, 199 final SnapshotRegionManifest manifest) { 200 String regionName = SnapshotManifest.getRegionNameFromManifest(manifest); 201 return new Path(snapshotDir, SNAPSHOT_MANIFEST_PREFIX + regionName); 202 } 203}