1 /**
2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements. See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership. The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License. You may obtain a copy of the License at
9 *
10 * http://www.apache.org/licenses/LICENSE-2.0
11 *
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
17 */
18 package org.apache.hadoop.hbase.io;
19
20 import java.io.IOException;
21 import java.io.InputStream;
22 import java.lang.reflect.InvocationTargetException;
23 import java.lang.reflect.Method;
24
25 import org.apache.commons.logging.Log;
26 import org.apache.commons.logging.LogFactory;
27 import org.apache.hadoop.fs.FSDataInputStream;
28 import org.apache.hadoop.fs.FileSystem;
29 import org.apache.hadoop.fs.Path;
30 import org.apache.hadoop.hbase.fs.HFileSystem;
31
32 import com.google.common.annotations.VisibleForTesting;
33
34 /**
35 * Wrapper for input stream(s) that takes care of the interaction of FS and HBase checksums,
36 * as well as closing streams. Initialization is not thread-safe, but normal operation is;
37 * see method comments.
38 */
39 public class FSDataInputStreamWrapper {
40 private static final Log LOG = LogFactory.getLog(FSDataInputStreamWrapper.class);
41 private static final boolean isLogTraceEnabled = LOG.isTraceEnabled();
42
43 private final HFileSystem hfs;
44 private final Path path;
45 private final FileLink link;
46 private final boolean doCloseStreams;
47
48 /** Two stream handles, one with and one without FS-level checksum.
49 * HDFS checksum setting is on FS level, not single read level, so you have to keep two
50 * FS objects and two handles open to interleave different reads freely, which is very sad.
51 * This is what we do:
52 * 1) First, we need to read the trailer of HFile to determine checksum parameters.
53 * We always use FS checksum to do that, so ctor opens {@link #stream}.
54 * 2.1) After that, if HBase checksum is not used, we'd just always use {@link #stream};
55 * 2.2) If HBase checksum can be used, we'll open {@link #streamNoFsChecksum},
56 * and close {@link #stream}. User MUST call prepareForBlockReader for that to happen;
57 * if they don't, (2.1) will be the default.
58 * 3) The users can call {@link #shouldUseHBaseChecksum()}, and pass its result to
59 * {@link #getStream(boolean)} to get stream (if Java had out/pointer params we could
60 * return both in one call). This stream is guaranteed to be set.
61 * 4) The first time HBase checksum fails, one would call {@link #fallbackToFsChecksum(int)}.
62 * That will take lock, and open {@link #stream}. While this is going on, others will
63 * continue to use the old stream; if they also want to fall back, they'll also call
64 * {@link #fallbackToFsChecksum(int)}, and block until {@link #stream} is set.
65 * 5) After some number of checksumOk() calls, we will go back to using HBase checksum.
66 * We will have 2 handles; however we presume checksums fail so rarely that we don't care.
67 */
68 private volatile FSDataInputStream stream = null;
69 private volatile FSDataInputStream streamNoFsChecksum = null;
70 private Object streamNoFsChecksumFirstCreateLock = new Object();
71
72 // The configuration states that we should validate hbase checksums
73 private boolean useHBaseChecksumConfigured;
74
75 // Record the current state of this reader with respect to
76 // validating checkums in HBase. This is originally set the same
77 // value as useHBaseChecksumConfigured, but can change state as and when
78 // we encounter checksum verification failures.
79 private volatile boolean useHBaseChecksum;
80
81 // In the case of a checksum failure, do these many succeeding
82 // reads without hbase checksum verification.
83 private volatile int hbaseChecksumOffCount = -1;
84
85 private Boolean instanceOfCanUnbuffer = null;
86 // Using reflection to get org.apache.hadoop.fs.CanUnbuffer#unbuffer method to avoid compilation
87 // errors against Hadoop pre 2.6.4 and 2.7.1 versions.
88 private Method unbuffer = null;
89
90 public FSDataInputStreamWrapper(FileSystem fs, Path path) throws IOException {
91 this(fs, null, path, false);
92 }
93
94 public FSDataInputStreamWrapper(FileSystem fs, Path path, boolean dropBehind) throws IOException {
95 this(fs, null, path, dropBehind);
96 }
97
98 public FSDataInputStreamWrapper(FileSystem fs, FileLink link) throws IOException {
99 this(fs, link, null, false);
100 }
101 public FSDataInputStreamWrapper(FileSystem fs, FileLink link,
102 boolean dropBehind) throws IOException {
103 this(fs, link, null, dropBehind);
104 }
105
106 private FSDataInputStreamWrapper(FileSystem fs, FileLink link,
107 Path path, boolean dropBehind) throws IOException {
108 assert (path == null) != (link == null);
109 this.path = path;
110 this.link = link;
111 this.doCloseStreams = true;
112 // If the fs is not an instance of HFileSystem, then create an instance of HFileSystem
113 // that wraps over the specified fs. In this case, we will not be able to avoid
114 // checksumming inside the filesystem.
115 this.hfs = (fs instanceof HFileSystem) ? (HFileSystem)fs : new HFileSystem(fs);
116
117 // Initially we are going to read the tail block. Open the reader w/FS checksum.
118 this.useHBaseChecksumConfigured = this.useHBaseChecksum = false;
119 this.stream = (link != null) ? link.open(hfs) : hfs.open(path);
120 try {
121 this.stream.setDropBehind(dropBehind);
122 } catch (Exception e) {
123 // Skipped.
124 }
125 }
126
127
128 /**
129 * Prepares the streams for block reader. NOT THREAD SAFE. Must be called once, after any
130 * reads finish and before any other reads start (what happens in reality is we read the
131 * tail, then call this based on what's in the tail, then read blocks).
132 * @param forceNoHBaseChecksum Force not using HBase checksum.
133 */
134 public void prepareForBlockReader(boolean forceNoHBaseChecksum) throws IOException {
135 if (hfs == null) return;
136 assert this.stream != null && !this.useHBaseChecksumConfigured;
137 boolean useHBaseChecksum =
138 !forceNoHBaseChecksum && hfs.useHBaseChecksum() && (hfs.getNoChecksumFs() != hfs);
139
140 if (useHBaseChecksum) {
141 FileSystem fsNc = hfs.getNoChecksumFs();
142 this.streamNoFsChecksum = (link != null) ? link.open(fsNc) : fsNc.open(path);
143 this.useHBaseChecksumConfigured = this.useHBaseChecksum = useHBaseChecksum;
144 // Close the checksum stream; we will reopen it if we get an HBase checksum failure.
145 this.stream.close();
146 this.stream = null;
147 }
148 }
149
150 /** For use in tests. */
151 @VisibleForTesting
152 public FSDataInputStreamWrapper(FSDataInputStream fsdis) {
153 this(fsdis, fsdis);
154 }
155
156 /** For use in tests. */
157 @VisibleForTesting
158 public FSDataInputStreamWrapper(FSDataInputStream fsdis, FSDataInputStream noChecksum) {
159 doCloseStreams = false;
160 stream = fsdis;
161 streamNoFsChecksum = noChecksum;
162 path = null;
163 link = null;
164 hfs = null;
165 useHBaseChecksumConfigured = useHBaseChecksum = false;
166 }
167
168 /**
169 * @return Whether we are presently using HBase checksum.
170 */
171 public boolean shouldUseHBaseChecksum() {
172 return this.useHBaseChecksum;
173 }
174
175 /**
176 * Get the stream to use. Thread-safe.
177 * @param useHBaseChecksum must be the value that shouldUseHBaseChecksum has returned
178 * at some point in the past, otherwise the result is undefined.
179 */
180 public FSDataInputStream getStream(boolean useHBaseChecksum) {
181 return useHBaseChecksum ? this.streamNoFsChecksum : this.stream;
182 }
183
184 /**
185 * Read from non-checksum stream failed, fall back to FS checksum. Thread-safe.
186 * @param offCount For how many checksumOk calls to turn off the HBase checksum.
187 */
188 public FSDataInputStream fallbackToFsChecksum(int offCount) throws IOException {
189 // checksumOffCount is speculative, but let's try to reset it less.
190 boolean partOfConvoy = false;
191 if (this.stream == null) {
192 synchronized (streamNoFsChecksumFirstCreateLock) {
193 partOfConvoy = (this.stream != null);
194 if (!partOfConvoy) {
195 this.stream = (link != null) ? link.open(hfs) : hfs.open(path);
196 }
197 }
198 }
199 if (!partOfConvoy) {
200 this.useHBaseChecksum = false;
201 this.hbaseChecksumOffCount = offCount;
202 }
203 return this.stream;
204 }
205
206 /** Report that checksum was ok, so we may ponder going back to HBase checksum. */
207 public void checksumOk() {
208 if (this.useHBaseChecksumConfigured && !this.useHBaseChecksum
209 && (this.hbaseChecksumOffCount-- < 0)) {
210 // The stream we need is already open (because we were using HBase checksum in the past).
211 assert this.streamNoFsChecksum != null;
212 this.useHBaseChecksum = true;
213 }
214 }
215
216 /** Close stream(s) if necessary. */
217 public void close() throws IOException {
218 if (!doCloseStreams) return;
219 try {
220 if (stream != streamNoFsChecksum && streamNoFsChecksum != null) {
221 streamNoFsChecksum.close();
222 streamNoFsChecksum = null;
223 }
224 } finally {
225 if (stream != null) {
226 stream.close();
227 stream = null;
228 }
229 }
230 }
231
232 public HFileSystem getHfs() {
233 return this.hfs;
234 }
235
236 /**
237 * This will free sockets and file descriptors held by the stream only when the stream implements
238 * org.apache.hadoop.fs.CanUnbuffer. NOT THREAD SAFE. Must be called only when all the clients
239 * using this stream to read the blocks have finished reading. If by chance the stream is
240 * unbuffered and there are clients still holding this stream for read then on next client read
241 * request a new socket will be opened by Datanode without client knowing about it and will serve
242 * its read request. Note: If this socket is idle for some time then the DataNode will close the
243 * socket and the socket will move into CLOSE_WAIT state and on the next client request on this
244 * stream, the current socket will be closed and a new socket will be opened to serve the
245 * requests.
246 */
247 @SuppressWarnings({ "rawtypes" })
248 public void unbuffer() {
249 FSDataInputStream stream = this.getStream(this.shouldUseHBaseChecksum());
250 if (stream != null) {
251 InputStream wrappedStream = stream.getWrappedStream();
252 // CanUnbuffer interface was added as part of HDFS-7694 and the fix is available in Hadoop
253 // 2.6.4+ and 2.7.1+ versions only so check whether the stream object implements the
254 // CanUnbuffer interface or not and based on that call the unbuffer api.
255 final Class<? extends InputStream> streamClass = wrappedStream.getClass();
256 if (this.instanceOfCanUnbuffer == null) {
257 // To ensure we compute whether the stream is instance of CanUnbuffer only once.
258 this.instanceOfCanUnbuffer = false;
259 Class<?>[] streamInterfaces = streamClass.getInterfaces();
260 for (Class c : streamInterfaces) {
261 if (c.getCanonicalName().toString().equals("org.apache.hadoop.fs.CanUnbuffer")) {
262 try {
263 this.unbuffer = streamClass.getDeclaredMethod("unbuffer");
264 } catch (NoSuchMethodException | SecurityException e) {
265 if (isLogTraceEnabled) {
266 LOG.trace("Failed to find 'unbuffer' method in class " + streamClass
267 + " . So there may be a TCP socket connection "
268 + "left open in CLOSE_WAIT state.", e);
269 }
270 return;
271 }
272 this.instanceOfCanUnbuffer = true;
273 break;
274 }
275 }
276 }
277 if (this.instanceOfCanUnbuffer) {
278 try {
279 this.unbuffer.invoke(wrappedStream);
280 } catch (IllegalAccessException | IllegalArgumentException | InvocationTargetException e) {
281 if (isLogTraceEnabled) {
282 LOG.trace("Failed to invoke 'unbuffer' method in class " + streamClass
283 + " . So there may be a TCP socket connection left open in CLOSE_WAIT state.", e);
284 }
285 }
286 } else {
287 if (isLogTraceEnabled) {
288 LOG.trace("Failed to find 'unbuffer' method in class " + streamClass
289 + " . So there may be a TCP socket connection "
290 + "left open in CLOSE_WAIT state. For more details check "
291 + "https://issues.apache.org/jira/browse/HBASE-9393");
292 }
293 }
294 }
295 }
296 }