001/**
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hdfs.protocol.datatransfer;
019
020import static org.apache.hadoop.hdfs.protocolPB.PBHelper.vintPrefixed;
021
022import java.io.IOException;
023import java.io.InputStream;
024import java.io.OutputStream;
025import java.util.ArrayList;
026import java.util.Arrays;
027import java.util.Collections;
028
029import com.google.common.collect.Lists;
030import org.apache.hadoop.classification.InterfaceAudience;
031import org.apache.hadoop.classification.InterfaceStability;
032import org.apache.hadoop.hdfs.HdfsConfiguration;
033import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_DATANODE_OOB_TIMEOUT_KEY;
034import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_DATANODE_OOB_TIMEOUT_DEFAULT;
035
036import org.apache.hadoop.hdfs.protocol.proto.DataTransferProtos.PipelineAckProto;
037import org.apache.hadoop.hdfs.protocol.proto.DataTransferProtos.Status;
038import com.google.protobuf.TextFormat;
039import org.apache.hadoop.hdfs.util.LongBitFormat;
040
041/** Pipeline Acknowledgment **/
042@InterfaceAudience.Private
043@InterfaceStability.Evolving
044public class PipelineAck {
045  PipelineAckProto proto;
046  public final static long UNKOWN_SEQNO = -2;
047  final static int OOB_START = Status.OOB_RESTART_VALUE; // the first OOB type
048  final static int OOB_END = Status.OOB_RESERVED3_VALUE; // the last OOB type
049  final static int NUM_OOB_TYPES = OOB_END - OOB_START + 1;
050  // place holder for timeout value of each OOB type
051  final static long[] OOB_TIMEOUT;
052
053  public enum ECN {
054    DISABLED(0),
055    SUPPORTED(1),
056    SUPPORTED2(2),
057    CONGESTED(3);
058
059    private final int value;
060    private static final ECN[] VALUES = values();
061    static ECN valueOf(int value) {
062      return VALUES[value];
063    }
064
065    ECN(int value) {
066      this.value = value;
067    }
068
069    public int getValue() {
070      return value;
071    }
072  }
073
074  private enum StatusFormat {
075    STATUS(null, 4),
076    RESERVED(STATUS.BITS, 1),
077    ECN_BITS(RESERVED.BITS, 2);
078
079    private final LongBitFormat BITS;
080
081    StatusFormat(LongBitFormat prev, int bits) {
082      BITS = new LongBitFormat(name(), prev, bits, 0);
083    }
084
085    static Status getStatus(int header) {
086      return Status.valueOf((int) STATUS.BITS.retrieve(header));
087    }
088
089    static ECN getECN(int header) {
090      return ECN.valueOf((int) ECN_BITS.BITS.retrieve(header));
091    }
092
093    public static int setStatus(int old, Status status) {
094      return (int) STATUS.BITS.combine(status.getNumber(), old);
095    }
096
097    public static int setECN(int old, ECN ecn) {
098      return (int) ECN_BITS.BITS.combine(ecn.getValue(), old);
099    }
100  }
101
102  static {
103    OOB_TIMEOUT = new long[NUM_OOB_TYPES];
104    HdfsConfiguration conf = new HdfsConfiguration();
105    String[] ele = conf.get(DFS_DATANODE_OOB_TIMEOUT_KEY,
106        DFS_DATANODE_OOB_TIMEOUT_DEFAULT).split(",");
107    for (int i = 0; i < NUM_OOB_TYPES; i++) {
108      OOB_TIMEOUT[i] = (i < ele.length) ? Long.parseLong(ele[i]) : 0;
109    }
110  }
111
112  /** default constructor **/
113  public PipelineAck() {
114  }
115  
116  /**
117   * Constructor assuming no next DN in pipeline
118   * @param seqno sequence number
119   * @param replies an array of replies
120   */
121  public PipelineAck(long seqno, int[] replies) {
122    this(seqno, replies, 0L);
123  }
124
125  /**
126   * Constructor
127   * @param seqno sequence number
128   * @param replies an array of replies
129   * @param downstreamAckTimeNanos ack RTT in nanoseconds, 0 if no next DN in pipeline
130   */
131  public PipelineAck(long seqno, int[] replies,
132                     long downstreamAckTimeNanos) {
133    ArrayList<Status> statusList = Lists.newArrayList();
134    ArrayList<Integer> flagList = Lists.newArrayList();
135    for (int r : replies) {
136      statusList.add(StatusFormat.getStatus(r));
137      flagList.add(r);
138    }
139    proto = PipelineAckProto.newBuilder()
140      .setSeqno(seqno)
141      .addAllReply(statusList)
142      .addAllFlag(flagList)
143      .setDownstreamAckTimeNanos(downstreamAckTimeNanos)
144      .build();
145  }
146  
147  /**
148   * Get the sequence number
149   * @return the sequence number
150   */
151  public long getSeqno() {
152    return proto.getSeqno();
153  }
154  
155  /**
156   * Get the number of replies
157   * @return the number of replies
158   */
159  public short getNumOfReplies() {
160    return (short)proto.getReplyCount();
161  }
162  
163  /**
164   * get the header flag of ith reply
165   */
166  public int getHeaderFlag(int i) {
167    if (proto.getFlagCount() > 0) {
168      return proto.getFlag(i);
169    } else {
170      return combineHeader(ECN.DISABLED, proto.getReply(i));
171    }
172  }
173
174  public int getFlag(int i) {
175    return proto.getFlag(i);
176  }
177
178  /**
179   * Get the time elapsed for downstream ack RTT in nanoseconds
180   * @return time elapsed for downstream ack in nanoseconds, 0 if no next DN in pipeline
181   */
182  public long getDownstreamAckTimeNanos() {
183    return proto.getDownstreamAckTimeNanos();
184  }
185
186  /**
187   * Check if this ack contains error status
188   * @return true if all statuses are SUCCESS
189   */
190  public boolean isSuccess() {
191    for (Status s : proto.getReplyList()) {
192      if (s != Status.SUCCESS) {
193        return false;
194      }
195    }
196    return true;
197  }
198
199  /**
200   * Returns the OOB status if this ack contains one. 
201   * @return null if it is not an OOB ack.
202   */
203  public Status getOOBStatus() {
204    // Normal data transfer acks will have a valid sequence number, so
205    // this will return right away in most cases.
206    if (getSeqno() != UNKOWN_SEQNO) {
207      return null;
208    }
209    for (Status s : proto.getReplyList()) {
210      // The following check is valid because protobuf guarantees to
211      // preserve the ordering of enum elements.
212      if (s.getNumber() >= OOB_START && s.getNumber() <= OOB_END) {
213        return s;
214      }
215    }
216    return null;
217  }
218
219  /**
220   * Get the timeout to be used for transmitting the OOB type
221   * @return the timeout in milliseconds
222   */
223  public static long getOOBTimeout(Status status) throws IOException {
224    int index = status.getNumber() - OOB_START;
225    if (index >= 0 && index < NUM_OOB_TYPES) {
226      return OOB_TIMEOUT[index];
227    } 
228    // Not an OOB.
229    throw new IOException("Not an OOB status: " + status);
230  }
231
232  /** Get the Restart OOB ack status */
233  public static Status getRestartOOBStatus() {
234    return Status.OOB_RESTART;
235  }
236
237  /** return true if it is the restart OOB status code  */
238  public static boolean isRestartOOBStatus(Status st) {
239    return st.equals(Status.OOB_RESTART);
240  }
241
242  /**** Writable interface ****/
243  public void readFields(InputStream in) throws IOException {
244    proto = PipelineAckProto.parseFrom(vintPrefixed(in));
245  }
246
247  public void write(OutputStream out) throws IOException {
248    proto.writeDelimitedTo(out);
249  }
250  
251  @Override //Object
252  public String toString() {
253    return TextFormat.shortDebugString(proto);
254  }
255
256  public static Status getStatusFromHeader(int header) {
257    return StatusFormat.getStatus(header);
258  }
259
260  public static int setStatusForHeader(int old, Status status) {
261    return StatusFormat.setStatus(old, status);
262  }
263
264  public static int combineHeader(ECN ecn, Status status) {
265    int header = 0;
266    header = StatusFormat.setStatus(header, status);
267    header = StatusFormat.setECN(header, ecn);
268    return header;
269  }
270}