This is an automated email from the ASF dual-hosted git repository.
zhangduo pushed a commit to branch branch-2.2
in repository https://gitbox.apache.org/repos/asf/hbase.git
The following commit(s) were added to refs/heads/branch-2.2 by this push:
new 160c229 Revert "HBASE-24625 AsyncFSWAL.getLogFileSizeIfBeingWritten
does not return the expected synced file length. (#1970)"
160c229 is described below
commit 160c2290bf6540a4a56df50eda6bfb77130a1351
Author: Duo Zhang <[email protected]>
AuthorDate: Tue Jul 7 18:17:20 2020 +0800
Revert "HBASE-24625 AsyncFSWAL.getLogFileSizeIfBeingWritten does not return
the expected synced file length. (#1970)"
This reverts commit bf587fa2411940cb47d3e207a67714f598b0d228.
---
.../hadoop/hbase/io/asyncfs/AsyncFSOutput.java | 5 ---
.../io/asyncfs/FanOutOneBlockAsyncDFSOutput.java | 5 ---
.../hbase/io/asyncfs/WrapperAsyncFSOutput.java | 13 +-----
.../hbase/regionserver/wal/AbstractFSWAL.java | 2 +-
.../regionserver/wal/AsyncProtobufLogWriter.java | 5 ---
.../hbase/regionserver/wal/ProtobufLogWriter.java | 11 -----
.../org/apache/hadoop/hbase/wal/WALProvider.java | 17 --------
.../regionserver/TestFailedAppendAndSync.java | 49 ++++++++++------------
.../hadoop/hbase/regionserver/TestHRegion.java | 5 ---
.../hadoop/hbase/regionserver/TestWALLockup.java | 10 -----
.../hbase/regionserver/wal/TestAsyncFSWAL.java | 5 ---
.../regionserver/wal/TestAsyncFSWALDurability.java | 5 ---
.../regionserver/wal/TestFSHLogDurability.java | 5 ---
13 files changed, 24 insertions(+), 113 deletions(-)
diff --git
a/hbase-server/src/main/java/org/apache/hadoop/hbase/io/asyncfs/AsyncFSOutput.java
b/hbase-server/src/main/java/org/apache/hadoop/hbase/io/asyncfs/AsyncFSOutput.java
index 059ca00..3c520b8 100644
---
a/hbase-server/src/main/java/org/apache/hadoop/hbase/io/asyncfs/AsyncFSOutput.java
+++
b/hbase-server/src/main/java/org/apache/hadoop/hbase/io/asyncfs/AsyncFSOutput.java
@@ -89,9 +89,4 @@ public interface AsyncFSOutput extends Closeable {
*/
@Override
void close() throws IOException;
-
- /**
- * @return byteSize success synced to underlying filesystem.
- */
- long getSyncedLength();
}
diff --git
a/hbase-server/src/main/java/org/apache/hadoop/hbase/io/asyncfs/FanOutOneBlockAsyncDFSOutput.java
b/hbase-server/src/main/java/org/apache/hadoop/hbase/io/asyncfs/FanOutOneBlockAsyncDFSOutput.java
index 5f70cfb..ed9da5f 100644
---
a/hbase-server/src/main/java/org/apache/hadoop/hbase/io/asyncfs/FanOutOneBlockAsyncDFSOutput.java
+++
b/hbase-server/src/main/java/org/apache/hadoop/hbase/io/asyncfs/FanOutOneBlockAsyncDFSOutput.java
@@ -577,9 +577,4 @@ public class FanOutOneBlockAsyncDFSOutput implements
AsyncFSOutput {
public boolean isBroken() {
return state == State.BROKEN;
}
-
- @Override
- public long getSyncedLength() {
- return this.ackedBlockLength;
- }
}
diff --git
a/hbase-server/src/main/java/org/apache/hadoop/hbase/io/asyncfs/WrapperAsyncFSOutput.java
b/hbase-server/src/main/java/org/apache/hadoop/hbase/io/asyncfs/WrapperAsyncFSOutput.java
index 39f1f71..bbb4e54 100644
---
a/hbase-server/src/main/java/org/apache/hadoop/hbase/io/asyncfs/WrapperAsyncFSOutput.java
+++
b/hbase-server/src/main/java/org/apache/hadoop/hbase/io/asyncfs/WrapperAsyncFSOutput.java
@@ -45,8 +45,6 @@ public class WrapperAsyncFSOutput implements AsyncFSOutput {
private final ExecutorService executor;
- private volatile long syncedLength = 0;
-
public WrapperAsyncFSOutput(Path file, FSDataOutputStream out) {
this.out = out;
this.executor = Executors.newSingleThreadExecutor(new
ThreadFactoryBuilder().setDaemon(true)
@@ -93,11 +91,7 @@ public class WrapperAsyncFSOutput implements AsyncFSOutput {
out.hflush();
}
}
- long pos = out.getPos();
- if(pos > this.syncedLength) {
- this.syncedLength = pos;
- }
- future.complete(pos);
+ future.complete(out.getPos());
} catch (IOException e) {
future.completeExceptionally(e);
return;
@@ -130,9 +124,4 @@ public class WrapperAsyncFSOutput implements AsyncFSOutput {
public boolean isBroken() {
return false;
}
-
- @Override
- public long getSyncedLength() {
- return this.syncedLength;
- }
}
diff --git
a/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/AbstractFSWAL.java
b/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/AbstractFSWAL.java
index 5621a19..2a7ebff 100644
---
a/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/AbstractFSWAL.java
+++
b/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/AbstractFSWAL.java
@@ -1014,7 +1014,7 @@ public abstract class AbstractFSWAL<W extends WriterBase>
implements WAL {
Path currentPath = getOldPath();
if (path.equals(currentPath)) {
W writer = this.writer;
- return writer != null ? OptionalLong.of(writer.getSyncedLength()) :
OptionalLong.empty();
+ return writer != null ? OptionalLong.of(writer.getLength()) :
OptionalLong.empty();
} else {
return OptionalLong.empty();
}
diff --git
a/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/AsyncProtobufLogWriter.java
b/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/AsyncProtobufLogWriter.java
index 8c944b1..e731611 100644
---
a/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/AsyncProtobufLogWriter.java
+++
b/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/AsyncProtobufLogWriter.java
@@ -231,9 +231,4 @@ public class AsyncProtobufLogWriter extends
AbstractProtobufLogWriter
protected OutputStream getOutputStreamForCellEncoder() {
return asyncOutputWrapper;
}
-
- @Override
- public long getSyncedLength() {
- return this.output.getSyncedLength();
- }
}
diff --git
a/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/ProtobufLogWriter.java
b/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/ProtobufLogWriter.java
index 4d8548c..5c8e0d2 100644
---
a/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/ProtobufLogWriter.java
+++
b/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/ProtobufLogWriter.java
@@ -19,13 +19,10 @@ package org.apache.hadoop.hbase.regionserver.wal;
import java.io.IOException;
import java.io.OutputStream;
-import java.util.concurrent.atomic.AtomicLong;
-
import org.apache.hadoop.fs.FSDataOutputStream;
import org.apache.hadoop.fs.FileSystem;
import org.apache.hadoop.fs.Path;
import org.apache.hadoop.hbase.Cell;
-import org.apache.hadoop.hbase.util.AtomicUtils;
import org.apache.hadoop.hbase.util.CommonFSUtils;
import
org.apache.hadoop.hbase.util.CommonFSUtils.StreamLacksCapabilityException;
import org.apache.hadoop.hbase.wal.FSHLogProvider;
@@ -48,8 +45,6 @@ public class ProtobufLogWriter extends
AbstractProtobufLogWriter
protected FSDataOutputStream output;
- private final AtomicLong syncedLength = new AtomicLong(0);
-
@Override
public void append(Entry entry) throws IOException {
entry.getKey().getBuilder(compressor).
@@ -89,12 +84,6 @@ public class ProtobufLogWriter extends
AbstractProtobufLogWriter
} else {
fsdos.hflush();
}
- AtomicUtils.updateMax(this.syncedLength, fsdos.getPos());
- }
-
- @Override
- public long getSyncedLength() {
- return this.syncedLength.get();
}
public FSDataOutputStream getStream() {
diff --git
a/hbase-server/src/main/java/org/apache/hadoop/hbase/wal/WALProvider.java
b/hbase-server/src/main/java/org/apache/hadoop/hbase/wal/WALProvider.java
index c3bd149..6f0b983 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/wal/WALProvider.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/wal/WALProvider.java
@@ -25,7 +25,6 @@ import java.util.OptionalLong;
import java.util.concurrent.CompletableFuture;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hbase.client.RegionInfo;
-import org.apache.hadoop.hbase.regionserver.wal.AsyncFSWAL;
import org.apache.hadoop.hbase.regionserver.wal.WALActionsListener;
import org.apache.hadoop.hbase.replication.regionserver.WALFileLengthProvider;
import org.apache.yetus.audience.InterfaceAudience;
@@ -75,22 +74,6 @@ public interface WALProvider {
interface WriterBase extends Closeable {
long getLength();
- /**
- * NOTE: We add this method for {@link WALFileLengthProvider} used for
replication,
- * considering the case if we use {@link AsyncFSWAL},we write to 3 DNs
concurrently,
- * according to the visibility guarantee of HDFS, the data will be
available immediately
- * when arriving at DN since all the DNs will be considered as the last
one in pipeline.
- * This means replication may read uncommitted data and replicate it to
the remote cluster
- * and cause data inconsistency.
- * The method {@link WriterBase#getLength} may return length which just in
hdfs client
- * buffer and not successfully synced to HDFS, so we use this method to
return the length
- * successfully synced to HDFS and replication thread could only read
writing WAL file
- * limited by this length.
- * see also HBASE-14004 and this document for more details:
- *
https://docs.google.com/document/d/11AyWtGhItQs6vsLRIx32PwTxmBY3libXwGXI25obVEY/edit#
- * @return byteSize successfully synced to underlying filesystem.
- */
- long getSyncedLength();
}
// Writers are used internally. Users outside of the WAL should be relying
on the
diff --git
a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestFailedAppendAndSync.java
b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestFailedAppendAndSync.java
index 2013778..4e6a1fe 100644
---
a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestFailedAppendAndSync.java
+++
b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestFailedAppendAndSync.java
@@ -130,40 +130,35 @@ public class TestFailedAppendAndSync {
@Override
protected Writer createWriterInstance(Path path) throws IOException {
final Writer w = super.createWriterInstance(path);
- return new Writer() {
- @Override
- public void close() throws IOException {
- w.close();
- }
-
- @Override
- public void sync(boolean forceSync) throws IOException {
- if (throwSyncException) {
- throw new IOException("FAKE! Failed to replace a bad
datanode...");
+ return new Writer() {
+ @Override
+ public void close() throws IOException {
+ w.close();
}
- w.sync(forceSync);
- }
- @Override
- public void append(Entry entry) throws IOException {
- if (throwAppendException) {
- throw new IOException("FAKE! Failed to replace a bad
datanode...");
+ @Override
+ public void sync(boolean forceSync) throws IOException {
+ if (throwSyncException) {
+ throw new IOException("FAKE! Failed to replace a bad
datanode...");
+ }
+ w.sync(forceSync);
}
- w.append(entry);
- }
- @Override
- public long getLength() {
- return w.getLength();
- }
+ @Override
+ public void append(Entry entry) throws IOException {
+ if (throwAppendException) {
+ throw new IOException("FAKE! Failed to replace a bad
datanode...");
+ }
+ w.append(entry);
+ }
- @Override
- public long getSyncedLength() {
- return w.getSyncedLength();
+ @Override
+ public long getLength() {
+ return w.getLength();
+ }
+ };
}
- };
}
- }
// Make up mocked server and services.
RegionServerServices services = mock(RegionServerServices.class);
diff --git
a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegion.java
b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegion.java
index a7fd62f..66edbd7 100644
---
a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegion.java
+++
b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegion.java
@@ -1250,11 +1250,6 @@ public class TestHRegion {
public long getLength() {
return w.getLength();
}
-
- @Override
- public long getSyncedLength() {
- return w.getSyncedLength();
- }
};
}
}
diff --git
a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestWALLockup.java
b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestWALLockup.java
index 21f1774..a50ef78 100644
---
a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestWALLockup.java
+++
b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestWALLockup.java
@@ -190,11 +190,6 @@ public class TestWALLockup {
public long getLength() {
return w.getLength();
}
-
- @Override
- public long getSyncedLength() {
- return w.getSyncedLength();
- }
};
}
}
@@ -379,11 +374,6 @@ public class TestWALLockup {
public long getLength() {
return w.getLength();
}
-
- @Override
- public long getSyncedLength() {
- return w.getSyncedLength();
- }
};
}
}
diff --git
a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/wal/TestAsyncFSWAL.java
b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/wal/TestAsyncFSWAL.java
index f31a908..704cdfa 100644
---
a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/wal/TestAsyncFSWAL.java
+++
b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/wal/TestAsyncFSWAL.java
@@ -156,11 +156,6 @@ public class TestAsyncFSWAL extends AbstractTestFSWAL {
}
@Override
- public long getSyncedLength() {
- return writer.getSyncedLength();
- }
-
- @Override
public CompletableFuture<Long> sync(boolean forceSync) {
CompletableFuture<Long> result = writer.sync(forceSync);
if (failedCount.incrementAndGet() < 1000) {
diff --git
a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/wal/TestAsyncFSWALDurability.java
b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/wal/TestAsyncFSWALDurability.java
index 2b1da95..f9dee07 100644
---
a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/wal/TestAsyncFSWALDurability.java
+++
b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/wal/TestAsyncFSWALDurability.java
@@ -110,11 +110,6 @@ class CustomAsyncFSWAL extends AsyncFSWAL {
}
@Override
- public long getSyncedLength() {
- return writer.getSyncedLength();
- }
-
- @Override
public CompletableFuture<Long> sync(boolean forceSync) {
writerSyncFlag = forceSync;
return writer.sync(forceSync);
diff --git
a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/wal/TestFSHLogDurability.java
b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/wal/TestFSHLogDurability.java
index 3c25044..9c46058 100644
---
a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/wal/TestFSHLogDurability.java
+++
b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/wal/TestFSHLogDurability.java
@@ -85,11 +85,6 @@ class CustomFSHLog extends FSHLog {
}
@Override
- public long getSyncedLength() {
- return writer.getSyncedLength();
- }
-
- @Override
public void sync(boolean forceSync) throws IOException {
writerSyncFlag = forceSync;
writer.sync(forceSync);