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.server.blockmanagement;
019
020import com.google.common.annotations.VisibleForTesting;
021import com.google.common.base.Preconditions;
022import org.apache.hadoop.hdfs.protocol.Block;
023import org.apache.hadoop.hdfs.protocol.HdfsConstants;
024import org.apache.hadoop.hdfs.server.common.GenerationStamp;
025import org.apache.hadoop.hdfs.server.common.HdfsServerConstants;
026
027import java.io.IOException;
028
029/**
030 * BlockIdManager allocates the generation stamps and the block ID. The
031 * {@see FSNamesystem} is responsible for persisting the allocations in the
032 * {@see EditLog}.
033 */
034public class BlockIdManager {
035  /**
036   * The global generation stamp for legacy blocks with randomly
037   * generated block IDs.
038   */
039  private final GenerationStamp generationStampV1 = new GenerationStamp();
040  /**
041   * The global generation stamp for this file system.
042   */
043  private final GenerationStamp generationStampV2 = new GenerationStamp();
044  /**
045   * The value of the generation stamp when the first switch to sequential
046   * block IDs was made. Blocks with generation stamps below this value
047   * have randomly allocated block IDs. Blocks with generation stamps above
048   * this value had sequentially allocated block IDs. Read from the fsImage
049   * (or initialized as an offset from the V1 (legacy) generation stamp on
050   * upgrade).
051   */
052  private long generationStampV1Limit;
053  /**
054   * The global block ID space for this file system.
055   */
056  private final SequentialBlockIdGenerator blockIdGenerator;
057
058  public BlockIdManager(BlockManager blockManager) {
059    this.generationStampV1Limit = HdfsConstants.GRANDFATHER_GENERATION_STAMP;
060    this.blockIdGenerator = new SequentialBlockIdGenerator(blockManager);
061  }
062
063  /**
064   * Upgrades the generation stamp for the filesystem
065   * by reserving a sufficient range for all existing blocks.
066   * Should be invoked only during the first upgrade to
067   * sequential block IDs.
068   */
069  public long upgradeGenerationStampToV2() {
070    Preconditions.checkState(generationStampV2.getCurrentValue() ==
071      GenerationStamp.LAST_RESERVED_STAMP);
072    generationStampV2.skipTo(generationStampV1.getCurrentValue() +
073      HdfsServerConstants.RESERVED_GENERATION_STAMPS_V1);
074
075    generationStampV1Limit = generationStampV2.getCurrentValue();
076    return generationStampV2.getCurrentValue();
077  }
078
079  /**
080   * Sets the generation stamp that delineates random and sequentially
081   * allocated block IDs.
082   *
083   * @param stamp set generation stamp limit to this value
084   */
085  public void setGenerationStampV1Limit(long stamp) {
086    Preconditions.checkState(generationStampV1Limit == HdfsConstants
087      .GRANDFATHER_GENERATION_STAMP);
088    generationStampV1Limit = stamp;
089  }
090
091  /**
092   * Gets the value of the generation stamp that delineates sequential
093   * and random block IDs.
094   */
095  public long getGenerationStampAtblockIdSwitch() {
096    return generationStampV1Limit;
097  }
098
099  @VisibleForTesting
100  SequentialBlockIdGenerator getBlockIdGenerator() {
101    return blockIdGenerator;
102  }
103
104  /**
105   * Sets the maximum allocated block ID for this filesystem. This is
106   * the basis for allocating new block IDs.
107   */
108  public void setLastAllocatedBlockId(long blockId) {
109    blockIdGenerator.skipTo(blockId);
110  }
111
112  /**
113   * Gets the maximum sequentially allocated block ID for this filesystem
114   */
115  public long getLastAllocatedBlockId() {
116    return blockIdGenerator.getCurrentValue();
117  }
118
119  /**
120   * Sets the current generation stamp for legacy blocks
121   */
122  public void setGenerationStampV1(long stamp) {
123    generationStampV1.setCurrentValue(stamp);
124  }
125
126  /**
127   * Gets the current generation stamp for legacy blocks
128   */
129  public long getGenerationStampV1() {
130    return generationStampV1.getCurrentValue();
131  }
132
133  /**
134   * Gets the current generation stamp for this filesystem
135   */
136  public void setGenerationStampV2(long stamp) {
137    generationStampV2.setCurrentValue(stamp);
138  }
139
140  public long getGenerationStampV2() {
141    return generationStampV2.getCurrentValue();
142  }
143
144  /**
145   * Increments, logs and then returns the stamp
146   */
147  public long nextGenerationStamp(boolean legacyBlock) throws IOException {
148    return legacyBlock ? getNextGenerationStampV1() :
149      getNextGenerationStampV2();
150  }
151
152  @VisibleForTesting
153  long getNextGenerationStampV1() throws IOException {
154    long genStampV1 = generationStampV1.nextValue();
155
156    if (genStampV1 >= generationStampV1Limit) {
157      // We ran out of generation stamps for legacy blocks. In practice, it
158      // is extremely unlikely as we reserved 1T v1 generation stamps. The
159      // result is that we can no longer append to the legacy blocks that
160      // were created before the upgrade to sequential block IDs.
161      throw new OutOfV1GenerationStampsException();
162    }
163
164    return genStampV1;
165  }
166
167  @VisibleForTesting
168  long getNextGenerationStampV2() {
169    return generationStampV2.nextValue();
170  }
171
172  public long getGenerationStampV1Limit() {
173    return generationStampV1Limit;
174  }
175
176  /**
177   * Determine whether the block ID was randomly generated (legacy) or
178   * sequentially generated. The generation stamp value is used to
179   * make the distinction.
180   *
181   * @return true if the block ID was randomly generated, false otherwise.
182   */
183  public boolean isLegacyBlock(Block block) {
184    return block.getGenerationStamp() < getGenerationStampV1Limit();
185  }
186
187  /**
188   * Increments, logs and then returns the block ID
189   */
190  public long nextBlockId() {
191    return blockIdGenerator.nextValue();
192  }
193
194  public boolean isGenStampInFuture(Block block) {
195    if (isLegacyBlock(block)) {
196      return block.getGenerationStamp() > getGenerationStampV1();
197    } else {
198      return block.getGenerationStamp() > getGenerationStampV2();
199    }
200  }
201
202  public void clear() {
203    generationStampV1.setCurrentValue(GenerationStamp.LAST_RESERVED_STAMP);
204    generationStampV2.setCurrentValue(GenerationStamp.LAST_RESERVED_STAMP);
205    getBlockIdGenerator().setCurrentValue(SequentialBlockIdGenerator
206      .LAST_RESERVED_BLOCK_ID);
207    generationStampV1Limit = HdfsConstants.GRANDFATHER_GENERATION_STAMP;
208  }
209}