|
23 | 23 | import java.util.ArrayList;
|
24 | 24 | import java.util.Collections;
|
25 | 25 | import java.util.List;
|
| 26 | +import java.util.Set; |
26 | 27 | import java.util.concurrent.CompletableFuture;
|
27 | 28 | import java.util.concurrent.CompletionException;
|
| 29 | +import java.util.concurrent.ConcurrentHashMap; |
28 | 30 | import java.util.concurrent.ConcurrentLinkedQueue;
|
| 31 | +import java.util.concurrent.atomic.AtomicLong; |
29 | 32 | import java.util.regex.Matcher;
|
30 | 33 | import java.util.regex.Pattern;
|
31 |
| -import java.util.stream.Collectors; |
32 | 34 | import org.apache.hadoop.conf.Configuration;
|
33 | 35 | import org.apache.hadoop.hbase.TableName;
|
34 | 36 | import org.apache.hadoop.hbase.util.Pair;
|
35 | 37 | import org.apache.yetus.audience.InterfaceAudience;
|
| 38 | +import org.slf4j.Logger; |
| 39 | +import org.slf4j.LoggerFactory; |
36 | 40 |
|
37 | 41 | /**
|
38 | 42 | * {@link BufferedMutator} implementation based on {@link AsyncBufferedMutator}.
|
39 | 43 | */
|
40 | 44 | @InterfaceAudience.Private
|
41 | 45 | class BufferedMutatorOverAsyncBufferedMutator implements BufferedMutator {
|
42 | 46 |
|
| 47 | + private static final Logger LOG = |
| 48 | + LoggerFactory.getLogger(BufferedMutatorOverAsyncBufferedMutator.class); |
| 49 | + |
43 | 50 | private final AsyncBufferedMutator mutator;
|
44 | 51 |
|
45 | 52 | private final ExceptionListener listener;
|
46 | 53 |
|
47 |
| - private List<CompletableFuture<Void>> futures = new ArrayList<>(); |
| 54 | + private final Set<CompletableFuture<Void>> futures = ConcurrentHashMap.newKeySet(); |
| 55 | + |
| 56 | + private final AtomicLong bufferedSize = new AtomicLong(0); |
48 | 57 |
|
49 | 58 | private final ConcurrentLinkedQueue<Pair<Mutation, Throwable>> errors =
|
50 | 59 | new ConcurrentLinkedQueue<>();
|
51 | 60 |
|
52 |
| - private final static int BUFFERED_FUTURES_THRESHOLD = 1024; |
53 |
| - |
54 | 61 | BufferedMutatorOverAsyncBufferedMutator(AsyncBufferedMutator mutator,
|
55 | 62 | ExceptionListener listener) {
|
56 | 63 | this.mutator = mutator;
|
@@ -100,62 +107,69 @@ private RetriesExhaustedWithDetailsException makeError() {
|
100 | 107 | return new RetriesExhaustedWithDetailsException(throwables, rows, hostnameAndPorts);
|
101 | 108 | }
|
102 | 109 |
|
| 110 | + private void internalFlush() throws RetriesExhaustedWithDetailsException { |
| 111 | + // should get the future array before calling mutator.flush, otherwise we may hit an infinite |
| 112 | + // wait, since someone may add new future to the map after we calling the flush. |
| 113 | + CompletableFuture<?>[] toWait = futures.toArray(new CompletableFuture<?>[0]); |
| 114 | + mutator.flush(); |
| 115 | + try { |
| 116 | + CompletableFuture.allOf(toWait).join(); |
| 117 | + } catch (CompletionException e) { |
| 118 | + // just ignore, we will record the actual error in the errors field |
| 119 | + LOG.debug("Flush failed, you should get an exception thrown to your code", e); |
| 120 | + } |
| 121 | + if (!errors.isEmpty()) { |
| 122 | + RetriesExhaustedWithDetailsException error = makeError(); |
| 123 | + listener.onException(error, this); |
| 124 | + } |
| 125 | + } |
| 126 | + |
103 | 127 | @Override
|
104 | 128 | public void mutate(List<? extends Mutation> mutations) throws IOException {
|
105 |
| - List<CompletableFuture<Void>> toBuffered = new ArrayList<>(); |
106 | 129 | List<CompletableFuture<Void>> fs = mutator.mutate(mutations);
|
107 | 130 | for (int i = 0, n = fs.size(); i < n; i++) {
|
108 | 131 | CompletableFuture<Void> toComplete = new CompletableFuture<>();
|
109 |
| - final int index = i; |
110 |
| - addListener(fs.get(index), (r, e) -> { |
| 132 | + futures.add(toComplete); |
| 133 | + Mutation mutation = mutations.get(i); |
| 134 | + long heapSize = mutation.heapSize(); |
| 135 | + bufferedSize.addAndGet(heapSize); |
| 136 | + addListener(fs.get(i), (r, e) -> { |
| 137 | + futures.remove(toComplete); |
| 138 | + bufferedSize.addAndGet(-heapSize); |
111 | 139 | if (e != null) {
|
112 |
| - errors.add(Pair.newPair(mutations.get(index), e)); |
| 140 | + errors.add(Pair.newPair(mutation, e)); |
113 | 141 | toComplete.completeExceptionally(e);
|
114 | 142 | } else {
|
115 | 143 | toComplete.complete(r);
|
116 | 144 | }
|
117 | 145 | });
|
118 |
| - toBuffered.add(toComplete); |
119 | 146 | }
|
120 | 147 | synchronized (this) {
|
121 |
| - futures.addAll(toBuffered); |
122 |
| - if (futures.size() > BUFFERED_FUTURES_THRESHOLD) { |
123 |
| - tryCompleteFuture(); |
124 |
| - } |
125 |
| - if (!errors.isEmpty()) { |
| 148 | + if (bufferedSize.get() > mutator.getWriteBufferSize() * 2) { |
| 149 | + // We have too many mutations which are not completed yet, let's call a flush to release the |
| 150 | + // memory to prevent OOM |
| 151 | + // We use buffer size * 2 is because that, the async buffered mutator will flush |
| 152 | + // automatically when the write buffer size limit is reached, so usually we do not need to |
| 153 | + // call flush explicitly if the buffered size is only a little larger than the buffer size |
| 154 | + // limit. But if the buffered size is too large(2 times of the buffer size), we still need |
| 155 | + // to block here to prevent OOM. |
| 156 | + internalFlush(); |
| 157 | + } else if (!errors.isEmpty()) { |
126 | 158 | RetriesExhaustedWithDetailsException error = makeError();
|
127 | 159 | listener.onException(error, this);
|
128 | 160 | }
|
129 | 161 | }
|
130 | 162 | }
|
131 | 163 |
|
132 |
| - private void tryCompleteFuture() { |
133 |
| - futures = futures.stream().filter(f -> !f.isDone()).collect(Collectors.toList()); |
134 |
| - } |
135 |
| - |
136 | 164 | @Override
|
137 |
| - public void close() throws IOException { |
138 |
| - flush(); |
| 165 | + public synchronized void close() throws IOException { |
| 166 | + internalFlush(); |
139 | 167 | mutator.close();
|
140 | 168 | }
|
141 | 169 |
|
142 | 170 | @Override
|
143 |
| - public void flush() throws IOException { |
144 |
| - mutator.flush(); |
145 |
| - synchronized (this) { |
146 |
| - List<CompletableFuture<Void>> toComplete = this.futures; |
147 |
| - this.futures = new ArrayList<>(); |
148 |
| - try { |
149 |
| - CompletableFuture.allOf(toComplete.toArray(new CompletableFuture<?>[toComplete.size()])) |
150 |
| - .join(); |
151 |
| - } catch (CompletionException e) { |
152 |
| - // just ignore, we will record the actual error in the errors field |
153 |
| - } |
154 |
| - if (!errors.isEmpty()) { |
155 |
| - RetriesExhaustedWithDetailsException error = makeError(); |
156 |
| - listener.onException(error, this); |
157 |
| - } |
158 |
| - } |
| 171 | + public synchronized void flush() throws IOException { |
| 172 | + internalFlush(); |
159 | 173 | }
|
160 | 174 |
|
161 | 175 | @Override
|
|
0 commit comments