|
27 | 27 |
|
28 | 28 | import com.amazonaws.services.s3.model.GetObjectRequest; |
29 | 29 | import com.amazonaws.services.s3.model.S3Object; |
| 30 | +import com.amazonaws.services.s3.model.S3ObjectInputStream; |
30 | 31 | import org.slf4j.Logger; |
31 | 32 | import org.slf4j.LoggerFactory; |
32 | 33 |
|
|
36 | 37 | import org.apache.hadoop.fs.s3a.S3AReadOpContext; |
37 | 38 | import org.apache.hadoop.fs.s3a.S3ObjectAttributes; |
38 | 39 | import org.apache.hadoop.fs.s3a.impl.ChangeTracker; |
| 40 | +import org.apache.hadoop.fs.s3a.impl.SDKStreamDrainer; |
39 | 41 | import org.apache.hadoop.fs.s3a.statistics.S3AInputStreamStatistics; |
40 | 42 | import org.apache.hadoop.fs.statistics.DurationTracker; |
41 | 43 |
|
42 | | -import static org.apache.hadoop.fs.statistics.impl.IOStatisticsBinding.invokeTrackingDuration; |
43 | | -import static org.apache.hadoop.io.IOUtils.cleanupWithLogger; |
44 | | - |
45 | 44 | /** |
46 | 45 | * Encapsulates low level interactions with S3 object on AWS. |
47 | 46 | */ |
@@ -79,7 +78,7 @@ public class S3ARemoteObject { |
79 | 78 | * Maps a stream returned by openForRead() to the associated S3 object. |
80 | 79 | * That allows us to close the object when closing the stream. |
81 | 80 | */ |
82 | | - private Map<InputStream, S3Object> s3Objects; |
| 81 | + private final Map<InputStream, S3Object> s3Objects; |
83 | 82 |
|
84 | 83 | /** |
85 | 84 | * uri of the object being read. |
@@ -225,104 +224,27 @@ public InputStream openForRead(long offset, int size) throws IOException { |
225 | 224 | void close(InputStream inputStream, int numRemainingBytes) { |
226 | 225 | S3Object obj; |
227 | 226 | synchronized (s3Objects) { |
228 | | - obj = s3Objects.get(inputStream); |
| 227 | + obj = s3Objects.remove(inputStream); |
229 | 228 | if (obj == null) { |
230 | 229 | throw new IllegalArgumentException("inputStream not found"); |
231 | 230 | } |
232 | | - s3Objects.remove(inputStream); |
233 | 231 | } |
234 | | - |
| 232 | + SDKStreamDrainer drainer = new SDKStreamDrainer( |
| 233 | + uri, |
| 234 | + obj, |
| 235 | + (S3ObjectInputStream)inputStream, |
| 236 | + false, |
| 237 | + numRemainingBytes, |
| 238 | + streamStatistics, |
| 239 | + "close() operation"); |
235 | 240 | if (numRemainingBytes <= context.getAsyncDrainThreshold()) { |
236 | 241 | // don't bother with async io. |
237 | | - drain(false, "close() operation", numRemainingBytes, obj, inputStream); |
| 242 | + drainer.apply(); |
238 | 243 | } else { |
239 | 244 | LOG.debug("initiating asynchronous drain of {} bytes", numRemainingBytes); |
240 | | - // schedule an async drain/abort with references to the fields so they |
241 | | - // can be reused |
242 | | - client.submit( |
243 | | - () -> drain(false, "close() operation", numRemainingBytes, obj, |
244 | | - inputStream)); |
| 245 | + // schedule an async drain/abort |
| 246 | + client.submit(drainer); |
245 | 247 | } |
246 | 248 | } |
247 | 249 |
|
248 | | - /** |
249 | | - * drain the stream. This method is intended to be |
250 | | - * used directly or asynchronously, and measures the |
251 | | - * duration of the operation in the stream statistics. |
252 | | - * |
253 | | - * @param shouldAbort force an abort; used if explicitly requested. |
254 | | - * @param reason reason for stream being closed; used in messages |
255 | | - * @param remaining remaining bytes |
256 | | - * @param requestObject http request object; |
257 | | - * @param inputStream stream to close. |
258 | | - * @return was the stream aborted? |
259 | | - */ |
260 | | - private boolean drain( |
261 | | - final boolean shouldAbort, |
262 | | - final String reason, |
263 | | - final long remaining, |
264 | | - final S3Object requestObject, |
265 | | - final InputStream inputStream) { |
266 | | - |
267 | | - try { |
268 | | - return invokeTrackingDuration( |
269 | | - streamStatistics.initiateInnerStreamClose(shouldAbort), |
270 | | - () -> drainOrAbortHttpStream(shouldAbort, reason, remaining, |
271 | | - requestObject, inputStream)); |
272 | | - } catch (IOException e) { |
273 | | - // this is only here because invokeTrackingDuration() has it in its |
274 | | - // signature |
275 | | - return shouldAbort; |
276 | | - } |
277 | | - } |
278 | | - |
279 | | - /** |
280 | | - * Drain or abort the inner stream. |
281 | | - * Exceptions are swallowed. |
282 | | - * If a close() is attempted and fails, the operation escalates to |
283 | | - * an abort. |
284 | | - * |
285 | | - * @param shouldAbort force an abort; used if explicitly requested. |
286 | | - * @param reason reason for stream being closed; used in messages |
287 | | - * @param remaining remaining bytes |
288 | | - * @param requestObject http request object |
289 | | - * @param inputStream stream to close. |
290 | | - * @return was the stream aborted? |
291 | | - */ |
292 | | - private boolean drainOrAbortHttpStream( |
293 | | - boolean shouldAbort, |
294 | | - final String reason, |
295 | | - final long remaining, |
296 | | - final S3Object requestObject, |
297 | | - final InputStream inputStream) { |
298 | | - |
299 | | - if (!shouldAbort && remaining > 0) { |
300 | | - try { |
301 | | - long drained = 0; |
302 | | - byte[] buffer = new byte[DRAIN_BUFFER_SIZE]; |
303 | | - while (true) { |
304 | | - final int count = inputStream.read(buffer); |
305 | | - if (count < 0) { |
306 | | - // no more data is left |
307 | | - break; |
308 | | - } |
309 | | - drained += count; |
310 | | - } |
311 | | - LOG.debug("Drained stream of {} bytes", drained); |
312 | | - } catch (Exception e) { |
313 | | - // exception escalates to an abort |
314 | | - LOG.debug("When closing {} stream for {}, will abort the stream", uri, |
315 | | - reason, e); |
316 | | - shouldAbort = true; |
317 | | - } |
318 | | - } |
319 | | - cleanupWithLogger(LOG, inputStream); |
320 | | - cleanupWithLogger(LOG, requestObject); |
321 | | - streamStatistics.streamClose(shouldAbort, remaining); |
322 | | - |
323 | | - LOG.debug("Stream {} {}: {}; remaining={}", uri, |
324 | | - (shouldAbort ? "aborted" : "closed"), reason, |
325 | | - remaining); |
326 | | - return shouldAbort; |
327 | | - } |
328 | 250 | } |
0 commit comments