@@ -1802,6 +1802,136 @@ class KafkaApisTest extends Logging {
1802
1802
}
1803
1803
}
1804
1804
1805
+ @Test
1806
+ def testInitProducerIdWithEnable2PcFailsWithoutTwoPhaseCommitAcl(): Unit = {
1807
+ val transactionalId = "txnId"
1808
+ addTopicToMetadataCache("topic", numPartitions = 1)
1809
+
1810
+ val initProducerIdRequest = new InitProducerIdRequest.Builder(
1811
+ new InitProducerIdRequestData()
1812
+ .setTransactionalId(transactionalId)
1813
+ .setTransactionTimeoutMs(TimeUnit.MINUTES.toMillis(15).toInt)
1814
+ .setEnable2Pc(true)
1815
+ .setProducerId(RecordBatch.NO_PRODUCER_ID)
1816
+ .setProducerEpoch(RecordBatch.NO_PRODUCER_EPOCH)
1817
+ ).build(6.toShort) // Use version 6 which supports enable2Pc
1818
+
1819
+ val request = buildRequest(initProducerIdRequest)
1820
+ val requestLocal = RequestLocal.withThreadConfinedCaching
1821
+ val authorizer: Authorizer = mock(classOf[Authorizer])
1822
+ kafkaApis = createKafkaApis(authorizer = Some(authorizer))
1823
+
1824
+ // Allow WRITE but deny TWO_PHASE_COMMIT
1825
+ when(authorizer.authorize(
1826
+ any(),
1827
+ ArgumentMatchers.eq(Collections.singletonList(new Action(
1828
+ AclOperation.WRITE,
1829
+ new ResourcePattern(ResourceType.TRANSACTIONAL_ID, transactionalId, PatternType.LITERAL),
1830
+ 1,
1831
+ true,
1832
+ true)))
1833
+ )).thenReturn(Collections.singletonList(AuthorizationResult.ALLOWED))
1834
+
1835
+ when(authorizer.authorize(
1836
+ any(),
1837
+ ArgumentMatchers.eq(Collections.singletonList(new Action(
1838
+ AclOperation.TWO_PHASE_COMMIT,
1839
+ new ResourcePattern(ResourceType.TRANSACTIONAL_ID, transactionalId, PatternType.LITERAL),
1840
+ 1,
1841
+ true,
1842
+ true)))
1843
+ )).thenReturn(Collections.singletonList(AuthorizationResult.DENIED))
1844
+
1845
+ val capturedResponse = ArgumentCaptor.forClass(classOf[InitProducerIdResponse])
1846
+
1847
+ kafkaApis.handleInitProducerIdRequest(request, requestLocal)
1848
+
1849
+ verify(requestChannel).sendResponse(
1850
+ ArgumentMatchers.eq(request),
1851
+ capturedResponse.capture(),
1852
+ ArgumentMatchers.eq(None)
1853
+ )
1854
+
1855
+ assertEquals(Errors.TRANSACTIONAL_ID_AUTHORIZATION_FAILED.code, capturedResponse.getValue.data.errorCode)
1856
+ }
1857
+
1858
+ @Test
1859
+ def testInitProducerIdWithEnable2PcSucceedsWithTwoPhaseCommitAcl(): Unit = {
1860
+ val transactionalId = "txnId"
1861
+ addTopicToMetadataCache("topic", numPartitions = 1)
1862
+
1863
+ val initProducerIdRequest = new InitProducerIdRequest.Builder(
1864
+ new InitProducerIdRequestData()
1865
+ .setTransactionalId(transactionalId)
1866
+ .setTransactionTimeoutMs(TimeUnit.MINUTES.toMillis(15).toInt)
1867
+ .setEnable2Pc(true)
1868
+ .setProducerId(RecordBatch.NO_PRODUCER_ID)
1869
+ .setProducerEpoch(RecordBatch.NO_PRODUCER_EPOCH)
1870
+ ).build(6.toShort) // Use version 6 which supports enable2Pc
1871
+
1872
+ val request = buildRequest(initProducerIdRequest)
1873
+ val requestLocal = RequestLocal.withThreadConfinedCaching
1874
+ val authorizer: Authorizer = mock(classOf[Authorizer])
1875
+ kafkaApis = createKafkaApis(authorizer = Some(authorizer))
1876
+
1877
+ // Both permissions are allowed
1878
+ when(authorizer.authorize(
1879
+ any(),
1880
+ ArgumentMatchers.eq(Collections.singletonList(new Action(
1881
+ AclOperation.WRITE,
1882
+ new ResourcePattern(ResourceType.TRANSACTIONAL_ID, transactionalId, PatternType.LITERAL),
1883
+ 1,
1884
+ true,
1885
+ true)))
1886
+ )).thenReturn(Collections.singletonList(AuthorizationResult.ALLOWED))
1887
+
1888
+ when(authorizer.authorize(
1889
+ any(),
1890
+ ArgumentMatchers.eq(Collections.singletonList(new Action(
1891
+ AclOperation.TWO_PHASE_COMMIT,
1892
+ new ResourcePattern(ResourceType.TRANSACTIONAL_ID, transactionalId, PatternType.LITERAL),
1893
+ 1,
1894
+ true,
1895
+ true)))
1896
+ )).thenReturn(Collections.singletonList(AuthorizationResult.ALLOWED))
1897
+
1898
+ val responseCallback = ArgumentCaptor.forClass(classOf[InitProducerIdResult => Unit])
1899
+
1900
+ when(txnCoordinator.handleInitProducerId(
1901
+ ArgumentMatchers.eq(transactionalId),
1902
+ anyInt(),
1903
+ ArgumentMatchers.eq(true), // enable2Pc = true
1904
+ anyBoolean(),
1905
+ any(),
1906
+ responseCallback.capture(),
1907
+ ArgumentMatchers.eq(requestLocal)
1908
+ )).thenAnswer(_ => responseCallback.getValue.apply(InitProducerIdResult(15L, 0.toShort, Errors.NONE)))
1909
+
1910
+ kafkaApis.handleInitProducerIdRequest(request, requestLocal)
1911
+
1912
+ // Verify coordinator was called with enable2Pc=true
1913
+ verify(txnCoordinator).handleInitProducerId(
1914
+ ArgumentMatchers.eq(transactionalId),
1915
+ anyInt(),
1916
+ ArgumentMatchers.eq(true), // enable2Pc = true
1917
+ anyBoolean(),
1918
+ any(),
1919
+ any(),
1920
+ ArgumentMatchers.eq(requestLocal)
1921
+ )
1922
+
1923
+ val capturedResponse = ArgumentCaptor.forClass(classOf[InitProducerIdResponse])
1924
+ verify(requestChannel).sendResponse(
1925
+ ArgumentMatchers.eq(request),
1926
+ capturedResponse.capture(),
1927
+ ArgumentMatchers.eq(None)
1928
+ )
1929
+
1930
+ assertEquals(Errors.NONE.code, capturedResponse.getValue.data.errorCode)
1931
+ assertEquals(15L, capturedResponse.getValue.data.producerId)
1932
+ assertEquals(0, capturedResponse.getValue.data.producerEpoch)
1933
+ }
1934
+
1805
1935
@Test
1806
1936
def testBatchedAddPartitionsToTxnRequest(): Unit = {
1807
1937
val topic = "topic"
0 commit comments