HBASE-26286: Add support for specifying store file tracker when restoring or cloning...
[hbase.git] / hbase-server / src / main / java / org / apache / hadoop / hbase / ipc / NettyRpcServerResponseEncoder.java
blob09589da16354da27deaa024576a10476f28f259b
1 /**
2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements. See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership. The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License. You may obtain a copy of the License at
10 * http://www.apache.org/licenses/LICENSE-2.0
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
18 package org.apache.hadoop.hbase.ipc;
20 import org.apache.hbase.thirdparty.io.netty.buffer.Unpooled;
21 import org.apache.hbase.thirdparty.io.netty.channel.ChannelHandlerContext;
22 import org.apache.hbase.thirdparty.io.netty.channel.ChannelOutboundHandlerAdapter;
23 import org.apache.hbase.thirdparty.io.netty.channel.ChannelPromise;
25 import org.apache.yetus.audience.InterfaceAudience;
27 /**
28 * Encoder for {@link RpcResponse}.
29 * @since 2.0.0
31 @InterfaceAudience.Private
32 class NettyRpcServerResponseEncoder extends ChannelOutboundHandlerAdapter {
34 private final MetricsHBaseServer metrics;
36 NettyRpcServerResponseEncoder(MetricsHBaseServer metrics) {
37 this.metrics = metrics;
40 @Override
41 public void write(ChannelHandlerContext ctx, Object msg, ChannelPromise promise)
42 throws Exception {
43 if (msg instanceof RpcResponse) {
44 RpcResponse resp = (RpcResponse) msg;
45 BufferChain buf = resp.getResponse();
46 ctx.write(Unpooled.wrappedBuffer(buf.getBuffers()), promise).addListener(f -> {
47 resp.done();
48 if (f.isSuccess()) {
49 metrics.sentBytes(buf.size());
51 });
52 } else {
53 ctx.write(msg, promise);