Skip to content

Commit

Permalink
[ALLUXIO-2977] Remove unnecessary AtomicBoolean usage
Browse files Browse the repository at this point in the history
  • Loading branch information
apc999 committed Jul 31, 2017
1 parent 95eb9ed commit 184867e
Show file tree
Hide file tree
Showing 3 changed files with 21 additions and 16 deletions.
Expand Up @@ -33,7 +33,6 @@

import java.io.IOException;
import java.util.concurrent.ExecutorService;
import java.util.concurrent.atomic.AtomicBoolean;
import java.util.concurrent.atomic.AtomicReference;
import java.util.concurrent.locks.ReentrantLock;

Expand Down Expand Up @@ -124,7 +123,7 @@ abstract class AbstractReadHandler<T extends ReadRequest>
private Error mError;

/** This is set when the SUCCESS or CANCEL response is sent. This is only for sanity check. */
private AtomicBoolean mDone = new AtomicBoolean(false);
private volatile boolean mDone;

/**
* A wrapper on an error used to pass error information from the netty I/O thread to the packet
Expand Down Expand Up @@ -299,7 +298,7 @@ private void reset() {
mCancel = false;
mError = null;
mRequest.set(null);
mDone.set(false);
mDone = false;
}
}

Expand Down Expand Up @@ -509,8 +508,8 @@ private void replyError(AlluxioStatusException e) {
* Writes a success response.
*/
private void replyEof() {
Preconditions.checkState(!mDone.get());
mDone.set(true);
Preconditions.checkState(!mDone);
mDone = true;
mChannel.writeAndFlush(RPCProtoMessage.createOkResponse(null))
.addListeners(ChannelFutureListener.CLOSE_ON_FAILURE);
}
Expand All @@ -519,8 +518,8 @@ private void replyEof() {
* Writes a cancel response.
*/
private void replyCancel() {
Preconditions.checkState(!mDone.get());
mDone.set(true);
Preconditions.checkState(!mDone);
mDone = true;
mChannel.writeAndFlush(RPCProtoMessage.createCancelResponse())
.addListeners(ChannelFutureListener.CLOSE_ON_FAILURE);
}
Expand Down
Expand Up @@ -167,8 +167,9 @@ protected BlockWriteRequest createRequest(RPCProtoMessage msg) throws Exception
@Override
protected void completeRequest(Channel channel) throws Exception {
BlockWriteRequest request = getRequest();
Preconditions.checkState(request != null);

if (request == null) {
return;
}
if (request.getContext().getBlockWriter() != null) {
request.getContext().getBlockWriter().close();
}
Expand All @@ -178,8 +179,9 @@ protected void completeRequest(Channel channel) throws Exception {
@Override
protected void cancelRequest() throws Exception {
BlockWriteRequest request = getRequest();
Preconditions.checkState(request != null);

if (request == null) {
return;
}
if (request.getContext().getBlockWriter() != null) {
request.getContext().getBlockWriter().close();
}
Expand All @@ -189,7 +191,9 @@ protected void cancelRequest() throws Exception {
@Override
protected void cleanupRequest() throws Exception {
BlockWriteRequest request = getRequest();
Preconditions.checkState(request != null);
if (request == null) {
return;
}
mWorker.cleanupSession(request.getSessionId());
}

Expand Down
Expand Up @@ -180,8 +180,9 @@ protected UfsFileWriteRequest createRequest(RPCProtoMessage msg) throws Exceptio
@Override
protected void completeRequest(Channel channel) throws Exception {
UfsFileWriteRequest request = getRequest();
Preconditions.checkState(request != null);

if (request == null) {
return;
}
if (request.getContext().getOutputStream() == null) {
createUfsFile(channel);
}
Expand All @@ -194,8 +195,9 @@ protected void completeRequest(Channel channel) throws Exception {
@Override
protected void cancelRequest() throws Exception {
UfsFileWriteRequest request = getRequest();
Preconditions.checkState(request != null);

if (request == null) {
return;
}
// TODO(calvin): Consider adding cancel to the ufs stream api.
if (request.getContext().getOutputStream() != null
&& request.getContext().getUnderFileSystem() != null) {
Expand Down

0 comments on commit 184867e

Please sign in to comment.