|
3 | 3 | import com.linkedin.davinci.repository.NativeMetadataRepositoryViewAdapter; |
4 | 4 | import com.linkedin.venice.exceptions.VeniceException; |
5 | 5 | import com.linkedin.venice.kafka.protocol.ControlMessage; |
| 6 | +import com.linkedin.venice.kafka.protocol.VersionSwap; |
6 | 7 | import com.linkedin.venice.kafka.protocol.enums.ControlMessageType; |
7 | 8 | import com.linkedin.venice.pubsub.api.DefaultPubSubMessage; |
8 | 9 | import com.linkedin.venice.pubsub.api.PubSubConsumerAdapter; |
@@ -73,7 +74,8 @@ protected VeniceAfterImageConsumerImpl( |
73 | 74 | storeRepository, |
74 | 75 | storeName, |
75 | 76 | changelogClientConfig.getConsumerName(), |
76 | | - this.changeCaptureStats); |
| 77 | + this.changeCaptureStats, |
| 78 | + changelogClientConfig.isVersionSwapByControlMessageEnabled()); |
77 | 79 | } |
78 | 80 |
|
79 | 81 | @Override |
@@ -178,6 +180,100 @@ protected static void adjustSeekCheckPointsBasedOnHeartbeats( |
178 | 180 | } |
179 | 181 | } |
180 | 182 |
|
| 183 | + /** |
| 184 | + * Similar to {@link #internalSeekToEndOfPush} this can also be optimized later for a faster find. |
| 185 | + */ |
| 186 | + @Override |
| 187 | + protected CompletableFuture<Void> internalFindNewVersionCheckpoints( |
| 188 | + String oldVersionTopic, |
| 189 | + String newVersionTopic, |
| 190 | + long generationId, |
| 191 | + Set<Integer> partitions) { |
| 192 | + if (partitions.isEmpty()) { |
| 193 | + return CompletableFuture.completedFuture(null); |
| 194 | + } |
| 195 | + return CompletableFuture.supplyAsync(() -> { |
| 196 | + boolean lockAcquired = false; |
| 197 | + Map<Integer, VeniceChangeCoordinate> checkpoints = new HashMap<>(); |
| 198 | + try { |
| 199 | + synchronized (internalSeekConsumer) { |
| 200 | + PubSubConsumerAdapter consumerAdapter = internalSeekConsumer.get(); |
| 201 | + consumerAdapter.batchUnsubscribe(consumerAdapter.getAssignment()); |
| 202 | + Map<PubSubTopicPartition, List<DefaultPubSubMessage>> polledResults; |
| 203 | + Map<Integer, Boolean> versionSwapConsumedPerPartitionMap = new HashMap<>(); |
| 204 | + for (Integer partition: partitions) { |
| 205 | + versionSwapConsumedPerPartitionMap.put(partition, false); |
| 206 | + } |
| 207 | + List<PubSubTopicPartition> topicPartitionList = getPartitionListToSubscribe( |
| 208 | + partitions, |
| 209 | + Collections.EMPTY_SET, |
| 210 | + pubSubTopicRepository.getTopic(newVersionTopic)); |
| 211 | + |
| 212 | + for (PubSubTopicPartition topicPartition: topicPartitionList) { |
| 213 | + consumerAdapter.subscribe(topicPartition, PubSubSymbolicPosition.EARLIEST); |
| 214 | + } |
| 215 | + |
| 216 | + // Poll until we receive the desired version swap message in the new version topic for each partition |
| 217 | + LOGGER.info( |
| 218 | + "Polling for version swap messages in: {} with generation id: {} for partitions: {}", |
| 219 | + newVersionTopic, |
| 220 | + generationId, |
| 221 | + partitions); |
| 222 | + while (!versionSwapConsumedPerPartitionMap.values().stream().allMatch(x -> x)) { |
| 223 | + polledResults = consumerAdapter.poll(5000L); |
| 224 | + for (Map.Entry<PubSubTopicPartition, List<DefaultPubSubMessage>> entry: polledResults.entrySet()) { |
| 225 | + PubSubTopicPartition pubSubTopicPartition = entry.getKey(); |
| 226 | + List<DefaultPubSubMessage> messageList = entry.getValue(); |
| 227 | + for (DefaultPubSubMessage message: messageList) { |
| 228 | + if (message.getKey().isControlMessage()) { |
| 229 | + ControlMessage controlMessage = (ControlMessage) message.getValue().getPayloadUnion(); |
| 230 | + ControlMessageType controlMessageType = ControlMessageType.valueOf(controlMessage); |
| 231 | + if (controlMessageType.equals(ControlMessageType.VERSION_SWAP)) { |
| 232 | + VersionSwap versionSwap = (VersionSwap) controlMessage.getControlMessageUnion(); |
| 233 | + // In theory just matching the generation id and source region should be sufficient but just to be |
| 234 | + // safe we will match all fields |
| 235 | + if (versionSwap.getGenerationId() == generationId |
| 236 | + && versionSwap.getSourceRegion().toString().equals(clientRegionName) |
| 237 | + && oldVersionTopic.equals(versionSwap.getOldServingVersionTopic().toString()) |
| 238 | + && newVersionTopic.equals(versionSwap.getNewServingVersionTopic().toString())) { |
| 239 | + LOGGER.info( |
| 240 | + "Found corresponding version swap message for partition: {}", |
| 241 | + pubSubTopicPartition.getPartitionNumber()); |
| 242 | + versionSwapConsumedPerPartitionMap.put(pubSubTopicPartition.getPartitionNumber(), true); |
| 243 | + VeniceChangeCoordinate coordinate = new VeniceChangeCoordinate( |
| 244 | + pubSubTopicPartition.getPubSubTopic().getName(), |
| 245 | + message.getPosition(), |
| 246 | + pubSubTopicPartition.getPartitionNumber()); |
| 247 | + checkpoints.put(pubSubTopicPartition.getPartitionNumber(), coordinate); |
| 248 | + // We are done with this partition |
| 249 | + consumerAdapter.unSubscribe(pubSubTopicPartition); |
| 250 | + break; |
| 251 | + } |
| 252 | + } |
| 253 | + } |
| 254 | + } |
| 255 | + } |
| 256 | + } |
| 257 | + LOGGER.info( |
| 258 | + "Found all version swap messages in: {} with generation id: {} for partitions: {}", |
| 259 | + newVersionTopic, |
| 260 | + generationId, |
| 261 | + partitions); |
| 262 | + } |
| 263 | + // We cannot change the subscription here because the consumer might not finish polling all the messages in the |
| 264 | + // old version topic yet. We can acquire the lock and update the VersionSwapMessageState. |
| 265 | + subscriptionLock.writeLock().lock(); |
| 266 | + lockAcquired = true; |
| 267 | + versionSwapMessageState.setNewTopicCheckpoints(checkpoints); |
| 268 | + } finally { |
| 269 | + if (lockAcquired) { |
| 270 | + subscriptionLock.writeLock().unlock(); |
| 271 | + } |
| 272 | + } |
| 273 | + return null; |
| 274 | + }, seekExecutorService); |
| 275 | + } |
| 276 | + |
181 | 277 | protected CompletableFuture<Void> internalSeekToEndOfPush( |
182 | 278 | Set<Integer> partitions, |
183 | 279 | PubSubTopic targetTopic, |
|
0 commit comments