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.hbase.procedure2.store.wal;
019
020import java.util.ArrayList;
021import java.util.Arrays;
022import java.util.List;
023import org.apache.hadoop.hbase.procedure2.Procedure;
024import org.apache.hadoop.hbase.procedure2.store.wal.ProcedureStoreTracker.DeleteState;
025import org.apache.yetus.audience.InterfaceAudience;
026
027import org.apache.hadoop.hbase.shaded.protobuf.generated.ProcedureProtos;
028
029/**
030 * A bitmap which can grow/merge with other {@link BitSetNode} (if certain conditions are met).
031 * Boundaries of bitmap are aligned to multiples of {@link BitSetNode#BITS_PER_WORD}. So the range
032 * of a {@link BitSetNode} is from [x * K, y * K) where x and y are integers, y > x and K is
033 * BITS_PER_WORD.
034 * <p/>
035 * We have two main bit sets to describe the state of procedures, the meanings are:
036 *
037 * <pre>
038 *  ----------------------
039 * | modified | deleted |  meaning
040 * |     0    |   0     |  proc exists, but hasn't been updated since last resetUpdates().
041 * |     1    |   0     |  proc was updated (but not deleted).
042 * |     1    |   1     |  proc was deleted.
043 * |     0    |   1     |  proc doesn't exist (maybe never created, maybe deleted in past).
044 * ----------------------
045 * </pre>
046 *
047 * The meaning of modified is that, we have modified the state of the procedure, no matter insert,
048 * update, or delete. And if it is an insert or update, we will set the deleted to 0, if not we will
049 * set the delete to 1.
050 * <p/>
051 * For a non-partial BitSetNode, the initial modified value is 0 and deleted value is 1. For the
052 * partial one, the initial modified value is 0 and the initial deleted value is also 0. In
053 * {@link #unsetPartialFlag()} we will reset the deleted to 1 if it is not modified.
054 * @deprecated Since 2.3.0, will be removed in 4.0.0. Keep here only for rolling upgrading, now we
055 *             use the new region based procedure store.
056 */
057@Deprecated
058@InterfaceAudience.Private
059class BitSetNode {
060  private static final long WORD_MASK = 0xffffffffffffffffL;
061  private static final int ADDRESS_BITS_PER_WORD = 6;
062  private static final int BITS_PER_WORD = 1 << ADDRESS_BITS_PER_WORD;
063  // The BitSetNode itself has 48 bytes overhead, which is the size of 6 longs, so here we use a max
064  // node size 4, which is 8 longs since we have an array for modified and also an array for
065  // deleted. The assumption here is that most procedures will be deleted soon so we'd better keep
066  // the BitSetNode small.
067  private static final int MAX_NODE_SIZE = 4 << ADDRESS_BITS_PER_WORD;
068
069  /**
070   * Mimics {@link ProcedureStoreTracker#partial}. It will effect how we fill the new deleted bits
071   * when growing.
072   */
073  private boolean partial;
074
075  /**
076   * Set of procedures which have been modified since last {@link #resetModified()}. Useful to track
077   * procedures which have been modified since last WAL write.
078   */
079  private long[] modified;
080
081  /**
082   * Keeps track of procedure ids which belong to this bitmap's range and have been deleted. This
083   * represents global state since it's not reset on WAL rolls.
084   */
085  private long[] deleted;
086  /**
087   * Offset of bitmap i.e. procedure id corresponding to first bit.
088   */
089  private long start;
090
091  public void dump() {
092    System.out.printf("%06d:%06d min=%d max=%d%n", getStart(), getEnd(), getActiveMinProcId(),
093      getActiveMaxProcId());
094    System.out.println("Modified:");
095    for (int i = 0; i < modified.length; ++i) {
096      for (int j = 0; j < BITS_PER_WORD; ++j) {
097        System.out.print((modified[i] & (1L << j)) != 0 ? "1" : "0");
098      }
099      System.out.println(" " + i);
100    }
101    System.out.println();
102    System.out.println("Delete:");
103    for (int i = 0; i < deleted.length; ++i) {
104      for (int j = 0; j < BITS_PER_WORD; ++j) {
105        System.out.print((deleted[i] & (1L << j)) != 0 ? "1" : "0");
106      }
107      System.out.println(" " + i);
108    }
109    System.out.println();
110  }
111
112  public BitSetNode(long procId, boolean partial) {
113    start = alignDown(procId);
114
115    int count = 1;
116    modified = new long[count];
117    deleted = new long[count];
118    if (!partial) {
119      Arrays.fill(deleted, WORD_MASK);
120    }
121
122    this.partial = partial;
123    updateState(procId, false);
124  }
125
126  public BitSetNode(ProcedureProtos.ProcedureStoreTracker.TrackerNode data) {
127    start = data.getStartId();
128    int size = data.getUpdatedCount();
129    assert size == data.getDeletedCount();
130    modified = new long[size];
131    deleted = new long[size];
132    for (int i = 0; i < size; ++i) {
133      modified[i] = data.getUpdated(i);
134      deleted[i] = data.getDeleted(i);
135    }
136    partial = false;
137  }
138
139  public BitSetNode(BitSetNode other, boolean resetDelete) {
140    this.start = other.start;
141    // The resetDelete will be set to true when building cleanup tracker.
142    // as we will reset deleted flags for all the unmodified bits to 1, the partial flag is useless
143    // so set it to false for not confusing the developers when debugging.
144    this.partial = resetDelete ? false : other.partial;
145    this.modified = other.modified.clone();
146    // The intention here is that, if a procedure is not modified in this tracker, then we do not
147    // need to take care of it, so we will set deleted to true for these bits, i.e, if modified is
148    // 0, then we set deleted to 1, otherwise keep it as is. So here, the equation is
149    // deleted |= ~modified, i.e,
150    if (resetDelete) {
151      this.deleted = new long[other.deleted.length];
152      for (int i = 0; i < this.deleted.length; ++i) {
153        this.deleted[i] |= ~(other.modified[i]);
154      }
155    } else {
156      this.deleted = other.deleted.clone();
157    }
158  }
159
160  public void insertOrUpdate(final long procId) {
161    updateState(procId, false);
162  }
163
164  public void delete(final long procId) {
165    updateState(procId, true);
166  }
167
168  public long getStart() {
169    return start;
170  }
171
172  /**
173   * Inclusive.
174   */
175  public long getEnd() {
176    return start + (modified.length << ADDRESS_BITS_PER_WORD) - 1;
177  }
178
179  public boolean contains(final long procId) {
180    return start <= procId && procId <= getEnd();
181  }
182
183  public DeleteState isDeleted(final long procId) {
184    int bitmapIndex = getBitmapIndex(procId);
185    int wordIndex = bitmapIndex >> ADDRESS_BITS_PER_WORD;
186    if (wordIndex >= deleted.length) {
187      return DeleteState.MAYBE;
188    }
189    // The left shift of java only takes care of the lowest several bits(5 for int and 6 for long),
190    // so here we can use bitmapIndex directly, without mod 64
191    return (deleted[wordIndex] & (1L << bitmapIndex)) != 0 ? DeleteState.YES : DeleteState.NO;
192  }
193
194  public boolean isModified(long procId) {
195    int bitmapIndex = getBitmapIndex(procId);
196    int wordIndex = bitmapIndex >> ADDRESS_BITS_PER_WORD;
197    if (wordIndex >= modified.length) {
198      return false;
199    }
200    // The left shift of java only takes care of the lowest several bits(5 for int and 6 for long),
201    // so here we can use bitmapIndex directly, without mod 64
202    return (modified[wordIndex] & (1L << bitmapIndex)) != 0;
203  }
204
205  /** Returns true, if all the procedures has been modified. */
206  public boolean isAllModified() {
207    // TODO: cache the value
208    for (int i = 0; i < modified.length; ++i) {
209      if ((modified[i] | deleted[i]) != WORD_MASK) {
210        return false;
211      }
212    }
213    return true;
214  }
215
216  /** Returns all the active procedure ids in this bit set. */
217  public long[] getActiveProcIds() {
218    List<Long> procIds = new ArrayList<>();
219    for (int wordIndex = 0; wordIndex < modified.length; wordIndex++) {
220      if (deleted[wordIndex] == WORD_MASK || modified[wordIndex] == 0) {
221        // This should be the common case, where most procedures has been deleted.
222        continue;
223      }
224      long baseProcId = getStart() + (wordIndex << ADDRESS_BITS_PER_WORD);
225      for (int i = 0; i < (1 << ADDRESS_BITS_PER_WORD); i++) {
226        long mask = 1L << i;
227        if ((deleted[wordIndex] & mask) == 0 && (modified[wordIndex] & mask) != 0) {
228          procIds.add(baseProcId + i);
229        }
230      }
231    }
232    return procIds.stream().mapToLong(Long::longValue).toArray();
233  }
234
235  /** Returns true, if there are no active procedures in this BitSetNode, else false. */
236  public boolean isEmpty() {
237    // TODO: cache the value
238    for (int i = 0; i < deleted.length; ++i) {
239      if (deleted[i] != WORD_MASK) {
240        return false;
241      }
242    }
243    return true;
244  }
245
246  public void resetModified() {
247    Arrays.fill(modified, 0);
248  }
249
250  public void unsetPartialFlag() {
251    partial = false;
252    for (int i = 0; i < modified.length; ++i) {
253      for (int j = 0; j < BITS_PER_WORD; ++j) {
254        if ((modified[i] & (1L << j)) == 0) {
255          deleted[i] |= (1L << j);
256        }
257      }
258    }
259  }
260
261  /**
262   * Convert to
263   * org.apache.hadoop.hbase.protobuf.generated.ProcedureProtos.ProcedureStoreTracker.TrackerNode
264   * protobuf.
265   */
266  public ProcedureProtos.ProcedureStoreTracker.TrackerNode convert() {
267    ProcedureProtos.ProcedureStoreTracker.TrackerNode.Builder builder =
268      ProcedureProtos.ProcedureStoreTracker.TrackerNode.newBuilder();
269    builder.setStartId(start);
270    for (int i = 0; i < modified.length; ++i) {
271      builder.addUpdated(modified[i]);
272      builder.addDeleted(deleted[i]);
273    }
274    return builder.build();
275  }
276
277  // ========================================================================
278  // Grow/Merge Helpers
279  // ========================================================================
280  public boolean canGrow(long procId) {
281    if (procId <= start) {
282      return getEnd() - procId < MAX_NODE_SIZE;
283    } else {
284      return procId - start < MAX_NODE_SIZE;
285    }
286  }
287
288  public boolean canMerge(BitSetNode rightNode) {
289    // Can just compare 'starts' since boundaries are aligned to multiples of BITS_PER_WORD.
290    assert start < rightNode.start;
291    return (rightNode.getEnd() - start) < MAX_NODE_SIZE;
292  }
293
294  public void grow(long procId) {
295    // make sure you have call canGrow first before calling this method
296    assert canGrow(procId);
297    if (procId < start) {
298      // grow to left
299      long newStart = alignDown(procId);
300      int delta = (int) (start - newStart) >> ADDRESS_BITS_PER_WORD;
301      start = newStart;
302      long[] newModified = new long[modified.length + delta];
303      System.arraycopy(modified, 0, newModified, delta, modified.length);
304      modified = newModified;
305      long[] newDeleted = new long[deleted.length + delta];
306      if (!partial) {
307        for (int i = 0; i < delta; i++) {
308          newDeleted[i] = WORD_MASK;
309        }
310      }
311      System.arraycopy(deleted, 0, newDeleted, delta, deleted.length);
312      deleted = newDeleted;
313    } else {
314      // grow to right
315      long newEnd = alignUp(procId + 1);
316      int delta = (int) (newEnd - getEnd()) >> ADDRESS_BITS_PER_WORD;
317      int newSize = modified.length + delta;
318      long[] newModified = Arrays.copyOf(modified, newSize);
319      modified = newModified;
320      long[] newDeleted = Arrays.copyOf(deleted, newSize);
321      if (!partial) {
322        for (int i = deleted.length; i < newSize; i++) {
323          newDeleted[i] = WORD_MASK;
324        }
325      }
326      deleted = newDeleted;
327    }
328  }
329
330  public void merge(BitSetNode rightNode) {
331    assert start < rightNode.start;
332    int newSize = (int) (rightNode.getEnd() - start + 1) >> ADDRESS_BITS_PER_WORD;
333    long[] newModified = Arrays.copyOf(modified, newSize);
334    System.arraycopy(rightNode.modified, 0, newModified, newSize - rightNode.modified.length,
335      rightNode.modified.length);
336    long[] newDeleted = Arrays.copyOf(deleted, newSize);
337    System.arraycopy(rightNode.deleted, 0, newDeleted, newSize - rightNode.deleted.length,
338      rightNode.deleted.length);
339    if (!partial) {
340      for (int i = deleted.length, n = newSize - rightNode.deleted.length; i < n; i++) {
341        newDeleted[i] = WORD_MASK;
342      }
343    }
344    modified = newModified;
345    deleted = newDeleted;
346  }
347
348  @Override
349  public String toString() {
350    return "BitSetNode(" + getStart() + "-" + getEnd() + ")";
351  }
352
353  // ========================================================================
354  // Min/Max Helpers
355  // ========================================================================
356  public long getActiveMinProcId() {
357    long minProcId = start;
358    for (int i = 0; i < deleted.length; ++i) {
359      if (deleted[i] == 0) {
360        return minProcId;
361      }
362
363      if (deleted[i] != WORD_MASK) {
364        for (int j = 0; j < BITS_PER_WORD; ++j) {
365          if ((deleted[i] & (1L << j)) == 0) {
366            return minProcId + j;
367          }
368        }
369      }
370
371      minProcId += BITS_PER_WORD;
372    }
373    return Procedure.NO_PROC_ID;
374  }
375
376  public long getActiveMaxProcId() {
377    long maxProcId = getEnd();
378    for (int i = deleted.length - 1; i >= 0; --i) {
379      if (deleted[i] == 0) {
380        return maxProcId;
381      }
382
383      if (deleted[i] != WORD_MASK) {
384        for (int j = BITS_PER_WORD - 1; j >= 0; --j) {
385          if ((deleted[i] & (1L << j)) == 0) {
386            return maxProcId - (BITS_PER_WORD - 1 - j);
387          }
388        }
389      }
390      maxProcId -= BITS_PER_WORD;
391    }
392    return Procedure.NO_PROC_ID;
393  }
394
395  // ========================================================================
396  // Bitmap Helpers
397  // ========================================================================
398  private int getBitmapIndex(final long procId) {
399    return (int) (procId - start);
400  }
401
402  void updateState(long procId, boolean isDeleted) {
403    int bitmapIndex = getBitmapIndex(procId);
404    int wordIndex = bitmapIndex >> ADDRESS_BITS_PER_WORD;
405    long value = (1L << bitmapIndex);
406
407    try {
408      modified[wordIndex] |= value;
409    } catch (ArrayIndexOutOfBoundsException aioobe) {
410      // We've gotten a AIOOBE in here; add detail to help debug.
411      ArrayIndexOutOfBoundsException aioobe2 =
412        new ArrayIndexOutOfBoundsException("pid=" + procId + ", deleted=" + isDeleted);
413      aioobe2.initCause(aioobe);
414      throw aioobe2;
415    }
416    if (isDeleted) {
417      deleted[wordIndex] |= value;
418    } else {
419      deleted[wordIndex] &= ~value;
420    }
421  }
422
423  // ========================================================================
424  // Helpers
425  // ========================================================================
426  /** Returns upper boundary (aligned to multiple of BITS_PER_WORD) of bitmap range x belongs to. */
427  private static long alignUp(final long x) {
428    return (x + (BITS_PER_WORD - 1)) & -BITS_PER_WORD;
429  }
430
431  /** Returns lower boundary (aligned to multiple of BITS_PER_WORD) of bitmap range x belongs to. */
432  private static long alignDown(final long x) {
433    return x & -BITS_PER_WORD;
434  }
435}