|
22 | 22 | import org.apache.hudi.common.fs.FSUtils; |
23 | 23 | import org.apache.hudi.common.model.HoodieWriteStat; |
24 | 24 | import org.apache.hudi.common.table.HoodieTableMetaClient; |
| 25 | +import org.apache.hudi.common.table.timeline.HoodieActiveTimeline; |
| 26 | +import org.apache.hudi.common.table.timeline.HoodieInstant; |
25 | 27 | import org.apache.hudi.common.table.timeline.HoodieTimeline; |
26 | 28 | import org.apache.hudi.configuration.FlinkOptions; |
27 | 29 | import org.apache.hudi.configuration.HadoopConfigurations; |
@@ -253,6 +255,49 @@ void testSyncMetadataTable() throws Exception { |
253 | 255 | assertThat(completedTimeline.nthFromLastInstant(1).get().getAction(), is(HoodieTimeline.COMMIT_ACTION)); |
254 | 256 | } |
255 | 257 |
|
| 258 | + @Test |
| 259 | + void testSyncMetadataTableWithReusedInstant() throws Exception { |
| 260 | + // reset |
| 261 | + reset(); |
| 262 | + // override the default configuration |
| 263 | + Configuration conf = TestConfigurations.getDefaultConf(tempFile.getAbsolutePath()); |
| 264 | + conf.setBoolean(FlinkOptions.METADATA_ENABLED, true); |
| 265 | + OperatorCoordinator.Context context = new MockOperatorCoordinatorContext(new OperatorID(), 1); |
| 266 | + coordinator = new StreamWriteOperatorCoordinator(conf, context); |
| 267 | + coordinator.start(); |
| 268 | + coordinator.setExecutor(new MockCoordinatorExecutor(context)); |
| 269 | + |
| 270 | + final WriteMetadataEvent event0 = WriteMetadataEvent.emptyBootstrap(0); |
| 271 | + |
| 272 | + coordinator.handleEventFromOperator(0, event0); |
| 273 | + |
| 274 | + String instant = coordinator.getInstant(); |
| 275 | + assertNotEquals("", instant); |
| 276 | + |
| 277 | + final String metadataTableBasePath = HoodieTableMetadata.getMetadataTableBasePath(tempFile.getAbsolutePath()); |
| 278 | + HoodieTableMetaClient metadataTableMetaClient = StreamerUtil.createMetaClient(metadataTableBasePath, HadoopConfigurations.getHadoopConf(conf)); |
| 279 | + HoodieTimeline completedTimeline = metadataTableMetaClient.getActiveTimeline().filterCompletedInstants(); |
| 280 | + assertThat("One instant need to sync to metadata table", completedTimeline.getInstants().count(), is(1L)); |
| 281 | + assertThat(completedTimeline.lastInstant().get().getTimestamp(), is(HoodieTableMetadata.SOLO_COMMIT_TIMESTAMP)); |
| 282 | + |
| 283 | + // writes a normal commit |
| 284 | + mockWriteWithMetadata(); |
| 285 | + instant = coordinator.getInstant(); |
| 286 | + // creates an inflight commit on the metadata timeline |
| 287 | + metadataTableMetaClient.getActiveTimeline() |
| 288 | + .createNewInstant(new HoodieInstant(HoodieInstant.State.REQUESTED, HoodieActiveTimeline.DELTA_COMMIT_ACTION, instant)); |
| 289 | + metadataTableMetaClient.getActiveTimeline().transitionRequestedToInflight(HoodieActiveTimeline.DELTA_COMMIT_ACTION, instant); |
| 290 | + metadataTableMetaClient.reloadActiveTimeline(); |
| 291 | + |
| 292 | + // write another commit with existing instant on the metadata timeline |
| 293 | + instant = mockWriteWithMetadata(); |
| 294 | + metadataTableMetaClient.reloadActiveTimeline(); |
| 295 | + |
| 296 | + completedTimeline = metadataTableMetaClient.getActiveTimeline().filterCompletedInstants(); |
| 297 | + assertThat("One instant need to sync to metadata table", completedTimeline.getInstants().count(), is(3L)); |
| 298 | + assertThat(completedTimeline.lastInstant().get().getTimestamp(), is(instant)); |
| 299 | + } |
| 300 | + |
256 | 301 | // ------------------------------------------------------------------------- |
257 | 302 | // Utilities |
258 | 303 | // ------------------------------------------------------------------------- |
|
0 commit comments