[
https://issues.apache.org/jira/browse/SPARK-17829?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15610399#comment-15610399
]
Cody Koeninger commented on SPARK-17829:
I'm not telling you to do it that way, just asking
[
https://issues.apache.org/jira/browse/SPARK-17829?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15609653#comment-15609653
]
Cody Koeninger commented on SPARK-17829:
Have you considered using a typeclass?
> Stable for
t;Done reading offsets from ZooKeeper. Took " +
>> (System.currentTimeMillis() - start))
>>
>> Some(offsets)
>> case None =>
>> LogHandler.log.info("No offsets found in ZooKeeper. Took " +
>> (System.currentTimeMillis() - start
I think only supporting 1 version of scala at any given time is not
sufficient, 2 probably is ok.
I.e. don't drop 2.10 before 2.12 is out + supported
On Tue, Oct 25, 2016 at 10:56 AM, Sean Owen wrote:
> The general forces are that new versions of things to support emerge,
Kafka consumers should be backwards compatible with kafka brokers, so
at the very least you should be able to use the
streaming-spark-kafka-0-10 to do what you're talking about.
On Tue, Oct 25, 2016 at 4:30 AM, Prabhu GS wrote:
> Hi,
>
> I would like to know if the same
Cody Koeninger created SPARK-18057:
--
Summary: Update structured streaming kafka from 10.0.1 to 10.1.0
Key: SPARK-18057
URL: https://issues.apache.org/jira/browse/SPARK-18057
Project: Spark
[
https://issues.apache.org/jira/browse/SPARK-18056?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger updated SPARK-18056:
---
Description: There are a couple of relevant KIPs here,
https://archive.apache.org/dist/kafka
Cody Koeninger created SPARK-18056:
--
Summary: Update KafkaDStreams from 10.0.1 to 10.1.0
Key: SPARK-18056
URL: https://issues.apache.org/jira/browse/SPARK-18056
Project: Spark
Issue Type
al.ms can be set differently.
> I'll leave it to you on how to add this to docs!
>
>
> On Thu, Oct 20, 2016 at 1:41 PM, Cody Koeninger <c...@koeninger.org> wrote:
>>
>> Right on, I put in a PR to make a note of that in the docs.
>>
>> On Thu, Oct 20, 2016 at 1
gain, I know what I have to do :)
>
> On Fri, Oct 21, 2016 at 5:05 PM, Cody Koeninger <c...@koeninger.org> wrote:
>>
>> 0. If your processing time is regularly greater than your batch
>> interval you're going to have problems anyway. Investigate this more,
>> set m
0. If your processing time is regularly greater than your batch
interval you're going to have problems anyway. Investigate this more,
set maxRatePerPartition, something.
1. That's personally what I tend to do.
2. Why are you relying on checkpoints if you're storing offset state
in the database?
[
https://issues.apache.org/jira/browse/SPARK-17829?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15593000#comment-15593000
]
Cody Koeninger commented on SPARK-17829:
At least with regard to kafka offsets, it might be good
Cody Koeninger created SPARK-18033:
--
Summary: Deprecate TaskContext.partitionId
Key: SPARK-18033
URL: https://issues.apache.org/jira/browse/SPARK-18033
Project: Spark
Issue Type
think that makes sense, I can start a
ticket.
On Thu, Oct 20, 2016 at 1:16 PM, Reynold Xin <r...@databricks.com> wrote:
> Seems like a good new API to add?
>
>
> On Thu, Oct 20, 2016 at 11:14 AM, Cody Koeninger <c...@koeninger.org> wrote:
>>
>> Access to the par
Access to the partition ID is necessary for basically every single one
of my jobs, and there isn't a foreachPartiionWithIndex equivalent.
You can kind of work around it with empty foreach after the map, but
it's really awkward to explain to people.
On Thu, Oct 20, 2016 at 12:52 PM, Reynold Xin
Right on, I put in a PR to make a note of that in the docs.
On Thu, Oct 20, 2016 at 12:13 PM, Srikanth <srikanth...@gmail.com> wrote:
> Yeah, setting those params helped.
>
> On Wed, Oct 19, 2016 at 1:32 PM, Cody Koeninger <c...@koeninger.org> wrote:
>>
>>
ncy ?
>>
> I think that the fact that they serve as an output trigger is a problem,
> but Structured Streaming seems to resolve this now.
>
>>
>> Thanks
>> Shivaram
>>
>> On Wed, Oct 19, 2016 at 1:29 PM, Michael Armbrust
>> <mich...@databricks
Is anyone seriously thinking about alternatives to microbatches?
On Wed, Oct 19, 2016 at 2:45 PM, Michael Armbrust
wrote:
> Anything that is actively being designed should be in JIRA, and it seems
> like you found most of it. In general, release windows can be found on
Iterator.next(KafkaRDD.scala:193)
> at scala.collection.Iterator$$anon$11.next(Iterator.scala:409)
> at scala.collection.Iterator$$anon$11.next(Iterator.scala:409)
> at scala.collection.Iterator$$anon$21.next(Iterator.scala:838)
>
>
> On Wed, Sep 7, 2016 at 3:55 PM, Cod
[
https://issues.apache.org/jira/browse/SPARK-17147?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15586417#comment-15586417
]
Cody Koeninger commented on SPARK-17147:
If that's something you're seeing regularly, probably
[
https://issues.apache.org/jira/browse/SPARK-17147?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15586397#comment-15586397
]
Cody Koeninger commented on SPARK-17147:
Then no, this issue is unlikely to affect you unless
[
https://issues.apache.org/jira/browse/SPARK-17147?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger updated SPARK-17147:
---
Summary: Spark Streaming Kafka 0.10 Consumer Can't Handle Non-consecutive
Offsets (i.e. Log
+1 to putting docs in one clear place.
On Oct 18, 2016 6:40 AM, "Sean Owen" wrote:
> I'm OK with that. The upside to the wiki is that it can be edited directly
> outside of a release cycle. However, in practice I find that the wiki is
> rarely changed. To me it also serves
[
https://issues.apache.org/jira/browse/SPARK-17147?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15584172#comment-15584172
]
Cody Koeninger commented on SPARK-17147:
Well, are you using compacted topics?
> Spark Stream
SPARK-17841 three line bugfix that has a week old PR
SPARK-17812 being able to specify starting offsets is a must have for
a Kafka mvp in my opinion, already has a PR
SPARK-17813 I can put in a PR for this tonight if it'll be considered
On Mon, Oct 17, 2016 at 12:28 AM, Reynold Xin
P. However I think that Spark should
>> have real-time streaming support. Currently I see many posts/comments
>> that "Spark has too big latency". Spark Streaming is doing very good
>> jobs with micro-batches, however I think it is possible to add also more
>
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15578894#comment-15578894
]
Cody Koeninger commented on SPARK-17812:
As you just said yourself, assign doesn't mean you
[
https://issues.apache.org/jira/browse/SPARK-17935?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15578141#comment-15578141
]
Cody Koeninger commented on SPARK-17935:
Why is this in kafka-0-8, when we haven't resolved
[
https://issues.apache.org/jira/browse/SPARK-17938?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15578133#comment-15578133
]
Cody Koeninger commented on SPARK-17938:
There was pretty extensive discussion of this on list
[
https://issues.apache.org/jira/browse/SPARK-17813?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15577037#comment-15577037
]
Cody Koeninger commented on SPARK-17813:
To be clear, the current direct stream (and as a result
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger updated SPARK-17812:
---
Description:
Right now you can only run a Streaming Query starting from either the earliest
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15577022#comment-15577022
]
Cody Koeninger commented on SPARK-17812:
Assign is useful, otherwise you have no way of consuming
nge[] offsets = ((HasOffsetRanges) rdd.rdd()).offsetRanges();
> CanCommitOffsets cco = (CanCommitOffsets) directKafkaStream.dstream();
> cco.commitAsync(offsets);
>
> I tried setting "max.poll.records" to 1000 but this did not help.
>
> Any idea what could be wrong?
>
[
https://issues.apache.org/jira/browse/SPARK-17937?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger updated SPARK-17937:
---
Description:
Possible events for which offsets are needed:
# New partition is discovered
[
https://issues.apache.org/jira/browse/SPARK-17937?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger updated SPARK-17937:
---
Description:
Possible events for which offsets are needed:
# New partition is discovered
[
https://issues.apache.org/jira/browse/SPARK-17937?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger updated SPARK-17937:
---
Description:
Possible events for which offsets are needed:
# New partition is discovered
[
https://issues.apache.org/jira/browse/SPARK-17937?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger updated SPARK-17937:
---
Description:
Possible events for which offsets are needed:
# New partition is discovered
utor: Exception in task 0.0 in stage 0.0 (TID 0)
> java.lang.AssertionError: assertion failed: Failed to get records for
> spark-executor-null mytopic2 0 2 after polling for 512
> ==
>
> -Original Message-
> From: Cody Koeninger [mailto:c...@koeninger.org]
> Sent: 201
I can't be sure, no.
On Fri, Oct 14, 2016 at 3:06 AM, Julian Keppel
<juliankeppel1...@gmail.com> wrote:
> Okay, thank you! Can you say, when this feature will be released?
>
> 2016-10-13 16:29 GMT+02:00 Cody Koeninger <c...@koeninger.org>:
>>
>> As Sean said, it'
[
https://issues.apache.org/jira/browse/SPARK-17937?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger updated SPARK-17937:
---
Description:
Possible events for which offsets are needed:
# New partition is discovered
[
https://issues.apache.org/jira/browse/SPARK-17937?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger updated SPARK-17937:
---
Description:
Possible events for which offsets are needed:
# New partition is discovered
[
https://issues.apache.org/jira/browse/SPARK-17937?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger updated SPARK-17937:
---
Description:
Possible events for which offsets are needed:
# New partition is discovered
[
https://issues.apache.org/jira/browse/SPARK-17937?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger updated SPARK-17937:
---
Description:
Possible events for which offsets are needed:
# *New partition* is discovered
[
https://issues.apache.org/jira/browse/SPARK-17937?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger updated SPARK-17937:
---
Description:
Possible events for which offsets are needed:
# *New partition* is discovered
[
https://issues.apache.org/jira/browse/SPARK-17937?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger updated SPARK-17937:
---
Description:
Possible events for which offsets are needed:
# New partition is discovered
[
https://issues.apache.org/jira/browse/SPARK-17937?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger updated SPARK-17937:
---
Description:
Possible events for which offsets are needed:
# New partition is discovered
Cody Koeninger created SPARK-17937:
--
Summary: Clarify Kafka offset semantics for Structured Streaming
Key: SPARK-17937
URL: https://issues.apache.org/jira/browse/SPARK-17937
Project: Spark
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15573843#comment-15573843
]
Cody Koeninger commented on SPARK-17812:
So I think this is what we're agreed on:
Mutually
[
https://issues.apache.org/jira/browse/SPARK-17813?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15573806#comment-15573806
]
Cody Koeninger commented on SPARK-17813:
So issues to be worked out here (assuming we're still
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15573766#comment-15573766
]
Cody Koeninger commented on SPARK-17812:
OK, failing on start is clear (it's really annoying
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15573563#comment-15573563
]
Cody Koeninger commented on SPARK-17812:
So a short term question
- with your proposed interface
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15573479#comment-15573479
]
Cody Koeninger commented on SPARK-17812:
While some decision is better than none, can you help me
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15573432#comment-15573432
]
Cody Koeninger edited comment on SPARK-17812 at 10/13/16 10:44 PM
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15573432#comment-15573432
]
Cody Koeninger commented on SPARK-17812:
If you're seriously worried that people are going to get
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15573395#comment-15573395
]
Cody Koeninger edited comment on SPARK-17812 at 10/13/16 10:25 PM:
---
1
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15573395#comment-15573395
]
Cody Koeninger commented on SPARK-17812:
1. we dont have lists, we have strings. regexes
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger updated SPARK-17812:
---
Comment: was deleted
(was: One other slightly ugly thing...
{noformat}
// starting
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15573166#comment-15573166
]
Cody Koeninger edited comment on SPARK-17812 at 10/13/16 9:17 PM
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15573166#comment-15573166
]
Cody Koeninger commented on SPARK-17812:
Here's my concrete suggestion:
3 mutually exclusive
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15572922#comment-15572922
]
Cody Koeninger edited comment on SPARK-17812 at 10/13/16 8:33 PM:
--
Sorry
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15573089#comment-15573089
]
Cody Koeninger commented on SPARK-17812:
One other slightly ugly thing...
{noformat}
// starting
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15572922#comment-15572922
]
Cody Koeninger commented on SPARK-17812:
Sorry, I didn't see this comment until just now.
X
I've always been confused as to why it would ever be a good idea to
put any streaming query system on the critical path for synchronous <
100msec requests. It seems to make a lot more sense to have a
streaming system do asynch updates of a store that has better latency
and quality of service
[
https://issues.apache.org/jira/browse/FLINK-3037?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15572133#comment-15572133
]
Cody Koeninger commented on FLINK-3037:
---
As a user, I want to be able to start low-value, high
[
https://issues.apache.org/jira/browse/SPARK-17900?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15572119#comment-15572119
]
Cody Koeninger commented on SPARK-17900:
Thanks for doing this, should make things clearer
fka.maxRatePerPartition", "10")
> conf.set("spark.streaming.backpressure.enabled", "true")
>
> That's not normal, is it? Do you notice anything odd in my logs?
>
> Thanks a lot.
>
>
>
> On 10/12/2016 07:31 PM, Cody Koeninger w
As Sean said, it's unreleased. If you want to try it out, build spark
http://spark.apache.org/docs/latest/building-spark.html
The easiest way to include the jar is probably to use mvn install to
put it in your local repository, then link it in your application's
mvn or sbt build file as
response.
>
>
>
> So Kafka direct stream actually has consumer on both the driver and
> executor? Can you please provide more details? Thank you very much!
>
>
>
> ________
>
> From: Cody Koeninger [mailto:c...@koeninger.org]
> Sent:
[
https://issues.apache.org/jira/browse/SPARK-15408?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger closed SPARK-15408.
--
Resolution: Cannot Reproduce
> Spark streaming app crashes with NotLeaderForPartitionExcept
[
https://issues.apache.org/jira/browse/SPARK-15272?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15570221#comment-15570221
]
Cody Koeninger commented on SPARK-15272:
Checking to see if the 0.10 consumer's handling
[
https://issues.apache.org/jira/browse/SPARK-15272?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15570221#comment-15570221
]
Cody Koeninger edited comment on SPARK-15272 at 10/12/16 11:33 PM
[
https://issues.apache.org/jira/browse/SPARK-11698?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15570208#comment-15570208
]
Cody Koeninger commented on SPARK-11698:
Would a custom ConsumerStrategy for the new consumer
[
https://issues.apache.org/jira/browse/SPARK-10320?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger resolved SPARK-10320.
Resolution: Fixed
Fix Version/s: 2.0.0
SPARK-12177 added the new consumer, which
[
https://issues.apache.org/jira/browse/SPARK-9947?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger closed SPARK-9947.
-
Resolution: Won't Fix
The direct DStream api already gives access to offsets, and it seems clear
[
https://issues.apache.org/jira/browse/SPARK-8337?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15570186#comment-15570186
]
Cody Koeninger commented on SPARK-8337:
---
Can this be closed, given that the subtasks are resolved
[
https://issues.apache.org/jira/browse/SPARK-5505?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger closed SPARK-5505.
-
Resolution: Won't Fix
The old kafka High Level Consumer has been abandoned at this point.
SPARK
[
https://issues.apache.org/jira/browse/SPARK-5718?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger resolved SPARK-5718.
---
Resolution: Fixed
Fix Version/s: 2.0.0
SPARK-12177 added support for the native kafka
[
https://issues.apache.org/jira/browse/SPARK-10815?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15570139#comment-15570139
]
Cody Koeninger commented on SPARK-10815:
Another unfortunate thing about the Sink api
rg/e730492453.png
notice the cutover point
On Wed, Oct 12, 2016 at 11:00 AM, Samy Dindane <s...@dindane.com> wrote:
> I am 100% sure.
>
> println(conf.get("spark.streaming.backpressure.enabled")) prints true.
>
>
> On 10/12/2016 05:48 PM, Cody Koeninger w
gt; Do you have any idea about why aren't backpressure working? How to debug
> this?
>
>
> On 10/11/2016 06:08 PM, Cody Koeninger wrote:
>>
>> http://spark.apache.org/docs/latest/configuration.html
>>
>> "This rate is upper bounded by the values
>&g
, are you talking about
> repartition using partitionBy()?
>
>
> On 10/11/2016 01:23 AM, Cody Koeninger wrote:
>>
>> Repartition almost always involves a shuffle.
>>
>> Let me see if I can explain the recovery stuff...
>>
>> Say you start with two kafka part
its set to none for the executors, because otherwise they wont do exactly
what the driver told them to do.
you should be able to set up the driver consumer to determine batches
however you want, though.
On Wednesday, October 12, 2016, Haopu Wang wrote:
> Hi,
>
>
>
> I want
[
https://issues.apache.org/jira/browse/SPARK-17344?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15567457#comment-15567457
]
Cody Koeninger commented on SPARK-17344:
Given the choice between rewriting underlying kafka
[
https://issues.apache.org/jira/browse/SPARK-17837?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Cody Koeninger closed SPARK-17837.
--
Resolution: Duplicate
Duplicate of SPARK-17829
> Disaster recovery of offsets from
[
https://issues.apache.org/jira/browse/FLINK-4280?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15567263#comment-15567263
]
Cody Koeninger commented on FLINK-4280:
---
FLINK-3123 would be pretty important to me, happy to help
[
https://issues.apache.org/jira/browse/SPARK-17344?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15566244#comment-15566244
]
Cody Koeninger commented on SPARK-17344:
How long would it take CDH to distribute 0.10
http://spark.apache.org/docs/latest/configuration.html
"This rate is upper bounded by the values
spark.streaming.receiver.maxRate and
spark.streaming.kafka.maxRatePerPartition if they are set (see
below)."
On Tue, Oct 11, 2016 at 10:57 AM, Samy Dindane wrote:
> Hi,
>
> Is it
involving mapWithState
> of course :) I'm just wondering why it doesn't support this use case yet.
>
> On Tue, Oct 11, 2016 at 3:41 PM, Cody Koeninger <c...@koeninger.org> wrote:
>>
>> They're telling you not to use the old function because it's linear on the
>> total
[
https://issues.apache.org/jira/browse/SPARK-17853?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15565559#comment-15565559
]
Cody Koeninger commented on SPARK-17853:
Good, will keep this ticket open at least until
They're telling you not to use the old function because it's linear on the
total number of keys, not keys in the batch, so it's slow.
But if that's what you really want, go ahead and do it, and see if it
performs well enough.
On Oct 11, 2016 6:28 AM, "DandyDev" wrote:
Hi
it works. Thanks!
Although changing this is ok for us, i am interested in the why :-) With
the old connector this was not a problem nor is it afaik with the pure
kafka consumer api
2016-10-11 14:30 GMT+02:00 Cody Koeninger <c...@koeninger.org>:
> Just out of curiosity, have you tried
[
https://issues.apache.org/jira/browse/SPARK-17853?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15565400#comment-15565400
]
Cody Koeninger commented on SPARK-17853:
Use a different group id.
Let me know if that addresses
c: String):
>> InputDStream[ConsumerRecord[String, Bytes]] = {
>> KafkaUtils.createDirectStream[String, Bytes](ssc,
>> LocationStrategies.PreferConsistent, ConsumerStrategies.Subscribe[String,
>> Bytes](Set(topic), kafkaParams))
>> }
>>
>>
ync(offsets);
>
> java.lang.ClassCastException:
> org.apache.spark.streaming.api.java.JavaInputDStream
> cannot be cast to org.apache.spark.streaming.kafka010.CanCommitOffsets
> at SparkTest.lambda$0(SparkTest.java:103)
>
> Best regards,
> Max
>
>
> 2016-10-10 20:18 GMT+02:00 Cody Koeninger <c...@koeninger.org>:
[
https://issues.apache.org/jira/browse/SPARK-17853?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15565278#comment-15565278
]
Cody Koeninger commented on SPARK-17853:
Which version of DStream are you using, 0-10 or 0-8
, starting
at topic-0 offsets 60, topic-1 offsets 66
Clear as mud?
On Mon, Oct 10, 2016 at 5:36 PM, Samy Dindane <s...@dindane.com> wrote:
>
>
> On 10/10/2016 8:14 PM, Cody Koeninger wrote:
>>
>> Glad it was helpful :)
>>
>> As far as executors, my e
ul, because we have run into some trouble in the past
> with some inside the ASF but essentially outside the Spark community who
> didn't like the way we were doing things.
>
> On Mon, Oct 10, 2016 at 3:53 PM, Cody Koeninger <c...@koeninger.org> wrote:
>>
>> Apache documents s
[
https://issues.apache.org/jira/browse/SPARK-17812?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=15563285#comment-15563285
]
Cody Koeninger commented on SPARK-17812:
No, it's not covered by strict assign. If you don't
ng lots of rules
>> from the beginning makes it confusing and can reduce contributions.
>> Although, as engineers, we believe that anything can be solved using
>> mechanical rules, in practice software development is a social process that
>> ultimately requires humans to tackle
community effort and I wouldn't want to move forward if up to half of the
> community thinks it's an untenable idea.
>
> rb
>
> On Mon, Oct 10, 2016 at 12:07 PM, Cody Koeninger <c...@koeninger.org> wrote:
>>
>> I think this is closer to a procedural issue than a code mod
301 - 400 of 1347 matches
Mail list logo