Re: [Discuss] FLIP-26 - SSL Mutual Authentication

2018-07-27 Thread Eron Wright
 As an update to this thread, Stephan opted to split the internal/external
configuration (by providing overrides for a common SSL configuration):
https://github.com/apache/flink/pull/6326

Note that Akka doesn't support hostname verification in its 'classic'
remoting implementation (though the new Artery implementation apparently
does), and such verification wouldn't apply to the client certificate
anyway.   So the reality is that one should use a limited truststore (never
the system truststore) for Akka communication.

On the question of routing external communication thru the YARN resource
proxy or Mesos/DCOS admin router, the value proposition is:
a) simplifies service discovery on the part of external clients,
b) permits single sign-on (SSO) be delegating authentication to a central
authority,
c) facilitates access from outside the cluster, via a public address.
The main challenge is that the Flink client code must support a more
diverse array of authentication methods, e.g. Kerberos when communicating
with the YARN proxy.

Given #6326, the next steps would be (unordered):
a) create an umbrella issue for the overall effort
b) dive into the authorization work for external communication
c) implement auto-generation of a certificate for internal communication
d) implement TLS on queryable state interface (FLINK-5029)

I'll take care of (a) unless there is any objection.
-Eron


On Sun, May 13, 2018 at 5:45 AM Stephan Ewen  wrote:

> Throwing in some more food for thought:
>
> An alternative to the above proposed separation of internal and external
> SSL would be the following:
>
>   - We separate channel encryption and authentication
>   - We use one common SSL layer (internal and external) that is in both
> cases only responsible for establishing an encrypted connection
>   - Authentication / authorization internally is done by SASL with
> username/password or shared secret.
>   - Authentication externally must be through a proxy and authorization
> based on a validating HTTP headers set by the proxy, as discussed above..
>
> Advantages:
>   - There is only one certificate needed, which could also be shared across
> applications
>   - One or two lines in the config authenticate and authorize internal
> communication
>   - One could possibly still fall back to the other mode by skipping
>
> Open Questions / Disadvantages
>   - Given that hostname verification during SSL handshake is not possible
> in many setups, the encrypted channel is vulnerable to man-in-the-middle
> attacks without mutual authentication. Not sure how serious that is,
> because it would need an attacker to have compromise network nodes of the
> cluster already. is that not a universal issue in the K8s world?
>
> This is anyways a bit hypothetical, because as long as we have akka beneath
> the RPC layer, we cannot go with that approach.
>
> However, if we want to at least keep the door open towards something like
> that in the future, we would need to set up configuration in such a way
> that we have a "common SSL" configuration (keystore, truststore, etc.) and
> internal/external options that override those. That would anyways be
> helpful for backwards compatibility.
>
> @Eron - what are your thoughts on that?
>
>
>
>
>
>
>
>
> On Sun, May 13, 2018 at 1:40 AM, Stephan Ewen 
> wrote:
>
> > Thank you for bringing this proposal up. It looks very good and we seem
> to
> > be thinking along very similar lines.
> >
> > Below are some comments and thoughts on the FLIP.
> >
> > *Internal vs. External Connectivity*
> >
> > That is a very helpful distinction, let's build on that.
> >
> >   - I would suggest to treat eventually all communication coming
> > potentially from users as external, meaning Client-to-Dispatcher,
> > Client-to-JobManager (trigger savepoint, change parallelism, ...), Web
> UI,
> > Queryable State.
> >
> >   - That leaves communication that is only between
> JobManager/TaskManager/
> > ResourceManager/Dispatcher/HistoryServer as internal.
> >
> >   - I am somewhat operating under the assumption that all external
> > communication will eventually be HTTP/REST. That works best with many
> > setups and is the basis for using service proxies that
> > handle  authentication/authorization.
> >
> >
> > In Flink 1.5 and future versions, we have the following update there:
> >
> >   - Akka is now strictly internal connectivity, the client (except legacy
> > client) do not use it any more.
> >
> >   - The Blob Server will move to purely internal connectivity in Flink
> > 1.6, where a POST of a job to the Dispatcher has the jars and the
> JobGraph.
> > That is important for Kubernetes setups, where exposing the BlobServer
> and
> > querying the blob port causes quite some friction.
> >
> >   - Treating queryable state as "internal connectivity" is fine for now.
> > We should treat it as "external" connectivity in the future if we move it
> > to HTTP/REST.
> >
> >
> > *Internal Connectivity and SSL Mutual Authentication*
> >
> > 

[jira] [Created] (FLINK-9988) job manager does not respect property jobmanager.web.address

2018-07-27 Thread Pavlo Petrychenko (JIRA)
Pavlo Petrychenko created FLINK-9988:


 Summary:   job manager does not respect property 
jobmanager.web.address
 Key: FLINK-9988
 URL: https://issues.apache.org/jira/browse/FLINK-9988
 Project: Flink
  Issue Type: Bug
Affects Versions: 1.5.1, 1.5.0
Reporter: Pavlo Petrychenko


As flink does not have any built in authentication mechanism, we used to setup 
nginx in front of it and start jobmanager on 127.0.0.1.

but starting from version 1.5.0 - it does not work anymore.

distespecting on jobmanager.web.address it always start on 0.0.0.0



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


[jira] [Created] (FLINK-9987) Rework ClassLoaderITCase to not rely on .version.properties file

2018-07-27 Thread Chesnay Schepler (JIRA)
Chesnay Schepler created FLINK-9987:
---

 Summary: Rework ClassLoaderITCase to not rely on 
.version.properties file
 Key: FLINK-9987
 URL: https://issues.apache.org/jira/browse/FLINK-9987
 Project: Flink
  Issue Type: Improvement
  Components: Tests
Affects Versions: 1.5.0, 1.6.0
Reporter: Chesnay Schepler
Assignee: Chesnay Schepler
 Fix For: 1.6.0


The classloading end-to-end test makes use of the {{.version.properties}} file 
that is bundled with flink-runtime to ensure classloading works properly.

In particular it access the {{git.remote.origin.url}} property.

This should be changed since this property isn't stable across builds, which 
results in test failures when running the e2e tests against a release that was 
built on a different machine.

The test furthermore relies on this file existing in the source flink-runtime 
module, which is a dangerous assumption as this requires flink-runtime to be 
built beforehand. It is also _yet another_ instance where we navigate out of a 
module which should never be done.



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


[jira] [Created] (FLINK-9986) Remove remote from .version.properties file

2018-07-27 Thread Chesnay Schepler (JIRA)
Chesnay Schepler created FLINK-9986:
---

 Summary: Remove remote from .version.properties file
 Key: FLINK-9986
 URL: https://issues.apache.org/jira/browse/FLINK-9986
 Project: Flink
  Issue Type: Improvement
  Components: Build System
Affects Versions: 1.5.1, 1.4.2, 1.6.0
Reporter: Chesnay Schepler
Assignee: Chesnay Schepler
 Fix For: 1.6.0


To log the revision {{flink-runtime}} creates a {{.version.properties}} file 
using the {{git-commit-id-plugin}} that is stored within the jar.

Here's an example:
{code}
git.commit.id.abbrev=1a9b648
git.commit.user.email=ches...@apache.org
git.commit.message.full=Commit for release 1.5.2\n
git.commit.id=1a9b6486a2d268d4fb8282c32d65fcc701d18e42
git.commit.message.short=Commit for release 1.5.2
git.commit.user.name=zentol
git.build.user.name=zentol
git.build.user.email=ches...@apache.org
git.branch=1a9b6486a2d268d4fb8282c32d65fcc701d18e42
git.commit.time=25.07.2018 @ 17\:10\:13 GMT
git.build.time=25.07.2018 @ 20\:47\:15 GMT
git.remote.origin.url=https\://github.com/zentol/flink.git
{code}

most of this information isn't used, as flink-runtime only access 
{{git.commit.id.abbrev}} and {{git.commit.time}}.
The build, remote and branch information should be removed as they are neither 
relevant, nor consistent, as releases can be created on any branch, under any 
git alias, against any remote.

To exclude properties we have to bump the plugin version to 2.1.9.



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


Re: Access generic pojo fields

2018-07-27 Thread Amol S - iProgrammer
Thanks Timo,

custom function worked for me with no further exceptions,

Thanks.



---
*Amol Suryawanshi*
Java Developer
am...@iprogrammer.com


*iProgrammer Solutions Pvt. Ltd.*



*Office 103, 104, 1st Floor Pride Portal,Shivaji Housing Society,
Bahiratwadi,Near Hotel JW Marriott, Off Senapati Bapat Road, Pune - 411016,
MH, INDIA.**Phone: +91 9689077510 | Skype: amols_iprogrammer*
www.iprogrammer.com 


On Fri, Jul 27, 2018 at 6:10 PM, Timo Walther  wrote:

> I tried to reproduce your error but everything worked fine. Which Flink
> version are you using?
>
> Inner joins are a Flink 1.5 feature.
>
>
> Am 27.07.18 um 13:28 schrieb Amol S - iProgrammer:
>
> Table master = table1.filter("ns === 'Master'").select("o as master,
>> 'accessBasicDBObject(applicationId,o)' as primaryKey");
>> Table child1 = table1.filter("ns === 'Child1'").select("o  as child1,
>> 'accessBasicDBObject(applicationId,o)' as foreignKey");
>> Table child2 = table1.filter("ns === 'Child2'").select("o  as child2,
>> 'accessBasicDBObject(applicationId,o)' as foreignKey2");
>>
>> Table result = master.join(child1).where("pri
>> maryKey==foreignKey").join(child2).where("primaryKey==foreignKey2");
>>
>
>
>


[jira] [Created] (FLINK-9985) Incorrect parameter order in document

2018-07-27 Thread zhangminglei (JIRA)
zhangminglei created FLINK-9985:
---

 Summary: Incorrect parameter order in document
 Key: FLINK-9985
 URL: https://issues.apache.org/jira/browse/FLINK-9985
 Project: Flink
  Issue Type: Bug
  Components: Documentation
Affects Versions: 1.5.1
Reporter: zhangminglei






--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


[jira] [Created] (FLINK-9984) Add a byte array table format factory

2018-07-27 Thread Timo Walther (JIRA)
Timo Walther created FLINK-9984:
---

 Summary: Add a byte array table format factory
 Key: FLINK-9984
 URL: https://issues.apache.org/jira/browse/FLINK-9984
 Project: Flink
  Issue Type: Sub-task
  Components: Table API  SQL
Reporter: Timo Walther


Sometimes it might be useful to just read or write a plain byte array into 
Kafka or other connectors. We should add a simple byte array 
SerializationSchemaFactory and DeserializationSchemaFactory.

See {{org.apache.flink.formats.json.JsonRowFormatFactory}} for a reference.



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


Re: Access generic pojo fields

2018-07-27 Thread Timo Walther
I tried to reproduce your error but everything worked fine. Which Flink 
version are you using?


Inner joins are a Flink 1.5 feature.


Am 27.07.18 um 13:28 schrieb Amol S - iProgrammer:

Table master = table1.filter("ns === 'Master'").select("o as master,
'accessBasicDBObject(applicationId,o)' as primaryKey");
Table child1 = table1.filter("ns === 'Child1'").select("o  as child1,
'accessBasicDBObject(applicationId,o)' as foreignKey");
Table child2 = table1.filter("ns === 'Child2'").select("o  as child2,
'accessBasicDBObject(applicationId,o)' as foreignKey2");

Table result = 
master.join(child1).where("primaryKey==foreignKey").join(child2).where("primaryKey==foreignKey2");





[jira] [Created] (FLINK-9983) Savepoints should count as checkpoints when recovering

2018-07-27 Thread Aljoscha Krettek (JIRA)
Aljoscha Krettek created FLINK-9983:
---

 Summary: Savepoints should count as checkpoints when recovering
 Key: FLINK-9983
 URL: https://issues.apache.org/jira/browse/FLINK-9983
 Project: Flink
  Issue Type: Bug
  Components: State Backends, Checkpointing
Reporter: Aljoscha Krettek
 Fix For: 1.6.0


If they are not used when recovering you can get intro problems with duplicate 
output data when a failure occurs after a savepoint was taken but before the 
next checkpoint occurs.

The fix, in the long run, is to differentiate between savepoints that have side 
effects and those that don't. The former would be used for a 
"stop-with-savepoint" scenario while the latter is for in-between savepoints. 
This is harder to implement, so I vote for the easy fix described for now for 
fixing the duplication problems.



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


Re: Access generic pojo fields

2018-07-27 Thread Amol S - iProgrammer
Hello Timo,

I have implemented my own scalar function as below

public class AccessBasicDBObject extends ScalarFunction {

public String eval(String key, BasicDBObject basicDBObject) {
if (basicDBObject.getString(key) != null)
return basicDBObject.getString(key);
else return "";
}

@Override
public TypeInformation getResultType(Class[] signature) {
return Types.STRING;
}
}

Table master = table1.filter("ns === 'Master'").select("o as master,
'accessBasicDBObject(applicationId,o)' as primaryKey");
Table child1 = table1.filter("ns === 'Child1'").select("o  as child1,
'accessBasicDBObject(applicationId,o)' as foreignKey");
Table child2 = table1.filter("ns === 'Child2'").select("o  as child2,
'accessBasicDBObject(applicationId,o)' as foreignKey2");

Table result = 
master.join(child1).where("primaryKey==foreignKey").join(child2).where("primaryKey==foreignKey2");

and it is giving me following error on line DataStream rowDataStream =
tableEnv.toDataStream(result, Row.class);

Exception in thread "Thread-27" org.apache.flink.table.api.TableException:
Cannot generate a valid execution plan for the given query:

LogicalJoin(condition=[true], joinType=[inner])
  LogicalJoin(condition=[true], joinType=[inner])
LogicalProject(master=[$2], primaryKey=[_UTF-16LE'accessBasicDBObject(
loanApplicationId,o)'])
  LogicalFilter(condition=[=($1, _UTF-16LE'analyticDB.
customerMISMaster')])
LogicalTableScan(table=[[_DataStreamTable_0]])
LogicalProject(child1=[$2], foreignKey=[_UTF-16LE'accessBasicDBObject(
loanApplicationId,o)'])
  LogicalFilter(condition=[=($1, _UTF-16LE'analyticDB.
customerMISChild1')])
LogicalTableScan(table=[[_DataStreamTable_0]])
  LogicalProject(child2=[$2], foreignKey2=[_UTF-16LE'accessBasicDBObject(
loanApplicationId,o)'])
LogicalFilter(condition=[=($1, _UTF-16LE'analyticDB.
customerMISChild2')])
  LogicalTableScan(table=[[_DataStreamTable_0]])

This exception indicates that the query uses an unsupported SQL feature.
Please check the documentation for the set of currently supported SQL
features.
at org.apache.flink.table.api.TableEnvironment.runVolcanoPlanner(
TableEnvironment.scala:274)
at org.apache.flink.table.api.StreamTableEnvironment.optimize(
StreamTableEnvironment.scala:722)
at org.apache.flink.table.api.StreamTableEnvironment.translate(
StreamTableEnvironment.scala:778)
at org.apache.flink.table.api.java.StreamTableEnvironment.toAppendStream(
StreamTableEnvironment.scala:308)
at org.apache.flink.table.api.java.StreamTableEnvironment.toAppendStream(
StreamTableEnvironment.scala:262)
at org.apache.flink.table.api.java.StreamTableEnvironment.toDataStream(
StreamTableEnvironment.scala:159)
at com.softcell.streaming.flink.StreamingOperations$2.run(
StreamingOperations.java:168)

---
*Amol Suryawanshi*
Java Developer
am...@iprogrammer.com


*iProgrammer Solutions Pvt. Ltd.*



*Office 103, 104, 1st Floor Pride Portal,Shivaji Housing Society,
Bahiratwadi,Near Hotel JW Marriott, Off Senapati Bapat Road, Pune - 411016,
MH, INDIA.**Phone: +91 9689077510 | Skype: amols_iprogrammer*
www.iprogrammer.com 


On Fri, Jul 27, 2018 at 3:13 PM, Timo Walther  wrote:

> Hi,
>
> I think the exception is self-explaining. BasicDBObject is not recognized
> as a POJO by Flink. A POJO is required such that the Table API knows the
> types of fields for following operations.
>
> The easiest way is to implement your own scalar function. E.g. a
> `accessBasicDBObject(obj, key)`.
>
> Regards,
> Timo
>
>
> Am 27.07.18 um 11:25 schrieb Amol S - iProgrammer:
>
>> Hello Timo,
>>
>> Thanks for quick reply. By using your suggestion Previous exception gone
>> but it is giving me following exception
>>
>> Expression 'o.get(_id) failed on input check: Cannot access field of
>> non-composite type 'GenericType'.
>>
>> ---
>> *Amol Suryawanshi*
>> Java Developer
>> am...@iprogrammer.com
>>
>>
>> *iProgrammer Solutions Pvt. Ltd.*
>>
>>
>>
>> *Office 103, 104, 1st Floor Pride Portal,Shivaji Housing Society,
>> Bahiratwadi,Near Hotel JW Marriott, Off Senapati Bapat Road, Pune -
>> 411016,
>> MH, INDIA.**Phone: +91 9689077510 | Skype: amols_iprogrammer*
>> www.iprogrammer.com 
>> 
>>
>> On Fri, Jul 27, 2018 at 1:08 PM, Timo Walther  wrote:
>>
>> Hi Amol,
>>>
>>> the dot operation is reserved for calling functions on fields. If you
>>> want
>>> to get a nested field in the Table API, use the `.get("applicationId")`
>>> operation. See also [1] under "Value access functions".
>>>
>>> Regards,
>>> Timo
>>>
>>> [1] https://ci.apache.org/projects/flink/flink-docs-release-1.5/
>>> dev/table/tableApi.html#built-in-functions
>>>
>>>
>>> Am 27.07.18 um 09:10 schrieb Amol S - iProgrammer:
>>>
>>> Hello Fabian,

 I am streaming my mongodb oplog using flink and want to 

[jira] [Created] (FLINK-9982) NPE in EnumValueSerializer#copy

2018-07-27 Thread zhangminglei (JIRA)
zhangminglei created FLINK-9982:
---

 Summary: NPE in EnumValueSerializer#copy
 Key: FLINK-9982
 URL: https://issues.apache.org/jira/browse/FLINK-9982
 Project: Flink
  Issue Type: Bug
Reporter: zhangminglei






--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


Re: Access generic pojo fields

2018-07-27 Thread Timo Walther

Hi,

I think the exception is self-explaining. BasicDBObject is not 
recognized as a POJO by Flink. A POJO is required such that the Table 
API knows the types of fields for following operations.


The easiest way is to implement your own scalar function. E.g. a 
`accessBasicDBObject(obj, key)`.


Regards,
Timo


Am 27.07.18 um 11:25 schrieb Amol S - iProgrammer:

Hello Timo,

Thanks for quick reply. By using your suggestion Previous exception gone
but it is giving me following exception

Expression 'o.get(_id) failed on input check: Cannot access field of
non-composite type 'GenericType'.

---
*Amol Suryawanshi*
Java Developer
am...@iprogrammer.com


*iProgrammer Solutions Pvt. Ltd.*



*Office 103, 104, 1st Floor Pride Portal,Shivaji Housing Society,
Bahiratwadi,Near Hotel JW Marriott, Off Senapati Bapat Road, Pune - 411016,
MH, INDIA.**Phone: +91 9689077510 | Skype: amols_iprogrammer*
www.iprogrammer.com 


On Fri, Jul 27, 2018 at 1:08 PM, Timo Walther  wrote:


Hi Amol,

the dot operation is reserved for calling functions on fields. If you want
to get a nested field in the Table API, use the `.get("applicationId")`
operation. See also [1] under "Value access functions".

Regards,
Timo

[1] https://ci.apache.org/projects/flink/flink-docs-release-1.5/
dev/table/tableApi.html#built-in-functions


Am 27.07.18 um 09:10 schrieb Amol S - iProgrammer:


Hello Fabian,

I am streaming my mongodb oplog using flink and want to use flink table
API
to join multiple tables.  My code looks like

DataStream streamSource = env
  .addSource(kafkaConsumer)
  .setParallelism(4);

StreamTableEnvironment tableEnv = TableEnvironment.getTableEnvir
onment(env);
// Convert the DataStream into a Table with default fields "f0", "f1"
Table table1 = tableEnv.fromDataStream(streamSource);

Table master = table1.filter("ns === 'Master'").select("o as master,
o.applicationId as primaryKey");
Table child1 = table1.filter("ns === 'Child1'").select("o  as child1,
o.applicationId as foreignKey");
Table child2 = table1.filter("ns === 'Child2'").select("o  as child2,
o.applicationId as foreignKey2");

Table result = master.join(child1).where("pri
maryKey==foreignKey").join(child2).where("primaryKey==foreignKey2");

it is throwing error "Method threw
'org.apache.flink.table.api.ValidationException' exception. Undefined
function: APPLICATIONID"

public class Oplog implements Serializable{
  private BasicDBObject o;
}

Where o is generic java type for fetching mongodb oplog and I can not
replace this generic type with static pojo's. please tell me any work
around on this.

BasicDBObject suffice following two rules


 -

 The class must be public.
 -

 It must have a public constructor without arguments (default
constructor)

and we can access class members through basicDBObject.getString("abc")




---
*Amol Suryawanshi*
Java Developer
am...@iprogrammer.com


*iProgrammer Solutions Pvt. Ltd.*



*Office 103, 104, 1st Floor Pride Portal,Shivaji Housing Society,
Bahiratwadi,Near Hotel JW Marriott, Off Senapati Bapat Road, Pune -
411016,
MH, INDIA.**Phone: +91 9689077510 | Skype: amols_iprogrammer*
www.iprogrammer.com 







Re: Access generic pojo fields

2018-07-27 Thread Amol S - iProgrammer
Hello Timo,

Thanks for quick reply. By using your suggestion Previous exception gone
but it is giving me following exception

Expression 'o.get(_id) failed on input check: Cannot access field of
non-composite type 'GenericType'.

---
*Amol Suryawanshi*
Java Developer
am...@iprogrammer.com


*iProgrammer Solutions Pvt. Ltd.*



*Office 103, 104, 1st Floor Pride Portal,Shivaji Housing Society,
Bahiratwadi,Near Hotel JW Marriott, Off Senapati Bapat Road, Pune - 411016,
MH, INDIA.**Phone: +91 9689077510 | Skype: amols_iprogrammer*
www.iprogrammer.com 


On Fri, Jul 27, 2018 at 1:08 PM, Timo Walther  wrote:

> Hi Amol,
>
> the dot operation is reserved for calling functions on fields. If you want
> to get a nested field in the Table API, use the `.get("applicationId")`
> operation. See also [1] under "Value access functions".
>
> Regards,
> Timo
>
> [1] https://ci.apache.org/projects/flink/flink-docs-release-1.5/
> dev/table/tableApi.html#built-in-functions
>
>
> Am 27.07.18 um 09:10 schrieb Amol S - iProgrammer:
>
>> Hello Fabian,
>>
>> I am streaming my mongodb oplog using flink and want to use flink table
>> API
>> to join multiple tables.  My code looks like
>>
>> DataStream streamSource = env
>>  .addSource(kafkaConsumer)
>>  .setParallelism(4);
>>
>> StreamTableEnvironment tableEnv = TableEnvironment.getTableEnvir
>> onment(env);
>> // Convert the DataStream into a Table with default fields "f0", "f1"
>> Table table1 = tableEnv.fromDataStream(streamSource);
>>
>> Table master = table1.filter("ns === 'Master'").select("o as master,
>> o.applicationId as primaryKey");
>> Table child1 = table1.filter("ns === 'Child1'").select("o  as child1,
>> o.applicationId as foreignKey");
>> Table child2 = table1.filter("ns === 'Child2'").select("o  as child2,
>> o.applicationId as foreignKey2");
>>
>> Table result = master.join(child1).where("pri
>> maryKey==foreignKey").join(child2).where("primaryKey==foreignKey2");
>>
>> it is throwing error "Method threw
>> 'org.apache.flink.table.api.ValidationException' exception. Undefined
>> function: APPLICATIONID"
>>
>> public class Oplog implements Serializable{
>>  private BasicDBObject o;
>> }
>>
>> Where o is generic java type for fetching mongodb oplog and I can not
>> replace this generic type with static pojo's. please tell me any work
>> around on this.
>>
>> BasicDBObject suffice following two rules
>>
>>
>> -
>>
>> The class must be public.
>> -
>>
>> It must have a public constructor without arguments (default
>> constructor)
>>
>> and we can access class members through basicDBObject.getString("abc")
>>
>>
>>
>>
>> ---
>> *Amol Suryawanshi*
>> Java Developer
>> am...@iprogrammer.com
>>
>>
>> *iProgrammer Solutions Pvt. Ltd.*
>>
>>
>>
>> *Office 103, 104, 1st Floor Pride Portal,Shivaji Housing Society,
>> Bahiratwadi,Near Hotel JW Marriott, Off Senapati Bapat Road, Pune -
>> 411016,
>> MH, INDIA.**Phone: +91 9689077510 | Skype: amols_iprogrammer*
>> www.iprogrammer.com 
>> 
>>
>>
>


[jira] [Created] (FLINK-9981) Tune performance of RocksDB implementation

2018-07-27 Thread Stefan Richter (JIRA)
Stefan Richter created FLINK-9981:
-

 Summary: Tune performance of RocksDB implementation
 Key: FLINK-9981
 URL: https://issues.apache.org/jira/browse/FLINK-9981
 Project: Flink
  Issue Type: Sub-task
  Components: State Backends, Checkpointing
Affects Versions: 1.6.0
Reporter: Stefan Richter
Assignee: Stefan Richter


General performance tuning/polishing for the RocksDB implementation. We can 
figure out how caching/seeking can be improved.



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


[jira] [Created] (FLINK-9980) wiki-edits quickstart example fails when run outside of IDE

2018-07-27 Thread David Anderson (JIRA)
David Anderson created FLINK-9980:
-

 Summary: wiki-edits quickstart example fails when run outside of 
IDE
 Key: FLINK-9980
 URL: https://issues.apache.org/jira/browse/FLINK-9980
 Project: Flink
  Issue Type: Bug
  Components: Examples
Affects Versions: 1.5.1
Reporter: David Anderson
 Fix For: 1.6.0


Following the instructions in the docs, I find that this example runs in 
intellij, but when run from the command line as instructed

{{mvn exec:java -Dexec.mainClass=wikiedits.WikipediaAnalysis}}

it fails with 

{{java.lang.NoClassDefFoundError: 
org/apache/flink/streaming/api/functions/source/SourceFunction}}

I discovered this when trying to reproduce a problem reported on stack 
overflow: 
[https://stackoverflow.com/questions/51550479/caused-by-java-io-ioexception-unable-to-serialize-default-value-of-type-tuple2]

That user is getting a different runtime error.



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


[jira] [Created] (FLINK-9979) Support a custom FlinkKafkaPartitioner for a Kafka table sink factory

2018-07-27 Thread Timo Walther (JIRA)
Timo Walther created FLINK-9979:
---

 Summary: Support a custom FlinkKafkaPartitioner for a Kafka table 
sink factory
 Key: FLINK-9979
 URL: https://issues.apache.org/jira/browse/FLINK-9979
 Project: Flink
  Issue Type: Sub-task
  Components: Table API  SQL
Reporter: Timo Walther
Assignee: Timo Walther


Currently, the Kafka table sink factory does not support a custom 
FlinkKafkaPartitioner. However, this is needed for many use cases.



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


Re: Access generic pojo fields

2018-07-27 Thread Timo Walther

Hi Amol,

the dot operation is reserved for calling functions on fields. If you 
want to get a nested field in the Table API, use the 
`.get("applicationId")` operation. See also [1] under "Value access 
functions".


Regards,
Timo

[1] 
https://ci.apache.org/projects/flink/flink-docs-release-1.5/dev/table/tableApi.html#built-in-functions



Am 27.07.18 um 09:10 schrieb Amol S - iProgrammer:

Hello Fabian,

I am streaming my mongodb oplog using flink and want to use flink table API
to join multiple tables.  My code looks like

DataStream streamSource = env
 .addSource(kafkaConsumer)
 .setParallelism(4);

StreamTableEnvironment tableEnv = TableEnvironment.getTableEnvironment(env);
// Convert the DataStream into a Table with default fields "f0", "f1"
Table table1 = tableEnv.fromDataStream(streamSource);

Table master = table1.filter("ns === 'Master'").select("o as master,
o.applicationId as primaryKey");
Table child1 = table1.filter("ns === 'Child1'").select("o  as child1,
o.applicationId as foreignKey");
Table child2 = table1.filter("ns === 'Child2'").select("o  as child2,
o.applicationId as foreignKey2");

Table result = 
master.join(child1).where("primaryKey==foreignKey").join(child2).where("primaryKey==foreignKey2");

it is throwing error "Method threw
'org.apache.flink.table.api.ValidationException' exception. Undefined
function: APPLICATIONID"

public class Oplog implements Serializable{
 private BasicDBObject o;
}

Where o is generic java type for fetching mongodb oplog and I can not
replace this generic type with static pojo's. please tell me any work
around on this.

BasicDBObject suffice following two rules


-

The class must be public.
-

It must have a public constructor without arguments (default constructor)

and we can access class members through basicDBObject.getString("abc")




---
*Amol Suryawanshi*
Java Developer
am...@iprogrammer.com


*iProgrammer Solutions Pvt. Ltd.*



*Office 103, 104, 1st Floor Pride Portal,Shivaji Housing Society,
Bahiratwadi,Near Hotel JW Marriott, Off Senapati Bapat Road, Pune - 411016,
MH, INDIA.**Phone: +91 9689077510 | Skype: amols_iprogrammer*
www.iprogrammer.com 






Fwd: [apache/flink-web] One of your dependencies may have a security vulnerability

2018-07-27 Thread Stephan Ewen
-- Forwarded message --
From: Apache Security Team 
Date: Thu, Jul 26, 2018 at 6:04 PM
Subject: Fwd: [apache/flink-web] One of your dependencies may have a
security vulnerability
To: priv...@flink.apache.org


Hi Flink PMC,

we are still receiving this notification from github.

Regards,
Yann.

-- Forwarded message --
From: GitHub 
Date: Wed, Jul 25, 2018 at 3:43 PM
Subject: [apache/flink-web] One of your dependencies may have a security
vulnerability
To: apache/flink-web 
Cc: Security alert 


We found a potential security vulnerabilty in one of your dependencies
[image: GitHub]  Sign in 
*asfsecurity,*

We found a potential security vulnerability in a repository for which you
have been granted security alert access.
[image: @apache] apache/flink-web 
Known * high severity* security vulnerability detected in yajl-ruby < 1.3.1
defined in Gemfile
.
Gemfile  update
suggested: yajl-ruby ~> 1.3.1.
Always verify the validity and compatibility of suggestions with your
codebase.
Review vulnerable dependency

--

Only users who have been assigned access to security alerts will receive
these notifications.
Unsubscribe

· Email preferences  · Terms
 · Privacy
 · Sign into GitHub


GitHub, Inc.
88 Colin P Kelly Jr St.
San Francisco, CA 94107



Access generic pojo fields

2018-07-27 Thread Amol S - iProgrammer
Hello Fabian,

I am streaming my mongodb oplog using flink and want to use flink table API
to join multiple tables.  My code looks like

DataStream streamSource = env
.addSource(kafkaConsumer)
.setParallelism(4);

StreamTableEnvironment tableEnv = TableEnvironment.getTableEnvironment(env);
// Convert the DataStream into a Table with default fields "f0", "f1"
Table table1 = tableEnv.fromDataStream(streamSource);

Table master = table1.filter("ns === 'Master'").select("o as master,
o.applicationId as primaryKey");
Table child1 = table1.filter("ns === 'Child1'").select("o  as child1,
o.applicationId as foreignKey");
Table child2 = table1.filter("ns === 'Child2'").select("o  as child2,
o.applicationId as foreignKey2");

Table result = 
master.join(child1).where("primaryKey==foreignKey").join(child2).where("primaryKey==foreignKey2");

it is throwing error "Method threw
'org.apache.flink.table.api.ValidationException' exception. Undefined
function: APPLICATIONID"

public class Oplog implements Serializable{
private BasicDBObject o;
}

Where o is generic java type for fetching mongodb oplog and I can not
replace this generic type with static pojo's. please tell me any work
around on this.

BasicDBObject suffice following two rules


   -

   The class must be public.
   -

   It must have a public constructor without arguments (default constructor)

and we can access class members through basicDBObject.getString("abc")




---
*Amol Suryawanshi*
Java Developer
am...@iprogrammer.com


*iProgrammer Solutions Pvt. Ltd.*



*Office 103, 104, 1st Floor Pride Portal,Shivaji Housing Society,
Bahiratwadi,Near Hotel JW Marriott, Off Senapati Bapat Road, Pune - 411016,
MH, INDIA.**Phone: +91 9689077510 | Skype: amols_iprogrammer*
www.iprogrammer.com 



Re: [VOTE] Release 1.5.2, release candidate #2

2018-07-27 Thread Chesnay Schepler

+1

- built flink from source
- ensured all examples are bundled in binaries
- uploaded an example through WebUI, generated the plan and ran the job
- verified that TM logs can be accessed through WebUI
- ran tests on travis:
basic tests: https://travis-ci.org/zentol/flink/builds/408590209
E2E tests: https://travis-ci.org/zentol/flink-ci/builds/408589651

On 27.07.2018 00:15, Yaz Sh wrote:

+1

- Checked “signatures" for all packages
- Checked “checksum" for all packages
- Started cluster with no exceptions on log files
- Stoped cluster with no exceptions on log file
- Added more taskManager by running “taskmanager.sh start”
- Added more slots via “flink-conf.yml"
- Checked for all examples to be available on the packages
- Tested several batch and stream examples via WebUI
- Tested several batch and stream examples via CLI
- Tested Kafka010Example
- Checked the website PR
  


Cheers,
Yaz


On Jul 26, 2018, at 9:31 AM, Timo Walther  wrote:

I agree with Chesnay. Only regressions in the release candidate should cancel 
minor releases.

Timo


Am 26.07.18 um 15:02 schrieb Chesnay Schepler:

Since the regression already existed in 1.5.0 I will not cancel the vote,
as there's no benefit to canceling the bugfix release.

If a fix is found within the voting period we should consider it so we
don't have to create a separate release shortly afterwards for this issue.

On 26.07.2018 14:29, Piotr Nowojski wrote:

Voting -1

Flip6 in 1.5.0 seems like introduced regression in resource management in batch: 
https://issues.apache.org/jira/browse/FLINK-9969 


It’s not a super new bug, yet still I would block the release until we either 
fix it or confirm that’s a some misconfiguration.

Piotrek


On 26 Jul 2018, at 10:39, Chesnay Schepler  wrote:

Hi everyone,
Please review and vote on the release candidate #2 for the version 1.5.2, as 
follows:
[ ] +1, Approve the release
[ ] -1, Do not approve the release (please provide specific comments)


The complete staging area is available for your review, which includes:
* JIRA release notes [1],
* the official Apache source release and binary convenience releases to be 
deployed to dist.apache.org [2], which are signed with the key with fingerprint 
11D464BA [3],
* all artifacts to be deployed to the Maven Central Repository [4],
* source code tag "release-1.5.1-rc2" [5],
* website pull request listing the new release and adding announcement blog 
post [6].

The vote will be open for at least 72 hours. It is adopted by majority 
approval, with at least 3 PMC affirmative votes.

Thanks,
Chesnay

[1] 
https://issues.apache.org/jira/secure/ReleaseNote.jspa?projectId=12315522=12343588
[2] https://dist.apache.org/repos/dist/dev/flink/flink-1.5.2/
[3] https://dist.apache.org/repos/dist/release/flink/KEYS
[4] https://repository.apache.org/content/repositories/orgapacheflink-1174
[5] 
https://git-wip-us.apache.org/repos/asf?p=flink.git;a=tag;h=refs/tags/release-1.5.2-rc2
[6] https://github.com/apache/flink-web/pull/114











Re: [VOTE] Release 1.5.2, release candidate #2

2018-07-27 Thread Chesnay Schepler
I've opened a PR to fix this from happening again: 
https://github.com/apache/flink/pull/6436


In the mean-time I will regenerate the sha for the source release.

On 27.07.2018 08:27, Chesnay Schepler wrote:
That is indeed weird. It only seems to affect the source release, will 
look into it. This also affects 1.6.0 RC1, so it's at least not an 
issue of my environment.


On 27.07.2018 02:16, Thomas Weise wrote:

+1 but see below

Following shasum verification failed:

shasum:
/mnt/c/Users/Zento/Documents/GitHub/flink/tools/releasing/release/flink-1.5.2-src.tgz: 

/mnt/c/Users/Zento/Documents/GitHub/flink/tools/releasing/release/flink-1.5.2-src.tgz: 


FAILED open or read

The problem can be corrected by fixing the file path in
*flink-1.5.2-src.tgz.sha512* - I would recommend doing that before
promoting the release.

Successfully run Beam pipeline against locally running Flink cluster.

Thanks




On Thu, Jul 26, 2018 at 1:39 AM Chesnay Schepler  
wrote:



Hi everyone,
Please review and vote on the release candidate #2 for the version
1.5.2, as follows:
[ ] +1, Approve the release
[ ] -1, Do not approve the release (please provide specific comments)


The complete staging area is available for your review, which includes:
* JIRA release notes [1],
* the official Apache source release and binary convenience releases to
be deployed to dist.apache.org [2], which are signed with the key with
fingerprint 11D464BA [3],
* all artifacts to be deployed to the Maven Central Repository [4],
* source code tag "release-1.5.1-rc2" [5],
* website pull request listing the new release and adding announcement
blog post [6].

The vote will be open for at least 72 hours. It is adopted by majority
approval, with at least 3 PMC affirmative votes.

Thanks,
Chesnay

[1]

https://issues.apache.org/jira/secure/ReleaseNote.jspa?projectId=12315522=12343588 


[2] https://dist.apache.org/repos/dist/dev/flink/flink-1.5.2/
[3] https://dist.apache.org/repos/dist/release/flink/KEYS
[4] 
https://repository.apache.org/content/repositories/orgapacheflink-1174

[5]

https://git-wip-us.apache.org/repos/asf?p=flink.git;a=tag;h=refs/tags/release-1.5.2-rc2 


[6] https://github.com/apache/flink-web/pull/114













[jira] [Created] (FLINK-9978) Source release sha contains absolute file path

2018-07-27 Thread Chesnay Schepler (JIRA)
Chesnay Schepler created FLINK-9978:
---

 Summary: Source release sha contains absolute file path
 Key: FLINK-9978
 URL: https://issues.apache.org/jira/browse/FLINK-9978
 Project: Flink
  Issue Type: Bug
  Components: Release System
Affects Versions: 1.5.2, 1.6.0
Reporter: Chesnay Schepler
Assignee: Chesnay Schepler
 Fix For: 1.5.2, 1.6.0


The sha file for source releases contain the absolute path to the file, causing 
failures during verification on other machines since the path cannot be found.



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


Re: [VOTE] Release 1.5.2, release candidate #2

2018-07-27 Thread Chesnay Schepler
That is indeed weird. It only seems to affect the source release, will 
look into it. This also affects 1.6.0 RC1, so it's at least not an issue 
of my environment.


On 27.07.2018 02:16, Thomas Weise wrote:

+1 but see below

Following shasum verification failed:

shasum:
/mnt/c/Users/Zento/Documents/GitHub/flink/tools/releasing/release/flink-1.5.2-src.tgz:
/mnt/c/Users/Zento/Documents/GitHub/flink/tools/releasing/release/flink-1.5.2-src.tgz:
FAILED open or read

The problem can be corrected by fixing the file path in
*flink-1.5.2-src.tgz.sha512* - I would recommend doing that before
promoting the release.

Successfully run Beam pipeline against locally running Flink cluster.

Thanks




On Thu, Jul 26, 2018 at 1:39 AM Chesnay Schepler  wrote:


Hi everyone,
Please review and vote on the release candidate #2 for the version
1.5.2, as follows:
[ ] +1, Approve the release
[ ] -1, Do not approve the release (please provide specific comments)


The complete staging area is available for your review, which includes:
* JIRA release notes [1],
* the official Apache source release and binary convenience releases to
be deployed to dist.apache.org [2], which are signed with the key with
fingerprint 11D464BA [3],
* all artifacts to be deployed to the Maven Central Repository [4],
* source code tag "release-1.5.1-rc2" [5],
* website pull request listing the new release and adding announcement
blog post [6].

The vote will be open for at least 72 hours. It is adopted by majority
approval, with at least 3 PMC affirmative votes.

Thanks,
Chesnay

[1]

https://issues.apache.org/jira/secure/ReleaseNote.jspa?projectId=12315522=12343588
[2] https://dist.apache.org/repos/dist/dev/flink/flink-1.5.2/
[3] https://dist.apache.org/repos/dist/release/flink/KEYS
[4] https://repository.apache.org/content/repositories/orgapacheflink-1174
[5]

https://git-wip-us.apache.org/repos/asf?p=flink.git;a=tag;h=refs/tags/release-1.5.2-rc2
[6] https://github.com/apache/flink-web/pull/114