-
Notifications
You must be signed in to change notification settings - Fork 50
HTTP-122 Retry for source lookup table #148
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
Changes from 2 commits
610d4b9
16ffc35
a547bde
e0c7bcc
b67567a
680723b
9485f27
853d23d
0c5097b
75eaa2d
65de8b8
9753dea
c3ed544
0a56ba0
3cf56da
b08575d
e497587
dfb1133
d9c4257
195f4c3
95aa14e
72dc8ab
File filter
Filter by extension
Conversations
Jump to
Diff view
Diff view
There are no files selected for viewing
Original file line number | Diff line number | Diff line change |
---|---|---|
|
@@ -13,3 +13,4 @@ bin | |
/src/test/test.iml | ||
/flink-http-connector.iml | ||
/dependency-reduced-pom.xml | ||
/.java-version |
Original file line number | Diff line number | Diff line change |
---|---|---|
|
@@ -3,6 +3,9 @@ | |
## [Unreleased] | ||
|
||
- OIDC token request to not flow during explain | ||
- Added support for auto-retry for source table. Auto retry on IOException and user-defined http codes - parameter `gid.connector.http.source.lookup.retry-codes`. | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. normally one pr would be one line here There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. ok, set as one item with subitems |
||
- Parameters `gid.connector.http.source.lookup.error.code.exclude"` and `gid.connector.http.source.lookup.error.code` are replaced by `gid.connector.http.source.lookup.ignored-response-codes`. | ||
- Added connection timeout for source table - `gid.connector.http.source.lookup.connection.timeout`. | ||
|
||
## [0.18.0] - 2025-01-15 | ||
|
||
|
Original file line number | Diff line number | Diff line change |
---|---|---|
|
@@ -170,8 +170,22 @@ The second one is set per individual HTTP requests by HTTP client. Its default v | |
Flink's current implementation of `AsyncTableFunction` does not allow specifying custom logic for handling Flink AsyncIO timeouts as it is for Java API. | ||
Because of that, if AsyncIO timer passes, Flink will throw TimeoutException which will cause job restart. | ||
|
||
The HTTP request timeouts on the other hand will not cause Job restart. In that case, exception will be logged into application logs. | ||
To avoid job restart on timeouts caused by Lookup queries, the value of `gid.connector.http.source.lookup.request.timeout` should be smaller than `table.exec.async-lookup.timeout`. | ||
#### Retries | ||
Lookup source handles auto-retries for two scenarios: | ||
1. IOException occurs (e.g. timeout) | ||
maciejmaciejko-gid marked this conversation as resolved.
Show resolved
Hide resolved
|
||
2. HTTP server returns response with code, which is marked as temporal error. These codes are defined in table configuration. | ||
maciejmaciejko-gid marked this conversation as resolved.
Show resolved
Hide resolved
|
||
Retries are executed silently, without job restart. After reaching max retries attempts (per request) function will fail and restart job. | ||
maciejmaciejko-gid marked this conversation as resolved.
Show resolved
Hide resolved
|
||
|
||
Notice that response codes are splitted into 3 groups: | ||
maciejmaciejko-gid marked this conversation as resolved.
Show resolved
Hide resolved
|
||
- successful responses - response is returned immediately for further processing | ||
- temporary errors - request will be re-sent | ||
maciejmaciejko-gid marked this conversation as resolved.
Show resolved
Hide resolved
|
||
- error responses - unexpected response, which will fail the job. Any code which is not marked as successful or temporary error is marked as error. | ||
maciejmaciejko-gid marked this conversation as resolved.
Show resolved
Hide resolved
|
||
|
||
#### Retry strategy | ||
User can choose retry strategy type for source table: | ||
- fixed-delay - http request will be re-sent after specified delay | ||
- exponential-delay - request will be re-sent with exponential backoff strategy, limited to max-retries attempts. | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. I see that the config option is lookup.max-retries (I suggest using the exact config parameter name) - do we need a separate config for max-retries for sinks? It would be worth defining exactly what we mean by exponential backoff strategy. There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. ok, good idea. I added explanation |
||
|
||
|
||
#### Lookup multiple results | ||
|
||
|
@@ -391,19 +405,32 @@ is provided. | |
|
||
|
||
## HTTP status code handler | ||
Http Sink and Lookup Source connectors allow defining list of HTTP status codes that should be treated as errors. | ||
### Sink table | ||
Http Sink allows defining list of HTTP status codes that should be treated as errors. | ||
maciejmaciejko-gid marked this conversation as resolved.
Show resolved
Hide resolved
|
||
By default all 400s and 500s response codes will be interpreted as error code. | ||
maciejmaciejko-gid marked this conversation as resolved.
Show resolved
Hide resolved
|
||
|
||
This behavior can be changed by using below properties in table definition (DDL) for Sink and Lookup Source or passing it via | ||
maciejmaciejko-gid marked this conversation as resolved.
Show resolved
Hide resolved
|
||
`setProperty' method from Sink's builder. The property names are: | ||
- `gid.connector.http.sink.error.code` and `gid.connector.http.source.lookup.error.code` used to defined HTTP status code value that should be treated as error for example 404. | ||
`setProperty' method from Sink's builder. The property name are: | ||
- `gid.connector.http.sink.error.code` used to defined HTTP status code value that should be treated as error for example 404. | ||
Many status codes can be defined in one value, where each code should be separated with comma, for example: | ||
`401, 402, 403`. User can use this property also to define a type code mask. In that case, all codes from given HTTP response type will be treated as errors. | ||
An example of such a mask would be `3XX, 4XX, 5XX`. In this case, all 300s, 400s and 500s status codes will be treated as errors. | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. is X the only mask character? There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. It's part of sink configuration, which wasn't changed. It allows only [1-5]XX or exact http code value. I reimplemented handling of http response code only for source table, which is connected with retry feature. |
||
- `gid.connector.http.sink.error.code.exclude` and `gid.connector.http.source.lookup.error.code.exclude` used to exclude a HTTP code from error list. | ||
- `gid.connector.http.sink.error.code.exclude` used to exclude a HTTP code from error list. | ||
Many status codes can be defined in one value, where each code should be separated with comma, for example: | ||
`401, 402, 403`. In this example, codes 401, 402 and 403 would not be interpreted as error codes. | ||
|
||
### Source table | ||
maciejmaciejko-gid marked this conversation as resolved.
Show resolved
Hide resolved
|
||
Http source requires success codes defined in parameter: `gid.connector.http.source.lookup.success-codes`. That list should contains all http status codes | ||
maciejmaciejko-gid marked this conversation as resolved.
Show resolved
Hide resolved
|
||
which are considered as success response. It may be 200 (ok) as well as 404 (not found). The first one is standard response and its content should be deserialized/parsed. | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. nit: response -> responses I am not sure what we mean by " It may be 200 (ok) as well as 404 (not found). The first one is standard response and its content should be deserialized/parsed." Is 200 and 404 the defaults or recommended settings? |
||
Processing of 404 request's content may be skipped by adding it to parameter `gid.connector.http.source.lookup.ignored-response-codes`. | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. What does skipped mean here - fail the job? There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. The section was edited. Could you check if it's clear now? |
||
|
||
Both parameters supports whitelisting and blacklisting (ops!). Sample configuration may look like: | ||
maciejmaciejko-gid marked this conversation as resolved.
Show resolved
Hide resolved
|
||
`2XX,404,!203` - meaning all codes from group 2XX (200-299), with 404 and without 203 ('!' character). Group blacklisting e.g. !2XX is not supported. | ||
Notice that ignored-response-codes has to be a subset of success-codes. | ||
|
||
The same format is used in parameter `gid.connector.http.source.lookup.retry-codes`. | ||
|
||
|
||
## TLS (more secure replacement for SSL) and mTLS support | ||
|
||
Both Http Sink and Lookup Source connectors support HTTPS communication using TLS 1.2 and mTLS. | ||
|
@@ -479,6 +506,15 @@ be requested if the current time is later than the cached token expiry time minu | |
| gid.connector.http.source.lookup.response.thread-pool.size | optional | Sets the size of pool thread for HTTP lookup response processing. Increasing this value would mean that more concurrent requests can be processed in the same time. If not specified, the default value of 4 threads will be used. | | ||
| gid.connector.http.source.lookup.use-raw-authorization-header | optional | If set to `'true'`, uses the raw value set for the `Authorization` header, without transformation for Basic Authentication (base64, addition of "Basic " prefix). If not specified, defaults to `'false'`. | | ||
| gid.connector.http.source.lookup.request-callback | optional | Specify which `HttpLookupPostRequestCallback` implementation to use. By default, it is set to `slf4j-lookup-logger` corresponding to `Slf4jHttpLookupPostRequestCallback`. | | ||
| gid.connector.http.source.lookup.connection.timeout | optional | Source table connection timeout. | | ||
maciejmaciejko-gid marked this conversation as resolved.
Show resolved
Hide resolved
|
||
| gid.connector.http.source.lookup.success-codes | optional | Comma separated http codes considered as success response. Use [1-5]XX for groups and '!' character for excluding. | | ||
| gid.connector.http.source.lookup.retry-codes | optional | Comma separated http codes considered as temporal errors. Use [1-5]XX for groups and '!' character for excluding. | | ||
maciejmaciejko-gid marked this conversation as resolved.
Show resolved
Hide resolved
|
||
| gid.connector.http.source.lookup.ignored-response-codes | optional | Comma separated http codes. Content for these responses will be ignored. Use [1-5]XX for groups and '!' character for excluding. Ignored response codes has to be a subset of `gid.connector.http.source.lookup.success-codes`. | | ||
| gid.connector.http.source.lookup.retry-strategy.type | optional | Auto retry strategy type: fixed_delay (default) or exponential_delay. | | ||
| gid.connector.http.source.lookup.fixed-delay.delay | optional | Fixed-delay interval between retries. Default 1 second. | | ||
maciejmaciejko-gid marked this conversation as resolved.
Show resolved
Hide resolved
|
||
| gid.connector.http.source.lookup.exponential-delay.initial-backoff | optional | Exponential-delay initial delay. Default 1 second. | | ||
| gid.connector.http.source.lookup.max-backoff | optional | Exponential-delay maximum delay. Default 1 minute. | | ||
| gid.connector.http.source.lookup.backoff-multiplier | optional | Exponential-delay multiplier. Default value 1.5 | | ||
|
||
### HTTP Sink | ||
|
||
|
Original file line number | Diff line number | Diff line change |
---|---|---|
|
@@ -78,7 +78,6 @@ under the License. | |
<log4j.version>2.17.2</log4j.version> | ||
<lombok.version>1.18.22</lombok.version> | ||
<jackson.version>2.18.1</jackson.version> | ||
<junit4.version>4.13.2</junit4.version> | ||
<junit5.version>5.10.1</junit5.version> | ||
<junit.jupiter.version>${junit5.version}</junit.jupiter.version> | ||
<assertj.core.version>3.21.0</assertj.core.version> | ||
|
@@ -87,6 +86,8 @@ under the License. | |
<jacoco.plugin.version>0.8.12</jacoco.plugin.version> | ||
<maven.shade.plugin.version>3.1.1</maven.shade.plugin.version> | ||
<mockito-inline.version>4.6.1</mockito-inline.version> | ||
<resilence4j.version>1.7.1</resilence4j.version> | ||
<slf4j.version>2.0.17</slf4j.version> | ||
</properties> | ||
|
||
<repositories> | ||
|
@@ -119,25 +120,17 @@ under the License. | |
<scope>provided</scope> | ||
</dependency> | ||
|
||
<!-- Add logging framework, to produce console output when running in the IDE. --> | ||
<!-- These dependencies are excluded from the application JAR by default. --> | ||
<dependency> | ||
<groupId>org.apache.logging.log4j</groupId> | ||
<artifactId>log4j-slf4j-impl</artifactId> | ||
<version>${log4j.version}</version> | ||
<scope>provided</scope> | ||
</dependency> | ||
<dependency> | ||
<groupId>org.apache.logging.log4j</groupId> | ||
<artifactId>log4j-api</artifactId> | ||
<version>${log4j.version}</version> | ||
<scope>provided</scope> | ||
<groupId>org.slf4j</groupId> | ||
<artifactId>slf4j-api</artifactId> | ||
<version>${slf4j.version}</version> | ||
</dependency> | ||
<!-- Add logging framework, to produce console output when running in the IDE. --> | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. can we have the logging change in a separate PR - it is easier to track the history then please There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Why this change is needed anyways? One of the "rule of thumbs" when we were starting this connector was to try not add any external libraries to the connector, that my or may not clash with any user code -> i.e that is why we use Java's 11 http client. You need the resilence4j for retry functionality right? There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Dedicated lib for retries might be an overkill now, but I think we can benefit in long term. The library provides Rate Limiter or Circuit Breaker. Both features might be worth adding. Or at least Rate Limiter. There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. "Why this change is needed anyways? Yes, I had to change to compile project with resilence4j. Notice that Flink use the same API: "You need the resilence4j for retry functionality right?"
Do you think it's ok to add them? Another option is to shadow them. There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more.
|
||
<dependency> | ||
<groupId>org.apache.logging.log4j</groupId> | ||
<artifactId>log4j-core</artifactId> | ||
<version>${log4j.version}</version> | ||
<scope>provided</scope> | ||
<groupId>org.slf4j</groupId> | ||
<artifactId>slf4j-simple</artifactId> | ||
<version>${slf4j.version}</version> | ||
<scope>test</scope> | ||
</dependency> | ||
|
||
<dependency> | ||
|
@@ -167,6 +160,12 @@ under the License. | |
<scope>provided</scope> | ||
</dependency> | ||
|
||
<dependency> | ||
<groupId>io.github.resilience4j</groupId> | ||
<artifactId>resilience4j-retry</artifactId> | ||
<version>${resilence4j.version}</version> | ||
</dependency> | ||
|
||
<!--TEST--> | ||
<dependency> | ||
<groupId>org.apache.httpcomponents</groupId> | ||
|
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,15 @@ | ||
package com.getindata.connectors.http; | ||
|
||
import lombok.Getter; | ||
|
||
import java.net.http.HttpResponse; | ||
|
||
@Getter | ||
public class HttpStatusCodeValidationFailedException extends Exception { | ||
private final HttpResponse<?> response; | ||
|
||
public HttpStatusCodeValidationFailedException(String message, HttpResponse<?> response) { | ||
super(message); | ||
this.response = response; | ||
} | ||
} |
Uh oh!
There was an error while loading. Please reload this page.