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.regionserver;
019
020import java.util.List;
021import org.apache.hadoop.hbase.Cell;
022import org.apache.hadoop.hbase.HBaseInterfaceAudience;
023import org.apache.hadoop.hbase.client.metrics.ServerSideScanMetrics;
024import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
025import org.apache.yetus.audience.InterfaceAudience;
026import org.apache.yetus.audience.InterfaceStability;
027
028/**
029 * ScannerContext instances encapsulate limit tracking AND progress towards those limits during
030 * invocations of {@link InternalScanner#next(java.util.List)} and
031 * {@link RegionScanner#next(java.util.List)}.
032 * <p>
033 * A ScannerContext instance should be updated periodically throughout execution whenever progress
034 * towards a limit has been made. Each limit can be checked via the appropriate checkLimit method.
035 * <p>
036 * Once a limit has been reached, the scan will stop. The invoker of
037 * {@link InternalScanner#next(java.util.List)} or {@link RegionScanner#next(java.util.List)} can
038 * use the appropriate check*Limit methods to see exactly which limits have been reached.
039 * Alternatively, {@link #checkAnyLimitReached(LimitScope)} is provided to see if ANY limit was
040 * reached
041 * <p>
042 * {@link NoLimitScannerContext#NO_LIMIT} is an immutable static definition that can be used
043 * whenever a {@link ScannerContext} is needed but limits do not need to be enforced.
044 * <p>
045 * NOTE: It is important that this class only ever expose setter methods that can be safely skipped
046 * when limits should be NOT enforced. This is because of the necessary immutability of the class
047 * {@link NoLimitScannerContext}. If a setter cannot be safely skipped, the immutable nature of
048 * {@link NoLimitScannerContext} will lead to incorrect behavior.
049 */
050@InterfaceAudience.LimitedPrivate(HBaseInterfaceAudience.COPROC)
051@InterfaceStability.Evolving
052public class ScannerContext {
053
054  LimitFields limits;
055  /**
056   * A different set of progress fields. Only include batch, dataSize and heapSize. Compare to
057   * LimitFields, ProgressFields doesn't contain time field. As we save a deadline in LimitFields,
058   * so use {@link EnvironmentEdgeManager.currentTime()} directly when check time limit.
059   */
060  ProgressFields progress;
061
062  /**
063   * The state of the scanner after the invocation of {@link InternalScanner#next(java.util.List)}
064   * or {@link RegionScanner#next(java.util.List)}.
065   */
066  NextState scannerState;
067  private static final NextState DEFAULT_STATE = NextState.MORE_VALUES;
068
069  /**
070   * Used as an indication to invocations of {@link InternalScanner#next(java.util.List)} and
071   * {@link RegionScanner#next(java.util.List)} that, if true, the progress tracked within this
072   * {@link ScannerContext} instance should be considered while evaluating the limits. Useful for
073   * enforcing a set of limits across multiple calls (i.e. the limit may not be reached in a single
074   * invocation, but any progress made should be considered in future invocations)
075   * <p>
076   * Defaulting this value to false means that, by default, any tracked progress will be wiped clean
077   * on invocations to {@link InternalScanner#next(java.util.List)} and
078   * {@link RegionScanner#next(java.util.List)} and the call will be treated as though no progress
079   * has been made towards the limits so far.
080   * <p>
081   * This is an important mechanism. Users of Internal/Region scanners expect that they can define
082   * some limits and then repeatedly invoke {@link InternalScanner#next(List)} or
083   * {@link RegionScanner#next(List)} where each invocation respects these limits separately.
084   * <p>
085   * For example:
086   *
087   * <pre>
088   *  {@code
089   * ScannerContext context = new ScannerContext.newBuilder().setBatchLimit(5).build();
090   * RegionScanner scanner = ...
091   * List<Cell> results = new ArrayList<Cell>();
092   * while(scanner.next(results, context)) {
093   *   // Do something with a batch of 5 cells
094   * }
095   * }
096   * </pre>
097   *
098   * However, in the case of RPCs, the server wants to be able to define a set of limits for a
099   * particular RPC request and have those limits respected across multiple invocations. This means
100   * that the progress made towards the limits in earlier calls will be saved and considered in
101   * future invocations
102   */
103  boolean keepProgress;
104  private static boolean DEFAULT_KEEP_PROGRESS = false;
105
106  private Cell lastPeekedCell = null;
107
108  // Set this to true will have the same behavior with reaching the time limit.
109  // This is used when you want to make the current RSRpcService.scan returns immediately. For
110  // example, when we want to switch from pread to stream, we can only do it after the rpc call is
111  // returned.
112  private boolean returnImmediately;
113
114  /**
115   * Tracks the relevant server side metrics during scans. null when metrics should not be tracked
116   */
117  final ServerSideScanMetrics metrics;
118
119  ScannerContext(boolean keepProgress, LimitFields limitsToCopy, boolean trackMetrics) {
120    this.limits = new LimitFields();
121    if (limitsToCopy != null) {
122      this.limits.copy(limitsToCopy);
123    }
124
125    // Progress fields are initialized to 0
126    progress = new ProgressFields(0, 0, 0);
127
128    this.keepProgress = keepProgress;
129    this.scannerState = DEFAULT_STATE;
130    this.metrics = trackMetrics ? new ServerSideScanMetrics() : null;
131  }
132
133  public boolean isTrackingMetrics() {
134    return this.metrics != null;
135  }
136
137  /**
138   * Get the metrics instance. Should only be called after a call to {@link #isTrackingMetrics()}
139   * has been made to confirm that metrics are indeed being tracked.
140   * @return {@link ServerSideScanMetrics} instance that is tracking metrics for this scan
141   */
142  public ServerSideScanMetrics getMetrics() {
143    assert isTrackingMetrics();
144    return this.metrics;
145  }
146
147  /**
148   * @return true if the progress tracked so far in this instance will be considered during an
149   *         invocation of {@link InternalScanner#next(java.util.List)} or
150   *         {@link RegionScanner#next(java.util.List)}. false when the progress tracked so far
151   *         should not be considered and should instead be wiped away via {@link #clearProgress()}
152   */
153  boolean getKeepProgress() {
154    return keepProgress;
155  }
156
157  void setKeepProgress(boolean keepProgress) {
158    this.keepProgress = keepProgress;
159  }
160
161  /**
162   * Progress towards the batch limit has been made. Increment internal tracking of batch progress
163   */
164  void incrementBatchProgress(int batch) {
165    int currentBatch = progress.getBatch();
166    progress.setBatch(currentBatch + batch);
167  }
168
169  /**
170   * Progress towards the size limit has been made. Increment internal tracking of size progress
171   */
172  void incrementSizeProgress(long dataSize, long heapSize) {
173    long curDataSize = progress.getDataSize();
174    progress.setDataSize(curDataSize + dataSize);
175    long curHeapSize = progress.getHeapSize();
176    progress.setHeapSize(curHeapSize + heapSize);
177  }
178
179  /**
180   * Update the time progress.
181   * @deprecated will be removed in 3.0
182   */
183  @Deprecated
184  void updateTimeProgress() {
185  }
186
187  int getBatchProgress() {
188    return progress.getBatch();
189  }
190
191  long getDataSizeProgress() {
192    return progress.getDataSize();
193  }
194
195  long getHeapSizeProgress() {
196    return progress.getHeapSize();
197  }
198
199  /**
200   * @deprecated will be removed in 3.0
201   */
202  @Deprecated
203  long getTimeProgress() {
204    return EnvironmentEdgeManager.currentTime();
205  }
206
207  /**
208   * @deprecated will be removed in 3.0
209   */
210  @Deprecated
211  void setProgress(int batchProgress, long sizeProgress, long heapSizeProgress, long timeProgress) {
212    setProgress(batchProgress, sizeProgress, heapSizeProgress);
213  }
214
215  void setProgress(int batchProgress, long sizeProgress, long heapSizeProgress) {
216    setBatchProgress(batchProgress);
217    setSizeProgress(sizeProgress, heapSizeProgress);
218  }
219
220  void setSizeProgress(long dataSizeProgress, long heapSizeProgress) {
221    progress.setDataSize(dataSizeProgress);
222    progress.setHeapSize(heapSizeProgress);
223  }
224
225  void setBatchProgress(int batchProgress) {
226    progress.setBatch(batchProgress);
227  }
228
229  /**
230   * @deprecated will be removed in 3.0
231   */
232  @Deprecated
233  void setTimeProgress(long timeProgress) {
234  }
235
236  /**
237   * Clear away any progress that has been made so far. All progress fields are reset to initial
238   * values
239   */
240  void clearProgress() {
241    progress.setFields(0, 0, 0);
242  }
243
244  /**
245   * Note that this is not a typical setter. This setter returns the {@link NextState} that was
246   * passed in so that methods can be invoked against the new state. Furthermore, this pattern
247   * allows the {@link NoLimitScannerContext} to cleanly override this setter and simply return the
248   * new state, thus preserving the immutability of {@link NoLimitScannerContext} n * @return The
249   * state that was passed in.
250   */
251  NextState setScannerState(NextState state) {
252    if (!NextState.isValidState(state)) {
253      throw new IllegalArgumentException("Cannot set to invalid state: " + state);
254    }
255
256    this.scannerState = state;
257    return state;
258  }
259
260  /**
261   * @return true when we have more cells for the current row. This usually because we have reached
262   *         a limit in the middle of a row
263   */
264  boolean mayHaveMoreCellsInRow() {
265    return scannerState == NextState.SIZE_LIMIT_REACHED_MID_ROW
266      || scannerState == NextState.TIME_LIMIT_REACHED_MID_ROW
267      || scannerState == NextState.BATCH_LIMIT_REACHED;
268  }
269
270  /**
271   * n * @return true if the batch limit can be enforced in the checker's scope
272   */
273  boolean hasBatchLimit(LimitScope checkerScope) {
274    return limits.canEnforceBatchLimitFromScope(checkerScope) && limits.getBatch() > 0;
275  }
276
277  /**
278   * n * @return true if the size limit can be enforced in the checker's scope
279   */
280  boolean hasSizeLimit(LimitScope checkerScope) {
281    return limits.canEnforceSizeLimitFromScope(checkerScope)
282      && (limits.getDataSize() > 0 || limits.getHeapSize() > 0);
283  }
284
285  /**
286   * n * @return true if the time limit can be enforced in the checker's scope
287   */
288  boolean hasTimeLimit(LimitScope checkerScope) {
289    return limits.canEnforceTimeLimitFromScope(checkerScope)
290      && (limits.getTime() > 0 || returnImmediately);
291  }
292
293  /**
294   * n * @return true if any limit can be enforced within the checker's scope
295   */
296  boolean hasAnyLimit(LimitScope checkerScope) {
297    return hasBatchLimit(checkerScope) || hasSizeLimit(checkerScope) || hasTimeLimit(checkerScope);
298  }
299
300  /**
301   * @param scope The scope in which the size limit will be enforced
302   */
303  void setSizeLimitScope(LimitScope scope) {
304    limits.setSizeScope(scope);
305  }
306
307  /**
308   * @param scope The scope in which the time limit will be enforced
309   */
310  void setTimeLimitScope(LimitScope scope) {
311    limits.setTimeScope(scope);
312  }
313
314  int getBatchLimit() {
315    return limits.getBatch();
316  }
317
318  long getDataSizeLimit() {
319    return limits.getDataSize();
320  }
321
322  long getTimeLimit() {
323    return limits.getTime();
324  }
325
326  /**
327   * @param checkerScope The scope that the limit is being checked from
328   * @return true when the limit is enforceable from the checker's scope and it has been reached
329   */
330  boolean checkBatchLimit(LimitScope checkerScope) {
331    return hasBatchLimit(checkerScope) && progress.getBatch() >= limits.getBatch();
332  }
333
334  /**
335   * @param checkerScope The scope that the limit is being checked from
336   * @return true when the limit is enforceable from the checker's scope and it has been reached
337   */
338  boolean checkSizeLimit(LimitScope checkerScope) {
339    return hasSizeLimit(checkerScope) && (progress.getDataSize() >= limits.getDataSize()
340      || progress.getHeapSize() >= limits.getHeapSize());
341  }
342
343  /**
344   * @param checkerScope The scope that the limit is being checked from. The time limit is always
345   *                     checked against {@link EnvironmentEdgeManager.currentTime}
346   * @return true when the limit is enforceable from the checker's scope and it has been reached
347   */
348  boolean checkTimeLimit(LimitScope checkerScope) {
349    return hasTimeLimit(checkerScope)
350      && (returnImmediately || EnvironmentEdgeManager.currentTime() >= limits.getTime());
351  }
352
353  /**
354   * @param checkerScope The scope that the limits are being checked from
355   * @return true when some limit is enforceable from the checker's scope and it has been reached
356   */
357  boolean checkAnyLimitReached(LimitScope checkerScope) {
358    return checkSizeLimit(checkerScope) || checkBatchLimit(checkerScope)
359      || checkTimeLimit(checkerScope);
360  }
361
362  Cell getLastPeekedCell() {
363    return lastPeekedCell;
364  }
365
366  void setLastPeekedCell(Cell lastPeekedCell) {
367    this.lastPeekedCell = lastPeekedCell;
368  }
369
370  void returnImmediately() {
371    this.returnImmediately = true;
372  }
373
374  @Override
375  public String toString() {
376    StringBuilder sb = new StringBuilder();
377    sb.append("{");
378
379    sb.append("limits:");
380    sb.append(limits);
381
382    sb.append(", progress:");
383    sb.append(progress);
384
385    sb.append(", keepProgress:");
386    sb.append(keepProgress);
387
388    sb.append(", state:");
389    sb.append(scannerState);
390
391    sb.append("}");
392    return sb.toString();
393  }
394
395  public static Builder newBuilder() {
396    return new Builder();
397  }
398
399  public static Builder newBuilder(boolean keepProgress) {
400    return new Builder(keepProgress);
401  }
402
403  public static final class Builder {
404    boolean keepProgress = DEFAULT_KEEP_PROGRESS;
405    boolean trackMetrics = false;
406    LimitFields limits = new LimitFields();
407
408    private Builder() {
409    }
410
411    private Builder(boolean keepProgress) {
412      this.keepProgress = keepProgress;
413    }
414
415    public Builder setKeepProgress(boolean keepProgress) {
416      this.keepProgress = keepProgress;
417      return this;
418    }
419
420    public Builder setTrackMetrics(boolean trackMetrics) {
421      this.trackMetrics = trackMetrics;
422      return this;
423    }
424
425    public Builder setSizeLimit(LimitScope sizeScope, long dataSizeLimit, long heapSizeLimit) {
426      limits.setDataSize(dataSizeLimit);
427      limits.setHeapSize(heapSizeLimit);
428      limits.setSizeScope(sizeScope);
429      return this;
430    }
431
432    public Builder setTimeLimit(LimitScope timeScope, long timeLimit) {
433      limits.setTime(timeLimit);
434      limits.setTimeScope(timeScope);
435      return this;
436    }
437
438    public Builder setBatchLimit(int batchLimit) {
439      limits.setBatch(batchLimit);
440      return this;
441    }
442
443    public ScannerContext build() {
444      return new ScannerContext(keepProgress, limits, trackMetrics);
445    }
446  }
447
448  /**
449   * The possible states a scanner may be in following a call to {@link InternalScanner#next(List)}
450   */
451  public enum NextState {
452    MORE_VALUES(true, false),
453    NO_MORE_VALUES(false, false),
454    SIZE_LIMIT_REACHED(true, true),
455
456    /**
457     * Special case of size limit reached to indicate that the size limit was reached in the middle
458     * of a row and thus a partial results was formed
459     */
460    SIZE_LIMIT_REACHED_MID_ROW(true, true),
461    TIME_LIMIT_REACHED(true, true),
462
463    /**
464     * Special case of time limit reached to indicate that the time limit was reached in the middle
465     * of a row and thus a partial results was formed
466     */
467    TIME_LIMIT_REACHED_MID_ROW(true, true),
468    BATCH_LIMIT_REACHED(true, true);
469
470    private final boolean moreValues;
471    private final boolean limitReached;
472
473    private NextState(boolean moreValues, boolean limitReached) {
474      this.moreValues = moreValues;
475      this.limitReached = limitReached;
476    }
477
478    /**
479     * @return true when the state indicates that more values may follow those that have been
480     *         returned
481     */
482    public boolean hasMoreValues() {
483      return this.moreValues;
484    }
485
486    /** Returns true when the state indicates that a limit has been reached and scan should stop */
487    public boolean limitReached() {
488      return this.limitReached;
489    }
490
491    public static boolean isValidState(NextState state) {
492      return state != null;
493    }
494
495    public static boolean hasMoreValues(NextState state) {
496      return isValidState(state) && state.hasMoreValues();
497    }
498  }
499
500  /**
501   * The various scopes where a limit can be enforced. Used to differentiate when a limit should be
502   * enforced or not.
503   */
504  public enum LimitScope {
505    /**
506     * Enforcing a limit between rows means that the limit will not be considered until all the
507     * cells for a particular row have been retrieved
508     */
509    BETWEEN_ROWS(0),
510
511    /**
512     * Enforcing a limit between cells means that the limit will be considered after each full cell
513     * has been retrieved
514     */
515    BETWEEN_CELLS(1);
516
517    /**
518     * When enforcing a limit, we must check that the scope is appropriate for enforcement.
519     * <p>
520     * To communicate this concept, each scope has a depth. A limit will be enforced if the depth of
521     * the checker's scope is less than or equal to the limit's scope. This means that when checking
522     * limits, the checker must know their own scope (i.e. are they checking the limits between
523     * rows, between cells, etc...)
524     */
525    final int depth;
526
527    LimitScope(int depth) {
528      this.depth = depth;
529    }
530
531    final int depth() {
532      return depth;
533    }
534
535    /**
536     * @param checkerScope The scope in which the limit is being checked
537     * @return true when the checker is in a scope that indicates the limit can be enforced. Limits
538     *         can be enforced from "higher or equal" scopes (i.e. the checker's scope is at a
539     *         lesser depth than the limit)
540     */
541    boolean canEnforceLimitFromScope(LimitScope checkerScope) {
542      return checkerScope != null && checkerScope.depth() <= depth;
543    }
544  }
545
546  /**
547   * The different fields that can be used as limits in calls to
548   * {@link InternalScanner#next(java.util.List)} and {@link RegionScanner#next(java.util.List)}
549   */
550  private static class LimitFields {
551    /**
552     * Default values of the limit fields. Defined such that if a field does NOT change from its
553     * default, it will not be enforced
554     */
555    private static int DEFAULT_BATCH = -1;
556    private static long DEFAULT_SIZE = -1L;
557    private static long DEFAULT_TIME = -1L;
558
559    /**
560     * Default scope that is assigned to a limit if a scope is not specified.
561     */
562    private static final LimitScope DEFAULT_SCOPE = LimitScope.BETWEEN_ROWS;
563
564    // The batch limit will always be enforced between cells, thus, there isn't a field to hold the
565    // batch scope
566    int batch = DEFAULT_BATCH;
567
568    LimitScope sizeScope = DEFAULT_SCOPE;
569    // The sum of cell data sizes(key + value). The Cell data might be in on heap or off heap area.
570    long dataSize = DEFAULT_SIZE;
571    // The sum of heap space occupied by all tracked cells. This includes Cell POJO's overhead as
572    // such AND data cells of Cells which are in on heap area.
573    long heapSize = DEFAULT_SIZE;
574
575    LimitScope timeScope = DEFAULT_SCOPE;
576    long time = DEFAULT_TIME;
577
578    /**
579     * Fields keep their default values.
580     */
581    LimitFields() {
582    }
583
584    void copy(LimitFields limitsToCopy) {
585      if (limitsToCopy != null) {
586        setFields(limitsToCopy.getBatch(), limitsToCopy.getSizeScope(), limitsToCopy.getDataSize(),
587          limitsToCopy.getHeapSize(), limitsToCopy.getTimeScope(), limitsToCopy.getTime());
588      }
589    }
590
591    /**
592     * Set all fields together.
593     */
594    void setFields(int batch, LimitScope sizeScope, long dataSize, long heapSize,
595      LimitScope timeScope, long time) {
596      setBatch(batch);
597      setSizeScope(sizeScope);
598      setDataSize(dataSize);
599      setHeapSize(heapSize);
600      setTimeScope(timeScope);
601      setTime(time);
602    }
603
604    int getBatch() {
605      return this.batch;
606    }
607
608    void setBatch(int batch) {
609      this.batch = batch;
610    }
611
612    /**
613     * n * @return true when the limit can be enforced from the scope of the checker
614     */
615    boolean canEnforceBatchLimitFromScope(LimitScope checkerScope) {
616      return LimitScope.BETWEEN_CELLS.canEnforceLimitFromScope(checkerScope);
617    }
618
619    long getDataSize() {
620      return this.dataSize;
621    }
622
623    long getHeapSize() {
624      return this.heapSize;
625    }
626
627    void setDataSize(long dataSize) {
628      this.dataSize = dataSize;
629    }
630
631    void setHeapSize(long heapSize) {
632      this.heapSize = heapSize;
633    }
634
635    /** Returns {@link LimitScope} indicating scope in which the size limit is enforced */
636    LimitScope getSizeScope() {
637      return this.sizeScope;
638    }
639
640    /**
641     * Change the scope in which the size limit is enforced
642     */
643    void setSizeScope(LimitScope scope) {
644      this.sizeScope = scope;
645    }
646
647    /**
648     * n * @return true when the limit can be enforced from the scope of the checker
649     */
650    boolean canEnforceSizeLimitFromScope(LimitScope checkerScope) {
651      return this.sizeScope.canEnforceLimitFromScope(checkerScope);
652    }
653
654    long getTime() {
655      return this.time;
656    }
657
658    void setTime(long time) {
659      this.time = time;
660    }
661
662    /** Returns {@link LimitScope} indicating scope in which the time limit is enforced */
663    LimitScope getTimeScope() {
664      return this.timeScope;
665    }
666
667    /**
668     * Change the scope in which the time limit is enforced
669     */
670    void setTimeScope(LimitScope scope) {
671      this.timeScope = scope;
672    }
673
674    /**
675     * n * @return true when the limit can be enforced from the scope of the checker
676     */
677    boolean canEnforceTimeLimitFromScope(LimitScope checkerScope) {
678      return this.timeScope.canEnforceLimitFromScope(checkerScope);
679    }
680
681    @Override
682    public String toString() {
683      StringBuilder sb = new StringBuilder();
684      sb.append("{");
685
686      sb.append("batch:");
687      sb.append(batch);
688
689      sb.append(", dataSize:");
690      sb.append(dataSize);
691
692      sb.append(", heapSize:");
693      sb.append(heapSize);
694
695      sb.append(", sizeScope:");
696      sb.append(sizeScope);
697
698      sb.append(", time:");
699      sb.append(time);
700
701      sb.append(", timeScope:");
702      sb.append(timeScope);
703
704      sb.append("}");
705      return sb.toString();
706    }
707  }
708
709  private static class ProgressFields {
710
711    private static int DEFAULT_BATCH = -1;
712    private static long DEFAULT_SIZE = -1L;
713
714    // The batch limit will always be enforced between cells, thus, there isn't a field to hold the
715    // batch scope
716    int batch = DEFAULT_BATCH;
717
718    // The sum of cell data sizes(key + value). The Cell data might be in on heap or off heap area.
719    long dataSize = DEFAULT_SIZE;
720    // The sum of heap space occupied by all tracked cells. This includes Cell POJO's overhead as
721    // such AND data cells of Cells which are in on heap area.
722    long heapSize = DEFAULT_SIZE;
723
724    ProgressFields(int batch, long size, long heapSize) {
725      setFields(batch, size, heapSize);
726    }
727
728    /**
729     * Set all fields together.
730     */
731    void setFields(int batch, long dataSize, long heapSize) {
732      setBatch(batch);
733      setDataSize(dataSize);
734      setHeapSize(heapSize);
735    }
736
737    int getBatch() {
738      return this.batch;
739    }
740
741    void setBatch(int batch) {
742      this.batch = batch;
743    }
744
745    long getDataSize() {
746      return this.dataSize;
747    }
748
749    long getHeapSize() {
750      return this.heapSize;
751    }
752
753    void setDataSize(long dataSize) {
754      this.dataSize = dataSize;
755    }
756
757    void setHeapSize(long heapSize) {
758      this.heapSize = heapSize;
759    }
760
761    @Override
762    public String toString() {
763      StringBuilder sb = new StringBuilder();
764      sb.append("{");
765
766      sb.append("batch:");
767      sb.append(batch);
768
769      sb.append(", dataSize:");
770      sb.append(dataSize);
771
772      sb.append(", heapSize:");
773      sb.append(heapSize);
774
775      sb.append("}");
776      return sb.toString();
777    }
778  }
779}