From 919865a34bd5c3c99603993a0410846a97975869 Mon Sep 17 00:00:00 2001 From: Aaron Fabbri Date: Mon, 30 Apr 2018 16:02:57 -0700 Subject: [PATCH] HADOOP-15239 S3ABlockOutputStream.flush() be no-op when stream closed. Contributed by Gabor Bota. --- .../hadoop/fs/s3a/S3ABlockOutputStream.java | 7 +- .../fs/s3a/TestS3ABlockOutputStream.java | 66 +++++++++++++++++++ 2 files changed, 72 insertions(+), 1 deletion(-) create mode 100644 hadoop-tools/hadoop-aws/src/test/java/org/apache/hadoop/fs/s3a/TestS3ABlockOutputStream.java diff --git a/hadoop-tools/hadoop-aws/src/main/java/org/apache/hadoop/fs/s3a/S3ABlockOutputStream.java b/hadoop-tools/hadoop-aws/src/main/java/org/apache/hadoop/fs/s3a/S3ABlockOutputStream.java index 96de8e438d..bdffed4b25 100644 --- a/hadoop-tools/hadoop-aws/src/main/java/org/apache/hadoop/fs/s3a/S3ABlockOutputStream.java +++ b/hadoop-tools/hadoop-aws/src/main/java/org/apache/hadoop/fs/s3a/S3ABlockOutputStream.java @@ -238,7 +238,12 @@ void checkOpen() throws IOException { */ @Override public synchronized void flush() throws IOException { - checkOpen(); + try { + checkOpen(); + } catch (IOException e) { + LOG.warn("Stream closed: " + e.getMessage()); + return; + } S3ADataBlocks.DataBlock dataBlock = getActiveBlock(); if (dataBlock != null) { dataBlock.flush(); diff --git a/hadoop-tools/hadoop-aws/src/test/java/org/apache/hadoop/fs/s3a/TestS3ABlockOutputStream.java b/hadoop-tools/hadoop-aws/src/test/java/org/apache/hadoop/fs/s3a/TestS3ABlockOutputStream.java new file mode 100644 index 0000000000..ff176f58da --- /dev/null +++ b/hadoop-tools/hadoop-aws/src/test/java/org/apache/hadoop/fs/s3a/TestS3ABlockOutputStream.java @@ -0,0 +1,66 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.hadoop.fs.s3a; + +import org.apache.hadoop.fs.s3a.commit.PutTracker; +import org.apache.hadoop.util.Progressable; +import org.junit.Before; +import org.junit.Test; + +import java.io.IOException; +import java.util.concurrent.ExecutorService; + +import static org.junit.Assert.*; +import static org.mockito.Mockito.doThrow; +import static org.mockito.Mockito.mock; +import static org.mockito.Mockito.spy; + +/** + * Unit tests for {@link S3ABlockOutputStream}. + */ +public class TestS3ABlockOutputStream extends AbstractS3AMockTest { + + private S3ABlockOutputStream stream; + + @Before + public void setUp() throws Exception { + ExecutorService executorService = mock(ExecutorService.class); + Progressable progressable = mock(Progressable.class); + S3ADataBlocks.BlockFactory blockFactory = + mock(S3ADataBlocks.BlockFactory.class); + long blockSize = Constants.DEFAULT_MULTIPART_SIZE; + S3AInstrumentation.OutputStreamStatistics statistics = null; + WriteOperationHelper oHelper = mock(WriteOperationHelper.class); + PutTracker putTracker = mock(PutTracker.class); + stream = spy(new S3ABlockOutputStream(fs, "", executorService, + progressable, blockSize, blockFactory, statistics, oHelper, + putTracker)); + } + + @Test + public void testFlushNoOpWhenStreamClosed() throws Exception { + doThrow(new IOException()).when(stream).checkOpen(); + + try { + stream.flush(); + } catch (Exception e){ + fail("Should not have any exception."); + } + } +}