From 6401e8bd46b29037ce33d7de01061c3650d3e32a Mon Sep 17 00:00:00 2001 From: Dmitriy Tverdiakov Date: Tue, 23 Jul 2024 16:26:27 +0100 Subject: [PATCH 1/2] Add Bolt 5.6 --- benchkit-backend/pom.xml | 2 +- bundle/pom.xml | 2 +- driver/pom.xml | 2 +- .../async/connection/BoltProtocolUtil.java | 4 +- .../internal/messaging/BoltProtocol.java | 3 + .../messaging/v56/BoltProtocolV56.java | 31 + .../internal/util/MetadataExtractor.java | 4 +- .../connection/BoltProtocolUtilTest.java | 6 +- .../messaging/v56/BoltProtocolV56Test.java | 603 ++++++++++++++++++ .../messaging/v56/MessageFormatV56Test.java | 46 ++ .../messaging/v56/MessageWriterV56Test.java | 212 ++++++ .../org/neo4j/driver/testutil/TestUtil.java | 6 +- examples/pom.xml | 2 +- pom.xml | 2 +- testkit-backend/pom.xml | 2 +- .../messages/requests/GetFeatures.java | 1 + testkit-tests/pom.xml | 2 +- 17 files changed, 916 insertions(+), 14 deletions(-) create mode 100644 driver/src/main/java/org/neo4j/driver/internal/messaging/v56/BoltProtocolV56.java create mode 100644 driver/src/test/java/org/neo4j/driver/internal/messaging/v56/BoltProtocolV56Test.java create mode 100644 driver/src/test/java/org/neo4j/driver/internal/messaging/v56/MessageFormatV56Test.java create mode 100644 driver/src/test/java/org/neo4j/driver/internal/messaging/v56/MessageWriterV56Test.java diff --git a/benchkit-backend/pom.xml b/benchkit-backend/pom.xml index e60ed9955d..837862dc79 100644 --- a/benchkit-backend/pom.xml +++ b/benchkit-backend/pom.xml @@ -7,7 +7,7 @@ neo4j-java-driver-parent org.neo4j.driver - 5.22-SNAPSHOT + 5.23-SNAPSHOT benchkit-backend diff --git a/bundle/pom.xml b/bundle/pom.xml index 29ba7e6df6..c69b8c7034 100644 --- a/bundle/pom.xml +++ b/bundle/pom.xml @@ -6,7 +6,7 @@ org.neo4j.driver neo4j-java-driver-parent - 5.22-SNAPSHOT + 5.23-SNAPSHOT .. diff --git a/driver/pom.xml b/driver/pom.xml index 80a223a54b..a30917bbdc 100644 --- a/driver/pom.xml +++ b/driver/pom.xml @@ -6,7 +6,7 @@ org.neo4j.driver neo4j-java-driver-parent - 5.22-SNAPSHOT + 5.23-SNAPSHOT neo4j-java-driver diff --git a/driver/src/main/java/org/neo4j/driver/internal/async/connection/BoltProtocolUtil.java b/driver/src/main/java/org/neo4j/driver/internal/async/connection/BoltProtocolUtil.java index 1f9e138cb1..23bbcd2db9 100644 --- a/driver/src/main/java/org/neo4j/driver/internal/async/connection/BoltProtocolUtil.java +++ b/driver/src/main/java/org/neo4j/driver/internal/async/connection/BoltProtocolUtil.java @@ -27,7 +27,7 @@ import org.neo4j.driver.internal.messaging.v42.BoltProtocolV42; import org.neo4j.driver.internal.messaging.v44.BoltProtocolV44; import org.neo4j.driver.internal.messaging.v5.BoltProtocolV5; -import org.neo4j.driver.internal.messaging.v55.BoltProtocolV55; +import org.neo4j.driver.internal.messaging.v56.BoltProtocolV56; public final class BoltProtocolUtil { public static final int BOLT_MAGIC_PREAMBLE = 0x6060B017; @@ -39,7 +39,7 @@ public final class BoltProtocolUtil { private static final ByteBuf HANDSHAKE_BUF = unreleasableBuffer(copyInt( BOLT_MAGIC_PREAMBLE, - BoltProtocolV55.VERSION.toIntRange(BoltProtocolV5.VERSION), + BoltProtocolV56.VERSION.toIntRange(BoltProtocolV5.VERSION), BoltProtocolV44.VERSION.toIntRange(BoltProtocolV42.VERSION), BoltProtocolV41.VERSION.toInt(), BoltProtocolV3.VERSION.toInt())) diff --git a/driver/src/main/java/org/neo4j/driver/internal/messaging/BoltProtocol.java b/driver/src/main/java/org/neo4j/driver/internal/messaging/BoltProtocol.java index 15181cc8cd..a561f4d375 100644 --- a/driver/src/main/java/org/neo4j/driver/internal/messaging/BoltProtocol.java +++ b/driver/src/main/java/org/neo4j/driver/internal/messaging/BoltProtocol.java @@ -50,6 +50,7 @@ import org.neo4j.driver.internal.messaging.v53.BoltProtocolV53; import org.neo4j.driver.internal.messaging.v54.BoltProtocolV54; import org.neo4j.driver.internal.messaging.v55.BoltProtocolV55; +import org.neo4j.driver.internal.messaging.v56.BoltProtocolV56; import org.neo4j.driver.internal.spi.Connection; public interface BoltProtocol { @@ -214,6 +215,8 @@ static BoltProtocol forVersion(BoltProtocolVersion version) { return BoltProtocolV54.INSTANCE; } else if (BoltProtocolV55.VERSION.equals(version)) { return BoltProtocolV55.INSTANCE; + } else if (BoltProtocolV56.VERSION.equals(version)) { + return BoltProtocolV56.INSTANCE; } throw new ClientException("Unknown protocol version: " + version); } diff --git a/driver/src/main/java/org/neo4j/driver/internal/messaging/v56/BoltProtocolV56.java b/driver/src/main/java/org/neo4j/driver/internal/messaging/v56/BoltProtocolV56.java new file mode 100644 index 0000000000..c5b6d36804 --- /dev/null +++ b/driver/src/main/java/org/neo4j/driver/internal/messaging/v56/BoltProtocolV56.java @@ -0,0 +1,31 @@ +/* + * Copyright (c) "Neo4j" + * Neo4j Sweden AB [https://neo4j.com] + * + * Licensed under the Apache License, Version 2.0 (the "License"); + * you may not use this file except in compliance with the License. + * You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ +package org.neo4j.driver.internal.messaging.v56; + +import org.neo4j.driver.internal.messaging.BoltProtocol; +import org.neo4j.driver.internal.messaging.BoltProtocolVersion; +import org.neo4j.driver.internal.messaging.v55.BoltProtocolV55; + +public class BoltProtocolV56 extends BoltProtocolV55 { + public static final BoltProtocolVersion VERSION = new BoltProtocolVersion(5, 6); + public static final BoltProtocol INSTANCE = new BoltProtocolV56(); + + @Override + public BoltProtocolVersion version() { + return VERSION; + } +} diff --git a/driver/src/main/java/org/neo4j/driver/internal/util/MetadataExtractor.java b/driver/src/main/java/org/neo4j/driver/internal/util/MetadataExtractor.java index 27ef64cfe2..cb8022a8d9 100644 --- a/driver/src/main/java/org/neo4j/driver/internal/util/MetadataExtractor.java +++ b/driver/src/main/java/org/neo4j/driver/internal/util/MetadataExtractor.java @@ -394,6 +394,8 @@ private static GqlStatusObject extractGqlStatusObject(Value value) { return new InternalGqlStatusObject(status, description, diagnosticRecord); } else { var title = value.get("title").asString(); + var notificationDescription = + value.containsKey("description") ? value.get("description").asString() : description; var positionValue = diagnosticRecord.get("_position"); InputPosition position = null; @@ -427,7 +429,7 @@ private static GqlStatusObject extractGqlStatusObject(Value value) { diagnosticRecord, neo4jCode, title, - description, + notificationDescription, severity, rawSeverity, classification, diff --git a/driver/src/test/java/org/neo4j/driver/internal/async/connection/BoltProtocolUtilTest.java b/driver/src/test/java/org/neo4j/driver/internal/async/connection/BoltProtocolUtilTest.java index e8513b72da..4570a39055 100644 --- a/driver/src/test/java/org/neo4j/driver/internal/async/connection/BoltProtocolUtilTest.java +++ b/driver/src/test/java/org/neo4j/driver/internal/async/connection/BoltProtocolUtilTest.java @@ -30,7 +30,7 @@ import org.neo4j.driver.internal.messaging.v3.BoltProtocolV3; import org.neo4j.driver.internal.messaging.v41.BoltProtocolV41; import org.neo4j.driver.internal.messaging.v44.BoltProtocolV44; -import org.neo4j.driver.internal.messaging.v55.BoltProtocolV55; +import org.neo4j.driver.internal.messaging.v56.BoltProtocolV56; class BoltProtocolUtilTest { @Test @@ -38,7 +38,7 @@ void shouldReturnHandshakeBuf() { assertByteBufContains( handshakeBuf(), BOLT_MAGIC_PREAMBLE, - (5 << 16) | BoltProtocolV55.VERSION.toInt(), + (6 << 16) | BoltProtocolV56.VERSION.toInt(), (2 << 16) | BoltProtocolV44.VERSION.toInt(), BoltProtocolV41.VERSION.toInt(), BoltProtocolV3.VERSION.toInt()); @@ -46,7 +46,7 @@ void shouldReturnHandshakeBuf() { @Test void shouldReturnHandshakeString() { - assertEquals("[0x6060b017, 328965, 132100, 260, 3]", handshakeString()); + assertEquals("[0x6060b017, 394757, 132100, 260, 3]", handshakeString()); } @Test diff --git a/driver/src/test/java/org/neo4j/driver/internal/messaging/v56/BoltProtocolV56Test.java b/driver/src/test/java/org/neo4j/driver/internal/messaging/v56/BoltProtocolV56Test.java new file mode 100644 index 0000000000..6c286a12fc --- /dev/null +++ b/driver/src/test/java/org/neo4j/driver/internal/messaging/v56/BoltProtocolV56Test.java @@ -0,0 +1,603 @@ +/* + * Copyright (c) "Neo4j" + * Neo4j Sweden AB [https://neo4j.com] + * + * Licensed under the Apache License, Version 2.0 (the "License"); + * you may not use this file except in compliance with the License. + * You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ +package org.neo4j.driver.internal.messaging.v56; + +import static java.time.Duration.ofSeconds; +import static java.util.Collections.emptyMap; +import static java.util.Collections.singletonMap; +import static org.hamcrest.MatcherAssert.assertThat; +import static org.hamcrest.Matchers.hasSize; +import static org.hamcrest.Matchers.instanceOf; +import static org.junit.jupiter.api.Assertions.assertDoesNotThrow; +import static org.junit.jupiter.api.Assertions.assertEquals; +import static org.junit.jupiter.api.Assertions.assertFalse; +import static org.junit.jupiter.api.Assertions.assertNotNull; +import static org.junit.jupiter.api.Assertions.assertNull; +import static org.junit.jupiter.api.Assertions.assertSame; +import static org.junit.jupiter.api.Assertions.assertThrows; +import static org.junit.jupiter.api.Assertions.assertTrue; +import static org.mockito.ArgumentMatchers.any; +import static org.mockito.ArgumentMatchers.eq; +import static org.mockito.BDDMockito.then; +import static org.mockito.Mockito.doAnswer; +import static org.mockito.Mockito.mock; +import static org.mockito.Mockito.never; +import static org.mockito.Mockito.times; +import static org.mockito.Mockito.verify; +import static org.mockito.Mockito.when; +import static org.neo4j.driver.AccessMode.WRITE; +import static org.neo4j.driver.Values.value; +import static org.neo4j.driver.internal.DatabaseNameUtil.database; +import static org.neo4j.driver.internal.DatabaseNameUtil.defaultDatabase; +import static org.neo4j.driver.internal.handlers.pulln.FetchSizeUtil.UNLIMITED_FETCH_SIZE; +import static org.neo4j.driver.testutil.TestUtil.await; +import static org.neo4j.driver.testutil.TestUtil.connectionMock; + +import io.netty.channel.embedded.EmbeddedChannel; +import java.time.Clock; +import java.util.Collections; +import java.util.HashMap; +import java.util.Map; +import java.util.Set; +import java.util.concurrent.CompletionStage; +import java.util.function.Consumer; +import org.junit.jupiter.api.AfterEach; +import org.junit.jupiter.api.BeforeEach; +import org.junit.jupiter.api.Test; +import org.junit.jupiter.params.ParameterizedTest; +import org.junit.jupiter.params.provider.EnumSource; +import org.mockito.ArgumentCaptor; +import org.mockito.Mockito; +import org.neo4j.driver.AccessMode; +import org.neo4j.driver.AuthTokens; +import org.neo4j.driver.Bookmark; +import org.neo4j.driver.Logging; +import org.neo4j.driver.Query; +import org.neo4j.driver.TransactionConfig; +import org.neo4j.driver.Value; +import org.neo4j.driver.internal.BoltAgentUtil; +import org.neo4j.driver.internal.DatabaseBookmark; +import org.neo4j.driver.internal.DatabaseName; +import org.neo4j.driver.internal.InternalBookmark; +import org.neo4j.driver.internal.async.UnmanagedTransaction; +import org.neo4j.driver.internal.async.connection.ChannelAttributes; +import org.neo4j.driver.internal.async.inbound.InboundMessageDispatcher; +import org.neo4j.driver.internal.cluster.RoutingContext; +import org.neo4j.driver.internal.cursor.AsyncResultCursor; +import org.neo4j.driver.internal.handlers.BeginTxResponseHandler; +import org.neo4j.driver.internal.handlers.CommitTxResponseHandler; +import org.neo4j.driver.internal.handlers.PullAllResponseHandler; +import org.neo4j.driver.internal.handlers.RollbackTxResponseHandler; +import org.neo4j.driver.internal.handlers.RunResponseHandler; +import org.neo4j.driver.internal.handlers.TelemetryResponseHandler; +import org.neo4j.driver.internal.messaging.BoltProtocol; +import org.neo4j.driver.internal.messaging.MessageFormat; +import org.neo4j.driver.internal.messaging.request.BeginMessage; +import org.neo4j.driver.internal.messaging.request.CommitMessage; +import org.neo4j.driver.internal.messaging.request.GoodbyeMessage; +import org.neo4j.driver.internal.messaging.request.PullMessage; +import org.neo4j.driver.internal.messaging.request.RollbackMessage; +import org.neo4j.driver.internal.messaging.request.RunWithMetadataMessage; +import org.neo4j.driver.internal.messaging.request.TelemetryMessage; +import org.neo4j.driver.internal.messaging.v54.MessageFormatV54; +import org.neo4j.driver.internal.security.InternalAuthToken; +import org.neo4j.driver.internal.spi.Connection; +import org.neo4j.driver.internal.spi.ResponseHandler; + +public class BoltProtocolV56Test { + protected static final String QUERY_TEXT = "RETURN $x"; + protected static final Map PARAMS = singletonMap("x", value(42)); + protected static final Query QUERY = new Query(QUERY_TEXT, value(PARAMS)); + + protected final BoltProtocol protocol = createProtocol(); + private final EmbeddedChannel channel = new EmbeddedChannel(); + private final InboundMessageDispatcher messageDispatcher = new InboundMessageDispatcher(channel, Logging.none()); + + private final TransactionConfig txConfig = TransactionConfig.builder() + .withTimeout(ofSeconds(12)) + .withMetadata(singletonMap("key", value(42))) + .build(); + + @SuppressWarnings("SameReturnValue") + protected BoltProtocol createProtocol() { + return BoltProtocolV56.INSTANCE; + } + + @BeforeEach + void beforeEach() { + ChannelAttributes.setMessageDispatcher(channel, messageDispatcher); + } + + @AfterEach + void afterEach() { + channel.finishAndReleaseAll(); + } + + @Test + void shouldCreateMessageFormat() { + assertThat(protocol.createMessageFormat(), instanceOf(expectedMessageFormatType())); + } + + @Test + void shouldInitializeChannel() { + var promise = channel.newPromise(); + + protocol.initializeChannel( + "MyDriver/0.0.1", + BoltAgentUtil.VALUE, + dummyAuthToken(), + RoutingContext.EMPTY, + promise, + null, + mock(Clock.class)); + + assertThat(channel.outboundMessages(), hasSize(0)); + assertEquals(1, messageDispatcher.queuedHandlersCount()); + assertTrue(promise.isDone()); + + Map metadata = new HashMap<>(); + metadata.put("server", value("Neo4j/4.4.0")); + metadata.put("connection_id", value("bolt-42")); + + messageDispatcher.handleSuccessMessage(metadata); + + channel.flush(); + assertTrue(promise.isDone()); + assertTrue(promise.isSuccess()); + } + + @Test + void shouldPrepareToCloseChannel() { + protocol.prepareToCloseChannel(channel); + + assertThat(channel.outboundMessages(), hasSize(1)); + assertThat(channel.outboundMessages().poll(), instanceOf(GoodbyeMessage.class)); + assertEquals(1, messageDispatcher.queuedHandlersCount()); + } + + @Test + void shouldBeginTransactionWithoutBookmark() { + var connection = connectionMock(protocol); + + var stage = protocol.beginTransaction( + connection, Collections.emptySet(), TransactionConfig.empty(), null, null, Logging.none(), true); + + verify(connection) + .writeAndFlush( + eq(new BeginMessage( + Collections.emptySet(), + TransactionConfig.empty(), + defaultDatabase(), + WRITE, + null, + null, + null, + true, + Logging.none())), + any(BeginTxResponseHandler.class)); + assertNull(await(stage)); + } + + @Test + void shouldBeginTransactionWithBookmarks() { + var connection = connectionMock(protocol); + var bookmarks = Collections.singleton(InternalBookmark.parse("neo4j:bookmark:v1:tx100")); + + var stage = protocol.beginTransaction( + connection, bookmarks, TransactionConfig.empty(), null, null, Logging.none(), true); + + verify(connection) + .writeAndFlush( + eq(new BeginMessage( + bookmarks, + TransactionConfig.empty(), + defaultDatabase(), + WRITE, + null, + null, + null, + true, + Logging.none())), + any(BeginTxResponseHandler.class)); + assertNull(await(stage)); + } + + @Test + void shouldBeginTransactionWithConfig() { + var connection = connectionMock(protocol); + + var stage = protocol.beginTransaction( + connection, Collections.emptySet(), txConfig, null, null, Logging.none(), true); + + verify(connection) + .writeAndFlush( + eq(new BeginMessage( + Collections.emptySet(), + txConfig, + defaultDatabase(), + WRITE, + null, + null, + null, + true, + Logging.none())), + any(BeginTxResponseHandler.class)); + assertNull(await(stage)); + } + + @Test + void shouldBeginTransactionWithBookmarksAndConfig() { + var connection = connectionMock(protocol); + var bookmarks = Collections.singleton(InternalBookmark.parse("neo4j:bookmark:v1:tx4242")); + + var stage = protocol.beginTransaction(connection, bookmarks, txConfig, null, null, Logging.none(), true); + + verify(connection) + .writeAndFlush( + eq(new BeginMessage( + bookmarks, txConfig, defaultDatabase(), WRITE, null, null, null, true, Logging.none())), + any(BeginTxResponseHandler.class)); + assertNull(await(stage)); + } + + @Test + void shouldCommitTransaction() { + var bookmarkString = "neo4j:bookmark:v1:tx4242"; + + var connection = connectionMock(protocol); + when(connection.protocol()).thenReturn(protocol); + doAnswer(invocation -> { + ResponseHandler commitHandler = invocation.getArgument(1); + commitHandler.onSuccess(singletonMap("bookmark", value(bookmarkString))); + return null; + }) + .when(connection) + .writeAndFlush(eq(CommitMessage.COMMIT), any()); + + var stage = protocol.commitTransaction(connection); + + verify(connection).writeAndFlush(eq(CommitMessage.COMMIT), any(CommitTxResponseHandler.class)); + assertEquals(InternalBookmark.parse(bookmarkString), await(stage).bookmark()); + } + + @Test + void shouldRollbackTransaction() { + var connection = connectionMock(protocol); + + var stage = protocol.rollbackTransaction(connection); + + verify(connection).writeAndFlush(eq(RollbackMessage.ROLLBACK), any(RollbackTxResponseHandler.class)); + assertNull(await(stage)); + } + + @ParameterizedTest + @EnumSource(AccessMode.class) + void shouldRunInAutoCommitTransactionAndWaitForRunResponse(AccessMode mode) throws Exception { + testRunAndWaitForRunResponse(true, TransactionConfig.empty(), mode); + } + + @ParameterizedTest + @EnumSource(AccessMode.class) + void shouldRunInAutoCommitWithConfigTransactionAndWaitForRunResponse(AccessMode mode) throws Exception { + testRunAndWaitForRunResponse(true, txConfig, mode); + } + + @ParameterizedTest + @EnumSource(AccessMode.class) + void shouldRunInAutoCommitTransactionAndWaitForSuccessRunResponse(AccessMode mode) throws Exception { + testSuccessfulRunInAutoCommitTxWithWaitingForResponse(Collections.emptySet(), TransactionConfig.empty(), mode); + } + + @ParameterizedTest + @EnumSource(AccessMode.class) + void shouldRunInAutoCommitTransactionWithBookmarkAndConfigAndWaitForSuccessRunResponse(AccessMode mode) + throws Exception { + testSuccessfulRunInAutoCommitTxWithWaitingForResponse( + Collections.singleton(InternalBookmark.parse("neo4j:bookmark:v1:tx65")), txConfig, mode); + } + + @ParameterizedTest + @EnumSource(AccessMode.class) + void shouldRunInAutoCommitTransactionAndWaitForFailureRunResponse(AccessMode mode) { + testFailedRunInAutoCommitTxWithWaitingForResponse(Collections.emptySet(), TransactionConfig.empty(), mode); + } + + @ParameterizedTest + @EnumSource(AccessMode.class) + void shouldRunInAutoCommitTransactionWithBookmarkAndConfigAndWaitForFailureRunResponse(AccessMode mode) { + testFailedRunInAutoCommitTxWithWaitingForResponse( + Collections.singleton(InternalBookmark.parse("neo4j:bookmark:v1:tx163")), txConfig, mode); + } + + @ParameterizedTest + @EnumSource(AccessMode.class) + void shouldRunInUnmanagedTransactionAndWaitForRunResponse(AccessMode mode) throws Exception { + testRunAndWaitForRunResponse(false, TransactionConfig.empty(), mode); + } + + @ParameterizedTest + @EnumSource(AccessMode.class) + void shouldRunInUnmanagedTransactionAndWaitForSuccessRunResponse(AccessMode mode) throws Exception { + testRunInUnmanagedTransactionAndWaitForRunResponse(true, mode); + } + + @ParameterizedTest + @EnumSource(AccessMode.class) + void shouldRunInUnmanagedTransactionAndWaitForFailureRunResponse(AccessMode mode) throws Exception { + testRunInUnmanagedTransactionAndWaitForRunResponse(false, mode); + } + + @Test + void databaseNameInBeginTransaction() { + testDatabaseNameSupport(false); + } + + @Test + void databaseNameForAutoCommitTransactions() { + testDatabaseNameSupport(true); + } + + @Test + void shouldSupportDatabaseNameInBeginTransaction() { + var txStage = protocol.beginTransaction( + connectionMock("foo", protocol), + Collections.emptySet(), + TransactionConfig.empty(), + null, + null, + Logging.none(), + true); + + assertDoesNotThrow(() -> await(txStage)); + } + + @Test + void shouldNotSupportDatabaseNameForAutoCommitTransactions() { + assertDoesNotThrow(() -> protocol.runInAutoCommitTransaction( + connectionMock("foo", protocol), + new Query("RETURN 1"), + Collections.emptySet(), + (ignored) -> {}, + TransactionConfig.empty(), + UNLIMITED_FETCH_SIZE, + null, + Logging.none())); + } + + @Test + void shouldTelemetrySendTelemetryMessage() { + var connection = connectionMock(); + doAnswer((invocationOnMock) -> { + var handler = (TelemetryResponseHandler) invocationOnMock.getArgument(1); + handler.onSuccess(Map.of()); + return null; + }) + .when(connection) + .write(Mockito.any(), Mockito.any()); + var expectedApi = 1; + + await(protocol.telemetry(connection, expectedApi)); + + verify(connection).write(Mockito.eq(new TelemetryMessage(expectedApi)), Mockito.any()); + verify(connection, never()).writeAndFlush(Mockito.any(), Mockito.any()); + } + + private Class expectedMessageFormatType() { + return MessageFormatV54.class; + } + + private void testFailedRunInAutoCommitTxWithWaitingForResponse( + Set bookmarks, TransactionConfig config, AccessMode mode) { + // Given + var connection = connectionMock(mode, protocol); + @SuppressWarnings("unchecked") + Consumer bookmarkConsumer = mock(Consumer.class); + + var cursorFuture = protocol.runInAutoCommitTransaction( + connection, + QUERY, + bookmarks, + bookmarkConsumer, + config, + UNLIMITED_FETCH_SIZE, + null, + Logging.none()) + .asyncResult() + .toCompletableFuture(); + + var runHandler = verifySessionRunInvoked(connection, bookmarks, config, mode, defaultDatabase()); + assertFalse(cursorFuture.isDone()); + + // When I response to Run message with a failure + Throwable error = new RuntimeException(); + runHandler.onFailure(error); + + // Then + then(bookmarkConsumer).should(times(0)).accept(any()); + assertTrue(cursorFuture.isDone()); + var actual = + assertThrows(error.getClass(), () -> await(cursorFuture.get().mapSuccessfulRunCompletionAsync())); + assertSame(error, actual); + } + + private void testSuccessfulRunInAutoCommitTxWithWaitingForResponse( + Set bookmarks, TransactionConfig config, AccessMode mode) throws Exception { + // Given + var connection = connectionMock(mode, protocol); + @SuppressWarnings("unchecked") + Consumer bookmarkConsumer = mock(Consumer.class); + + var cursorFuture = protocol.runInAutoCommitTransaction( + connection, + QUERY, + bookmarks, + bookmarkConsumer, + config, + UNLIMITED_FETCH_SIZE, + null, + Logging.none()) + .asyncResult() + .toCompletableFuture(); + + var runHandler = verifySessionRunInvoked(connection, bookmarks, config, mode, defaultDatabase()); + assertFalse(cursorFuture.isDone()); + + // When I response to the run message + runHandler.onSuccess(emptyMap()); + + // Then + then(bookmarkConsumer).should(times(0)).accept(any()); + assertTrue(cursorFuture.isDone()); + assertNotNull(cursorFuture.get()); + } + + private void testRunInUnmanagedTransactionAndWaitForRunResponse(boolean success, AccessMode mode) throws Exception { + // Given + var connection = connectionMock(mode, protocol); + + var cursorFuture = protocol.runInUnmanagedTransaction( + connection, QUERY, mock(UnmanagedTransaction.class), UNLIMITED_FETCH_SIZE) + .asyncResult() + .toCompletableFuture(); + + var runHandler = verifyTxRunInvoked(connection); + assertFalse(cursorFuture.isDone()); + Throwable error = new RuntimeException(); + + if (success) { + runHandler.onSuccess(emptyMap()); + } else { + // When responded with a failure + runHandler.onFailure(error); + } + + // Then + assertTrue(cursorFuture.isDone()); + if (success) { + assertNotNull(await(cursorFuture.get().mapSuccessfulRunCompletionAsync())); + } else { + var actual = assertThrows( + error.getClass(), () -> await(cursorFuture.get().mapSuccessfulRunCompletionAsync())); + assertSame(error, actual); + } + } + + private void testRunAndWaitForRunResponse(boolean autoCommitTx, TransactionConfig config, AccessMode mode) + throws Exception { + // Given + var connection = connectionMock(mode, protocol); + var initialBookmarks = Collections.singleton(InternalBookmark.parse("neo4j:bookmark:v1:tx987")); + + CompletionStage cursorStage; + if (autoCommitTx) { + cursorStage = protocol.runInAutoCommitTransaction( + connection, + QUERY, + initialBookmarks, + (ignored) -> {}, + config, + UNLIMITED_FETCH_SIZE, + null, + Logging.none()) + .asyncResult(); + } else { + cursorStage = protocol.runInUnmanagedTransaction( + connection, QUERY, mock(UnmanagedTransaction.class), UNLIMITED_FETCH_SIZE) + .asyncResult(); + } + + // When & Then + var cursorFuture = cursorStage.toCompletableFuture(); + assertFalse(cursorFuture.isDone()); + + var runResponseHandler = autoCommitTx + ? verifySessionRunInvoked(connection, initialBookmarks, config, mode, defaultDatabase()) + : verifyTxRunInvoked(connection); + runResponseHandler.onSuccess(emptyMap()); + + assertTrue(cursorFuture.isDone()); + assertNotNull(cursorFuture.get()); + } + + private void testDatabaseNameSupport(boolean autoCommitTx) { + var connection = connectionMock("foo", protocol); + if (autoCommitTx) { + var factory = protocol.runInAutoCommitTransaction( + connection, + QUERY, + Collections.emptySet(), + (ignored) -> {}, + TransactionConfig.empty(), + UNLIMITED_FETCH_SIZE, + null, + Logging.none()); + var resultStage = factory.asyncResult(); + var runHandler = verifySessionRunInvoked( + connection, Collections.emptySet(), TransactionConfig.empty(), AccessMode.WRITE, database("foo")); + runHandler.onSuccess(emptyMap()); + await(resultStage); + verifySessionRunInvoked( + connection, Collections.emptySet(), TransactionConfig.empty(), AccessMode.WRITE, database("foo")); + } else { + var txStage = protocol.beginTransaction( + connection, Collections.emptySet(), TransactionConfig.empty(), null, null, Logging.none(), true); + await(txStage); + verifyBeginInvoked(connection, Collections.emptySet(), TransactionConfig.empty(), database("foo")); + } + } + + private ResponseHandler verifyTxRunInvoked(Connection connection) { + return verifyRunInvoked(connection, RunWithMetadataMessage.unmanagedTxRunMessage(QUERY)); + } + + private ResponseHandler verifySessionRunInvoked( + Connection connection, + Set bookmark, + TransactionConfig config, + AccessMode mode, + DatabaseName databaseName) { + var runMessage = RunWithMetadataMessage.autoCommitTxRunMessage( + QUERY, config, databaseName, mode, bookmark, null, null, true, Logging.none()); + return verifyRunInvoked(connection, runMessage); + } + + private ResponseHandler verifyRunInvoked(Connection connection, RunWithMetadataMessage runMessage) { + var runHandlerCaptor = ArgumentCaptor.forClass(ResponseHandler.class); + var pullHandlerCaptor = ArgumentCaptor.forClass(ResponseHandler.class); + + verify(connection).write(eq(runMessage), runHandlerCaptor.capture()); + verify(connection).writeAndFlush(any(PullMessage.class), pullHandlerCaptor.capture()); + + assertThat(runHandlerCaptor.getValue(), instanceOf(RunResponseHandler.class)); + assertThat(pullHandlerCaptor.getValue(), instanceOf(PullAllResponseHandler.class)); + + return runHandlerCaptor.getValue(); + } + + private void verifyBeginInvoked( + Connection connection, Set bookmarks, TransactionConfig config, DatabaseName databaseName) { + var beginHandlerCaptor = ArgumentCaptor.forClass(ResponseHandler.class); + var beginMessage = new BeginMessage( + bookmarks, config, databaseName, AccessMode.WRITE, null, null, null, true, Logging.none()); + verify(connection).writeAndFlush(eq(beginMessage), beginHandlerCaptor.capture()); + assertThat(beginHandlerCaptor.getValue(), instanceOf(BeginTxResponseHandler.class)); + } + + private static InternalAuthToken dummyAuthToken() { + return (InternalAuthToken) AuthTokens.basic("hello", "world"); + } +} diff --git a/driver/src/test/java/org/neo4j/driver/internal/messaging/v56/MessageFormatV56Test.java b/driver/src/test/java/org/neo4j/driver/internal/messaging/v56/MessageFormatV56Test.java new file mode 100644 index 0000000000..dc5f6ce0de --- /dev/null +++ b/driver/src/test/java/org/neo4j/driver/internal/messaging/v56/MessageFormatV56Test.java @@ -0,0 +1,46 @@ +/* + * Copyright (c) "Neo4j" + * Neo4j Sweden AB [https://neo4j.com] + * + * Licensed under the Apache License, Version 2.0 (the "License"); + * you may not use this file except in compliance with the License. + * You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ +package org.neo4j.driver.internal.messaging.v56; + +import static org.hamcrest.MatcherAssert.assertThat; +import static org.hamcrest.Matchers.instanceOf; +import static org.mockito.Mockito.mock; + +import org.junit.jupiter.api.Test; +import org.neo4j.driver.internal.messaging.MessageFormat; +import org.neo4j.driver.internal.messaging.v5.MessageReaderV5; +import org.neo4j.driver.internal.messaging.v54.MessageWriterV54; +import org.neo4j.driver.internal.packstream.PackInput; +import org.neo4j.driver.internal.packstream.PackOutput; + +class MessageFormatV56Test { + private static final MessageFormat format = BoltProtocolV56.INSTANCE.createMessageFormat(); + + @Test + void shouldCreateCorrectWriter() { + var writer = format.newWriter(mock(PackOutput.class)); + + assertThat(writer, instanceOf(MessageWriterV54.class)); + } + + @Test + void shouldCreateCorrectReader() { + var reader = format.newReader(mock(PackInput.class)); + + assertThat(reader, instanceOf(MessageReaderV5.class)); + } +} diff --git a/driver/src/test/java/org/neo4j/driver/internal/messaging/v56/MessageWriterV56Test.java b/driver/src/test/java/org/neo4j/driver/internal/messaging/v56/MessageWriterV56Test.java new file mode 100644 index 0000000000..4802e97e2f --- /dev/null +++ b/driver/src/test/java/org/neo4j/driver/internal/messaging/v56/MessageWriterV56Test.java @@ -0,0 +1,212 @@ +/* + * Copyright (c) "Neo4j" + * Neo4j Sweden AB [https://neo4j.com] + * + * Licensed under the Apache License, Version 2.0 (the "License"); + * you may not use this file except in compliance with the License. + * You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ +package org.neo4j.driver.internal.messaging.v56; + +import static java.time.Duration.ofSeconds; +import static java.util.Calendar.DECEMBER; +import static java.util.Collections.emptyMap; +import static java.util.Collections.singletonMap; +import static org.neo4j.driver.AccessMode.READ; +import static org.neo4j.driver.AccessMode.WRITE; +import static org.neo4j.driver.AuthTokens.basic; +import static org.neo4j.driver.Values.point; +import static org.neo4j.driver.Values.value; +import static org.neo4j.driver.internal.DatabaseNameUtil.database; +import static org.neo4j.driver.internal.DatabaseNameUtil.defaultDatabase; +import static org.neo4j.driver.internal.messaging.request.CommitMessage.COMMIT; +import static org.neo4j.driver.internal.messaging.request.DiscardAllMessage.DISCARD_ALL; +import static org.neo4j.driver.internal.messaging.request.GoodbyeMessage.GOODBYE; +import static org.neo4j.driver.internal.messaging.request.PullAllMessage.PULL_ALL; +import static org.neo4j.driver.internal.messaging.request.ResetMessage.RESET; +import static org.neo4j.driver.internal.messaging.request.RollbackMessage.ROLLBACK; +import static org.neo4j.driver.internal.messaging.request.RunWithMetadataMessage.autoCommitTxRunMessage; +import static org.neo4j.driver.internal.messaging.request.RunWithMetadataMessage.unmanagedTxRunMessage; + +import java.time.LocalDate; +import java.time.LocalDateTime; +import java.time.LocalTime; +import java.time.OffsetTime; +import java.time.ZoneId; +import java.time.ZoneOffset; +import java.time.ZonedDateTime; +import java.util.Collections; +import java.util.HashMap; +import java.util.Map; +import java.util.stream.Stream; +import org.neo4j.driver.Logging; +import org.neo4j.driver.Query; +import org.neo4j.driver.Value; +import org.neo4j.driver.Values; +import org.neo4j.driver.internal.BoltAgentUtil; +import org.neo4j.driver.internal.InternalBookmark; +import org.neo4j.driver.internal.messaging.Message; +import org.neo4j.driver.internal.messaging.MessageFormat; +import org.neo4j.driver.internal.messaging.request.BeginMessage; +import org.neo4j.driver.internal.messaging.request.DiscardMessage; +import org.neo4j.driver.internal.messaging.request.HelloMessage; +import org.neo4j.driver.internal.messaging.request.PullMessage; +import org.neo4j.driver.internal.messaging.request.RouteMessage; +import org.neo4j.driver.internal.messaging.request.TelemetryMessage; +import org.neo4j.driver.internal.packstream.PackOutput; +import org.neo4j.driver.internal.security.InternalAuthToken; +import org.neo4j.driver.internal.util.messaging.AbstractMessageWriterTestBase; + +public class MessageWriterV56Test extends AbstractMessageWriterTestBase { + @Override + protected MessageFormat.Writer newWriter(PackOutput output) { + return BoltProtocolV56.INSTANCE.createMessageFormat().newWriter(output); + } + + @Override + protected Stream supportedMessages() { + return Stream.of( + // Bolt V2 Data Types + unmanagedTxRunMessage(new Query("RETURN $point", singletonMap("point", point(42, 12.99, -180.0)))), + unmanagedTxRunMessage( + new Query("RETURN $point", singletonMap("point", point(42, 0.51, 2.99, 100.123)))), + unmanagedTxRunMessage( + new Query("RETURN $date", singletonMap("date", value(LocalDate.ofEpochDay(2147483650L))))), + unmanagedTxRunMessage(new Query( + "RETURN $time", singletonMap("time", value(OffsetTime.of(4, 16, 20, 999, ZoneOffset.MIN))))), + unmanagedTxRunMessage( + new Query("RETURN $time", singletonMap("time", value(LocalTime.of(12, 9, 18, 999_888))))), + unmanagedTxRunMessage(new Query( + "RETURN $dateTime", + singletonMap("dateTime", value(LocalDateTime.of(2049, DECEMBER, 12, 17, 25, 49, 199))))), + unmanagedTxRunMessage(new Query( + "RETURN $dateTime", + singletonMap( + "dateTime", + value(ZonedDateTime.of( + 2000, 1, 10, 12, 2, 49, 300, ZoneOffset.ofHoursMinutes(9, 30)))))), + unmanagedTxRunMessage(new Query( + "RETURN $dateTime", + singletonMap( + "dateTime", + value(ZonedDateTime.of(2000, 1, 10, 12, 2, 49, 300, ZoneId.of("Europe/Stockholm")))))), + + // New Bolt V4 messages + new PullMessage(100, 200), + new DiscardMessage(300, 400), + + // Bolt V3 messages + new HelloMessage( + "MyDriver/1.2.3", + BoltAgentUtil.VALUE, + ((InternalAuthToken) basic("neo4j", "neo4j")).toMap(), + Collections.emptyMap(), + false, + null, + true), + GOODBYE, + new BeginMessage( + Collections.singleton(InternalBookmark.parse("neo4j:bookmark:v1:tx123")), + ofSeconds(5), + singletonMap("key", value(42)), + READ, + defaultDatabase(), + null, + null, + null, + true, + Logging.none()), + new BeginMessage( + Collections.singleton(InternalBookmark.parse("neo4j:bookmark:v1:tx123")), + ofSeconds(5), + singletonMap("key", value(42)), + WRITE, + database("foo"), + null, + null, + null, + true, + Logging.none()), + COMMIT, + ROLLBACK, + RESET, + autoCommitTxRunMessage( + new Query("RETURN 1"), + ofSeconds(5), + singletonMap("key", value(42)), + defaultDatabase(), + READ, + Collections.singleton(InternalBookmark.parse("neo4j:bookmark:v1:tx1")), + null, + null, + true, + Logging.none()), + autoCommitTxRunMessage( + new Query("RETURN 1"), + ofSeconds(5), + singletonMap("key", value(42)), + database("foo"), + WRITE, + Collections.singleton(InternalBookmark.parse("neo4j:bookmark:v1:tx1")), + null, + null, + true, + Logging.none()), + unmanagedTxRunMessage(new Query("RETURN 1")), + + // Bolt V3 messages with struct values + autoCommitTxRunMessage( + new Query("RETURN $x", singletonMap("x", value(ZonedDateTime.now()))), + ofSeconds(1), + emptyMap(), + defaultDatabase(), + READ, + Collections.emptySet(), + null, + null, + true, + Logging.none()), + autoCommitTxRunMessage( + new Query("RETURN $x", singletonMap("x", value(ZonedDateTime.now()))), + ofSeconds(1), + emptyMap(), + database("foo"), + WRITE, + Collections.emptySet(), + null, + null, + true, + Logging.none()), + unmanagedTxRunMessage(new Query("RETURN $x", singletonMap("x", point(42, 1, 2, 3)))), + + // New 4.3 Messages + routeMessage(), + // New 5.4 message + telemetryMessage()); + } + + @Override + protected Stream unsupportedMessages() { + return Stream.of( + // Bolt V1, V2 and V3 messages + PULL_ALL, DISCARD_ALL); + } + + private RouteMessage routeMessage() { + Map routeContext = new HashMap<>(); + routeContext.put("someContext", Values.value(124)); + return new RouteMessage(routeContext, Collections.emptySet(), "dbName", null); + } + + private TelemetryMessage telemetryMessage() { + return new TelemetryMessage(1); + } +} diff --git a/driver/src/test/java/org/neo4j/driver/testutil/TestUtil.java b/driver/src/test/java/org/neo4j/driver/testutil/TestUtil.java index fb77ac9ac9..f897df97fe 100644 --- a/driver/src/test/java/org/neo4j/driver/testutil/TestUtil.java +++ b/driver/src/test/java/org/neo4j/driver/testutil/TestUtil.java @@ -97,6 +97,8 @@ import org.neo4j.driver.internal.messaging.v52.BoltProtocolV52; import org.neo4j.driver.internal.messaging.v53.BoltProtocolV53; import org.neo4j.driver.internal.messaging.v54.BoltProtocolV54; +import org.neo4j.driver.internal.messaging.v55.BoltProtocolV55; +import org.neo4j.driver.internal.messaging.v56.BoltProtocolV56; import org.neo4j.driver.internal.retry.RetryLogic; import org.neo4j.driver.internal.spi.Connection; import org.neo4j.driver.internal.spi.ConnectionProvider; @@ -465,7 +467,9 @@ public static Connection connectionMock(String databaseName, AccessMode mode, Bo BoltProtocolV51.VERSION, BoltProtocolV52.VERSION, BoltProtocolV53.VERSION, - BoltProtocolV54.VERSION) + BoltProtocolV54.VERSION, + BoltProtocolV55.VERSION, + BoltProtocolV56.VERSION) .contains(version)) { setupSuccessResponse(connection, CommitMessage.class); setupSuccessResponse(connection, RollbackMessage.class); diff --git a/examples/pom.xml b/examples/pom.xml index 9e9ea08ac1..7a363970c7 100644 --- a/examples/pom.xml +++ b/examples/pom.xml @@ -6,7 +6,7 @@ org.neo4j.driver neo4j-java-driver-parent - 5.22-SNAPSHOT + 5.23-SNAPSHOT org.neo4j.doc.driver diff --git a/pom.xml b/pom.xml index 423f714350..033f6b5d5c 100644 --- a/pom.xml +++ b/pom.xml @@ -5,7 +5,7 @@ org.neo4j.driver neo4j-java-driver-parent - 5.22-SNAPSHOT + 5.23-SNAPSHOT pom Neo4j Java Driver Project diff --git a/testkit-backend/pom.xml b/testkit-backend/pom.xml index 0517cc192a..7d7b69ed2c 100644 --- a/testkit-backend/pom.xml +++ b/testkit-backend/pom.xml @@ -7,7 +7,7 @@ neo4j-java-driver-parent org.neo4j.driver - 5.22-SNAPSHOT + 5.23-SNAPSHOT testkit-backend diff --git a/testkit-backend/src/main/java/neo4j/org/testkit/backend/messages/requests/GetFeatures.java b/testkit-backend/src/main/java/neo4j/org/testkit/backend/messages/requests/GetFeatures.java index 2b7814d32a..b7cc0d7bf4 100644 --- a/testkit-backend/src/main/java/neo4j/org/testkit/backend/messages/requests/GetFeatures.java +++ b/testkit-backend/src/main/java/neo4j/org/testkit/backend/messages/requests/GetFeatures.java @@ -42,6 +42,7 @@ public class GetFeatures implements TestkitRequest { "Feature:Bolt:5.3", "Feature:Bolt:5.4", "Feature:Bolt:5.5", + "Feature:Bolt:5.6", "AuthorizationExpiredTreatment", "ConfHint:connection.recv_timeout_seconds", "Feature:Auth:Bearer", diff --git a/testkit-tests/pom.xml b/testkit-tests/pom.xml index d67ca5db16..9beca13074 100644 --- a/testkit-tests/pom.xml +++ b/testkit-tests/pom.xml @@ -6,7 +6,7 @@ org.neo4j.driver neo4j-java-driver-parent - 5.22-SNAPSHOT + 5.23-SNAPSHOT .. From 396ff87ebbac3db36f1b0d0b03fbd7df45d6a9af Mon Sep 17 00:00:00 2001 From: Dmitriy Tverdiakov Date: Tue, 23 Jul 2024 17:36:25 +0100 Subject: [PATCH 2/2] Update shouldBuildResultSummaryWithGqlStatusObjects test --- .../org/neo4j/driver/internal/util/MetadataExtractorTest.java | 4 +++- 1 file changed, 3 insertions(+), 1 deletion(-) diff --git a/driver/src/test/java/org/neo4j/driver/internal/util/MetadataExtractorTest.java b/driver/src/test/java/org/neo4j/driver/internal/util/MetadataExtractorTest.java index 115f660a8c..afd8b5affd 100644 --- a/driver/src/test/java/org/neo4j/driver/internal/util/MetadataExtractorTest.java +++ b/driver/src/test/java/org/neo4j/driver/internal/util/MetadataExtractorTest.java @@ -356,6 +356,8 @@ void shouldBuildResultSummaryWithGqlStatusObjects() { "neo4j_code", "title", "title", + "description", + "notification_description", "diagnostic_record", parameters( "_severity", @@ -388,7 +390,7 @@ void shouldBuildResultSummaryWithGqlStatusObjects() { assertEquals("gql_status", firstGqlStatusObject.gqlStatus()); assertEquals("status_description", firstGqlStatusObject.statusDescription()); - assertEquals("status_description", firstGqlStatusObject.description()); + assertEquals("notification_description", firstGqlStatusObject.description()); assertEquals("neo4j_code", firstGqlStatusObject.code()); assertEquals("title", firstGqlStatusObject.title()); assertEquals("WARNING", firstGqlStatusObject.severity());