|
68 | 68 | import java.nio.ByteBuffer;
|
69 | 69 | import java.nio.charset.StandardCharsets;
|
70 | 70 | import java.time.Duration;
|
| 71 | +import java.time.LocalDate; |
71 | 72 | import java.util.Collection;
|
72 | 73 | import java.util.HashMap;
|
73 | 74 | import java.util.Iterator;
|
@@ -154,16 +155,27 @@ public static void initBeforeClass() throws Exception {
|
154 | 155 |
|
155 | 156 | @AfterAll
|
156 | 157 | public static void closeAfterAll() {
|
157 |
| - cassandraContainer1.close(); |
158 |
| - cassandraContainer2.close(); |
159 |
| - pulsarContainer.close(); |
| 158 | + if (cassandraContainer1 != null) { |
| 159 | + cassandraContainer1.close(); |
| 160 | + } |
| 161 | + if (cassandraContainer2 != null) { |
| 162 | + cassandraContainer2.close(); |
| 163 | + } |
| 164 | + if (pulsarContainer != null) { |
| 165 | + pulsarContainer.close(); |
| 166 | + } |
160 | 167 | }
|
161 | 168 |
|
162 | 169 | @Test
|
163 | 170 | public void testSinglePk() throws InterruptedException, IOException {
|
164 | 171 | testSinglePk("ks1");
|
165 | 172 | }
|
166 | 173 |
|
| 174 | + @Test |
| 175 | + public void testClusteringKey() throws InterruptedException, IOException { |
| 176 | + testClusteringKey("ks1"); |
| 177 | + } |
| 178 | + |
167 | 179 | @Test
|
168 | 180 | public void testCompoundPk() throws InterruptedException, IOException {
|
169 | 181 | testCompoundPk("ks1");
|
@@ -322,6 +334,118 @@ public void testSinglePk(String ksName) throws InterruptedException, IOException
|
322 | 334 | }
|
323 | 335 | }
|
324 | 336 |
|
| 337 | + // docker exec -it pulsar cat /pulsar/logs/functions/public/default/cassandra-source-ks1-table5/cassandra-source-ks1-table5-0.log |
| 338 | + public void testClusteringKey(String ksName) throws InterruptedException, IOException { |
| 339 | + try { |
| 340 | + try (CqlSession cqlSession = cassandraContainer1.getCqlSession()) { |
| 341 | + cqlSession.execute("CREATE KEYSPACE IF NOT EXISTS " + ksName + |
| 342 | + " WITH replication = {'class':'SimpleStrategy','replication_factor':'2'};"); |
| 343 | + cqlSession.execute("CREATE TABLE IF NOT EXISTS " + ksName + ".table5 (pk text, c1 date, c2 uuid, val int, PRIMARY KEY (pk, c1, c2)) WITH cdc=true"); |
| 344 | + cqlSession.execute("INSERT INTO " + ksName + ".table5 (pk, c1, c2, val) VALUES('1','2021-01-10', 016b123d-f732-4173-9225-c6717066c7af, 1)"); |
| 345 | + cqlSession.execute("INSERT INTO " + ksName + ".table5 (pk, c1, c2, val) VALUES('2','2021-01-10', 016b123d-f732-4173-9225-c6717066c7af, 1)"); |
| 346 | + cqlSession.execute("INSERT INTO " + ksName + ".table5 (pk, c1, c2, val) VALUES('3','2021-01-10', 016b123d-f732-4173-9225-c6717066c7af, 1)"); |
| 347 | + } |
| 348 | + deployConnector(ksName, "table5"); |
| 349 | + |
| 350 | + try (PulsarClient pulsarClient = PulsarClient.builder().serviceUrl(pulsarContainer.getPulsarBrokerUrl()).build()) { |
| 351 | + Map<String, Integer> mutationTable5 = new HashMap<>(); |
| 352 | + try (Consumer<GenericRecord> consumer = pulsarClient.newConsumer(org.apache.pulsar.client.api.Schema.AUTO_CONSUME()) |
| 353 | + .topic(String.format(Locale.ROOT, "data-%s.table5", ksName)) |
| 354 | + .subscriptionName("sub1") |
| 355 | + .subscriptionType(SubscriptionType.Key_Shared) |
| 356 | + .subscriptionMode(SubscriptionMode.Durable) |
| 357 | + .subscriptionInitialPosition(SubscriptionInitialPosition.Earliest) |
| 358 | + .subscribe()) { |
| 359 | + Message<GenericRecord> msg; |
| 360 | + while ((msg = consumer.receive(90, TimeUnit.SECONDS)) != null && |
| 361 | + mutationTable5.values().stream().mapToInt(i -> i).sum() < 4) { |
| 362 | + GenericRecord record = msg.getValue(); |
| 363 | + assertEquals(this.schemaType, record.getSchemaType()); |
| 364 | + Object key = getKey(msg); |
| 365 | + GenericRecord value = getValue(record); |
| 366 | + assertEquals((Integer) 0, mutationTable5.computeIfAbsent(getAndAssertKeyFieldAsString(key, "pk"), k -> 0)); |
| 367 | + assertEquals((int) LocalDate.parse("2021-01-10").toEpochDay(), getAndAssertKeyFieldAsInt(key, "c1")); |
| 368 | + assertEquals("016b123d-f732-4173-9225-c6717066c7af", getAndAssertKeyFieldAsString(key, "c2")); |
| 369 | + assertEquals(1, value.getField("val")); |
| 370 | + mutationTable5.compute(getAndAssertKeyFieldAsString(key, "pk"), (k, v) -> v + 1); |
| 371 | + consumer.acknowledge(msg); |
| 372 | + } |
| 373 | + assertEquals((Integer) 1, mutationTable5.get("1")); |
| 374 | + assertEquals((Integer) 1, mutationTable5.get("2")); |
| 375 | + assertEquals((Integer) 1, mutationTable5.get("3")); |
| 376 | + |
| 377 | + // trigger a schema update |
| 378 | + try (CqlSession cqlSession = cassandraContainer1.getCqlSession()) { |
| 379 | + cqlSession.execute("ALTER TABLE " + ksName + ".table5 ADD d double"); |
| 380 | + cqlSession.execute("INSERT INTO " + ksName + ".table5 (pk,c1,c2,val,d) VALUES('1','2021-01-10', 016b123d-f732-4173-9225-c6717066c7af,1,1.0)"); |
| 381 | + } |
| 382 | + while ((msg = consumer.receive(90, TimeUnit.SECONDS)) != null && |
| 383 | + mutationTable5.values().stream().mapToInt(i -> i).sum() < 5) { |
| 384 | + GenericRecord record = msg.getValue(); |
| 385 | + assertEquals(this.schemaType, record.getSchemaType()); |
| 386 | + Object key = getKey(msg); |
| 387 | + GenericRecord value = getValue(record); |
| 388 | + assertEquals("1", getAndAssertKeyFieldAsString(key, "pk")); |
| 389 | + assertEquals((int) LocalDate.parse("2021-01-10").toEpochDay(), getAndAssertKeyFieldAsInt(key, "c1")); |
| 390 | + assertEquals("016b123d-f732-4173-9225-c6717066c7af", getAndAssertKeyFieldAsString(key, "c2")); |
| 391 | + assertEquals(1, value.getField("val")); |
| 392 | + mutationTable5.compute(getAndAssertKeyFieldAsString(key, "pk"), (k, v) -> v + 1); |
| 393 | + consumer.acknowledge(msg); |
| 394 | + } |
| 395 | + assertEquals((Integer) 2, mutationTable5.get("1")); // 2 inserts for pk=1 |
| 396 | + assertEquals((Integer) 1, mutationTable5.get("2")); |
| 397 | + assertEquals((Integer) 1, mutationTable5.get("3")); |
| 398 | + |
| 399 | + // delete a row by partition key only |
| 400 | + try (CqlSession cqlSession = cassandraContainer1.getCqlSession()) { |
| 401 | + cqlSession.execute("DELETE FROM " + ksName + ".table5 WHERE pk = '1'"); |
| 402 | + } |
| 403 | + while ((msg = consumer.receive(60, TimeUnit.SECONDS)) != null && |
| 404 | + mutationTable5.values().stream().mapToInt(i -> i).sum() < 6) { |
| 405 | + GenericRecord record = msg.getValue(); |
| 406 | + assertEquals(this.schemaType, record.getSchemaType()); |
| 407 | + Object key = getKey(msg); |
| 408 | + GenericRecord value = getValue(record); |
| 409 | + assertEquals("1", getAndAssertKeyFieldAsString(key, "pk")); |
| 410 | + // clustering keys are null because we deleted by partition key only |
| 411 | + assertKeyFieldIsNull(key, "c1"); |
| 412 | + assertKeyFieldIsNull(key, "c2"); |
| 413 | + assertNullValue(value); |
| 414 | + mutationTable5.compute(getAndAssertKeyFieldAsString(key, "pk"), (k, v) -> v + 1); |
| 415 | + consumer.acknowledge(msg); |
| 416 | + } |
| 417 | + assertEquals((Integer) 3, mutationTable5.get("1")); // 2 inserts and 1 delete for pk=1 |
| 418 | + assertEquals((Integer) 1, mutationTable5.get("2")); |
| 419 | + assertEquals((Integer) 1, mutationTable5.get("3")); |
| 420 | + |
| 421 | + // delete a row by partition and clustering keys |
| 422 | + try (CqlSession cqlSession = cassandraContainer1.getCqlSession()) { |
| 423 | + cqlSession.execute("DELETE FROM " + ksName + ".table5 WHERE pk = '2' and c1 = '2021-01-10' and c2 = 016b123d-f732-4173-9225-c6717066c7af"); |
| 424 | + } |
| 425 | + while ((msg = consumer.receive(60, TimeUnit.SECONDS)) != null && |
| 426 | + mutationTable5.values().stream().mapToInt(i -> i).sum() < 6) { |
| 427 | + GenericRecord record = msg.getValue(); |
| 428 | + assertEquals(this.schemaType, record.getSchemaType()); |
| 429 | + Object key = getKey(msg); |
| 430 | + GenericRecord value = getValue(record); |
| 431 | + assertEquals("2", getAndAssertKeyFieldAsString(key, "pk")); |
| 432 | + assertEquals((int) LocalDate.parse("2021-01-10").toEpochDay(), getAndAssertKeyFieldAsInt(key, "c1")); |
| 433 | + assertEquals("016b123d-f732-4173-9225-c6717066c7af", getAndAssertKeyFieldAsString(key, "c2")); |
| 434 | + assertNullValue(value); |
| 435 | + mutationTable5.compute(getAndAssertKeyFieldAsString(key, "pk"), (k, v) -> v + 1); |
| 436 | + consumer.acknowledge(msg); |
| 437 | + } |
| 438 | + assertEquals((Integer) 3, mutationTable5.get("1")); // 2 inserts and 1 delete for pk=1 |
| 439 | + assertEquals((Integer) 2, mutationTable5.get("2")); // 1 insert and 1 delete for pk=2 |
| 440 | + assertEquals((Integer) 1, mutationTable5.get("3")); |
| 441 | + } |
| 442 | + } |
| 443 | + } finally { |
| 444 | + dumpFunctionLogs("cassandra-source-" + ksName + "-table5"); |
| 445 | + undeployConnector(ksName, "table5"); |
| 446 | + } |
| 447 | + } |
| 448 | + |
325 | 449 | // docker exec -it pulsar cat /pulsar/logs/functions/public/default/cassandra-source-ks1-table2/cassandra-source-ks1-table2-0.log
|
326 | 450 | public void testCompoundPk(String ksName) throws InterruptedException, IOException {
|
327 | 451 | try {
|
|
0 commit comments