Skip to content

Commit

Permalink
HBASE-21684 Throw DNRIOE when connection or rpc client is closed
Browse files Browse the repository at this point in the history
  • Loading branch information
Apache9 committed Jan 8, 2019
1 parent e9dae2a commit d957f0f
Show file tree
Hide file tree
Showing 2 changed files with 6 additions and 5 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -17,11 +17,11 @@
*/
package org.apache.hadoop.hbase.ipc;

import org.apache.hadoop.hbase.HBaseIOException;
import org.apache.hadoop.hbase.DoNotRetryIOException;
import org.apache.yetus.audience.InterfaceAudience;

@InterfaceAudience.Public
public class StoppedRpcClientException extends HBaseIOException {
public class StoppedRpcClientException extends DoNotRetryIOException {
public StoppedRpcClientException() {
super();
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -30,6 +30,7 @@
import java.util.concurrent.TimeUnit;
import java.util.concurrent.atomic.AtomicBoolean;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hbase.DoNotRetryIOException;
import org.apache.hadoop.hbase.util.Threads;
import org.apache.yetus.audience.InterfaceAudience;
import org.apache.zookeeper.KeeperException;
Expand Down Expand Up @@ -259,7 +260,7 @@ private static <T> CompletableFuture<T> failed(Throwable e) {

public CompletableFuture<byte[]> get(String path) {
if (closed.get()) {
return failed(new IOException("Client already closed"));
return failed(new DoNotRetryIOException("Client already closed"));
}
CompletableFuture<byte[]> future = new CompletableFuture<>();
tasks.add(new ZKTask<byte[]>(path, future, "get") {
Expand All @@ -275,7 +276,7 @@ protected void doExec(ZooKeeper zk) {

public CompletableFuture<Stat> exists(String path) {
if (closed.get()) {
return failed(new IOException("Client already closed"));
return failed(new DoNotRetryIOException("Client already closed"));
}
CompletableFuture<Stat> future = new CompletableFuture<>();
tasks.add(new ZKTask<Stat>(path, future, "exists") {
Expand Down Expand Up @@ -339,7 +340,7 @@ private void run() {
}
}
closeZk();
IOException error = new IOException("Client already closed");
DoNotRetryIOException error = new DoNotRetryIOException("Client already closed");
Arrays.stream(tasks.toArray(new Task[0])).forEach(t -> t.closed(error));
tasks.clear();
}
Expand Down

0 comments on commit d957f0f

Please sign in to comment.