Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

KAFKA-14812: ProducerPerformance still counting successful sending in … #13404

Merged
merged 1 commit into from Mar 21, 2023

Conversation

hudeqi
Copy link
Collaborator

@hudeqi hudeqi commented Mar 16, 2023

When using ProducerPerformance, I found that when the sending fails, it is still counted as successfully sent by stat and the metrics are printed in console. For example, when there is no write permission and cannot be written in, the sending success rate is still magically displayed.

absolute jira

@hudeqi
Copy link
Collaborator Author

hudeqi commented Mar 16, 2023

About issues in #13348 , my solution is: remove the use of the iteration variable of stats from the main thread, the read and write it completely by the callback of a separate producer thread, so that the previous problems will not occur. According to the test case given by @robobario, I Tried it several times and it seems to be ok. Thank you for reviewing again, thank you. @showuon @chia7712

Copy link
Contributor

@chia7712 chia7712 left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@hudeqi thanks for updating the solution. one small comment left. PTAL

@chia7712
Copy link
Contributor

@robobario please take a look if you have free cycle.

@robobario
Copy link
Contributor

robobario commented Mar 16, 2023

@robobario please take a look if you have free cycle.

LGTM. If you expose the internals of Stats a bit more we could test it with something like:

    @Test
    public void testStatsThreading() throws Exception {
        ExecutorService singleThreaded = Executors.newSingleThreadExecutor();
        final int numRecords = 1000000;
        ProducerPerformance.Stats stats = new ProducerPerformance.Stats(numRecords, 5000);
        for (int i = 0; i < numRecords; i++) {
            final Callback callback = stats.nextCompletion(0, 100);
            CompletableFuture.runAsync(() -> {
                callback.onCompletion(null, null);
            }, singleThreaded);
        }

        singleThreaded.shutdown();
        final boolean success = singleThreaded.awaitTermination(60, TimeUnit.SECONDS);
        assertTrue(success, "should have terminated");

        assertEquals(numRecords, stats.totalCount());
        assertEquals(numRecords, stats.iteration());
        assertEquals(500000, Arrays.stream(stats.latencies()).filter(value -> value != 0).count());
    }

@hudeqi
Copy link
Collaborator Author

hudeqi commented Mar 17, 2023

Thank you both for your valuable comments! I have added optimization. @chia7712 @robobario

@hudeqi
Copy link
Collaborator Author

hudeqi commented Mar 17, 2023

Do you have any suggestions here? @showuon

@hudeqi hudeqi requested a review from chia7712 March 20, 2023 06:08
@chia7712
Copy link
Contributor

[2023-03-17T04:47:45.614Z] BUILD SUCCESSFUL in 1h 56m 35s
[2023-03-17T04:47:45.614Z] 224 actionable tasks: 120 executed, 104 up-to-date

tests pass. will merge it later

Copy link
Contributor

@chia7712 chia7712 left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

LGTM

@chia7712 chia7712 merged commit aef004e into apache:trunk Mar 21, 2023
@hudeqi hudeqi deleted the hdq_fix_perf branch March 21, 2023 09:19
Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment
Labels
None yet
Projects
None yet
3 participants