From 3d2ef3bb96b96229e7d6295550662b7a959ab963 Mon Sep 17 00:00:00 2001 From: Tomoyuki Morita Date: Tue, 11 Jun 2024 13:54:44 -0700 Subject: [PATCH] Move classes from spark to async-query-core and async-query Signed-off-by: Tomoyuki Morita --- async-query-core/build.gradle | 19 ++++- .../src/main/antlr/FlintSparkSqlExtensions.g4 | 0 .../src/main/antlr/SparkSqlBase.g4 | 0 .../src/main/antlr/SqlBaseLexer.g4 | 44 +++++++++-- .../src/main/antlr/SqlBaseParser.g4 | 77 ++++++++++++------- .../org/opensearch/sql/asyncquery/Dummy.java | 13 ---- .../asyncquery/AsyncQueryExecutorService.java | 0 .../AsyncQueryExecutorServiceImpl.java | 0 .../AsyncQueryJobMetadataStorageService.java | 0 .../AsyncQueryNotFoundException.java | 0 .../model/AsyncQueryExecutionResponse.java | 0 .../spark/asyncquery/model/AsyncQueryId.java | 0 .../model/AsyncQueryJobMetadata.java | 0 .../model/AsyncQueryRequestContext.java | 0 .../asyncquery/model/AsyncQueryResult.java | 0 .../model/NullAsyncQueryRequestContext.java | 0 .../model/SparkSubmitParameters.java | 0 .../sql/spark/client/EMRServerlessClient.java | 0 .../client/EMRServerlessClientFactory.java | 0 .../EMRServerlessClientFactoryImpl.java | 0 .../spark/client/EmrServerlessClientImpl.java | 0 .../sql/spark/client/StartJobRequest.java | 0 .../config/SparkExecutionEngineConfig.java | 0 ...rkExecutionEngineConfigClusterSetting.java | 0 .../SparkExecutionEngineConfigSupplier.java | 0 .../config/SparkSubmitParameterModifier.java | 0 .../spark/data/constants/SparkConstants.java | 7 -- .../spark/dispatcher/AsyncQueryHandler.java | 0 .../spark/dispatcher/BatchQueryHandler.java | 0 .../DatasourceEmbeddedQueryIdProvider.java | 0 .../sql/spark/dispatcher/IndexDMLHandler.java | 0 .../dispatcher/InteractiveQueryHandler.java | 0 .../spark/dispatcher/QueryHandlerFactory.java | 0 .../sql/spark/dispatcher/QueryIdProvider.java | 0 .../spark/dispatcher/RefreshQueryHandler.java | 0 .../dispatcher/SparkQueryDispatcher.java | 0 .../dispatcher/StreamingQueryHandler.java | 0 .../model/DispatchQueryContext.java | 0 .../model/DispatchQueryRequest.java | 0 .../model/DispatchQueryResponse.java | 0 .../dispatcher/model/FlintIndexOptions.java | 0 .../model/FullyQualifiedTableName.java | 0 .../dispatcher/model/IndexDMLResult.java | 0 .../model/IndexQueryActionType.java | 0 .../dispatcher/model/IndexQueryDetails.java | 0 .../sql/spark/dispatcher/model/JobType.java | 0 .../session/CreateSessionRequest.java | 0 .../execution/session/InteractiveSession.java | 0 .../sql/spark/execution/session/Session.java | 0 .../session/SessionConfigSupplier.java | 0 .../spark/execution/session/SessionId.java | 0 .../execution/session/SessionManager.java | 0 .../spark/execution/session/SessionModel.java | 0 .../spark/execution/session/SessionState.java | 0 .../spark/execution/session/SessionType.java | 0 .../execution/statement/QueryRequest.java | 0 .../spark/execution/statement/Statement.java | 0 .../execution/statement/StatementId.java | 0 .../execution/statement/StatementModel.java | 0 .../execution/statement/StatementState.java | 0 .../execution/statestore/CopyBuilder.java | 0 .../statestore/OpenSearchStateStoreUtil.java | 0 .../statestore/SessionStorageService.java | 0 .../statestore/StateCopyBuilder.java | 0 .../execution/statestore/StateModel.java | 0 .../statestore/StatementStorageService.java | 0 .../xcontent/XContentSerializerUtil.java | 0 .../sql/spark/flint/FlintIndexMetadata.java | 0 .../flint/FlintIndexMetadataService.java | 0 .../sql/spark/flint/FlintIndexState.java | 0 .../sql/spark/flint/FlintIndexStateModel.java | 0 .../flint/FlintIndexStateModelService.java | 0 .../sql/spark/flint/FlintIndexType.java | 0 .../flint/IndexDMLResultStorageService.java | 0 .../spark/flint/operation/FlintIndexOp.java | 0 .../flint/operation/FlintIndexOpAlter.java | 0 .../flint/operation/FlintIndexOpCancel.java | 0 .../flint/operation/FlintIndexOpDrop.java | 0 .../flint/operation/FlintIndexOpFactory.java | 0 .../flint/operation/FlintIndexOpVacuum.java | 0 .../ConcurrencyLimitExceededException.java | 0 .../sql/spark/leasemanager/LeaseManager.java | 0 .../leasemanager/model/LeaseRequest.java | 0 .../response/JobExecutionResponseReader.java | 0 .../rest/model/CreateAsyncQueryRequest.java | 0 .../rest/model/CreateAsyncQueryResponse.java | 0 .../sql/spark/rest/model/LangType.java | 0 .../opensearch/sql/spark/utils/IDUtils.java | 0 .../sql/spark/utils/RealTimeProvider.java | 0 .../sql/spark/utils/SQLQueryUtils.java | 0 .../sql/spark/utils/TimeProvider.java | 0 .../opensearch/sql/asyncquery/DummyTest.java | 18 ----- .../AsyncQueryExecutorServiceImplTest.java | 8 +- .../model/SparkSubmitParametersTest.java | 0 .../EMRServerlessClientFactoryImplTest.java | 0 .../client/EmrServerlessClientImplTest.java | 0 .../sql/spark/client/StartJobRequestTest.java | 0 ...ecutionEngineConfigClusterSettingTest.java | 0 .../sql/spark/constants/TestConstants.java | 23 ++++++ .../spark/dispatcher/IndexDMLHandlerTest.java | 0 .../dispatcher/SparkQueryDispatcherTest.java | 0 .../execution/session/SessionManagerTest.java | 0 .../execution/session/SessionStateTest.java | 0 .../execution/session/SessionTypeTest.java | 0 .../statement/StatementStateTest.java | 0 .../OpenSearchStateStoreUtilTest.java | 0 .../execution/statestore/StateModelTest.java | 0 .../xcontent/XContentSerializerUtilTest.java | 0 .../sql/spark/flint/FlintIndexStateTest.java | 0 .../spark/flint/IndexQueryDetailsTest.java | 0 .../flint/operation/FlintIndexOpTest.java | 0 ...ConcurrencyLimitExceededExceptionTest.java | 19 +++++ .../model/CreateAsyncQueryRequestTest.java | 0 .../sql/spark/utils/IDUtilsTest.java | 33 ++++++++ .../sql/spark/utils/MockTimeProvider.java | 0 .../sql/spark/utils/RealTimeProviderTest.java | 19 +++++ .../sql/spark/utils/SQLQueryUtilsTest.java | 0 .../opensearch/sql/spark/utils/TestUtils.java | 44 +++++++++++ .../src/test/resources/invalid_response.json | 0 .../org.mockito.plugins.MockMaker | 1 + .../test/resources/select_query_response.json | 12 +++ async-query/build.gradle | 2 +- .../sql/asyncquery/DummyConsumer.java | 18 ----- ...chAsyncQueryJobMetadataStorageService.java | 0 .../cluster/ClusterManagerEventListener.java | 0 .../spark/cluster/FlintIndexRetention.java | 0 .../FlintStreamingJobHouseKeeperTask.java | 0 .../sql/spark/cluster/IndexCleanup.java | 0 ...penSearchSparkSubmitParameterModifier.java | 0 ...parkExecutionEngineConfigSupplierImpl.java | 0 .../OpenSearchSessionConfigSupplier.java | 0 .../execution/statestore/FromXContent.java | 0 .../OpenSearchSessionStorageService.java | 0 .../OpenSearchStatementStorageService.java | 0 .../execution/statestore/StateStore.java | 0 ...yncQueryJobMetadataXContentSerializer.java | 0 ...lintIndexStateModelXContentSerializer.java | 0 .../IndexDMLResultXContentSerializer.java | 0 .../SessionModelXContentSerializer.java | 0 .../StatementModelXContentSerializer.java | 0 .../xcontent/XContentCommonAttributes.java | 0 .../xcontent/XContentSerializer.java | 0 .../flint/FlintIndexMetadataServiceImpl.java | 0 ...OpenSearchFlintIndexStateModelService.java | 0 ...penSearchIndexDMLResultStorageService.java | 0 .../leasemanager/DefaultLeaseManager.java | 0 .../OpenSearchJobExecutionResponseReader.java | 0 .../rest/RestAsyncQueryManagementAction.java | 0 ...ransportCancelAsyncQueryRequestAction.java | 0 ...ransportCreateAsyncQueryRequestAction.java | 0 .../TransportGetAsyncQueryResultAction.java | 0 .../config/AsyncExecutorServiceModule.java | 0 .../AsyncQueryResultResponseFormatter.java | 0 .../model/CancelAsyncQueryActionRequest.java | 0 .../model/CancelAsyncQueryActionResponse.java | 0 .../model/CreateAsyncQueryActionRequest.java | 0 .../model/CreateAsyncQueryActionResponse.java | 0 .../GetAsyncQueryResultActionRequest.java | 0 .../GetAsyncQueryResultActionResponse.java | 0 .../sql/asyncquery/DummyConsumerTest.java | 28 ------- ...AsyncQueryExecutorServiceImplSpecTest.java | 0 .../AsyncQueryExecutorServiceSpec.java | 0 .../AsyncQueryGetResultSpecTest.java | 17 ++-- .../asyncquery/IndexQuerySpecAlterTest.java | 2 + .../spark/asyncquery/IndexQuerySpecTest.java | 13 ++-- .../asyncquery/IndexQuerySpecVacuumTest.java | 0 ...yncQueryJobMetadataStorageServiceTest.java | 0 .../asyncquery/model/MockFlintIndex.java | 0 .../asyncquery/model/MockFlintSparkJob.java | 0 .../FlintStreamingJobHouseKeeperTaskTest.java | 44 +++++++---- ...ExecutionEngineConfigSupplierImplTest.java | 0 .../sql/spark/constants/TestConstants.java | 17 ++++ .../session/InteractiveSessionTest.java | 17 ++-- .../execution/session/SessionTestUtil.java | 0 .../session/TestEMRServerlessClient.java | 0 .../execution/statement/StatementTest.java | 0 ...ueryJobMetadataXContentSerializerTest.java | 0 ...IndexStateModelXContentSerializerTest.java | 4 +- .../IndexDMLResultXContentSerializerTest.java | 0 .../SessionModelXContentSerializerTest.java | 4 +- .../StatementModelXContentSerializerTest.java | 4 +- .../xcontent/XContentSerializerTestUtil.java | 0 .../FlintIndexMetadataServiceImplTest.java | 6 +- ...SearchFlintIndexStateModelServiceTest.java | 0 .../leasemanager/DefaultLeaseManagerTest.java | 0 ...nSearchJobExecutionResponseReaderTest.java | 0 ...portCancelAsyncQueryRequestActionTest.java | 0 ...portCreateAsyncQueryRequestActionTest.java | 0 ...ransportGetAsyncQueryResultActionTest.java | 0 .../AsyncExecutorServiceModuleTest.java | 0 ...AsyncQueryResultResponseFormatterTest.java | 0 .../opensearch/sql/spark/utils/TestUtils.java | 44 +++++++++++ .../0.1.1/flint_covering_index.json | 0 .../flint-index-mappings/0.1.1/flint_mv.json | 0 .../0.1.1/flint_skipping_index.json | 0 .../0.1.1/flint_special_character_index.json | 0 .../flint_covering_index.json | 0 .../flint-index-mappings/flint_mv.json | 0 ...logs_covering_corrupted_index_mapping.json | 0 ...ttp_logs_covering_error_index_mapping.json | 0 ...mydb_http_logs_covering_index_mapping.json | 0 ...mydb_http_logs_skipping_index_mapping.json | 0 .../flint_my_glue_mydb_mv_mapping.json | 0 ...lint_mys3_default_http_logs_cv1_index.json | 0 ...mys3_default_http_logs_skipping_index.json | 0 .../flint_skipping_index.json | 0 .../flint_special_character_index.json | 0 .../flint-index-mappings/npe_mapping.json | 0 .../org.mockito.plugins.MockMaker | 1 + .../query_execution_result_mapping.json | 0 plugin/build.gradle | 1 + spark/build.gradle | 73 +----------------- .../sql/spark/client/EmrClientImpl.java | 5 +- .../sql/spark/helper/FlintHelper.java | 16 ++-- .../spark/storage/SparkStorageFactory.java | 6 +- .../spark/data/type/SparkDataTypeTest.java | 19 +++++ .../opensearch/sql/spark/utils/TestUtils.java | 20 ----- 217 files changed, 432 insertions(+), 266 deletions(-) rename {spark => async-query-core}/src/main/antlr/FlintSparkSqlExtensions.g4 (100%) rename {spark => async-query-core}/src/main/antlr/SparkSqlBase.g4 (100%) rename {spark => async-query-core}/src/main/antlr/SqlBaseLexer.g4 (90%) rename {spark => async-query-core}/src/main/antlr/SqlBaseParser.g4 (95%) delete mode 100644 async-query-core/src/main/java/org/opensearch/sql/asyncquery/Dummy.java rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorService.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImpl.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/asyncquery/AsyncQueryJobMetadataStorageService.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/asyncquery/exceptions/AsyncQueryNotFoundException.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryExecutionResponse.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryId.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryJobMetadata.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryRequestContext.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryResult.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/asyncquery/model/NullAsyncQueryRequestContext.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/asyncquery/model/SparkSubmitParameters.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/client/EMRServerlessClient.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/client/EMRServerlessClientFactory.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/client/EMRServerlessClientFactoryImpl.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/client/EmrServerlessClientImpl.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/client/StartJobRequest.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfig.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigClusterSetting.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigSupplier.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/config/SparkSubmitParameterModifier.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/data/constants/SparkConstants.java (94%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/AsyncQueryHandler.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/BatchQueryHandler.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/DatasourceEmbeddedQueryIdProvider.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/IndexDMLHandler.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/InteractiveQueryHandler.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/QueryHandlerFactory.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/QueryIdProvider.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/RefreshQueryHandler.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/SparkQueryDispatcher.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/StreamingQueryHandler.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/model/DispatchQueryContext.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/model/DispatchQueryRequest.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/model/DispatchQueryResponse.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/model/FlintIndexOptions.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/model/FullyQualifiedTableName.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/model/IndexDMLResult.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/model/IndexQueryActionType.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/model/IndexQueryDetails.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/dispatcher/model/JobType.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/session/CreateSessionRequest.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/session/InteractiveSession.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/session/Session.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/session/SessionConfigSupplier.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/session/SessionId.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/session/SessionManager.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/session/SessionModel.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/session/SessionState.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/session/SessionType.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/statement/QueryRequest.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/statement/Statement.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/statement/StatementId.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/statement/StatementModel.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/statement/StatementState.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/statestore/CopyBuilder.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/statestore/OpenSearchStateStoreUtil.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/statestore/SessionStorageService.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/statestore/StateCopyBuilder.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/statestore/StateModel.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/statestore/StatementStorageService.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializerUtil.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/flint/FlintIndexMetadata.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/flint/FlintIndexMetadataService.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/flint/FlintIndexState.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/flint/FlintIndexStateModel.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/flint/FlintIndexStateModelService.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/flint/FlintIndexType.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/flint/IndexDMLResultStorageService.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOp.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpAlter.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpCancel.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpDrop.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpFactory.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpVacuum.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/leasemanager/ConcurrencyLimitExceededException.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/leasemanager/LeaseManager.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/leasemanager/model/LeaseRequest.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/response/JobExecutionResponseReader.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/rest/model/CreateAsyncQueryRequest.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/rest/model/CreateAsyncQueryResponse.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/rest/model/LangType.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/utils/IDUtils.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/utils/RealTimeProvider.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/utils/SQLQueryUtils.java (100%) rename {spark => async-query-core}/src/main/java/org/opensearch/sql/spark/utils/TimeProvider.java (100%) delete mode 100644 async-query-core/src/test/java/org/opensearch/sql/asyncquery/DummyTest.java rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImplTest.java (97%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/asyncquery/model/SparkSubmitParametersTest.java (100%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/client/EMRServerlessClientFactoryImplTest.java (100%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/client/EmrServerlessClientImplTest.java (100%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/client/StartJobRequestTest.java (100%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigClusterSettingTest.java (100%) create mode 100644 async-query-core/src/test/java/org/opensearch/sql/spark/constants/TestConstants.java rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/dispatcher/IndexDMLHandlerTest.java (100%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/dispatcher/SparkQueryDispatcherTest.java (100%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/execution/session/SessionManagerTest.java (100%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/execution/session/SessionStateTest.java (100%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/execution/session/SessionTypeTest.java (100%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/execution/statement/StatementStateTest.java (100%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/execution/statestore/OpenSearchStateStoreUtilTest.java (100%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/execution/statestore/StateModelTest.java (100%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializerUtilTest.java (100%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/flint/FlintIndexStateTest.java (100%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/flint/IndexQueryDetailsTest.java (100%) rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpTest.java (100%) create mode 100644 async-query-core/src/test/java/org/opensearch/sql/spark/leasemanager/ConcurrencyLimitExceededExceptionTest.java rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/rest/model/CreateAsyncQueryRequestTest.java (100%) create mode 100644 async-query-core/src/test/java/org/opensearch/sql/spark/utils/IDUtilsTest.java rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/utils/MockTimeProvider.java (100%) create mode 100644 async-query-core/src/test/java/org/opensearch/sql/spark/utils/RealTimeProviderTest.java rename {spark => async-query-core}/src/test/java/org/opensearch/sql/spark/utils/SQLQueryUtilsTest.java (100%) create mode 100644 async-query-core/src/test/java/org/opensearch/sql/spark/utils/TestUtils.java rename {spark => async-query-core}/src/test/resources/invalid_response.json (100%) create mode 100644 async-query-core/src/test/resources/mockito-extensions/org.mockito.plugins.MockMaker create mode 100644 async-query-core/src/test/resources/select_query_response.json delete mode 100644 async-query/src/main/java/org/opensearch/sql/asyncquery/DummyConsumer.java rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/asyncquery/OpenSearchAsyncQueryJobMetadataStorageService.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/cluster/ClusterManagerEventListener.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/cluster/FlintIndexRetention.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/cluster/FlintStreamingJobHouseKeeperTask.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/cluster/IndexCleanup.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/config/OpenSearchSparkSubmitParameterModifier.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigSupplierImpl.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/execution/session/OpenSearchSessionConfigSupplier.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/execution/statestore/FromXContent.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/execution/statestore/OpenSearchSessionStorageService.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/execution/statestore/OpenSearchStatementStorageService.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/execution/statestore/StateStore.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/execution/xcontent/AsyncQueryJobMetadataXContentSerializer.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/execution/xcontent/FlintIndexStateModelXContentSerializer.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/execution/xcontent/IndexDMLResultXContentSerializer.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/execution/xcontent/SessionModelXContentSerializer.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/execution/xcontent/StatementModelXContentSerializer.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/execution/xcontent/XContentCommonAttributes.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializer.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/flint/FlintIndexMetadataServiceImpl.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/flint/OpenSearchFlintIndexStateModelService.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/flint/OpenSearchIndexDMLResultStorageService.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/leasemanager/DefaultLeaseManager.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/response/OpenSearchJobExecutionResponseReader.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/rest/RestAsyncQueryManagementAction.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/transport/TransportCancelAsyncQueryRequestAction.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/transport/TransportCreateAsyncQueryRequestAction.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/transport/TransportGetAsyncQueryResultAction.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/transport/config/AsyncExecutorServiceModule.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/transport/format/AsyncQueryResultResponseFormatter.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/transport/model/CancelAsyncQueryActionRequest.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/transport/model/CancelAsyncQueryActionResponse.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/transport/model/CreateAsyncQueryActionRequest.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/transport/model/CreateAsyncQueryActionResponse.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/transport/model/GetAsyncQueryResultActionRequest.java (100%) rename {spark => async-query}/src/main/java/org/opensearch/sql/spark/transport/model/GetAsyncQueryResultActionResponse.java (100%) delete mode 100644 async-query/src/test/java/org/opensearch/sql/asyncquery/DummyConsumerTest.java rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImplSpecTest.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceSpec.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryGetResultSpecTest.java (97%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecAlterTest.java (99%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecTest.java (99%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecVacuumTest.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/asyncquery/OpenSearchAsyncQueryJobMetadataStorageServiceTest.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/asyncquery/model/MockFlintIndex.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/asyncquery/model/MockFlintSparkJob.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/cluster/FlintStreamingJobHouseKeeperTaskTest.java (92%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigSupplierImplTest.java (100%) create mode 100644 async-query/src/test/java/org/opensearch/sql/spark/constants/TestConstants.java rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/execution/session/InteractiveSessionTest.java (92%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/execution/session/SessionTestUtil.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/execution/session/TestEMRServerlessClient.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/execution/statement/StatementTest.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/execution/xcontent/AsyncQueryJobMetadataXContentSerializerTest.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/execution/xcontent/FlintIndexStateModelXContentSerializerTest.java (97%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/execution/xcontent/IndexDMLResultXContentSerializerTest.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/execution/xcontent/SessionModelXContentSerializerTest.java (97%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/execution/xcontent/StatementModelXContentSerializerTest.java (98%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializerTestUtil.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/flint/FlintIndexMetadataServiceImplTest.java (97%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/flint/OpenSearchFlintIndexStateModelServiceTest.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/leasemanager/DefaultLeaseManagerTest.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/response/OpenSearchJobExecutionResponseReaderTest.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/transport/TransportCancelAsyncQueryRequestActionTest.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/transport/TransportCreateAsyncQueryRequestActionTest.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/transport/TransportGetAsyncQueryResultActionTest.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/transport/config/AsyncExecutorServiceModuleTest.java (100%) rename {spark => async-query}/src/test/java/org/opensearch/sql/spark/transport/format/AsyncQueryResultResponseFormatterTest.java (100%) create mode 100644 async-query/src/test/java/org/opensearch/sql/spark/utils/TestUtils.java rename {spark => async-query}/src/test/resources/flint-index-mappings/0.1.1/flint_covering_index.json (100%) rename {spark => async-query}/src/test/resources/flint-index-mappings/0.1.1/flint_mv.json (100%) rename {spark => async-query}/src/test/resources/flint-index-mappings/0.1.1/flint_skipping_index.json (100%) rename {spark => async-query}/src/test/resources/flint-index-mappings/0.1.1/flint_special_character_index.json (100%) rename {spark => async-query}/src/test/resources/flint-index-mappings/flint_covering_index.json (100%) rename {spark => async-query}/src/test/resources/flint-index-mappings/flint_mv.json (100%) rename {spark => async-query}/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_covering_corrupted_index_mapping.json (100%) rename {spark => async-query}/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_covering_error_index_mapping.json (100%) rename {spark => async-query}/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_covering_index_mapping.json (100%) rename {spark => async-query}/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_skipping_index_mapping.json (100%) rename {spark => async-query}/src/test/resources/flint-index-mappings/flint_my_glue_mydb_mv_mapping.json (100%) rename {spark => async-query}/src/test/resources/flint-index-mappings/flint_mys3_default_http_logs_cv1_index.json (100%) rename {spark => async-query}/src/test/resources/flint-index-mappings/flint_mys3_default_http_logs_skipping_index.json (100%) rename {spark => async-query}/src/test/resources/flint-index-mappings/flint_skipping_index.json (100%) rename {spark => async-query}/src/test/resources/flint-index-mappings/flint_special_character_index.json (100%) rename {spark => async-query}/src/test/resources/flint-index-mappings/npe_mapping.json (100%) create mode 100644 async-query/src/test/resources/mockito-extensions/org.mockito.plugins.MockMaker rename {spark => async-query}/src/test/resources/query_execution_result_mapping.json (100%) create mode 100644 spark/src/test/java/org/opensearch/sql/spark/data/type/SparkDataTypeTest.java diff --git a/async-query-core/build.gradle b/async-query-core/build.gradle index 3673872988..5cd681c9ff 100644 --- a/async-query-core/build.gradle +++ b/async-query-core/build.gradle @@ -44,6 +44,11 @@ generateGrammarSource.dependsOn downloadG4Files dependencies { antlr "org.antlr:antlr4:4.7.1" + implementation project(':core') + implementation project(':spark') // TODO: dependency to spark should be eliminated + implementation project(':datasources') // TODO: dependency to datasources should be eliminated + implementation project(':legacy') // TODO: dependency to legacy should be eliminated + implementation group: 'org.json', name: 'json', version: '20231013' implementation group: 'com.google.guava', name: 'guava', version: '32.0.1-jre' implementation group: 'com.fasterxml.jackson.core', name: 'jackson-core', version: "${versions.jackson}" implementation group: 'com.fasterxml.jackson.core', name: 'jackson-databind', version: "${versions.jackson_databind}" @@ -108,7 +113,19 @@ jacocoTestCoverageVerification { violationRules { rule { element = 'CLASS' - excludes = [] + // TODO: Add unit tests in async-query-core and remove exclusions + excludes = [ + 'org.opensearch.sql.spark.asyncquery.model.*', + 'org.opensearch.sql.spark.data.constants.*', + 'org.opensearch.sql.spark.dispatcher.model.*', + 'org.opensearch.sql.spark.dispatcher.*', + 'org.opensearch.sql.spark.execution.session.*', + 'org.opensearch.sql.spark.execution.statement.*', + 'org.opensearch.sql.spark.flint.*', + 'org.opensearch.sql.spark.flint.operation.*', + 'org.opensearch.sql.spark.rest.*', + 'org.opensearch.sql.spark.utils.SQLQueryUtils.*' + ] limit { counter = 'LINE' minimum = 1.0 diff --git a/spark/src/main/antlr/FlintSparkSqlExtensions.g4 b/async-query-core/src/main/antlr/FlintSparkSqlExtensions.g4 similarity index 100% rename from spark/src/main/antlr/FlintSparkSqlExtensions.g4 rename to async-query-core/src/main/antlr/FlintSparkSqlExtensions.g4 diff --git a/spark/src/main/antlr/SparkSqlBase.g4 b/async-query-core/src/main/antlr/SparkSqlBase.g4 similarity index 100% rename from spark/src/main/antlr/SparkSqlBase.g4 rename to async-query-core/src/main/antlr/SparkSqlBase.g4 diff --git a/spark/src/main/antlr/SqlBaseLexer.g4 b/async-query-core/src/main/antlr/SqlBaseLexer.g4 similarity index 90% rename from spark/src/main/antlr/SqlBaseLexer.g4 rename to async-query-core/src/main/antlr/SqlBaseLexer.g4 index 83e40c4a20..a9705c1733 100644 --- a/spark/src/main/antlr/SqlBaseLexer.g4 +++ b/async-query-core/src/main/antlr/SqlBaseLexer.g4 @@ -69,6 +69,35 @@ lexer grammar SqlBaseLexer; public void markUnclosedComment() { has_unclosed_bracketed_comment = true; } + + /** + * When greater than zero, it's in the middle of parsing ARRAY/MAP/STRUCT type. + */ + public int complex_type_level_counter = 0; + + /** + * Increase the counter by one when hits KEYWORD 'ARRAY', 'MAP', 'STRUCT'. + */ + public void incComplexTypeLevelCounter() { + complex_type_level_counter++; + } + + /** + * Decrease the counter by one when hits close tag '>' && the counter greater than zero + * which means we are in the middle of complex type parsing. Otherwise, it's a dangling + * GT token and we do nothing. + */ + public void decComplexTypeLevelCounter() { + if (complex_type_level_counter > 0) complex_type_level_counter--; + } + + /** + * If the counter is zero, it's a shift right operator. It can be closing tags of an complex + * type definition, such as MAP>. + */ + public boolean isShiftRightOperator() { + return complex_type_level_counter == 0 ? true : false; + } } SEMICOLON: ';'; @@ -100,7 +129,7 @@ ANTI: 'ANTI'; ANY: 'ANY'; ANY_VALUE: 'ANY_VALUE'; ARCHIVE: 'ARCHIVE'; -ARRAY: 'ARRAY'; +ARRAY: 'ARRAY' {incComplexTypeLevelCounter();}; AS: 'AS'; ASC: 'ASC'; AT: 'AT'; @@ -108,6 +137,7 @@ AUTHORIZATION: 'AUTHORIZATION'; BETWEEN: 'BETWEEN'; BIGINT: 'BIGINT'; BINARY: 'BINARY'; +BINDING: 'BINDING'; BOOLEAN: 'BOOLEAN'; BOTH: 'BOTH'; BUCKET: 'BUCKET'; @@ -137,6 +167,7 @@ COMMENT: 'COMMENT'; COMMIT: 'COMMIT'; COMPACT: 'COMPACT'; COMPACTIONS: 'COMPACTIONS'; +COMPENSATION: 'COMPENSATION'; COMPUTE: 'COMPUTE'; CONCATENATE: 'CONCATENATE'; CONSTRAINT: 'CONSTRAINT'; @@ -257,7 +288,7 @@ LOCKS: 'LOCKS'; LOGICAL: 'LOGICAL'; LONG: 'LONG'; MACRO: 'MACRO'; -MAP: 'MAP'; +MAP: 'MAP' {incComplexTypeLevelCounter();}; MATCHED: 'MATCHED'; MERGE: 'MERGE'; MICROSECOND: 'MICROSECOND'; @@ -298,8 +329,6 @@ OVERWRITE: 'OVERWRITE'; PARTITION: 'PARTITION'; PARTITIONED: 'PARTITIONED'; PARTITIONS: 'PARTITIONS'; -PERCENTILE_CONT: 'PERCENTILE_CONT'; -PERCENTILE_DISC: 'PERCENTILE_DISC'; PERCENTLIT: 'PERCENT'; PIVOT: 'PIVOT'; PLACING: 'PLACING'; @@ -362,7 +391,7 @@ STATISTICS: 'STATISTICS'; STORED: 'STORED'; STRATIFY: 'STRATIFY'; STRING: 'STRING'; -STRUCT: 'STRUCT'; +STRUCT: 'STRUCT' {incComplexTypeLevelCounter();}; SUBSTR: 'SUBSTR'; SUBSTRING: 'SUBSTRING'; SYNC: 'SYNC'; @@ -439,8 +468,11 @@ NEQ : '<>'; NEQJ: '!='; LT : '<'; LTE : '<=' | '!>'; -GT : '>'; +GT : '>' {decComplexTypeLevelCounter();}; GTE : '>=' | '!<'; +SHIFT_LEFT: '<<'; +SHIFT_RIGHT: '>>' {isShiftRightOperator()}?; +SHIFT_RIGHT_UNSIGNED: '>>>' {isShiftRightOperator()}?; PLUS: '+'; MINUS: '-'; diff --git a/spark/src/main/antlr/SqlBaseParser.g4 b/async-query-core/src/main/antlr/SqlBaseParser.g4 similarity index 95% rename from spark/src/main/antlr/SqlBaseParser.g4 rename to async-query-core/src/main/antlr/SqlBaseParser.g4 index 60b67b0802..4552c17e0c 100644 --- a/spark/src/main/antlr/SqlBaseParser.g4 +++ b/async-query-core/src/main/antlr/SqlBaseParser.g4 @@ -77,7 +77,7 @@ statement | USE identifierReference #use | USE namespace identifierReference #useNamespace | SET CATALOG (errorCapturingIdentifier | stringLit) #setCatalog - | CREATE namespace (IF NOT EXISTS)? identifierReference + | CREATE namespace (IF errorCapturingNot EXISTS)? identifierReference (commentSpec | locationSpec | (WITH (DBPROPERTIES | PROPERTIES) propertyList))* #createNamespace @@ -92,7 +92,7 @@ statement | createTableHeader (LEFT_PAREN createOrReplaceTableColTypeList RIGHT_PAREN)? tableProvider? createTableClauses (AS? query)? #createTable - | CREATE TABLE (IF NOT EXISTS)? target=tableIdentifier + | CREATE TABLE (IF errorCapturingNot EXISTS)? target=tableIdentifier LIKE source=tableIdentifier (tableProvider | rowFormat | @@ -141,7 +141,7 @@ statement SET SERDE stringLit (WITH SERDEPROPERTIES propertyList)? #setTableSerDe | ALTER TABLE identifierReference (partitionSpec)? SET SERDEPROPERTIES propertyList #setTableSerDe - | ALTER (TABLE | VIEW) identifierReference ADD (IF NOT EXISTS)? + | ALTER (TABLE | VIEW) identifierReference ADD (IF errorCapturingNot EXISTS)? partitionSpecLocation+ #addTablePartition | ALTER TABLE identifierReference from=partitionSpec RENAME TO to=partitionSpec #renameTablePartition @@ -153,9 +153,10 @@ statement | DROP TABLE (IF EXISTS)? identifierReference PURGE? #dropTable | DROP VIEW (IF EXISTS)? identifierReference #dropView | CREATE (OR REPLACE)? (GLOBAL? TEMPORARY)? - VIEW (IF NOT EXISTS)? identifierReference + VIEW (IF errorCapturingNot EXISTS)? identifierReference identifierCommentList? (commentSpec | + schemaBinding | (PARTITIONED ON identifierList) | (TBLPROPERTIES propertyList))* AS query #createView @@ -163,7 +164,8 @@ statement tableIdentifier (LEFT_PAREN colTypeList RIGHT_PAREN)? tableProvider (OPTIONS propertyList)? #createTempViewUsing | ALTER VIEW identifierReference AS? query #alterViewQuery - | CREATE (OR REPLACE)? TEMPORARY? FUNCTION (IF NOT EXISTS)? + | ALTER VIEW identifierReference schemaBinding #alterViewSchemaBinding + | CREATE (OR REPLACE)? TEMPORARY? FUNCTION (IF errorCapturingNot EXISTS)? identifierReference AS className=stringLit (USING resource (COMMA resource)*)? #createFunction | DROP TEMPORARY? FUNCTION (IF EXISTS)? identifierReference #dropFunction @@ -224,7 +226,7 @@ statement | SET .*? #setConfiguration | RESET configKey #resetQuotedConfiguration | RESET .*? #resetConfiguration - | CREATE INDEX (IF NOT EXISTS)? identifier ON TABLE? + | CREATE INDEX (IF errorCapturingNot EXISTS)? identifier ON TABLE? identifierReference (USING indexType=identifier)? LEFT_PAREN columns=multipartIdentifierPropertyList RIGHT_PAREN (OPTIONS options=propertyList)? #createIndex @@ -315,7 +317,7 @@ unsupportedHiveNativeCommands ; createTableHeader - : CREATE TEMPORARY? EXTERNAL? TABLE (IF NOT EXISTS)? identifierReference + : CREATE TEMPORARY? EXTERNAL? TABLE (IF errorCapturingNot EXISTS)? identifierReference ; replaceTableHeader @@ -342,6 +344,10 @@ locationSpec : LOCATION stringLit ; +schemaBinding + : WITH SCHEMA (BINDING | COMPENSATION | EVOLUTION | TYPE EVOLUTION) + ; + commentSpec : COMMENT stringLit ; @@ -351,8 +357,8 @@ query ; insertInto - : INSERT OVERWRITE TABLE? identifierReference (partitionSpec (IF NOT EXISTS)?)? ((BY NAME) | identifierList)? #insertOverwriteTable - | INSERT INTO TABLE? identifierReference partitionSpec? (IF NOT EXISTS)? ((BY NAME) | identifierList)? #insertIntoTable + : INSERT OVERWRITE TABLE? identifierReference (partitionSpec (IF errorCapturingNot EXISTS)?)? ((BY NAME) | identifierList)? #insertOverwriteTable + | INSERT INTO TABLE? identifierReference partitionSpec? (IF errorCapturingNot EXISTS)? ((BY NAME) | identifierList)? #insertIntoTable | INSERT INTO TABLE? identifierReference REPLACE whereClause #insertIntoReplaceWhere | INSERT OVERWRITE LOCAL? DIRECTORY path=stringLit rowFormat? createFileFormat? #insertOverwriteHiveDir | INSERT OVERWRITE LOCAL? DIRECTORY (path=stringLit)? tableProvider (OPTIONS options=propertyList)? #insertOverwriteDir @@ -389,6 +395,7 @@ describeFuncName | comparisonOperator | arithmeticOperator | predicateOperator + | shiftOperator | BANG ; @@ -588,11 +595,11 @@ matchedClause : WHEN MATCHED (AND matchedCond=booleanExpression)? THEN matchedAction ; notMatchedClause - : WHEN NOT MATCHED (BY TARGET)? (AND notMatchedCond=booleanExpression)? THEN notMatchedAction + : WHEN errorCapturingNot MATCHED (BY TARGET)? (AND notMatchedCond=booleanExpression)? THEN notMatchedAction ; notMatchedBySourceClause - : WHEN NOT MATCHED BY SOURCE (AND notMatchedBySourceCond=booleanExpression)? THEN notMatchedBySourceAction + : WHEN errorCapturingNot MATCHED BY SOURCE (AND notMatchedBySourceCond=booleanExpression)? THEN notMatchedBySourceAction ; matchedAction @@ -838,9 +845,11 @@ tableArgumentPartitioning : ((WITH SINGLE PARTITION) | ((PARTITION | DISTRIBUTE) BY (((LEFT_PAREN partition+=expression (COMMA partition+=expression)* RIGHT_PAREN)) + | (expression (COMMA invalidMultiPartitionExpression=expression)+) | partition+=expression))) ((ORDER | SORT) BY (((LEFT_PAREN sortItem (COMMA sortItem)* RIGHT_PAREN) + | (sortItem (COMMA invalidMultiSortItem=sortItem)+) | sortItem)))? ; @@ -956,15 +965,20 @@ booleanExpression ; predicate - : NOT? kind=BETWEEN lower=valueExpression AND upper=valueExpression - | NOT? kind=IN LEFT_PAREN expression (COMMA expression)* RIGHT_PAREN - | NOT? kind=IN LEFT_PAREN query RIGHT_PAREN - | NOT? kind=RLIKE pattern=valueExpression - | NOT? kind=(LIKE | ILIKE) quantifier=(ANY | SOME | ALL) (LEFT_PAREN RIGHT_PAREN | LEFT_PAREN expression (COMMA expression)* RIGHT_PAREN) - | NOT? kind=(LIKE | ILIKE) pattern=valueExpression (ESCAPE escapeChar=stringLit)? - | IS NOT? kind=NULL - | IS NOT? kind=(TRUE | FALSE | UNKNOWN) - | IS NOT? kind=DISTINCT FROM right=valueExpression + : errorCapturingNot? kind=BETWEEN lower=valueExpression AND upper=valueExpression + | errorCapturingNot? kind=IN LEFT_PAREN expression (COMMA expression)* RIGHT_PAREN + | errorCapturingNot? kind=IN LEFT_PAREN query RIGHT_PAREN + | errorCapturingNot? kind=RLIKE pattern=valueExpression + | errorCapturingNot? kind=(LIKE | ILIKE) quantifier=(ANY | SOME | ALL) (LEFT_PAREN RIGHT_PAREN | LEFT_PAREN expression (COMMA expression)* RIGHT_PAREN) + | errorCapturingNot? kind=(LIKE | ILIKE) pattern=valueExpression (ESCAPE escapeChar=stringLit)? + | IS errorCapturingNot? kind=NULL + | IS errorCapturingNot? kind=(TRUE | FALSE | UNKNOWN) + | IS errorCapturingNot? kind=DISTINCT FROM right=valueExpression + ; + +errorCapturingNot + : NOT + | BANG ; valueExpression @@ -972,12 +986,19 @@ valueExpression | operator=(MINUS | PLUS | TILDE) valueExpression #arithmeticUnary | left=valueExpression operator=(ASTERISK | SLASH | PERCENT | DIV) right=valueExpression #arithmeticBinary | left=valueExpression operator=(PLUS | MINUS | CONCAT_PIPE) right=valueExpression #arithmeticBinary + | left=valueExpression shiftOperator right=valueExpression #shiftExpression | left=valueExpression operator=AMPERSAND right=valueExpression #arithmeticBinary | left=valueExpression operator=HAT right=valueExpression #arithmeticBinary | left=valueExpression operator=PIPE right=valueExpression #arithmeticBinary | left=valueExpression comparisonOperator right=valueExpression #comparison ; +shiftOperator + : SHIFT_LEFT + | SHIFT_RIGHT + | SHIFT_RIGHT_UNSIGNED + ; + datetimeUnit : YEAR | QUARTER | MONTH | WEEK | DAY | DAYOFYEAR @@ -1143,7 +1164,7 @@ qualifiedColTypeWithPosition ; colDefinitionDescriptorWithPosition - : NOT NULL + : errorCapturingNot NULL | defaultExpression | commentSpec | colPosition @@ -1162,7 +1183,7 @@ colTypeList ; colType - : colName=errorCapturingIdentifier dataType (NOT NULL)? commentSpec? + : colName=errorCapturingIdentifier dataType (errorCapturingNot NULL)? commentSpec? ; createOrReplaceTableColTypeList @@ -1174,7 +1195,7 @@ createOrReplaceTableColType ; colDefinitionOption - : NOT NULL + : errorCapturingNot NULL | defaultExpression | generationExpression | commentSpec @@ -1189,7 +1210,7 @@ complexColTypeList ; complexColType - : errorCapturingIdentifier COLON? dataType (NOT NULL)? commentSpec? + : errorCapturingIdentifier COLON? dataType (errorCapturingNot NULL)? commentSpec? ; whenClause @@ -1296,7 +1317,7 @@ alterColumnAction : TYPE dataType | commentSpec | colPosition - | setOrDrop=(SET | DROP) NOT NULL + | setOrDrop=(SET | DROP) errorCapturingNot NULL | SET defaultExpression | dropDefault=DROP DEFAULT ; @@ -1343,6 +1364,7 @@ ansiNonReserved | BIGINT | BINARY | BINARY_HEX + | BINDING | BOOLEAN | BUCKET | BUCKETS @@ -1365,6 +1387,7 @@ ansiNonReserved | COMMIT | COMPACT | COMPACTIONS + | COMPENSATION | COMPUTE | CONCATENATE | COST @@ -1643,6 +1666,7 @@ nonReserved | BIGINT | BINARY | BINARY_HEX + | BINDING | BOOLEAN | BOTH | BUCKET @@ -1672,6 +1696,7 @@ nonReserved | COMMIT | COMPACT | COMPACTIONS + | COMPENSATION | COMPUTE | CONCATENATE | CONSTRAINT @@ -1824,8 +1849,6 @@ nonReserved | PARTITION | PARTITIONED | PARTITIONS - | PERCENTILE_CONT - | PERCENTILE_DISC | PERCENTLIT | PIVOT | PLACING diff --git a/async-query-core/src/main/java/org/opensearch/sql/asyncquery/Dummy.java b/async-query-core/src/main/java/org/opensearch/sql/asyncquery/Dummy.java deleted file mode 100644 index b7ab572f2a..0000000000 --- a/async-query-core/src/main/java/org/opensearch/sql/asyncquery/Dummy.java +++ /dev/null @@ -1,13 +0,0 @@ -/* - * Copyright OpenSearch Contributors - * SPDX-License-Identifier: Apache-2.0 - */ - -package org.opensearch.sql.asyncquery; - -// This is a dummy class for scaffolding and should be deleted later -public class Dummy { - public String hello() { - return "Hello!"; - } -} diff --git a/spark/src/main/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorService.java b/async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorService.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorService.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorService.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImpl.java b/async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImpl.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImpl.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImpl.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/asyncquery/AsyncQueryJobMetadataStorageService.java b/async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/AsyncQueryJobMetadataStorageService.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/asyncquery/AsyncQueryJobMetadataStorageService.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/AsyncQueryJobMetadataStorageService.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/asyncquery/exceptions/AsyncQueryNotFoundException.java b/async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/exceptions/AsyncQueryNotFoundException.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/asyncquery/exceptions/AsyncQueryNotFoundException.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/exceptions/AsyncQueryNotFoundException.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryExecutionResponse.java b/async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryExecutionResponse.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryExecutionResponse.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryExecutionResponse.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryId.java b/async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryId.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryId.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryId.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryJobMetadata.java b/async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryJobMetadata.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryJobMetadata.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryJobMetadata.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryRequestContext.java b/async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryRequestContext.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryRequestContext.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryRequestContext.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryResult.java b/async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryResult.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryResult.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/model/AsyncQueryResult.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/asyncquery/model/NullAsyncQueryRequestContext.java b/async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/model/NullAsyncQueryRequestContext.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/asyncquery/model/NullAsyncQueryRequestContext.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/model/NullAsyncQueryRequestContext.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/asyncquery/model/SparkSubmitParameters.java b/async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/model/SparkSubmitParameters.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/asyncquery/model/SparkSubmitParameters.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/asyncquery/model/SparkSubmitParameters.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/client/EMRServerlessClient.java b/async-query-core/src/main/java/org/opensearch/sql/spark/client/EMRServerlessClient.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/client/EMRServerlessClient.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/client/EMRServerlessClient.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/client/EMRServerlessClientFactory.java b/async-query-core/src/main/java/org/opensearch/sql/spark/client/EMRServerlessClientFactory.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/client/EMRServerlessClientFactory.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/client/EMRServerlessClientFactory.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/client/EMRServerlessClientFactoryImpl.java b/async-query-core/src/main/java/org/opensearch/sql/spark/client/EMRServerlessClientFactoryImpl.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/client/EMRServerlessClientFactoryImpl.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/client/EMRServerlessClientFactoryImpl.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/client/EmrServerlessClientImpl.java b/async-query-core/src/main/java/org/opensearch/sql/spark/client/EmrServerlessClientImpl.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/client/EmrServerlessClientImpl.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/client/EmrServerlessClientImpl.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/client/StartJobRequest.java b/async-query-core/src/main/java/org/opensearch/sql/spark/client/StartJobRequest.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/client/StartJobRequest.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/client/StartJobRequest.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfig.java b/async-query-core/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfig.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfig.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfig.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigClusterSetting.java b/async-query-core/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigClusterSetting.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigClusterSetting.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigClusterSetting.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigSupplier.java b/async-query-core/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigSupplier.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigSupplier.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigSupplier.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/config/SparkSubmitParameterModifier.java b/async-query-core/src/main/java/org/opensearch/sql/spark/config/SparkSubmitParameterModifier.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/config/SparkSubmitParameterModifier.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/config/SparkSubmitParameterModifier.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/data/constants/SparkConstants.java b/async-query-core/src/main/java/org/opensearch/sql/spark/data/constants/SparkConstants.java similarity index 94% rename from spark/src/main/java/org/opensearch/sql/spark/data/constants/SparkConstants.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/data/constants/SparkConstants.java index b9436b0801..5b25bc175a 100644 --- a/spark/src/main/java/org/opensearch/sql/spark/data/constants/SparkConstants.java +++ b/async-query-core/src/main/java/org/opensearch/sql/spark/data/constants/SparkConstants.java @@ -6,8 +6,6 @@ package org.opensearch.sql.spark.data.constants; public class SparkConstants { - public static final String EMR = "emr"; - public static final String STEP_ID_FIELD = "stepId.keyword"; public static final String JOB_ID_FIELD = "jobRunId"; @@ -21,16 +19,11 @@ public class SparkConstants { public static final String SPARK_SQL_APPLICATION_JAR = "file:///home/hadoop/.ivy2/jars/org.opensearch_opensearch-spark-sql-application_2.12-0.3.0-SNAPSHOT.jar"; public static final String SPARK_REQUEST_BUFFER_INDEX_NAME = ".query_execution_request"; - // TODO should be replaced with mvn jar. - public static final String FLINT_INTEGRATION_JAR = - "s3://spark-datasource/flint-spark-integration-assembly-0.3.0-SNAPSHOT.jar"; - // TODO should be replaced with mvn jar. public static final String FLINT_DEFAULT_CLUSTER_NAME = "opensearch-cluster"; public static final String FLINT_DEFAULT_HOST = "localhost"; public static final String FLINT_DEFAULT_PORT = "9200"; public static final String FLINT_DEFAULT_SCHEME = "http"; public static final String FLINT_DEFAULT_AUTH = "noauth"; - public static final String FLINT_DEFAULT_REGION = "us-west-2"; public static final String DEFAULT_CLASS_NAME = "org.apache.spark.sql.FlintJob"; public static final String S3_AWS_CREDENTIALS_PROVIDER_KEY = "spark.hadoop.fs.s3.customAWSCredentialsProvider"; diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/AsyncQueryHandler.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/AsyncQueryHandler.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/AsyncQueryHandler.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/AsyncQueryHandler.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/BatchQueryHandler.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/BatchQueryHandler.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/BatchQueryHandler.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/BatchQueryHandler.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/DatasourceEmbeddedQueryIdProvider.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/DatasourceEmbeddedQueryIdProvider.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/DatasourceEmbeddedQueryIdProvider.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/DatasourceEmbeddedQueryIdProvider.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/IndexDMLHandler.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/IndexDMLHandler.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/IndexDMLHandler.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/IndexDMLHandler.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/InteractiveQueryHandler.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/InteractiveQueryHandler.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/InteractiveQueryHandler.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/InteractiveQueryHandler.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/QueryHandlerFactory.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/QueryHandlerFactory.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/QueryHandlerFactory.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/QueryHandlerFactory.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/QueryIdProvider.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/QueryIdProvider.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/QueryIdProvider.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/QueryIdProvider.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/RefreshQueryHandler.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/RefreshQueryHandler.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/RefreshQueryHandler.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/RefreshQueryHandler.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/SparkQueryDispatcher.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/SparkQueryDispatcher.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/SparkQueryDispatcher.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/SparkQueryDispatcher.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/StreamingQueryHandler.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/StreamingQueryHandler.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/StreamingQueryHandler.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/StreamingQueryHandler.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/DispatchQueryContext.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/DispatchQueryContext.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/DispatchQueryContext.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/DispatchQueryContext.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/DispatchQueryRequest.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/DispatchQueryRequest.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/DispatchQueryRequest.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/DispatchQueryRequest.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/DispatchQueryResponse.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/DispatchQueryResponse.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/DispatchQueryResponse.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/DispatchQueryResponse.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/FlintIndexOptions.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/FlintIndexOptions.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/FlintIndexOptions.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/FlintIndexOptions.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/FullyQualifiedTableName.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/FullyQualifiedTableName.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/FullyQualifiedTableName.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/FullyQualifiedTableName.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/IndexDMLResult.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/IndexDMLResult.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/IndexDMLResult.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/IndexDMLResult.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/IndexQueryActionType.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/IndexQueryActionType.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/IndexQueryActionType.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/IndexQueryActionType.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/IndexQueryDetails.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/IndexQueryDetails.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/IndexQueryDetails.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/IndexQueryDetails.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/JobType.java b/async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/JobType.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/dispatcher/model/JobType.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/dispatcher/model/JobType.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/session/CreateSessionRequest.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/CreateSessionRequest.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/session/CreateSessionRequest.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/CreateSessionRequest.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/session/InteractiveSession.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/InteractiveSession.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/session/InteractiveSession.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/InteractiveSession.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/session/Session.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/Session.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/session/Session.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/Session.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/session/SessionConfigSupplier.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/SessionConfigSupplier.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/session/SessionConfigSupplier.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/SessionConfigSupplier.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/session/SessionId.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/SessionId.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/session/SessionId.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/SessionId.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/session/SessionManager.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/SessionManager.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/session/SessionManager.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/SessionManager.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/session/SessionModel.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/SessionModel.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/session/SessionModel.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/SessionModel.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/session/SessionState.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/SessionState.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/session/SessionState.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/SessionState.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/session/SessionType.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/SessionType.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/session/SessionType.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/session/SessionType.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/statement/QueryRequest.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/statement/QueryRequest.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/statement/QueryRequest.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/statement/QueryRequest.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/statement/Statement.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/statement/Statement.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/statement/Statement.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/statement/Statement.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/statement/StatementId.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/statement/StatementId.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/statement/StatementId.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/statement/StatementId.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/statement/StatementModel.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/statement/StatementModel.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/statement/StatementModel.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/statement/StatementModel.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/statement/StatementState.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/statement/StatementState.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/statement/StatementState.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/statement/StatementState.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/statestore/CopyBuilder.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/statestore/CopyBuilder.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/statestore/CopyBuilder.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/statestore/CopyBuilder.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/statestore/OpenSearchStateStoreUtil.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/statestore/OpenSearchStateStoreUtil.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/statestore/OpenSearchStateStoreUtil.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/statestore/OpenSearchStateStoreUtil.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/statestore/SessionStorageService.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/statestore/SessionStorageService.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/statestore/SessionStorageService.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/statestore/SessionStorageService.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/statestore/StateCopyBuilder.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/statestore/StateCopyBuilder.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/statestore/StateCopyBuilder.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/statestore/StateCopyBuilder.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/statestore/StateModel.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/statestore/StateModel.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/statestore/StateModel.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/statestore/StateModel.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/statestore/StatementStorageService.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/statestore/StatementStorageService.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/statestore/StatementStorageService.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/statestore/StatementStorageService.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializerUtil.java b/async-query-core/src/main/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializerUtil.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializerUtil.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializerUtil.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/FlintIndexMetadata.java b/async-query-core/src/main/java/org/opensearch/sql/spark/flint/FlintIndexMetadata.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/FlintIndexMetadata.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/flint/FlintIndexMetadata.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/FlintIndexMetadataService.java b/async-query-core/src/main/java/org/opensearch/sql/spark/flint/FlintIndexMetadataService.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/FlintIndexMetadataService.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/flint/FlintIndexMetadataService.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/FlintIndexState.java b/async-query-core/src/main/java/org/opensearch/sql/spark/flint/FlintIndexState.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/FlintIndexState.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/flint/FlintIndexState.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/FlintIndexStateModel.java b/async-query-core/src/main/java/org/opensearch/sql/spark/flint/FlintIndexStateModel.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/FlintIndexStateModel.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/flint/FlintIndexStateModel.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/FlintIndexStateModelService.java b/async-query-core/src/main/java/org/opensearch/sql/spark/flint/FlintIndexStateModelService.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/FlintIndexStateModelService.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/flint/FlintIndexStateModelService.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/FlintIndexType.java b/async-query-core/src/main/java/org/opensearch/sql/spark/flint/FlintIndexType.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/FlintIndexType.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/flint/FlintIndexType.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/IndexDMLResultStorageService.java b/async-query-core/src/main/java/org/opensearch/sql/spark/flint/IndexDMLResultStorageService.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/IndexDMLResultStorageService.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/flint/IndexDMLResultStorageService.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOp.java b/async-query-core/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOp.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOp.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOp.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpAlter.java b/async-query-core/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpAlter.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpAlter.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpAlter.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpCancel.java b/async-query-core/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpCancel.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpCancel.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpCancel.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpDrop.java b/async-query-core/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpDrop.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpDrop.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpDrop.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpFactory.java b/async-query-core/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpFactory.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpFactory.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpFactory.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpVacuum.java b/async-query-core/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpVacuum.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpVacuum.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpVacuum.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/leasemanager/ConcurrencyLimitExceededException.java b/async-query-core/src/main/java/org/opensearch/sql/spark/leasemanager/ConcurrencyLimitExceededException.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/leasemanager/ConcurrencyLimitExceededException.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/leasemanager/ConcurrencyLimitExceededException.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/leasemanager/LeaseManager.java b/async-query-core/src/main/java/org/opensearch/sql/spark/leasemanager/LeaseManager.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/leasemanager/LeaseManager.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/leasemanager/LeaseManager.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/leasemanager/model/LeaseRequest.java b/async-query-core/src/main/java/org/opensearch/sql/spark/leasemanager/model/LeaseRequest.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/leasemanager/model/LeaseRequest.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/leasemanager/model/LeaseRequest.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/response/JobExecutionResponseReader.java b/async-query-core/src/main/java/org/opensearch/sql/spark/response/JobExecutionResponseReader.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/response/JobExecutionResponseReader.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/response/JobExecutionResponseReader.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/rest/model/CreateAsyncQueryRequest.java b/async-query-core/src/main/java/org/opensearch/sql/spark/rest/model/CreateAsyncQueryRequest.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/rest/model/CreateAsyncQueryRequest.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/rest/model/CreateAsyncQueryRequest.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/rest/model/CreateAsyncQueryResponse.java b/async-query-core/src/main/java/org/opensearch/sql/spark/rest/model/CreateAsyncQueryResponse.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/rest/model/CreateAsyncQueryResponse.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/rest/model/CreateAsyncQueryResponse.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/rest/model/LangType.java b/async-query-core/src/main/java/org/opensearch/sql/spark/rest/model/LangType.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/rest/model/LangType.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/rest/model/LangType.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/utils/IDUtils.java b/async-query-core/src/main/java/org/opensearch/sql/spark/utils/IDUtils.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/utils/IDUtils.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/utils/IDUtils.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/utils/RealTimeProvider.java b/async-query-core/src/main/java/org/opensearch/sql/spark/utils/RealTimeProvider.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/utils/RealTimeProvider.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/utils/RealTimeProvider.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/utils/SQLQueryUtils.java b/async-query-core/src/main/java/org/opensearch/sql/spark/utils/SQLQueryUtils.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/utils/SQLQueryUtils.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/utils/SQLQueryUtils.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/utils/TimeProvider.java b/async-query-core/src/main/java/org/opensearch/sql/spark/utils/TimeProvider.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/utils/TimeProvider.java rename to async-query-core/src/main/java/org/opensearch/sql/spark/utils/TimeProvider.java diff --git a/async-query-core/src/test/java/org/opensearch/sql/asyncquery/DummyTest.java b/async-query-core/src/test/java/org/opensearch/sql/asyncquery/DummyTest.java deleted file mode 100644 index 8fa1cf49ec..0000000000 --- a/async-query-core/src/test/java/org/opensearch/sql/asyncquery/DummyTest.java +++ /dev/null @@ -1,18 +0,0 @@ -/* - * Copyright OpenSearch Contributors - * SPDX-License-Identifier: Apache-2.0 - */ - -package org.opensearch.sql.asyncquery; - -import static org.junit.jupiter.api.Assertions.assertEquals; - -import org.junit.jupiter.api.Test; - -public class DummyTest { - @Test - public void test() { - Dummy dummy = new Dummy(); - assertEquals("Hello!", dummy.hello()); - } -} diff --git a/spark/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImplTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImplTest.java similarity index 97% rename from spark/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImplTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImplTest.java index b87fb0dad7..8325a10fbc 100644 --- a/spark/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImplTest.java +++ b/async-query-core/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImplTest.java @@ -33,7 +33,7 @@ import org.opensearch.sql.spark.asyncquery.model.AsyncQueryExecutionResponse; import org.opensearch.sql.spark.asyncquery.model.AsyncQueryJobMetadata; import org.opensearch.sql.spark.asyncquery.model.AsyncQueryRequestContext; -import org.opensearch.sql.spark.config.OpenSearchSparkSubmitParameterModifier; +import org.opensearch.sql.spark.asyncquery.model.SparkSubmitParameters; import org.opensearch.sql.spark.config.SparkExecutionEngineConfig; import org.opensearch.sql.spark.config.SparkExecutionEngineConfigSupplier; import org.opensearch.sql.spark.config.SparkSubmitParameterModifier; @@ -114,8 +114,10 @@ void testCreateAsyncQuery() { @Test void testCreateAsyncQueryWithExtraSparkSubmitParameter() { - OpenSearchSparkSubmitParameterModifier modifier = - new OpenSearchSparkSubmitParameterModifier("--conf spark.dynamicAllocation.enabled=false"); + SparkSubmitParameterModifier modifier = + (SparkSubmitParameters parameters) -> { + parameters.setExtraParameters("--conf spark.dynamicAllocation.enabled=false"); + }; when(sparkExecutionEngineConfigSupplier.getSparkExecutionEngineConfig(any())) .thenReturn( SparkExecutionEngineConfig.builder() diff --git a/spark/src/test/java/org/opensearch/sql/spark/asyncquery/model/SparkSubmitParametersTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/asyncquery/model/SparkSubmitParametersTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/asyncquery/model/SparkSubmitParametersTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/asyncquery/model/SparkSubmitParametersTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/client/EMRServerlessClientFactoryImplTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/client/EMRServerlessClientFactoryImplTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/client/EMRServerlessClientFactoryImplTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/client/EMRServerlessClientFactoryImplTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/client/EmrServerlessClientImplTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/client/EmrServerlessClientImplTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/client/EmrServerlessClientImplTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/client/EmrServerlessClientImplTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/client/StartJobRequestTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/client/StartJobRequestTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/client/StartJobRequestTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/client/StartJobRequestTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigClusterSettingTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigClusterSettingTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigClusterSettingTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigClusterSettingTest.java diff --git a/async-query-core/src/test/java/org/opensearch/sql/spark/constants/TestConstants.java b/async-query-core/src/test/java/org/opensearch/sql/spark/constants/TestConstants.java new file mode 100644 index 0000000000..295c74dcee --- /dev/null +++ b/async-query-core/src/test/java/org/opensearch/sql/spark/constants/TestConstants.java @@ -0,0 +1,23 @@ +/* + * Copyright OpenSearch Contributors + * SPDX-License-Identifier: Apache-2.0 + */ + +package org.opensearch.sql.spark.constants; + +public class TestConstants { + public static final String QUERY = "select 1"; + public static final String EMR_JOB_ID = "job-123xxx"; + public static final String EMRS_APPLICATION_ID = "app-xxxxx"; + public static final String EMRS_EXECUTION_ROLE = "execution_role"; + public static final String EMRS_JOB_NAME = "job_name"; + public static final String SPARK_SUBMIT_PARAMETERS = "--conf org.flint.sql.SQLJob"; + public static final String TEST_CLUSTER_NAME = "TEST_CLUSTER"; + public static final String MOCK_SESSION_ID = "s-0123456"; + public static final String MOCK_STATEMENT_ID = "st-0123456"; + public static final String ENTRY_POINT_START_JAR = + "file:///home/hadoop/.ivy2/jars/org.opensearch_opensearch-spark-sql-application_2.12-0.3.0-SNAPSHOT.jar"; + public static final String DEFAULT_RESULT_INDEX = "query_execution_result_ds1"; + public static final String US_EAST_REGION = "us-east-1"; + public static final String US_WEST_REGION = "us-west-1"; +} diff --git a/spark/src/test/java/org/opensearch/sql/spark/dispatcher/IndexDMLHandlerTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/dispatcher/IndexDMLHandlerTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/dispatcher/IndexDMLHandlerTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/dispatcher/IndexDMLHandlerTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/dispatcher/SparkQueryDispatcherTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/dispatcher/SparkQueryDispatcherTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/dispatcher/SparkQueryDispatcherTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/dispatcher/SparkQueryDispatcherTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/session/SessionManagerTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/execution/session/SessionManagerTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/execution/session/SessionManagerTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/execution/session/SessionManagerTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/session/SessionStateTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/execution/session/SessionStateTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/execution/session/SessionStateTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/execution/session/SessionStateTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/session/SessionTypeTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/execution/session/SessionTypeTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/execution/session/SessionTypeTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/execution/session/SessionTypeTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/statement/StatementStateTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/execution/statement/StatementStateTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/execution/statement/StatementStateTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/execution/statement/StatementStateTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/statestore/OpenSearchStateStoreUtilTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/execution/statestore/OpenSearchStateStoreUtilTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/execution/statestore/OpenSearchStateStoreUtilTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/execution/statestore/OpenSearchStateStoreUtilTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/statestore/StateModelTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/execution/statestore/StateModelTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/execution/statestore/StateModelTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/execution/statestore/StateModelTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializerUtilTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializerUtilTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializerUtilTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializerUtilTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/flint/FlintIndexStateTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/flint/FlintIndexStateTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/flint/FlintIndexStateTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/flint/FlintIndexStateTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/flint/IndexQueryDetailsTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/flint/IndexQueryDetailsTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/flint/IndexQueryDetailsTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/flint/IndexQueryDetailsTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/flint/operation/FlintIndexOpTest.java diff --git a/async-query-core/src/test/java/org/opensearch/sql/spark/leasemanager/ConcurrencyLimitExceededExceptionTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/leasemanager/ConcurrencyLimitExceededExceptionTest.java new file mode 100644 index 0000000000..c0591eaf66 --- /dev/null +++ b/async-query-core/src/test/java/org/opensearch/sql/spark/leasemanager/ConcurrencyLimitExceededExceptionTest.java @@ -0,0 +1,19 @@ +/* + * Copyright OpenSearch Contributors + * SPDX-License-Identifier: Apache-2.0 + */ + +package org.opensearch.sql.spark.leasemanager; + +import static org.junit.jupiter.api.Assertions.assertEquals; + +import org.junit.jupiter.api.Test; + +class ConcurrencyLimitExceededExceptionTest { + @Test + public void test() { + ConcurrencyLimitExceededException e = new ConcurrencyLimitExceededException("Test"); + + assertEquals("Test", e.getMessage()); + } +} diff --git a/spark/src/test/java/org/opensearch/sql/spark/rest/model/CreateAsyncQueryRequestTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/rest/model/CreateAsyncQueryRequestTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/rest/model/CreateAsyncQueryRequestTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/rest/model/CreateAsyncQueryRequestTest.java diff --git a/async-query-core/src/test/java/org/opensearch/sql/spark/utils/IDUtilsTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/utils/IDUtilsTest.java new file mode 100644 index 0000000000..1893256c39 --- /dev/null +++ b/async-query-core/src/test/java/org/opensearch/sql/spark/utils/IDUtilsTest.java @@ -0,0 +1,33 @@ +/* + * Copyright OpenSearch Contributors + * SPDX-License-Identifier: Apache-2.0 + */ + +package org.opensearch.sql.spark.utils; + +import static org.junit.jupiter.api.Assertions.assertEquals; +import static org.junit.jupiter.api.Assertions.assertNotEquals; +import static org.junit.jupiter.api.Assertions.assertTrue; + +import org.junit.jupiter.api.Test; + +class IDUtilsTest { + public static final String DATASOURCE_NAME = "DATASOURCE_NAME"; + + @Test + public void encodeAndDecode() { + String id = IDUtils.encode(DATASOURCE_NAME); + String decoded = IDUtils.decode(id); + + assertTrue(id.length() > IDUtils.PREFIX_LEN); + assertEquals(DATASOURCE_NAME, decoded); + } + + @Test + public void generateUniqueIds() { + String id1 = IDUtils.encode(DATASOURCE_NAME); + String id2 = IDUtils.encode(DATASOURCE_NAME); + + assertNotEquals(id1, id2); + } +} diff --git a/spark/src/test/java/org/opensearch/sql/spark/utils/MockTimeProvider.java b/async-query-core/src/test/java/org/opensearch/sql/spark/utils/MockTimeProvider.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/utils/MockTimeProvider.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/utils/MockTimeProvider.java diff --git a/async-query-core/src/test/java/org/opensearch/sql/spark/utils/RealTimeProviderTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/utils/RealTimeProviderTest.java new file mode 100644 index 0000000000..7eb5a56cfe --- /dev/null +++ b/async-query-core/src/test/java/org/opensearch/sql/spark/utils/RealTimeProviderTest.java @@ -0,0 +1,19 @@ +/* + * Copyright OpenSearch Contributors + * SPDX-License-Identifier: Apache-2.0 + */ + +package org.opensearch.sql.spark.utils; + +import static org.junit.jupiter.api.Assertions.assertTrue; + +import org.junit.jupiter.api.Test; + +class RealTimeProviderTest { + @Test + public void testCurrentEpochMillis() { + RealTimeProvider realTimeProvider = new RealTimeProvider(); + + assertTrue(realTimeProvider.currentEpochMillis() > 0); + } +} diff --git a/spark/src/test/java/org/opensearch/sql/spark/utils/SQLQueryUtilsTest.java b/async-query-core/src/test/java/org/opensearch/sql/spark/utils/SQLQueryUtilsTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/utils/SQLQueryUtilsTest.java rename to async-query-core/src/test/java/org/opensearch/sql/spark/utils/SQLQueryUtilsTest.java diff --git a/async-query-core/src/test/java/org/opensearch/sql/spark/utils/TestUtils.java b/async-query-core/src/test/java/org/opensearch/sql/spark/utils/TestUtils.java new file mode 100644 index 0000000000..6566ce424a --- /dev/null +++ b/async-query-core/src/test/java/org/opensearch/sql/spark/utils/TestUtils.java @@ -0,0 +1,44 @@ +/* + * Copyright OpenSearch Contributors + * SPDX-License-Identifier: Apache-2.0 + */ + +package org.opensearch.sql.spark.utils; + +import com.google.common.base.Charsets; +import com.google.common.io.Resources; +import java.io.IOException; +import java.net.URL; +import java.util.Objects; +import lombok.SneakyThrows; +import org.opensearch.action.admin.indices.create.CreateIndexRequest; +import org.opensearch.client.Client; +import org.opensearch.common.xcontent.XContentType; + +public class TestUtils { + /** + * Get Json document from the files in resources folder. + * + * @param filename filename. + * @return String. + * @throws IOException IOException. + */ + public static String getJson(String filename) throws IOException { + ClassLoader classLoader = TestUtils.class.getClassLoader(); + return new String( + Objects.requireNonNull(classLoader.getResourceAsStream(filename)).readAllBytes()); + } + + @SneakyThrows + public static String loadMappings(String path) { + URL url = Resources.getResource(path); + return Resources.toString(url, Charsets.UTF_8); + } + + public static void createIndexWithMappings( + Client client, String indexName, String metadataFileLocation) { + CreateIndexRequest request = new CreateIndexRequest(indexName); + request.mapping(loadMappings(metadataFileLocation), XContentType.JSON); + client.admin().indices().create(request).actionGet(); + } +} diff --git a/spark/src/test/resources/invalid_response.json b/async-query-core/src/test/resources/invalid_response.json similarity index 100% rename from spark/src/test/resources/invalid_response.json rename to async-query-core/src/test/resources/invalid_response.json diff --git a/async-query-core/src/test/resources/mockito-extensions/org.mockito.plugins.MockMaker b/async-query-core/src/test/resources/mockito-extensions/org.mockito.plugins.MockMaker new file mode 100644 index 0000000000..ca6ee9cea8 --- /dev/null +++ b/async-query-core/src/test/resources/mockito-extensions/org.mockito.plugins.MockMaker @@ -0,0 +1 @@ +mock-maker-inline \ No newline at end of file diff --git a/async-query-core/src/test/resources/select_query_response.json b/async-query-core/src/test/resources/select_query_response.json new file mode 100644 index 0000000000..24cb06b49e --- /dev/null +++ b/async-query-core/src/test/resources/select_query_response.json @@ -0,0 +1,12 @@ +{ + "data": { + "result": [ + "{'1':1}" + ], + "schema": [ + "{'column_name':'1','data_type':'integer'}" + ], + "stepId": "s-123456789", + "applicationId": "application-abc" + } +} diff --git a/async-query/build.gradle b/async-query/build.gradle index ee40e5b366..bf8cafe1e7 100644 --- a/async-query/build.gradle +++ b/async-query/build.gradle @@ -17,7 +17,7 @@ repositories { dependencies { api project(':core') - implementation project(':async-query-core') + api project(':async-query-core') implementation project(':protocol') implementation project(':datasources') implementation project(':legacy') diff --git a/async-query/src/main/java/org/opensearch/sql/asyncquery/DummyConsumer.java b/async-query/src/main/java/org/opensearch/sql/asyncquery/DummyConsumer.java deleted file mode 100644 index 9b1641e559..0000000000 --- a/async-query/src/main/java/org/opensearch/sql/asyncquery/DummyConsumer.java +++ /dev/null @@ -1,18 +0,0 @@ -/* - * Copyright OpenSearch Contributors - * SPDX-License-Identifier: Apache-2.0 - */ - -package org.opensearch.sql.asyncquery; - -import lombok.AllArgsConstructor; - -// This is a dummy class for scaffolding and should be deleted later -@AllArgsConstructor -public class DummyConsumer { - Dummy dummy; - - public String hello() { - return dummy.hello(); - } -} diff --git a/spark/src/main/java/org/opensearch/sql/spark/asyncquery/OpenSearchAsyncQueryJobMetadataStorageService.java b/async-query/src/main/java/org/opensearch/sql/spark/asyncquery/OpenSearchAsyncQueryJobMetadataStorageService.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/asyncquery/OpenSearchAsyncQueryJobMetadataStorageService.java rename to async-query/src/main/java/org/opensearch/sql/spark/asyncquery/OpenSearchAsyncQueryJobMetadataStorageService.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/cluster/ClusterManagerEventListener.java b/async-query/src/main/java/org/opensearch/sql/spark/cluster/ClusterManagerEventListener.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/cluster/ClusterManagerEventListener.java rename to async-query/src/main/java/org/opensearch/sql/spark/cluster/ClusterManagerEventListener.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/cluster/FlintIndexRetention.java b/async-query/src/main/java/org/opensearch/sql/spark/cluster/FlintIndexRetention.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/cluster/FlintIndexRetention.java rename to async-query/src/main/java/org/opensearch/sql/spark/cluster/FlintIndexRetention.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/cluster/FlintStreamingJobHouseKeeperTask.java b/async-query/src/main/java/org/opensearch/sql/spark/cluster/FlintStreamingJobHouseKeeperTask.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/cluster/FlintStreamingJobHouseKeeperTask.java rename to async-query/src/main/java/org/opensearch/sql/spark/cluster/FlintStreamingJobHouseKeeperTask.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/cluster/IndexCleanup.java b/async-query/src/main/java/org/opensearch/sql/spark/cluster/IndexCleanup.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/cluster/IndexCleanup.java rename to async-query/src/main/java/org/opensearch/sql/spark/cluster/IndexCleanup.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/config/OpenSearchSparkSubmitParameterModifier.java b/async-query/src/main/java/org/opensearch/sql/spark/config/OpenSearchSparkSubmitParameterModifier.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/config/OpenSearchSparkSubmitParameterModifier.java rename to async-query/src/main/java/org/opensearch/sql/spark/config/OpenSearchSparkSubmitParameterModifier.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigSupplierImpl.java b/async-query/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigSupplierImpl.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigSupplierImpl.java rename to async-query/src/main/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigSupplierImpl.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/session/OpenSearchSessionConfigSupplier.java b/async-query/src/main/java/org/opensearch/sql/spark/execution/session/OpenSearchSessionConfigSupplier.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/session/OpenSearchSessionConfigSupplier.java rename to async-query/src/main/java/org/opensearch/sql/spark/execution/session/OpenSearchSessionConfigSupplier.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/statestore/FromXContent.java b/async-query/src/main/java/org/opensearch/sql/spark/execution/statestore/FromXContent.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/statestore/FromXContent.java rename to async-query/src/main/java/org/opensearch/sql/spark/execution/statestore/FromXContent.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/statestore/OpenSearchSessionStorageService.java b/async-query/src/main/java/org/opensearch/sql/spark/execution/statestore/OpenSearchSessionStorageService.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/statestore/OpenSearchSessionStorageService.java rename to async-query/src/main/java/org/opensearch/sql/spark/execution/statestore/OpenSearchSessionStorageService.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/statestore/OpenSearchStatementStorageService.java b/async-query/src/main/java/org/opensearch/sql/spark/execution/statestore/OpenSearchStatementStorageService.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/statestore/OpenSearchStatementStorageService.java rename to async-query/src/main/java/org/opensearch/sql/spark/execution/statestore/OpenSearchStatementStorageService.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/statestore/StateStore.java b/async-query/src/main/java/org/opensearch/sql/spark/execution/statestore/StateStore.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/statestore/StateStore.java rename to async-query/src/main/java/org/opensearch/sql/spark/execution/statestore/StateStore.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/AsyncQueryJobMetadataXContentSerializer.java b/async-query/src/main/java/org/opensearch/sql/spark/execution/xcontent/AsyncQueryJobMetadataXContentSerializer.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/AsyncQueryJobMetadataXContentSerializer.java rename to async-query/src/main/java/org/opensearch/sql/spark/execution/xcontent/AsyncQueryJobMetadataXContentSerializer.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/FlintIndexStateModelXContentSerializer.java b/async-query/src/main/java/org/opensearch/sql/spark/execution/xcontent/FlintIndexStateModelXContentSerializer.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/FlintIndexStateModelXContentSerializer.java rename to async-query/src/main/java/org/opensearch/sql/spark/execution/xcontent/FlintIndexStateModelXContentSerializer.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/IndexDMLResultXContentSerializer.java b/async-query/src/main/java/org/opensearch/sql/spark/execution/xcontent/IndexDMLResultXContentSerializer.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/IndexDMLResultXContentSerializer.java rename to async-query/src/main/java/org/opensearch/sql/spark/execution/xcontent/IndexDMLResultXContentSerializer.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/SessionModelXContentSerializer.java b/async-query/src/main/java/org/opensearch/sql/spark/execution/xcontent/SessionModelXContentSerializer.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/SessionModelXContentSerializer.java rename to async-query/src/main/java/org/opensearch/sql/spark/execution/xcontent/SessionModelXContentSerializer.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/StatementModelXContentSerializer.java b/async-query/src/main/java/org/opensearch/sql/spark/execution/xcontent/StatementModelXContentSerializer.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/StatementModelXContentSerializer.java rename to async-query/src/main/java/org/opensearch/sql/spark/execution/xcontent/StatementModelXContentSerializer.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/XContentCommonAttributes.java b/async-query/src/main/java/org/opensearch/sql/spark/execution/xcontent/XContentCommonAttributes.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/XContentCommonAttributes.java rename to async-query/src/main/java/org/opensearch/sql/spark/execution/xcontent/XContentCommonAttributes.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializer.java b/async-query/src/main/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializer.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializer.java rename to async-query/src/main/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializer.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/FlintIndexMetadataServiceImpl.java b/async-query/src/main/java/org/opensearch/sql/spark/flint/FlintIndexMetadataServiceImpl.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/FlintIndexMetadataServiceImpl.java rename to async-query/src/main/java/org/opensearch/sql/spark/flint/FlintIndexMetadataServiceImpl.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/OpenSearchFlintIndexStateModelService.java b/async-query/src/main/java/org/opensearch/sql/spark/flint/OpenSearchFlintIndexStateModelService.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/OpenSearchFlintIndexStateModelService.java rename to async-query/src/main/java/org/opensearch/sql/spark/flint/OpenSearchFlintIndexStateModelService.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/flint/OpenSearchIndexDMLResultStorageService.java b/async-query/src/main/java/org/opensearch/sql/spark/flint/OpenSearchIndexDMLResultStorageService.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/flint/OpenSearchIndexDMLResultStorageService.java rename to async-query/src/main/java/org/opensearch/sql/spark/flint/OpenSearchIndexDMLResultStorageService.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/leasemanager/DefaultLeaseManager.java b/async-query/src/main/java/org/opensearch/sql/spark/leasemanager/DefaultLeaseManager.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/leasemanager/DefaultLeaseManager.java rename to async-query/src/main/java/org/opensearch/sql/spark/leasemanager/DefaultLeaseManager.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/response/OpenSearchJobExecutionResponseReader.java b/async-query/src/main/java/org/opensearch/sql/spark/response/OpenSearchJobExecutionResponseReader.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/response/OpenSearchJobExecutionResponseReader.java rename to async-query/src/main/java/org/opensearch/sql/spark/response/OpenSearchJobExecutionResponseReader.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/rest/RestAsyncQueryManagementAction.java b/async-query/src/main/java/org/opensearch/sql/spark/rest/RestAsyncQueryManagementAction.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/rest/RestAsyncQueryManagementAction.java rename to async-query/src/main/java/org/opensearch/sql/spark/rest/RestAsyncQueryManagementAction.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/transport/TransportCancelAsyncQueryRequestAction.java b/async-query/src/main/java/org/opensearch/sql/spark/transport/TransportCancelAsyncQueryRequestAction.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/transport/TransportCancelAsyncQueryRequestAction.java rename to async-query/src/main/java/org/opensearch/sql/spark/transport/TransportCancelAsyncQueryRequestAction.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/transport/TransportCreateAsyncQueryRequestAction.java b/async-query/src/main/java/org/opensearch/sql/spark/transport/TransportCreateAsyncQueryRequestAction.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/transport/TransportCreateAsyncQueryRequestAction.java rename to async-query/src/main/java/org/opensearch/sql/spark/transport/TransportCreateAsyncQueryRequestAction.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/transport/TransportGetAsyncQueryResultAction.java b/async-query/src/main/java/org/opensearch/sql/spark/transport/TransportGetAsyncQueryResultAction.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/transport/TransportGetAsyncQueryResultAction.java rename to async-query/src/main/java/org/opensearch/sql/spark/transport/TransportGetAsyncQueryResultAction.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/transport/config/AsyncExecutorServiceModule.java b/async-query/src/main/java/org/opensearch/sql/spark/transport/config/AsyncExecutorServiceModule.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/transport/config/AsyncExecutorServiceModule.java rename to async-query/src/main/java/org/opensearch/sql/spark/transport/config/AsyncExecutorServiceModule.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/transport/format/AsyncQueryResultResponseFormatter.java b/async-query/src/main/java/org/opensearch/sql/spark/transport/format/AsyncQueryResultResponseFormatter.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/transport/format/AsyncQueryResultResponseFormatter.java rename to async-query/src/main/java/org/opensearch/sql/spark/transport/format/AsyncQueryResultResponseFormatter.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/transport/model/CancelAsyncQueryActionRequest.java b/async-query/src/main/java/org/opensearch/sql/spark/transport/model/CancelAsyncQueryActionRequest.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/transport/model/CancelAsyncQueryActionRequest.java rename to async-query/src/main/java/org/opensearch/sql/spark/transport/model/CancelAsyncQueryActionRequest.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/transport/model/CancelAsyncQueryActionResponse.java b/async-query/src/main/java/org/opensearch/sql/spark/transport/model/CancelAsyncQueryActionResponse.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/transport/model/CancelAsyncQueryActionResponse.java rename to async-query/src/main/java/org/opensearch/sql/spark/transport/model/CancelAsyncQueryActionResponse.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/transport/model/CreateAsyncQueryActionRequest.java b/async-query/src/main/java/org/opensearch/sql/spark/transport/model/CreateAsyncQueryActionRequest.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/transport/model/CreateAsyncQueryActionRequest.java rename to async-query/src/main/java/org/opensearch/sql/spark/transport/model/CreateAsyncQueryActionRequest.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/transport/model/CreateAsyncQueryActionResponse.java b/async-query/src/main/java/org/opensearch/sql/spark/transport/model/CreateAsyncQueryActionResponse.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/transport/model/CreateAsyncQueryActionResponse.java rename to async-query/src/main/java/org/opensearch/sql/spark/transport/model/CreateAsyncQueryActionResponse.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/transport/model/GetAsyncQueryResultActionRequest.java b/async-query/src/main/java/org/opensearch/sql/spark/transport/model/GetAsyncQueryResultActionRequest.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/transport/model/GetAsyncQueryResultActionRequest.java rename to async-query/src/main/java/org/opensearch/sql/spark/transport/model/GetAsyncQueryResultActionRequest.java diff --git a/spark/src/main/java/org/opensearch/sql/spark/transport/model/GetAsyncQueryResultActionResponse.java b/async-query/src/main/java/org/opensearch/sql/spark/transport/model/GetAsyncQueryResultActionResponse.java similarity index 100% rename from spark/src/main/java/org/opensearch/sql/spark/transport/model/GetAsyncQueryResultActionResponse.java rename to async-query/src/main/java/org/opensearch/sql/spark/transport/model/GetAsyncQueryResultActionResponse.java diff --git a/async-query/src/test/java/org/opensearch/sql/asyncquery/DummyConsumerTest.java b/async-query/src/test/java/org/opensearch/sql/asyncquery/DummyConsumerTest.java deleted file mode 100644 index a08dbae736..0000000000 --- a/async-query/src/test/java/org/opensearch/sql/asyncquery/DummyConsumerTest.java +++ /dev/null @@ -1,28 +0,0 @@ -/* - * Copyright OpenSearch Contributors - * SPDX-License-Identifier: Apache-2.0 - */ - -package org.opensearch.sql.asyncquery; - -import static org.junit.jupiter.api.Assertions.assertEquals; -import static org.mockito.Mockito.when; - -import org.junit.jupiter.api.Test; -import org.junit.jupiter.api.extension.ExtendWith; -import org.mockito.Mock; -import org.mockito.junit.jupiter.MockitoExtension; - -@ExtendWith(MockitoExtension.class) -class DummyConsumerTest { - - @Mock Dummy dummy; - - @Test - public void test() { - DummyConsumer dummyConsumer = new DummyConsumer(dummy); - when(dummy.hello()).thenReturn("Hello from mock"); - - assertEquals("Hello from mock", dummyConsumer.hello()); - } -} diff --git a/spark/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImplSpecTest.java b/async-query/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImplSpecTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImplSpecTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceImplSpecTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceSpec.java b/async-query/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceSpec.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceSpec.java rename to async-query/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryExecutorServiceSpec.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryGetResultSpecTest.java b/async-query/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryGetResultSpecTest.java similarity index 97% rename from spark/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryGetResultSpecTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryGetResultSpecTest.java index 12fa8043ea..94719509d2 100644 --- a/spark/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryGetResultSpecTest.java +++ b/async-query/src/test/java/org/opensearch/sql/spark/asyncquery/AsyncQueryGetResultSpecTest.java @@ -56,7 +56,10 @@ public class AsyncQueryGetResultSpecTest extends AsyncQueryExecutorServiceSpec { @Before public void doSetUp() { mockIndexState = - new MockFlintSparkJob(flintIndexStateModelService, mockIndex.latestId, MYS3_DATASOURCE); + new MockFlintSparkJob( + flintIndexStateModelService, + mockIndex.latestId, + AsyncQueryExecutorServiceSpec.MYS3_DATASOURCE); } @Test @@ -439,7 +442,8 @@ public JSONObject getResultWithQueryId(String queryId, String resultIndex) { }); this.createQueryResponse = queryService.createAsyncQuery( - new CreateAsyncQueryRequest(query, MYS3_DATASOURCE, LangType.SQL, null), + new CreateAsyncQueryRequest( + query, AsyncQueryExecutorServiceSpec.MYS3_DATASOURCE, LangType.SQL, null), asyncQueryRequestContext); } @@ -515,7 +519,10 @@ void emrJobWriteResultDoc(Map resultDoc) { /** Simulate EMR-S updates query_execution_request with state */ void emrJobUpdateStatementState(StatementState newState) { - StatementModel stmt = statementStorageService.getStatement(queryId, MYS3_DATASOURCE).get(); + StatementModel stmt = + statementStorageService + .getStatement(queryId, AsyncQueryExecutorServiceSpec.MYS3_DATASOURCE) + .get(); statementStorageService.updateStatementState(stmt, newState); } @@ -530,7 +537,7 @@ private Map createEmptyResultDoc(String queryId) { document.put("schema", ImmutableList.of()); document.put("jobRunId", "XXX"); document.put("applicationId", "YYY"); - document.put("dataSourceName", MYS3_DATASOURCE); + document.put("dataSourceName", AsyncQueryExecutorServiceSpec.MYS3_DATASOURCE); document.put("status", "SUCCESS"); document.put("error", ""); document.put("queryId", queryId); @@ -555,7 +562,7 @@ private Map createResultDoc( document.put("schema", schema); document.put("jobRunId", "XXX"); document.put("applicationId", "YYY"); - document.put("dataSourceName", MYS3_DATASOURCE); + document.put("dataSourceName", AsyncQueryExecutorServiceSpec.MYS3_DATASOURCE); document.put("status", "SUCCESS"); document.put("error", ""); document.put("queryId", queryId); diff --git a/spark/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecAlterTest.java b/async-query/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecAlterTest.java similarity index 99% rename from spark/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecAlterTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecAlterTest.java index 801a24922f..4697a2d880 100644 --- a/spark/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecAlterTest.java +++ b/async-query/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecAlterTest.java @@ -1,5 +1,7 @@ package org.opensearch.sql.spark.asyncquery; +import static org.junit.Assert.assertEquals; + import com.amazonaws.services.emrserverless.model.CancelJobRunResult; import com.amazonaws.services.emrserverless.model.GetJobRunResult; import com.amazonaws.services.emrserverless.model.JobRun; diff --git a/spark/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecTest.java b/async-query/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecTest.java similarity index 99% rename from spark/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecTest.java index b4962240f5..9c03208152 100644 --- a/spark/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecTest.java +++ b/async-query/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecTest.java @@ -10,7 +10,6 @@ import com.amazonaws.services.emrserverless.model.JobRun; import com.amazonaws.services.emrserverless.model.ValidationException; import com.google.common.collect.ImmutableList; -import org.junit.Assert; import org.junit.Test; import org.junit.jupiter.api.Assertions; import org.opensearch.sql.spark.asyncquery.model.AsyncQueryExecutionResponse; @@ -634,13 +633,13 @@ public void dropIndexWithIndexInDeletedState() { @Override public CancelJobRunResult cancelJobRun( String applicationId, String jobId, boolean allowExceptionPropagation) { - Assert.fail("should not call cancelJobRun"); + fail("should not call cancelJobRun"); return null; } @Override public GetJobRunResult getJobRunResult(String applicationId, String jobId) { - Assert.fail("should not call getJobRunResult"); + fail("should not call getJobRunResult"); return null; } }; @@ -732,13 +731,13 @@ public void edgeCaseNoIndexStateDoc() { @Override public CancelJobRunResult cancelJobRun( String applicationId, String jobId, boolean allowExceptionPropagation) { - Assert.fail("should not call cancelJobRun"); + fail("should not call cancelJobRun"); return null; } @Override public GetJobRunResult getJobRunResult(String applicationId, String jobId) { - Assert.fail("should not call getJobRunResult"); + fail("should not call getJobRunResult"); return null; } }; @@ -878,13 +877,13 @@ public void cancelAutoRefreshCreateFlintIndexShouldThrowException() { @Override public CancelJobRunResult cancelJobRun( String applicationId, String jobId, boolean allowExceptionPropagation) { - Assert.fail("should not call cancelJobRun"); + fail("should not call cancelJobRun"); return null; } @Override public GetJobRunResult getJobRunResult(String applicationId, String jobId) { - Assert.fail("should not call getJobRunResult"); + fail("should not call getJobRunResult"); return null; } }; diff --git a/spark/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecVacuumTest.java b/async-query/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecVacuumTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecVacuumTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/asyncquery/IndexQuerySpecVacuumTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/asyncquery/OpenSearchAsyncQueryJobMetadataStorageServiceTest.java b/async-query/src/test/java/org/opensearch/sql/spark/asyncquery/OpenSearchAsyncQueryJobMetadataStorageServiceTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/asyncquery/OpenSearchAsyncQueryJobMetadataStorageServiceTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/asyncquery/OpenSearchAsyncQueryJobMetadataStorageServiceTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/asyncquery/model/MockFlintIndex.java b/async-query/src/test/java/org/opensearch/sql/spark/asyncquery/model/MockFlintIndex.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/asyncquery/model/MockFlintIndex.java rename to async-query/src/test/java/org/opensearch/sql/spark/asyncquery/model/MockFlintIndex.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/asyncquery/model/MockFlintSparkJob.java b/async-query/src/test/java/org/opensearch/sql/spark/asyncquery/model/MockFlintSparkJob.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/asyncquery/model/MockFlintSparkJob.java rename to async-query/src/test/java/org/opensearch/sql/spark/asyncquery/model/MockFlintSparkJob.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/cluster/FlintStreamingJobHouseKeeperTaskTest.java b/async-query/src/test/java/org/opensearch/sql/spark/cluster/FlintStreamingJobHouseKeeperTaskTest.java similarity index 92% rename from spark/src/test/java/org/opensearch/sql/spark/cluster/FlintStreamingJobHouseKeeperTaskTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/cluster/FlintStreamingJobHouseKeeperTaskTest.java index aa4684811f..50419591d0 100644 --- a/spark/src/test/java/org/opensearch/sql/spark/cluster/FlintStreamingJobHouseKeeperTaskTest.java +++ b/async-query/src/test/java/org/opensearch/sql/spark/cluster/FlintStreamingJobHouseKeeperTaskTest.java @@ -40,7 +40,9 @@ public void testStreamingJobHouseKeeperWhenDataSourceDisabled() { INDEX.createIndex(); MockFlintSparkJob flintIndexJob = new MockFlintSparkJob( - flintIndexStateModelService, INDEX.getLatestId(), MYGLUE_DATASOURCE); + flintIndexStateModelService, + INDEX.getLatestId(), + AsyncQueryExecutorServiceSpec.MYGLUE_DATASOURCE); indexJobMapping.put(INDEX, flintIndexJob); HashMap existingOptions = new HashMap<>(); existingOptions.put("auto_refresh", "true"); @@ -48,7 +50,7 @@ public void testStreamingJobHouseKeeperWhenDataSourceDisabled() { INDEX.updateIndexOptions(existingOptions, false); flintIndexJob.refreshing(); }); - changeDataSourceStatus(MYGLUE_DATASOURCE, DISABLED); + changeDataSourceStatus(AsyncQueryExecutorServiceSpec.MYGLUE_DATASOURCE, DISABLED); LocalEMRSClient emrsClient = getCancelledLocalEmrsClient(); FlintIndexMetadataService flintIndexMetadataService = new FlintIndexMetadataServiceImpl(client); FlintStreamingJobHouseKeeperTask flintStreamingJobHouseKeeperTask = @@ -119,7 +121,9 @@ public void testStreamingJobHouseKeeperWhenCancelJobGivesTimeout() { INDEX.createIndex(); MockFlintSparkJob flintIndexJob = new MockFlintSparkJob( - flintIndexStateModelService, INDEX.getLatestId(), MYGLUE_DATASOURCE); + flintIndexStateModelService, + INDEX.getLatestId(), + AsyncQueryExecutorServiceSpec.MYGLUE_DATASOURCE); indexJobMapping.put(INDEX, flintIndexJob); HashMap existingOptions = new HashMap<>(); existingOptions.put("auto_refresh", "true"); @@ -127,7 +131,7 @@ public void testStreamingJobHouseKeeperWhenCancelJobGivesTimeout() { INDEX.updateIndexOptions(existingOptions, false); flintIndexJob.refreshing(); }); - changeDataSourceStatus(MYGLUE_DATASOURCE, DISABLED); + changeDataSourceStatus(AsyncQueryExecutorServiceSpec.MYGLUE_DATASOURCE, DISABLED); LocalEMRSClient emrsClient = new LocalEMRSClient(); FlintIndexMetadataService flintIndexMetadataService = new FlintIndexMetadataServiceImpl(client); FlintStreamingJobHouseKeeperTask flintStreamingJobHouseKeeperTask = @@ -167,7 +171,9 @@ public void testSimulateConcurrentJobHouseKeeperExecution() { INDEX.createIndex(); MockFlintSparkJob flintIndexJob = new MockFlintSparkJob( - flintIndexStateModelService, INDEX.getLatestId(), MYGLUE_DATASOURCE); + flintIndexStateModelService, + INDEX.getLatestId(), + AsyncQueryExecutorServiceSpec.MYGLUE_DATASOURCE); indexJobMapping.put(INDEX, flintIndexJob); HashMap existingOptions = new HashMap<>(); existingOptions.put("auto_refresh", "true"); @@ -175,7 +181,7 @@ public void testSimulateConcurrentJobHouseKeeperExecution() { INDEX.updateIndexOptions(existingOptions, false); flintIndexJob.refreshing(); }); - changeDataSourceStatus(MYGLUE_DATASOURCE, DISABLED); + changeDataSourceStatus(AsyncQueryExecutorServiceSpec.MYGLUE_DATASOURCE, DISABLED); LocalEMRSClient emrsClient = new LocalEMRSClient(); FlintIndexMetadataService flintIndexMetadataService = new FlintIndexMetadataServiceImpl(client); FlintStreamingJobHouseKeeperTask flintStreamingJobHouseKeeperTask = @@ -217,7 +223,9 @@ public void testStreamingJobClearnerWhenDataSourceIsDeleted() { INDEX.createIndex(); MockFlintSparkJob flintIndexJob = new MockFlintSparkJob( - flintIndexStateModelService, INDEX.getLatestId(), MYGLUE_DATASOURCE); + flintIndexStateModelService, + INDEX.getLatestId(), + AsyncQueryExecutorServiceSpec.MYGLUE_DATASOURCE); indexJobMapping.put(INDEX, flintIndexJob); HashMap existingOptions = new HashMap<>(); existingOptions.put("auto_refresh", "true"); @@ -225,7 +233,7 @@ public void testStreamingJobClearnerWhenDataSourceIsDeleted() { INDEX.updateIndexOptions(existingOptions, false); flintIndexJob.refreshing(); }); - this.dataSourceService.deleteDataSource(MYGLUE_DATASOURCE); + this.dataSourceService.deleteDataSource(AsyncQueryExecutorServiceSpec.MYGLUE_DATASOURCE); LocalEMRSClient emrsClient = getCancelledLocalEmrsClient(); FlintIndexMetadataService flintIndexMetadataService = new FlintIndexMetadataServiceImpl(client); FlintStreamingJobHouseKeeperTask flintStreamingJobHouseKeeperTask = @@ -265,7 +273,9 @@ public void testStreamingJobHouseKeeperWhenDataSourceIsNeitherDisabledNorDeleted INDEX.createIndex(); MockFlintSparkJob flintIndexJob = new MockFlintSparkJob( - flintIndexStateModelService, INDEX.getLatestId(), MYGLUE_DATASOURCE); + flintIndexStateModelService, + INDEX.getLatestId(), + AsyncQueryExecutorServiceSpec.MYGLUE_DATASOURCE); indexJobMapping.put(INDEX, flintIndexJob); HashMap existingOptions = new HashMap<>(); existingOptions.put("auto_refresh", "true"); @@ -305,7 +315,7 @@ public void testStreamingJobHouseKeeperWhenDataSourceIsNeitherDisabledNorDeleted @Test public void testStreamingJobHouseKeeperWhenS3GlueIsDisabledButNotStreamingJobQueries() throws InterruptedException { - changeDataSourceStatus(MYGLUE_DATASOURCE, DISABLED); + changeDataSourceStatus(AsyncQueryExecutorServiceSpec.MYGLUE_DATASOURCE, DISABLED); LocalEMRSClient emrsClient = new LocalEMRSClient() { @Override @@ -341,7 +351,7 @@ public void testStreamingJobHouseKeeperWhenFlintIndexIsCorrupted() throws Interr MockFlintIndex mockFlintIndex = new MockFlintIndex(client(), indexName, FlintIndexType.COVERING, null); mockFlintIndex.createIndex(); - changeDataSourceStatus(MYGLUE_DATASOURCE, DISABLED); + changeDataSourceStatus(AsyncQueryExecutorServiceSpec.MYGLUE_DATASOURCE, DISABLED); LocalEMRSClient emrsClient = getCancelledLocalEmrsClient(); FlintIndexMetadataService flintIndexMetadataService = new FlintIndexMetadataServiceImpl(client); FlintStreamingJobHouseKeeperTask flintStreamingJobHouseKeeperTask = @@ -415,7 +425,9 @@ public void testStreamingJobHouseKeeperMultipleTimesWhenDataSourceDisabled() { INDEX.createIndex(); MockFlintSparkJob flintIndexJob = new MockFlintSparkJob( - flintIndexStateModelService, INDEX.getLatestId(), MYGLUE_DATASOURCE); + flintIndexStateModelService, + INDEX.getLatestId(), + AsyncQueryExecutorServiceSpec.MYGLUE_DATASOURCE); indexJobMapping.put(INDEX, flintIndexJob); HashMap existingOptions = new HashMap<>(); existingOptions.put("auto_refresh", "true"); @@ -423,7 +435,7 @@ public void testStreamingJobHouseKeeperMultipleTimesWhenDataSourceDisabled() { INDEX.updateIndexOptions(existingOptions, false); flintIndexJob.refreshing(); }); - changeDataSourceStatus(MYGLUE_DATASOURCE, DISABLED); + changeDataSourceStatus(AsyncQueryExecutorServiceSpec.MYGLUE_DATASOURCE, DISABLED); LocalEMRSClient emrsClient = getCancelledLocalEmrsClient(); FlintIndexMetadataService flintIndexMetadataService = new FlintIndexMetadataServiceImpl(client); FlintStreamingJobHouseKeeperTask flintStreamingJobHouseKeeperTask = @@ -487,7 +499,9 @@ public void testRunStreamingJobHouseKeeperWhenDataSourceIsDeleted() { INDEX.createIndex(); MockFlintSparkJob flintIndexJob = new MockFlintSparkJob( - flintIndexStateModelService, INDEX.getLatestId(), MYGLUE_DATASOURCE); + flintIndexStateModelService, + INDEX.getLatestId(), + AsyncQueryExecutorServiceSpec.MYGLUE_DATASOURCE); indexJobMapping.put(INDEX, flintIndexJob); HashMap existingOptions = new HashMap<>(); existingOptions.put("auto_refresh", "true"); @@ -495,7 +509,7 @@ public void testRunStreamingJobHouseKeeperWhenDataSourceIsDeleted() { INDEX.updateIndexOptions(existingOptions, false); flintIndexJob.refreshing(); }); - this.dataSourceService.deleteDataSource(MYGLUE_DATASOURCE); + this.dataSourceService.deleteDataSource(AsyncQueryExecutorServiceSpec.MYGLUE_DATASOURCE); LocalEMRSClient emrsClient = getCancelledLocalEmrsClient(); FlintIndexMetadataService flintIndexMetadataService = new FlintIndexMetadataServiceImpl(client); FlintStreamingJobHouseKeeperTask flintStreamingJobHouseKeeperTask = diff --git a/spark/src/test/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigSupplierImplTest.java b/async-query/src/test/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigSupplierImplTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigSupplierImplTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/config/SparkExecutionEngineConfigSupplierImplTest.java diff --git a/async-query/src/test/java/org/opensearch/sql/spark/constants/TestConstants.java b/async-query/src/test/java/org/opensearch/sql/spark/constants/TestConstants.java new file mode 100644 index 0000000000..5b4ffbea2c --- /dev/null +++ b/async-query/src/test/java/org/opensearch/sql/spark/constants/TestConstants.java @@ -0,0 +1,17 @@ +/* + * Copyright OpenSearch Contributors + * SPDX-License-Identifier: Apache-2.0 + */ + +package org.opensearch.sql.spark.constants; + +public class TestConstants { + public static final String TEST_DATASOURCE_NAME = "test_datasource_name"; + public static final String EMR_JOB_ID = "job-123xxx"; + public static final String EMRS_APPLICATION_ID = "app-xxxxx"; + public static final String EMRS_EXECUTION_ROLE = "execution_role"; + public static final String SPARK_SUBMIT_PARAMETERS = "--conf org.flint.sql.SQLJob"; + public static final String TEST_CLUSTER_NAME = "TEST_CLUSTER"; + public static final String MOCK_SESSION_ID = "s-0123456"; + public static final String US_WEST_REGION = "us-west-1"; +} diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/session/InteractiveSessionTest.java b/async-query/src/test/java/org/opensearch/sql/spark/execution/session/InteractiveSessionTest.java similarity index 92% rename from spark/src/test/java/org/opensearch/sql/spark/execution/session/InteractiveSessionTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/execution/session/InteractiveSessionTest.java index 7d8da14011..16900dde41 100644 --- a/spark/src/test/java/org/opensearch/sql/spark/execution/session/InteractiveSessionTest.java +++ b/async-query/src/test/java/org/opensearch/sql/spark/execution/session/InteractiveSessionTest.java @@ -8,7 +8,6 @@ import static org.opensearch.sql.spark.constants.TestConstants.TEST_CLUSTER_NAME; import static org.opensearch.sql.spark.constants.TestConstants.TEST_DATASOURCE_NAME; import static org.opensearch.sql.spark.execution.session.SessionState.NOT_STARTED; -import static org.opensearch.sql.spark.execution.session.SessionTestUtil.createSessionRequest; import java.util.HashMap; import java.util.Optional; @@ -86,7 +85,7 @@ public void openCloseSession() { SessionAssertions assertions = new SessionAssertions(session); assertions - .open(createSessionRequest()) + .open(SessionTestUtil.createSessionRequest()) .assertSessionState(NOT_STARTED) .assertAppId("appId") .assertJobId("jobId"); @@ -109,7 +108,7 @@ public void openSessionFailedConflict() { .statementStorageService(statementStorageService) .serverlessClient(emrsClient) .build(); - session.open(createSessionRequest(), asyncQueryRequestContext); + session.open(SessionTestUtil.createSessionRequest(), asyncQueryRequestContext); InteractiveSession duplicateSession = InteractiveSession.builder() @@ -121,7 +120,9 @@ public void openSessionFailedConflict() { IllegalStateException exception = assertThrows( IllegalStateException.class, - () -> duplicateSession.open(createSessionRequest(), asyncQueryRequestContext)); + () -> + duplicateSession.open( + SessionTestUtil.createSessionRequest(), asyncQueryRequestContext)); assertEquals("session already exist. " + sessionId, exception.getMessage()); } @@ -135,7 +136,7 @@ public void closeNotExistSession() { .statementStorageService(statementStorageService) .serverlessClient(emrsClient) .build(); - session.open(createSessionRequest(), asyncQueryRequestContext); + session.open(SessionTestUtil.createSessionRequest(), asyncQueryRequestContext); client().delete(new DeleteRequest(indexName, sessionId.getSessionId())).actionGet(); @@ -147,7 +148,8 @@ public void closeNotExistSession() { @Test public void sessionManagerCreateSession() { Session session = - sessionManager.createSession(createSessionRequest(), asyncQueryRequestContext); + sessionManager.createSession( + SessionTestUtil.createSessionRequest(), asyncQueryRequestContext); new SessionAssertions(session) .assertSessionState(NOT_STARTED) @@ -158,7 +160,8 @@ public void sessionManagerCreateSession() { @Test public void sessionManagerGetSession() { Session session = - sessionManager.createSession(createSessionRequest(), asyncQueryRequestContext); + sessionManager.createSession( + SessionTestUtil.createSessionRequest(), asyncQueryRequestContext); Optional managerSession = sessionManager.getSession(session.getSessionId()); assertTrue(managerSession.isPresent()); diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/session/SessionTestUtil.java b/async-query/src/test/java/org/opensearch/sql/spark/execution/session/SessionTestUtil.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/execution/session/SessionTestUtil.java rename to async-query/src/test/java/org/opensearch/sql/spark/execution/session/SessionTestUtil.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/session/TestEMRServerlessClient.java b/async-query/src/test/java/org/opensearch/sql/spark/execution/session/TestEMRServerlessClient.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/execution/session/TestEMRServerlessClient.java rename to async-query/src/test/java/org/opensearch/sql/spark/execution/session/TestEMRServerlessClient.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/statement/StatementTest.java b/async-query/src/test/java/org/opensearch/sql/spark/execution/statement/StatementTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/execution/statement/StatementTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/execution/statement/StatementTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/AsyncQueryJobMetadataXContentSerializerTest.java b/async-query/src/test/java/org/opensearch/sql/spark/execution/xcontent/AsyncQueryJobMetadataXContentSerializerTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/AsyncQueryJobMetadataXContentSerializerTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/execution/xcontent/AsyncQueryJobMetadataXContentSerializerTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/FlintIndexStateModelXContentSerializerTest.java b/async-query/src/test/java/org/opensearch/sql/spark/execution/xcontent/FlintIndexStateModelXContentSerializerTest.java similarity index 97% rename from spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/FlintIndexStateModelXContentSerializerTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/execution/xcontent/FlintIndexStateModelXContentSerializerTest.java index 0d6d5f3119..a7c3381893 100644 --- a/spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/FlintIndexStateModelXContentSerializerTest.java +++ b/async-query/src/test/java/org/opensearch/sql/spark/execution/xcontent/FlintIndexStateModelXContentSerializerTest.java @@ -8,11 +8,11 @@ import static org.junit.jupiter.api.Assertions.assertEquals; import static org.junit.jupiter.api.Assertions.assertNull; import static org.junit.jupiter.api.Assertions.assertThrows; -import static org.mockito.Mockito.mock; import org.json.JSONObject; import org.junit.jupiter.api.Test; import org.junit.jupiter.api.extension.ExtendWith; +import org.mockito.Mockito; import org.mockito.junit.jupiter.MockitoExtension; import org.opensearch.core.xcontent.ToXContent; import org.opensearch.core.xcontent.XContentBuilder; @@ -108,7 +108,7 @@ private JSONObject getBaseJson() { @Test void fromXContentThrowsExceptionWhenParsingInvalidContent() { - XContentParser parser = mock(XContentParser.class); + XContentParser parser = Mockito.mock(XContentParser.class); assertThrows(RuntimeException.class, () -> serializer.fromXContent(parser, 0, 0)); } diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/IndexDMLResultXContentSerializerTest.java b/async-query/src/test/java/org/opensearch/sql/spark/execution/xcontent/IndexDMLResultXContentSerializerTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/IndexDMLResultXContentSerializerTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/execution/xcontent/IndexDMLResultXContentSerializerTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/SessionModelXContentSerializerTest.java b/async-query/src/test/java/org/opensearch/sql/spark/execution/xcontent/SessionModelXContentSerializerTest.java similarity index 97% rename from spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/SessionModelXContentSerializerTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/execution/xcontent/SessionModelXContentSerializerTest.java index 36c019485f..9a260aee5e 100644 --- a/spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/SessionModelXContentSerializerTest.java +++ b/async-query/src/test/java/org/opensearch/sql/spark/execution/xcontent/SessionModelXContentSerializerTest.java @@ -8,10 +8,10 @@ import static org.junit.Assert.assertNull; import static org.junit.jupiter.api.Assertions.assertEquals; import static org.junit.jupiter.api.Assertions.assertThrows; -import static org.mockito.Mockito.mock; import org.json.JSONObject; import org.junit.jupiter.api.Test; +import org.mockito.Mockito; import org.opensearch.core.xcontent.ToXContent; import org.opensearch.core.xcontent.XContentBuilder; import org.opensearch.core.xcontent.XContentParser; @@ -111,7 +111,7 @@ private JSONObject getBaseJson() { @Test void fromXContentThrowsExceptionWhenParsingInvalidContent() { - XContentParser parser = mock(XContentParser.class); + XContentParser parser = Mockito.mock(XContentParser.class); assertThrows(RuntimeException.class, () -> serializer.fromXContent(parser, 0, 0)); } diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/StatementModelXContentSerializerTest.java b/async-query/src/test/java/org/opensearch/sql/spark/execution/xcontent/StatementModelXContentSerializerTest.java similarity index 98% rename from spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/StatementModelXContentSerializerTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/execution/xcontent/StatementModelXContentSerializerTest.java index cdca39d051..e8021f252a 100644 --- a/spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/StatementModelXContentSerializerTest.java +++ b/async-query/src/test/java/org/opensearch/sql/spark/execution/xcontent/StatementModelXContentSerializerTest.java @@ -8,11 +8,11 @@ import static org.junit.jupiter.api.Assertions.assertEquals; import static org.junit.jupiter.api.Assertions.assertNull; import static org.junit.jupiter.api.Assertions.assertThrows; -import static org.mockito.Mockito.mock; import org.json.JSONObject; import org.junit.jupiter.api.Test; import org.junit.jupiter.api.extension.ExtendWith; +import org.mockito.Mockito; import org.mockito.junit.jupiter.MockitoExtension; import org.opensearch.core.xcontent.ToXContent; import org.opensearch.core.xcontent.XContentBuilder; @@ -92,7 +92,7 @@ void fromXContentShouldDeserializeStatementModelWithoutAccountId() throws Except @Test void fromXContentThrowsExceptionWhenParsingInvalidContent() { - XContentParser parser = mock(XContentParser.class); + XContentParser parser = Mockito.mock(XContentParser.class); assertThrows(RuntimeException.class, () -> serializer.fromXContent(parser, 0, 0)); } diff --git a/spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializerTestUtil.java b/async-query/src/test/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializerTestUtil.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializerTestUtil.java rename to async-query/src/test/java/org/opensearch/sql/spark/execution/xcontent/XContentSerializerTestUtil.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/flint/FlintIndexMetadataServiceImplTest.java b/async-query/src/test/java/org/opensearch/sql/spark/flint/FlintIndexMetadataServiceImplTest.java similarity index 97% rename from spark/src/test/java/org/opensearch/sql/spark/flint/FlintIndexMetadataServiceImplTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/flint/FlintIndexMetadataServiceImplTest.java index f6baa82dd2..fa7c908160 100644 --- a/spark/src/test/java/org/opensearch/sql/spark/flint/FlintIndexMetadataServiceImplTest.java +++ b/async-query/src/test/java/org/opensearch/sql/spark/flint/FlintIndexMetadataServiceImplTest.java @@ -6,7 +6,6 @@ package org.opensearch.sql.spark.flint; import static org.mockito.Answers.RETURNS_DEEP_STUBS; -import static org.mockito.Mockito.mock; import static org.mockito.Mockito.when; import com.google.common.base.Charsets; @@ -20,6 +19,7 @@ import org.junit.jupiter.api.Test; import org.junit.jupiter.api.extension.ExtendWith; import org.mockito.Mock; +import org.mockito.Mockito; import org.mockito.junit.jupiter.MockitoExtension; import org.opensearch.action.admin.indices.mapping.get.GetMappingsResponse; import org.opensearch.client.Client; @@ -158,7 +158,7 @@ void testGetJobIDWithNPEExceptionForMultipleIndices() { @SneakyThrows public void mockNodeClientIndicesMappings(String indexName, String mappings) { - GetMappingsResponse mockResponse = mock(GetMappingsResponse.class); + GetMappingsResponse mockResponse = Mockito.mock(GetMappingsResponse.class); when(client.admin().indices().prepareGetMappings().setIndices(indexName).get()) .thenReturn(mockResponse); Map metadata; @@ -169,7 +169,7 @@ public void mockNodeClientIndicesMappings(String indexName, String mappings) { @SneakyThrows public void mockNodeClientIndicesMappings( String indexPattern, HashMap indexMappingsMap) { - GetMappingsResponse mockResponse = mock(GetMappingsResponse.class); + GetMappingsResponse mockResponse = Mockito.mock(GetMappingsResponse.class); when(client.admin().indices().prepareGetMappings().setIndices(indexPattern).get()) .thenReturn(mockResponse); Map metadataMap = new HashMap<>(); diff --git a/spark/src/test/java/org/opensearch/sql/spark/flint/OpenSearchFlintIndexStateModelServiceTest.java b/async-query/src/test/java/org/opensearch/sql/spark/flint/OpenSearchFlintIndexStateModelServiceTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/flint/OpenSearchFlintIndexStateModelServiceTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/flint/OpenSearchFlintIndexStateModelServiceTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/leasemanager/DefaultLeaseManagerTest.java b/async-query/src/test/java/org/opensearch/sql/spark/leasemanager/DefaultLeaseManagerTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/leasemanager/DefaultLeaseManagerTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/leasemanager/DefaultLeaseManagerTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/response/OpenSearchJobExecutionResponseReaderTest.java b/async-query/src/test/java/org/opensearch/sql/spark/response/OpenSearchJobExecutionResponseReaderTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/response/OpenSearchJobExecutionResponseReaderTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/response/OpenSearchJobExecutionResponseReaderTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/transport/TransportCancelAsyncQueryRequestActionTest.java b/async-query/src/test/java/org/opensearch/sql/spark/transport/TransportCancelAsyncQueryRequestActionTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/transport/TransportCancelAsyncQueryRequestActionTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/transport/TransportCancelAsyncQueryRequestActionTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/transport/TransportCreateAsyncQueryRequestActionTest.java b/async-query/src/test/java/org/opensearch/sql/spark/transport/TransportCreateAsyncQueryRequestActionTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/transport/TransportCreateAsyncQueryRequestActionTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/transport/TransportCreateAsyncQueryRequestActionTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/transport/TransportGetAsyncQueryResultActionTest.java b/async-query/src/test/java/org/opensearch/sql/spark/transport/TransportGetAsyncQueryResultActionTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/transport/TransportGetAsyncQueryResultActionTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/transport/TransportGetAsyncQueryResultActionTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/transport/config/AsyncExecutorServiceModuleTest.java b/async-query/src/test/java/org/opensearch/sql/spark/transport/config/AsyncExecutorServiceModuleTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/transport/config/AsyncExecutorServiceModuleTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/transport/config/AsyncExecutorServiceModuleTest.java diff --git a/spark/src/test/java/org/opensearch/sql/spark/transport/format/AsyncQueryResultResponseFormatterTest.java b/async-query/src/test/java/org/opensearch/sql/spark/transport/format/AsyncQueryResultResponseFormatterTest.java similarity index 100% rename from spark/src/test/java/org/opensearch/sql/spark/transport/format/AsyncQueryResultResponseFormatterTest.java rename to async-query/src/test/java/org/opensearch/sql/spark/transport/format/AsyncQueryResultResponseFormatterTest.java diff --git a/async-query/src/test/java/org/opensearch/sql/spark/utils/TestUtils.java b/async-query/src/test/java/org/opensearch/sql/spark/utils/TestUtils.java new file mode 100644 index 0000000000..6566ce424a --- /dev/null +++ b/async-query/src/test/java/org/opensearch/sql/spark/utils/TestUtils.java @@ -0,0 +1,44 @@ +/* + * Copyright OpenSearch Contributors + * SPDX-License-Identifier: Apache-2.0 + */ + +package org.opensearch.sql.spark.utils; + +import com.google.common.base.Charsets; +import com.google.common.io.Resources; +import java.io.IOException; +import java.net.URL; +import java.util.Objects; +import lombok.SneakyThrows; +import org.opensearch.action.admin.indices.create.CreateIndexRequest; +import org.opensearch.client.Client; +import org.opensearch.common.xcontent.XContentType; + +public class TestUtils { + /** + * Get Json document from the files in resources folder. + * + * @param filename filename. + * @return String. + * @throws IOException IOException. + */ + public static String getJson(String filename) throws IOException { + ClassLoader classLoader = TestUtils.class.getClassLoader(); + return new String( + Objects.requireNonNull(classLoader.getResourceAsStream(filename)).readAllBytes()); + } + + @SneakyThrows + public static String loadMappings(String path) { + URL url = Resources.getResource(path); + return Resources.toString(url, Charsets.UTF_8); + } + + public static void createIndexWithMappings( + Client client, String indexName, String metadataFileLocation) { + CreateIndexRequest request = new CreateIndexRequest(indexName); + request.mapping(loadMappings(metadataFileLocation), XContentType.JSON); + client.admin().indices().create(request).actionGet(); + } +} diff --git a/spark/src/test/resources/flint-index-mappings/0.1.1/flint_covering_index.json b/async-query/src/test/resources/flint-index-mappings/0.1.1/flint_covering_index.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/0.1.1/flint_covering_index.json rename to async-query/src/test/resources/flint-index-mappings/0.1.1/flint_covering_index.json diff --git a/spark/src/test/resources/flint-index-mappings/0.1.1/flint_mv.json b/async-query/src/test/resources/flint-index-mappings/0.1.1/flint_mv.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/0.1.1/flint_mv.json rename to async-query/src/test/resources/flint-index-mappings/0.1.1/flint_mv.json diff --git a/spark/src/test/resources/flint-index-mappings/0.1.1/flint_skipping_index.json b/async-query/src/test/resources/flint-index-mappings/0.1.1/flint_skipping_index.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/0.1.1/flint_skipping_index.json rename to async-query/src/test/resources/flint-index-mappings/0.1.1/flint_skipping_index.json diff --git a/spark/src/test/resources/flint-index-mappings/0.1.1/flint_special_character_index.json b/async-query/src/test/resources/flint-index-mappings/0.1.1/flint_special_character_index.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/0.1.1/flint_special_character_index.json rename to async-query/src/test/resources/flint-index-mappings/0.1.1/flint_special_character_index.json diff --git a/spark/src/test/resources/flint-index-mappings/flint_covering_index.json b/async-query/src/test/resources/flint-index-mappings/flint_covering_index.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/flint_covering_index.json rename to async-query/src/test/resources/flint-index-mappings/flint_covering_index.json diff --git a/spark/src/test/resources/flint-index-mappings/flint_mv.json b/async-query/src/test/resources/flint-index-mappings/flint_mv.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/flint_mv.json rename to async-query/src/test/resources/flint-index-mappings/flint_mv.json diff --git a/spark/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_covering_corrupted_index_mapping.json b/async-query/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_covering_corrupted_index_mapping.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_covering_corrupted_index_mapping.json rename to async-query/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_covering_corrupted_index_mapping.json diff --git a/spark/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_covering_error_index_mapping.json b/async-query/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_covering_error_index_mapping.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_covering_error_index_mapping.json rename to async-query/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_covering_error_index_mapping.json diff --git a/spark/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_covering_index_mapping.json b/async-query/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_covering_index_mapping.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_covering_index_mapping.json rename to async-query/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_covering_index_mapping.json diff --git a/spark/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_skipping_index_mapping.json b/async-query/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_skipping_index_mapping.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_skipping_index_mapping.json rename to async-query/src/test/resources/flint-index-mappings/flint_my_glue_mydb_http_logs_skipping_index_mapping.json diff --git a/spark/src/test/resources/flint-index-mappings/flint_my_glue_mydb_mv_mapping.json b/async-query/src/test/resources/flint-index-mappings/flint_my_glue_mydb_mv_mapping.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/flint_my_glue_mydb_mv_mapping.json rename to async-query/src/test/resources/flint-index-mappings/flint_my_glue_mydb_mv_mapping.json diff --git a/spark/src/test/resources/flint-index-mappings/flint_mys3_default_http_logs_cv1_index.json b/async-query/src/test/resources/flint-index-mappings/flint_mys3_default_http_logs_cv1_index.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/flint_mys3_default_http_logs_cv1_index.json rename to async-query/src/test/resources/flint-index-mappings/flint_mys3_default_http_logs_cv1_index.json diff --git a/spark/src/test/resources/flint-index-mappings/flint_mys3_default_http_logs_skipping_index.json b/async-query/src/test/resources/flint-index-mappings/flint_mys3_default_http_logs_skipping_index.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/flint_mys3_default_http_logs_skipping_index.json rename to async-query/src/test/resources/flint-index-mappings/flint_mys3_default_http_logs_skipping_index.json diff --git a/spark/src/test/resources/flint-index-mappings/flint_skipping_index.json b/async-query/src/test/resources/flint-index-mappings/flint_skipping_index.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/flint_skipping_index.json rename to async-query/src/test/resources/flint-index-mappings/flint_skipping_index.json diff --git a/spark/src/test/resources/flint-index-mappings/flint_special_character_index.json b/async-query/src/test/resources/flint-index-mappings/flint_special_character_index.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/flint_special_character_index.json rename to async-query/src/test/resources/flint-index-mappings/flint_special_character_index.json diff --git a/spark/src/test/resources/flint-index-mappings/npe_mapping.json b/async-query/src/test/resources/flint-index-mappings/npe_mapping.json similarity index 100% rename from spark/src/test/resources/flint-index-mappings/npe_mapping.json rename to async-query/src/test/resources/flint-index-mappings/npe_mapping.json diff --git a/async-query/src/test/resources/mockito-extensions/org.mockito.plugins.MockMaker b/async-query/src/test/resources/mockito-extensions/org.mockito.plugins.MockMaker new file mode 100644 index 0000000000..ca6ee9cea8 --- /dev/null +++ b/async-query/src/test/resources/mockito-extensions/org.mockito.plugins.MockMaker @@ -0,0 +1 @@ +mock-maker-inline \ No newline at end of file diff --git a/spark/src/test/resources/query_execution_result_mapping.json b/async-query/src/test/resources/query_execution_result_mapping.json similarity index 100% rename from spark/src/test/resources/query_execution_result_mapping.json rename to async-query/src/test/resources/query_execution_result_mapping.json diff --git a/plugin/build.gradle b/plugin/build.gradle index 49cb52721d..ed6fe53247 100644 --- a/plugin/build.gradle +++ b/plugin/build.gradle @@ -149,6 +149,7 @@ dependencies { api project(':prometheus') api project(':datasources') api project(':spark') + api project(':async-query') testImplementation group: 'net.bytebuddy', name: 'byte-buddy-agent', version: '1.14.9' testImplementation group: 'org.hamcrest', name: 'hamcrest-library', version: '2.1' diff --git a/spark/build.gradle b/spark/build.gradle index c221c4e36c..d9d5c96413 100644 --- a/spark/build.gradle +++ b/spark/build.gradle @@ -7,45 +7,15 @@ plugins { id 'java-library' id "io.freefair.lombok" id 'jacoco' - id 'antlr' } repositories { mavenCentral() } -tasks.register('downloadG4Files', Exec) { - description = 'Download remote .g4 files from GitHub' - - executable 'curl' - - args '-o', 'src/main/antlr/FlintSparkSqlExtensions.g4', 'https://raw.githubusercontent.com/opensearch-project/opensearch-spark/main/flint-spark-integration/src/main/antlr4/FlintSparkSqlExtensions.g4' - args '-o', 'src/main/antlr/SparkSqlBase.g4', 'https://raw.githubusercontent.com/opensearch-project/opensearch-spark/main/flint-spark-integration/src/main/antlr4/SparkSqlBase.g4' - args '-o', 'src/main/antlr/SqlBaseParser.g4', 'https://raw.githubusercontent.com/apache/spark/master/sql/api/src/main/antlr4/org/apache/spark/sql/catalyst/parser/SqlBaseParser.g4' - args '-o', 'src/main/antlr/SqlBaseLexer.g4', 'https://raw.githubusercontent.com/apache/spark/master/sql/api/src/main/antlr4/org/apache/spark/sql/catalyst/parser/SqlBaseLexer.g4' -} - -generateGrammarSource { - arguments += ['-visitor', '-package', 'org.opensearch.sql.spark.antlr.parser'] - source = sourceSets.main.antlr - outputDirectory = file("build/generated-src/antlr/main/org/opensearch/sql/spark/antlr/parser") -} -configurations { - compile { - extendsFrom = extendsFrom.findAll { it != configurations.antlr } - } -} - -// Make sure the downloadG4File task runs before the generateGrammarSource task -generateGrammarSource.dependsOn downloadG4Files - dependencies { - antlr "org.antlr:antlr4:4.7.1" - api project(':core') - implementation project(':protocol') implementation project(':datasources') - implementation project(':legacy') implementation group: 'org.opensearch', name: 'opensearch', version: "${opensearch_version}" implementation group: 'org.json', name: 'json', version: '20231013' @@ -59,20 +29,12 @@ dependencies { testImplementation group: 'org.mockito', name: 'mockito-core', version: '5.7.0' testImplementation group: 'org.mockito', name: 'mockito-junit-jupiter', version: '5.7.0' - testCompileOnly('junit:junit:4.13.1') { - exclude group: 'org.hamcrest', module: 'hamcrest-core' - } - testRuntimeOnly("org.junit.vintage:junit-vintage-engine") { - exclude group: 'org.hamcrest', module: 'hamcrest-core' - } testRuntimeOnly("org.junit.jupiter:junit-jupiter-engine") { exclude group: 'org.hamcrest', module: 'hamcrest-core' } testRuntimeOnly("org.junit.platform:junit-platform-launcher") { because 'allows tests to run from IDEs that bundle older version of launcher' } - testImplementation("org.opensearch.test:framework:${opensearch_version}") - testImplementation project(':opensearch') } test { @@ -84,54 +46,28 @@ test { exceptionFormat "full" } } -task junit4(type: Test) { - useJUnitPlatform { - includeEngines("junit-vintage") - } - systemProperty 'tests.security.manager', 'false' - testLogging { - events "failed" - exceptionFormat "full" - } -} jacocoTestReport { - dependsOn test, junit4 - executionData test, junit4 + dependsOn test + executionData test reports { html.required = true xml.required = true } afterEvaluate { classDirectories.setFrom(files(classDirectories.files.collect { - fileTree(dir: it, exclude: ['**/antlr/parser/**']) })) } } jacocoTestCoverageVerification { - dependsOn test, junit4 - executionData test, junit4 + dependsOn test + executionData test violationRules { rule { element = 'CLASS' excludes = [ 'org.opensearch.sql.spark.data.constants.*', - 'org.opensearch.sql.spark.rest.*', - 'org.opensearch.sql.spark.transport.model.*', - 'org.opensearch.sql.spark.asyncquery.model.*', - 'org.opensearch.sql.spark.asyncquery.exceptions.*', - 'org.opensearch.sql.spark.dispatcher.model.*', - 'org.opensearch.sql.spark.flint.FlintIndexType', - // ignore because XContext IOException - 'org.opensearch.sql.spark.execution.statestore.StateStore', - 'org.opensearch.sql.spark.execution.session.SessionModel', - 'org.opensearch.sql.spark.execution.statement.StatementModel', - 'org.opensearch.sql.spark.flint.FlintIndexStateModel', - // TODO: add tests for purging flint indices - 'org.opensearch.sql.spark.cluster.ClusterManagerEventListener*', - 'org.opensearch.sql.spark.cluster.FlintIndexRetention', - 'org.opensearch.sql.spark.cluster.IndexCleanup' ] limit { counter = 'LINE' @@ -145,7 +81,6 @@ jacocoTestCoverageVerification { } afterEvaluate { classDirectories.setFrom(files(classDirectories.files.collect { - fileTree(dir: it, exclude: ['**/antlr/parser/**']) })) } } diff --git a/spark/src/main/java/org/opensearch/sql/spark/client/EmrClientImpl.java b/spark/src/main/java/org/opensearch/sql/spark/client/EmrClientImpl.java index 87f35bbc1e..3ef911c8d8 100644 --- a/spark/src/main/java/org/opensearch/sql/spark/client/EmrClientImpl.java +++ b/spark/src/main/java/org/opensearch/sql/spark/client/EmrClientImpl.java @@ -6,7 +6,6 @@ package org.opensearch.sql.spark.client; import static org.opensearch.sql.datasource.model.DataSourceMetadata.DEFAULT_RESULT_INDEX; -import static org.opensearch.sql.spark.data.constants.SparkConstants.SPARK_SQL_APPLICATION_JAR; import com.amazonaws.services.elasticmapreduce.AmazonElasticMapReduce; import com.amazonaws.services.elasticmapreduce.model.ActionOnFailure; @@ -26,6 +25,10 @@ import org.opensearch.sql.spark.response.SparkResponse; public class EmrClientImpl implements SparkClient { + // EMR-S will download JAR to local maven + public static final String SPARK_SQL_APPLICATION_JAR = + "file:///home/hadoop/.ivy2/jars/org.opensearch_opensearch-spark-sql-application_2.12-0.3.0-SNAPSHOT.jar"; + private final AmazonElasticMapReduce emr; private final String emrCluster; private final FlintHelper flint; diff --git a/spark/src/main/java/org/opensearch/sql/spark/helper/FlintHelper.java b/spark/src/main/java/org/opensearch/sql/spark/helper/FlintHelper.java index 10d880187f..206ff4aed4 100644 --- a/spark/src/main/java/org/opensearch/sql/spark/helper/FlintHelper.java +++ b/spark/src/main/java/org/opensearch/sql/spark/helper/FlintHelper.java @@ -5,16 +5,18 @@ package org.opensearch.sql.spark.helper; -import static org.opensearch.sql.spark.data.constants.SparkConstants.FLINT_DEFAULT_AUTH; -import static org.opensearch.sql.spark.data.constants.SparkConstants.FLINT_DEFAULT_HOST; -import static org.opensearch.sql.spark.data.constants.SparkConstants.FLINT_DEFAULT_PORT; -import static org.opensearch.sql.spark.data.constants.SparkConstants.FLINT_DEFAULT_REGION; -import static org.opensearch.sql.spark.data.constants.SparkConstants.FLINT_DEFAULT_SCHEME; -import static org.opensearch.sql.spark.data.constants.SparkConstants.FLINT_INTEGRATION_JAR; - import lombok.Getter; public class FlintHelper { + // TODO should be replaced with mvn jar. + public static final String FLINT_INTEGRATION_JAR = + "s3://spark-datasource/flint-spark-integration-assembly-0.3.0-SNAPSHOT.jar"; + public static final String FLINT_DEFAULT_HOST = "localhost"; + public static final String FLINT_DEFAULT_PORT = "9200"; + public static final String FLINT_DEFAULT_SCHEME = "http"; + public static final String FLINT_DEFAULT_AUTH = "noauth"; + public static final String FLINT_DEFAULT_REGION = "us-west-2"; + @Getter private final String flintIntegrationJar; @Getter private final String flintHost; @Getter private final String flintPort; diff --git a/spark/src/main/java/org/opensearch/sql/spark/storage/SparkStorageFactory.java b/spark/src/main/java/org/opensearch/sql/spark/storage/SparkStorageFactory.java index 467bacbaea..4495eb0fac 100644 --- a/spark/src/main/java/org/opensearch/sql/spark/storage/SparkStorageFactory.java +++ b/spark/src/main/java/org/opensearch/sql/spark/storage/SparkStorageFactory.java @@ -5,9 +5,6 @@ package org.opensearch.sql.spark.storage; -import static org.opensearch.sql.spark.data.constants.SparkConstants.EMR; -import static org.opensearch.sql.spark.data.constants.SparkConstants.STEP_ID_FIELD; - import com.amazonaws.auth.AWSStaticCredentialsProvider; import com.amazonaws.auth.BasicAWSCredentials; import com.amazonaws.services.elasticmapreduce.AmazonElasticMapReduce; @@ -36,6 +33,8 @@ public class SparkStorageFactory implements DataSourceFactory { private final Client client; private final Settings settings; + public static final String EMR = "emr"; + public static final String STEP_ID_FIELD = "stepId.keyword"; // Spark datasource configuration properties public static final String CONNECTOR_TYPE = "spark.connector"; public static final String SPARK_SQL_APPLICATION = "spark.sql.application"; @@ -44,7 +43,6 @@ public class SparkStorageFactory implements DataSourceFactory { public static final String EMR_CLUSTER = "emr.cluster"; public static final String EMR_AUTH_TYPE = "emr.auth.type"; public static final String EMR_REGION = "emr.auth.region"; - public static final String EMR_ROLE_ARN = "emr.auth.role_arn"; public static final String EMR_ACCESS_KEY = "emr.auth.access_key"; public static final String EMR_SECRET_KEY = "emr.auth.secret_key"; diff --git a/spark/src/test/java/org/opensearch/sql/spark/data/type/SparkDataTypeTest.java b/spark/src/test/java/org/opensearch/sql/spark/data/type/SparkDataTypeTest.java new file mode 100644 index 0000000000..ff6cee2a5e --- /dev/null +++ b/spark/src/test/java/org/opensearch/sql/spark/data/type/SparkDataTypeTest.java @@ -0,0 +1,19 @@ +/* + * Copyright OpenSearch Contributors + * SPDX-License-Identifier: Apache-2.0 + */ + +package org.opensearch.sql.spark.data.type; + +import static org.junit.jupiter.api.Assertions.assertEquals; + +import org.junit.jupiter.api.Test; + +class SparkDataTypeTest { + @Test + public void testTypeName() { + SparkDataType sparkDataType = new SparkDataType("TYPE_NAME"); + + assertEquals("TYPE_NAME", sparkDataType.typeName()); + } +} diff --git a/spark/src/test/java/org/opensearch/sql/spark/utils/TestUtils.java b/spark/src/test/java/org/opensearch/sql/spark/utils/TestUtils.java index 4cab6afa9c..ca77006d9c 100644 --- a/spark/src/test/java/org/opensearch/sql/spark/utils/TestUtils.java +++ b/spark/src/test/java/org/opensearch/sql/spark/utils/TestUtils.java @@ -5,15 +5,8 @@ package org.opensearch.sql.spark.utils; -import com.google.common.base.Charsets; -import com.google.common.io.Resources; import java.io.IOException; -import java.net.URL; import java.util.Objects; -import lombok.SneakyThrows; -import org.opensearch.action.admin.indices.create.CreateIndexRequest; -import org.opensearch.client.Client; -import org.opensearch.common.xcontent.XContentType; public class TestUtils { @@ -29,17 +22,4 @@ public static String getJson(String filename) throws IOException { return new String( Objects.requireNonNull(classLoader.getResourceAsStream(filename)).readAllBytes()); } - - @SneakyThrows - public static String loadMappings(String path) { - URL url = Resources.getResource(path); - return Resources.toString(url, Charsets.UTF_8); - } - - public static void createIndexWithMappings( - Client client, String indexName, String metadataFileLocation) { - CreateIndexRequest request = new CreateIndexRequest(indexName); - request.mapping(loadMappings(metadataFileLocation), XContentType.JSON); - client.admin().indices().create(request).actionGet(); - } }