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 fs; 073 074 public ManifestBuilder(final Configuration conf, final FileSystem fs, final Path snapshotDir) { 075 this.snapshotDir = snapshotDir; 076 this.conf = conf; 077 this.fs = fs; 078 } 079 080 @Override 081 public SnapshotRegionManifest.Builder regionOpen(final RegionInfo regionInfo) { 082 SnapshotRegionManifest.Builder manifest = SnapshotRegionManifest.newBuilder(); 083 manifest.setRegionInfo(ProtobufUtil.toRegionInfo(regionInfo)); 084 return manifest; 085 } 086 087 @Override 088 public void regionClose(final SnapshotRegionManifest.Builder region) throws IOException { 089 // we should ensure the snapshot dir exist, maybe it has been deleted by master 090 // see HBASE-16464 091 if (fs.exists(snapshotDir)) { 092 SnapshotRegionManifest manifest = region.build(); 093 FSDataOutputStream stream = fs.create(getRegionManifestPath(snapshotDir, manifest)); 094 try { 095 manifest.writeTo(stream); 096 } finally { 097 stream.close(); 098 } 099 } else { 100 LOG.warn("can't write manifest without parent dir, maybe it has been deleted by master?"); 101 } 102 } 103 104 @Override 105 public SnapshotRegionManifest.FamilyFiles.Builder familyOpen( 106 final SnapshotRegionManifest.Builder region, final byte[] familyName) { 107 SnapshotRegionManifest.FamilyFiles.Builder family = 108 SnapshotRegionManifest.FamilyFiles.newBuilder(); 109 family.setFamilyName(UnsafeByteOperations.unsafeWrap(familyName)); 110 return family; 111 } 112 113 @Override 114 public void familyClose(final SnapshotRegionManifest.Builder region, 115 final SnapshotRegionManifest.FamilyFiles.Builder family) { 116 region.addFamilyFiles(family.build()); 117 } 118 119 @Override 120 public void storeFile(final SnapshotRegionManifest.Builder region, 121 final SnapshotRegionManifest.FamilyFiles.Builder family, final StoreFileInfo storeFile) 122 throws IOException { 123 SnapshotRegionManifest.StoreFile.Builder sfManifest = 124 SnapshotRegionManifest.StoreFile.newBuilder(); 125 sfManifest.setName(storeFile.getPath().getName()); 126 if (storeFile.isReference()) { 127 sfManifest.setReference(storeFile.getReference().convert()); 128 } 129 sfManifest.setFileSize(storeFile.getReferencedFileStatus(fs).getLen()); 130 family.addStoreFiles(sfManifest.build()); 131 } 132 } 133 134 static List<SnapshotRegionManifest> loadRegionManifests(final Configuration conf, 135 final Executor executor, final FileSystem fs, final Path snapshotDir, 136 final SnapshotDescription desc, final int manifestSizeLimit) throws IOException { 137 FileStatus[] manifestFiles = FSUtils.listStatus(fs, snapshotDir, new PathFilter() { 138 @Override 139 public boolean accept(Path path) { 140 return path.getName().startsWith(SNAPSHOT_MANIFEST_PREFIX); 141 } 142 }); 143 144 if (manifestFiles == null || manifestFiles.length == 0) return null; 145 146 final ExecutorCompletionService<SnapshotRegionManifest> completionService = 147 new ExecutorCompletionService<>(executor); 148 for (final FileStatus st: manifestFiles) { 149 completionService.submit(new Callable<SnapshotRegionManifest>() { 150 @Override 151 public SnapshotRegionManifest call() throws IOException { 152 FSDataInputStream stream = fs.open(st.getPath()); 153 CodedInputStream cin = CodedInputStream.newInstance(stream); 154 cin.setSizeLimit(manifestSizeLimit); 155 156 try { 157 return SnapshotRegionManifest.parseFrom(cin); 158 } finally { 159 stream.close(); 160 } 161 } 162 }); 163 } 164 165 ArrayList<SnapshotRegionManifest> regionsManifest = new ArrayList<>(manifestFiles.length); 166 try { 167 for (int i = 0; i < manifestFiles.length; ++i) { 168 regionsManifest.add(completionService.take().get()); 169 } 170 } catch (InterruptedException e) { 171 throw new InterruptedIOException(e.getMessage()); 172 } catch (ExecutionException e) { 173 Throwable t = e.getCause(); 174 175 if(t instanceof InvalidProtocolBufferException) { 176 throw (InvalidProtocolBufferException)t; 177 } else { 178 IOException ex = new IOException("ExecutionException"); 179 ex.initCause(e.getCause()); 180 throw ex; 181 } 182 } 183 return regionsManifest; 184 } 185 186 static void deleteRegionManifest(final FileSystem fs, final Path snapshotDir, 187 final SnapshotRegionManifest manifest) throws IOException { 188 fs.delete(getRegionManifestPath(snapshotDir, manifest), true); 189 } 190 191 private static Path getRegionManifestPath(final Path snapshotDir, 192 final SnapshotRegionManifest manifest) { 193 String regionName = SnapshotManifest.getRegionNameFromManifest(manifest); 194 return new Path(snapshotDir, SNAPSHOT_MANIFEST_PREFIX + regionName); 195 } 196}